Uploaded image for project: 'Kafka'
  1. Kafka
  2. KAFKA-14246

Update threading model for Consumer

    XMLWordPrintableJSON

Details

    Description

      Hi community,

       

      We are refactoring the current KafkaConsumer and making it more asynchronous.  This is the master Jira to track the project's progress; subtasks will be linked to this ticket.  Please review the design document and feel free to use this thread for discussion. 

       

      The design document is here: https://cwiki.apache.org/confluence/display/KAFKA/Proposal%3A+Consumer+Threading+Model+Refactor

       

      The original email thread is here: https://lists.apache.org/thread/13jvwzkzmb8c6t7drs4oj2kgkjzcn52l

       

      I will continue to update the 1pager as reviews and comments come.

       

      Thanks, 

      P

      Attachments

        Issue Links

          1.
          The background thread should try to process the remaining task until the shutdown timer is expired Sub-task Open Philip Nee  
          2.
          Propagate client response time when timeout to the request handler Sub-task Resolved Philip Nee  
          3.
          Timeout request might retry forever even if the user API times out in PrototypeAsyncConsumer Sub-task Open Philip Nee  
          4.
          Review exception caching logic used for reset/validate positions in async consumer Sub-task Resolved Unassigned  
          5.
          Remove duplicated integration tests for new consumer Sub-task Open Unassigned  
          6.
          CommitRequestManager not calling RequestState callbacks Sub-task Resolved Philip Nee  
          7.
          Fix for async consumer access to committed offsets with multiple consumers Sub-task Resolved Unassigned  
          8.
          Implement the groupMetadata Consumer API Sub-task Patch Available Bruno Cadonna

          0%

          Original Estimate - 504h
          Remaining Estimate - 504h
          9.
          Implement more of the remaining PrototypeAsyncConsumer APIs Sub-task Resolved Kirk True  
          10.
          DefaultBackgroundThread is running tight loop Sub-task Open Philip Nee  
          11.
          New consumer internals refactoring and clean up Sub-task Resolved Kirk True  
          12.
          Extract common logic from Fetcher Sub-task Resolved Kirk True  
          13.
          Extract inner classes from Fetcher for reuse in refactoring Sub-task Resolved Kirk True  
          14.
          Extract metadata-related tasks from Fetcher into MetadataFetcher Sub-task Resolved Kirk True  
          15.
          Document event queueing patterns Sub-task Open Kirk True  
          16.
          Refactoring for client code to reduce boilerplate Sub-task Resolved Kirk True  
          17.
          KafkaConsumer incorrectly passes locally-scoped deserializers to FetchConfig Sub-task Resolved Kirk True

          0%

          Original Estimate - 24h
          Remaining Estimate - 24h
          18.
          Port tests in FetcherTest to FetchRequestManagerTest Sub-task Resolved Kirk True  
          19.
          Introduce FetchRequestManager to integrate fetch into new consumer threading refactor Sub-task Resolved Kirk True  
          20.
          Consumer event queues should be bounded Sub-task Open Kirk True  
          21.
          Extract reusable logic from OffsetFetcher Sub-task Resolved Lianet Magrans  
          22.
          Introduce OffsetsRequestManager to integrate ListOffsets requests into new consumer threading refactor Sub-task Resolved Lianet Magrans  
          23.
          Implement new consumer offsetsForTimes Sub-task Resolved Lianet Magrans  
          24.
          Generalize integration tests to change use of KafkaConsumer to Consumer Sub-task Resolved Kirk True  
          25.
          Implement ConsumerGroupProtocolVersionResolver to determine consumer group protocol Sub-task Open Kirk True  
          26.
          Metadata Request Manager and listTopics/partitionsFor API Sub-task Resolved Philip Nee  
          27.
          Implement Wakeup() Sub-task Resolved Philip Nee  
          28.
          CompletableApplicationEvents aren't being completed when the consumer is closing Sub-task Resolved Philip Nee  
          29.
          Integration Tests for the New Consumer Implementation Sub-task Resolved Philip Nee  
          30.
          Design & implement support for internal Consumer delegates Sub-task Resolved Kirk True  
          31.
          Send GroupConsumerHeartbeatRequest on Consumer.close() Sub-task Open Philip Nee  
          32.
          OffsetsForTimes validation for negative timestamps in new consumer Sub-task Resolved Lianet Magrans  
          33.
          Evaluate conditions for short circuiting consumer API calls Sub-task Open Philip Nee  
          34.
          Review committed offset refresh logic Sub-task Open Philip Nee  
          35.
          Method always return the same value ApplicationEventProcessor.java Sub-task Resolved Kirk True  
          36.
          Ensure wakeups are handled correctly in PrototypeAsyncConsumer.poll() Sub-task Resolved Bruno Cadonna  
          37.
          Remove NetworkClientDelegate methods isUnavailable, maybeThrowAuthFailure, and tryConnect Sub-task In Progress Phuc Hong Tran  
          38.
          Determine if Timer should be used elsewhere in PrototypeAsyncConsumer.updateFetchPositions() Sub-task Open Unassigned  
          39.
          Ensure fetch offset and commit offset handler handles both timeout and various error types Sub-task Resolved Philip Nee  
          40.
          Improve handling of fetching during metadata updates Sub-task Open Kirk True  
          41.
          Investigate FetcherTest's/FetchRequestManager's testCompletedFetchRemoval Sub-task Open Kirk True  
          42.
          Investigate FetcherTest's/FetchRequestManager's duplicate metadata update in assignFromUserNoId Sub-task Open Kirk True  
          43.
          Investigate FetcherTest's/FetchRequestManager's testFetchingPendingPartitions and testInflightFetchOnPendingPartitions overlap Sub-task Open Kirk True  
          44.
          Investigate FetcherTest's/FetchRequestManager's testQuotaMetrics Sub-task Open Kirk True  
          45.
          Investigate FetcherTest's/FetchRequestManager's testFetcherLeadMetric Sub-task Open Kirk True  
          46.
          Investigate FetcherTest's/FetchRequestManager's testFetchResponseMetrics Sub-task Open Kirk True  
          47.
          Investigate FetcherTest's/FetchRequestManager's testFetchCompletedBeforeHandlerAdded Sub-task Open Kirk True  
          48.
          Investigate ConsumerNetworkThreadTest's testPollResultTimer Sub-task Open Philip Nee  
          49.
          Investigate auto commit guarantees during Consumer.assign() Sub-task Resolved Kirk True  
          50.
          Add tests to verify OffsetFetcherUtils.getOffsetResetTimestamp() Sub-task Open Unassigned  
          51.
          Create background thread skeleton for new Consumer threading model Sub-task Resolved Philip Nee  
          52.
          Ensure the correct thread is executing the callbacks Sub-task Resolved Philip Nee  
          53.
          Extract reusable logic from OffsetsForLeaderEpochClient Sub-task Resolved Lianet Magrans  
          54.
          Implement EventHandler interface and DefaultEventHandler for Consumer Sub-task Resolved Philip Nee  
          55.
          Implement assign() and assignment() Sub-task Resolved Philip Nee  
          56.
          Implement resetPositions functionality in OffsetsRequestManager Sub-task Resolved Lianet Magrans  
          57.
          Implement validatePositions functionality for new KafkaConsumer Sub-task Resolved Lianet Magrans  
          58.
          Investigate ConsumerNetworkThreadTest's testResetPositionsProcessFailureIsIgnored Sub-task Open Philip Nee  
          59.
          Integration tests for AsyncConsumer simple consume case Sub-task Resolved Lianet Magrans  
          60.
          Integrate committed offsets logic when updating fetching positions Sub-task Resolved Lianet Magrans  
          61.
          Refactor Commit Logic Sub-task Resolved Philip Nee  
          62.
          Refactor coordinator code Sub-task Resolved Philip Nee  
          63.
          Revoke partitions on Consumer.close() Sub-task Open Philip Nee  
          64.
          Update last seen epoch during commit Sub-task Resolved Lucas Brutschy  
          65.
          Implement max poll interval Sub-task Open Unassigned  
          66.
          Restrict Consumer API to be used from one thread Sub-task Resolved Lucas Brutschy  
          67.
          Ensure consumer.poll() execute autocommit callback Sub-task Patch Available Lucas Brutschy  
          68.
          Refactor OffsetFetchRequestState Error handling to be more consistent with OffsetCommitRequestState Sub-task Open Unassigned  
          69.
          Should ConsumerNetworkThread wrap the exception and notify the polling thread? Sub-task Open Unassigned  
          70.
          Investigate autocommit retry logic Sub-task Open Unassigned  
          71.
          Autocommit during close consistently fails with exception in background thread Sub-task Resolved Philip Nee  
          72.
          Remove excessive use of spy in CosumerTestBuilder Sub-task Open Unassigned  

          Activity

            People

              pnee Philip Nee
              pnee Philip Nee
              Votes:
              0 Vote for this issue
              Watchers:
              4 Start watching this issue

              Dates

                Created:
                Updated:

                Time Tracking

                  Estimated:
                  Original Estimate - 528h
                  528h
                  Remaining:
                  Remaining Estimate - 528h
                  528h
                  Logged:
                  Time Spent - Not Specified
                  Not Specified