[ https://issues.apache.org/jira/browse/KAFKA-15305?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17850778#comment-17850778 ]
Lianet Magrans commented on KAFKA-15305: ---------------------------------------- Agree with this missing bit, but I wonder if we need a bit more than just "wait for pending requests", because I don't see that we'll ever generate one on the close situation. With KAFKA-16639 we ensure that HB request to leave is created, but that happens only IF the HB manager is polled when closing the consumer. On close, looks to me that may not happen, because [here|https://github.com/apache/kafka/blob/32b2b73f673ecd41d17c03e99db3746c517990c4/clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java#L1232] is where the close operation triggers the leave event (that will just leave the state machine indicating LEAVING, to be used by the HB manager next poll), but right after we shutdown the network thread [here|https://github.com/apache/kafka/blob/32b2b73f673ecd41d17c03e99db3746c517990c4/clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java#L1237], so not polling managers anymore. At this point, even with the inflight suggestion above, the networkClient.poll [here|https://github.com/apache/kafka/blob/cc269b0d438534ae8fef16b39354da1d78332a2c/clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerNetworkThread.java#L299] would return empty I expect. So seems we also need to make sure that the HB manager is actually polled when closing the consumer (the HBManager.pollOnClose you had suggested at some point). With that, at [this|https://github.com/apache/kafka/blob/cc269b0d438534ae8fef16b39354da1d78332a2c/clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerNetworkThread.java#L216-L217] point we would actually generate the HB to leave, add then we would hit the logic you mentioned above with a request to send (not empty), [here|https://github.com/apache/kafka/blob/cc269b0d438534ae8fef16b39354da1d78332a2c/clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerNetworkThread.java#L299]. Makes sense? > The background thread should try to process the remaining task until the > shutdown timer is expired > -------------------------------------------------------------------------------------------------- > > Key: KAFKA-15305 > URL: https://issues.apache.org/jira/browse/KAFKA-15305 > Project: Kafka > Issue Type: Bug > Components: clients, consumer > Reporter: Philip Nee > Assignee: Chia-Ping Tsai > Priority: Major > Labels: consumer-threading-refactor, timeout > Fix For: 3.8.0 > > > While working on https://issues.apache.org/jira/browse/KAFKA-15304 > close() API supplies a timeout parameter so that the consumer can have a > grace period to process things before shutting down. The background thread > currently doesn't do that, when close() is initiated, it will immediately > close all of its dependencies. > > This might not be desirable because there could be remaining tasks to be > processed before closing. Maybe the correct things to do is to first stop > accepting API request, second, let the runOnce() continue to run before the > shutdown timer expires, then we can force closing all of its dependencies. -- This message was sent by Atlassian Jira (v8.20.10#820010)