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


##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/AbstractMembershipManager.java:
##########
@@ -1135,7 +1134,9 @@ CompletableFuture<Void> 
revokePartitions(Set<TopicPartition> partitionsToRevoke)
         // Ensure the set of partitions to revoke are still assigned
         Set<TopicPartition> revokedPartitions = new 
HashSet<>(partitionsToRevoke);
         revokedPartitions.retainAll(subscriptions.assignedPartitions());
-        log.info("Revoking previously assigned partitions {}", 
revokedPartitions.stream().map(TopicPartition::toString).collect(Collectors.joining(",
 ")));
+
+        if (log.isInfoEnabled())
+            log.info("Revoking previously assigned partitions {}", 
Utils.topicPartitionString(revokedPartitions));

Review Comment:
   Yep. Changed.



##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerNetworkThread.java:
##########
@@ -362,12 +368,18 @@ void cleanup() {
      * If there is a metadata error, complete all uncompleted events that 
require subscription metadata.
      */
     private void maybeFailOnMetadataError(List<CompletableEvent<?>> events) {
-        List<? extends CompletableApplicationEvent<?>> 
subscriptionMetadataEvent = events.stream()
-                .filter(e -> e instanceof CompletableApplicationEvent<?>)
-                .map(e -> (CompletableApplicationEvent<?>) e)
-                
.filter(CompletableApplicationEvent::requireSubscriptionMetadata)
-                .collect(Collectors.toList());
-        
+        List<CompletableApplicationEvent<?>> subscriptionMetadataEvent = new 
ArrayList<>();
+
+        for (CompletableEvent<?> ce : events) {
+            if (!(ce instanceof CompletableApplicationEvent))
+                continue;
+
+            CompletableApplicationEvent<?> cae = 
(CompletableApplicationEvent<?>) ce;
+
+            if (cae.requireSubscriptionMetadata())
+                subscriptionMetadataEvent.add(cae);

Review Comment:
   Done.



##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerRebalanceListenerInvoker.java:
##########
@@ -77,11 +81,14 @@ public Exception invokePartitionsAssigned(final 
SortedSet<TopicPartition> assign
     }
 
     public Exception invokePartitionsRevoked(final SortedSet<TopicPartition> 
revokedPartitions) {
-        log.info("Revoke previously assigned partitions {}", 
revokedPartitions.stream().map(TopicPartition::toString).collect(Collectors.joining(",
 ")));
+        if (log.isInfoEnabled())
+            log.info("Revoke previously assigned partitions {}", 
Utils.topicPartitionString(revokedPartitions));
+
         Set<TopicPartition> revokePausedPartitions = 
subscriptions.pausedPartitions();
         revokePausedPartitions.retainAll(revokedPartitions);
-        if (!revokePausedPartitions.isEmpty())
-                log.info("The pause flag in partitions [{}] will be removed 
due to revocation.", 
revokePausedPartitions.stream().map(TopicPartition::toString).collect(Collectors.joining(",
 ")));
+
+        if (!revokePausedPartitions.isEmpty() && log.isInfoEnabled())
+            log.info("The pause flag in partitions [{}] will be removed due to 
revocation.", Utils.topicPartitionString(revokePausedPartitions));

Review Comment:
   Done.



-- 
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