kirktrue commented on code in PR #17199:
URL: https://github.com/apache/kafka/pull/17199#discussion_r1849317079


##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java:
##########
@@ -213,10 +212,11 @@ private void process(final 
ConsumerRebalanceListenerCallbackNeededEvent event) {
     private final ApplicationEventHandler applicationEventHandler;
     private final Time time;
     private final AtomicReference<Optional<ConsumerGroupMetadata>> 
groupMetadata = new AtomicReference<>(Optional.empty());
-    private final KafkaConsumerMetrics kafkaConsumerMetrics;
+    private final KafkaAsyncConsumerMetrics kafkaAsyncConsumerMetrics;

Review Comment:
   `<bike-shedding>`I'm not staunchly against changing it, but I think keeping 
the variable as `kafkaConsumerMetrics` is fine.`</bike-shedding>`



##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/events/ApplicationEventHandler.java:
##########
@@ -39,29 +43,36 @@
  * An event handler that receives {@link ApplicationEvent application events} 
from the application thread which
  * are then readable from the {@link ApplicationEventProcessor} in the {@link 
ConsumerNetworkThread network thread}.
  */
+@SuppressWarnings("this-escape")
 public class ApplicationEventHandler implements Closeable {
 
     private final Logger log;
+    private final Time time;
     private final BlockingQueue<ApplicationEvent> applicationEventQueue;
     private final ConsumerNetworkThread networkThread;
     private final IdempotentCloser closer = new IdempotentCloser();
+    private final Optional<KafkaAsyncConsumerMetrics> 
kafkaAsyncConsumerMetrics;
 
     public ApplicationEventHandler(final LogContext logContext,
                                    final Time time,
                                    final BlockingQueue<ApplicationEvent> 
applicationEventQueue,
                                    final CompletableEventReaper 
applicationEventReaper,
                                    final Supplier<ApplicationEventProcessor> 
applicationEventProcessorSupplier,
                                    final Supplier<NetworkClientDelegate> 
networkClientDelegateSupplier,
-                                   final Supplier<RequestManagers> 
requestManagersSupplier) {
+                                   final Supplier<RequestManagers> 
requestManagersSupplier,
+                                   final Optional<KafkaAsyncConsumerMetrics> 
kafkaAsyncConsumerMetrics) {
         this.log = logContext.logger(ApplicationEventHandler.class);
+        this.time = time;
         this.applicationEventQueue = applicationEventQueue;
+        this.kafkaAsyncConsumerMetrics = kafkaAsyncConsumerMetrics;
         this.networkThread = new ConsumerNetworkThread(logContext,
                 time,
-                applicationEventQueue,
+                this,

Review Comment:
   I'm not crazy about this. (Haha, pun intended.)
   
   With this change there's now a bidirectional relationship between the event 
handler and the background thread. I'd hope we could find a way to maintain the 
decoupled nature of the event queue from the event handler 🤔



##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/NetworkClientDelegate.java:
##########
@@ -267,7 +275,9 @@ public void addAll(final List<UnsentRequest> requests) {
     public void add(final UnsentRequest r) {
         Objects.requireNonNull(r);
         r.setTimer(this.time, this.requestTimeoutMs);
+        r.setEnqueuedMs(this.time.milliseconds());

Review Comment:
   Is it OK to use `this.time.milliseconds()` here? Either way, I think we can 
call `time.milliseconds()` once and reuse it in the metrics update too, right?



##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java:
##########
@@ -1913,6 +1923,7 @@ private boolean processBackgroundEvents() {
                     log.warn("An error occurred when processing the background 
event: {}", e.getMessage(), e);
             }
         }
+        
kafkaAsyncConsumerMetrics.recordBackgroundEventQueueProcessingTime(time.milliseconds()
 - startMs);

Review Comment:
   Good point. It'll artificially skew the averages lower.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: jira-unsubscr...@kafka.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org

Reply via email to