mjsax commented on code in PR #21110:
URL: https://github.com/apache/kafka/pull/21110#discussion_r2601142138


##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/StreamsMembershipManager.java:
##########
@@ -699,17 +710,42 @@ public void 
onHeartbeatSuccess(StreamsGroupHeartbeatResponse response) {
             processAssignmentReceived(
                 toTasksAssignment(activeTasks),
                 toTasksAssignment(standbyTasks),
-                toTasksAssignment(warmupTasks)
+                toTasksAssignment(warmupTasks),
+                isGroupReady
             );
-        } else {
-            if (responseData.activeTasks() != null ||
-                responseData.standbyTasks() != null ||
-                responseData.warmupTasks() != null) {
+        } else if (responseData.activeTasks() != null ||
+            responseData.standbyTasks() != null ||
+            responseData.warmupTasks() != null) {
+
+            throw new IllegalStateException("Invalid response data, task 
collections must be all null or all non-null: "
+                + responseData);
+        } else if (isGroupReady != targetAssignment.isGroupReady) {

Review Comment:
   Why `isGroupReady != targetAssignment.isGroupReady` -- could we just say 
`if(isGroupReady)` instead?
   
   If `isGroupReady == false` and `targetAssignment.isGroupReady == true` the 
current condition would say "true" but this seems incorrect?



##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/StreamsMembershipManager.java:
##########
@@ -699,17 +710,42 @@ public void 
onHeartbeatSuccess(StreamsGroupHeartbeatResponse response) {
             processAssignmentReceived(
                 toTasksAssignment(activeTasks),
                 toTasksAssignment(standbyTasks),
-                toTasksAssignment(warmupTasks)
+                toTasksAssignment(warmupTasks),
+                isGroupReady
             );
-        } else {
-            if (responseData.activeTasks() != null ||
-                responseData.standbyTasks() != null ||
-                responseData.warmupTasks() != null) {
+        } else if (responseData.activeTasks() != null ||
+            responseData.standbyTasks() != null ||
+            responseData.warmupTasks() != null) {
+
+            throw new IllegalStateException("Invalid response data, task 
collections must be all null or all non-null: "
+                + responseData);
+        } else if (isGroupReady != targetAssignment.isGroupReady) {
+            // If the client did not provide a new assignment, but the group 
is now ready, update the target
+            // assignment and reconcile it.
+            processAssignmentReceived(
+                targetAssignment.activeTasks,
+                targetAssignment.standbyTasks,
+                targetAssignment.warmupTasks,
+                isGroupReady
+            );
+        }
+    }
 
-                throw new IllegalStateException("Invalid response data, task 
collections must be all null or all non-null: "
-                    + responseData);
+    private boolean 
isGroupReady(List<StreamsGroupHeartbeatResponseData.Status> statuses) {
+        if (statuses != null) {
+            for (final StreamsGroupHeartbeatResponseData.Status status : 
statuses) {
+                switch 
(StreamsGroupHeartbeatResponse.Status.fromCode(status.statusCode())) {
+                    case MISSING_SOURCE_TOPICS:
+                    case MISSING_INTERNAL_TOPICS:
+                    case INCORRECTLY_PARTITIONED_TOPICS:
+                    case ASSIGNMENT_DELAYED:

Review Comment:
   Not sure form the top of my head if we handle any of these error code 
somewhere else too? Or if there is a difference to "classic" protocol?
   
   But in "classic" we treat `MISSNG_SOURCE_TOPIC` as fatal. Not sure what 
`MISSING_INTERNAL_TOPIC` exactly mean (is this a transient error, just saying, 
topic not created yet)? -- INCORRECTLY_PARTITION_TOPICS does also sound fatal?
   
   Sure for a fatal error the group is not ready either. Just double checking 
if this is all as intended.



##########
clients/src/main/java/org/apache/kafka/common/requests/StreamsGroupHeartbeatResponse.java:
##########
@@ -81,7 +81,19 @@ public enum Status {
         MISSING_SOURCE_TOPICS((byte) 1, "One or more source topics are missing 
or a source topic regex resolves to zero topics."),
         INCORRECTLY_PARTITIONED_TOPICS((byte) 2, "One or more topics expected 
to be copartitioned are not copartitioned."),
         MISSING_INTERNAL_TOPICS((byte) 3, "One or more internal topics are 
missing."),
-        SHUTDOWN_APPLICATION((byte) 4, "A client requested the shutdown of the 
whole application.");
+        SHUTDOWN_APPLICATION((byte) 4, "A client requested the shutdown of the 
whole application."),
+        ASSIGNMENT_DELAYED((byte) 5, "The assignment was delayed by the 
coordinator."),
+        UNKNOWN_STATUS((byte) 255, "Status unrecognized.");
+
+        private static final Map<Byte, Status> CODE_TO_STATUS;
+
+        static {
+            Map<Byte, Status> map = new java.util.HashMap<>();

Review Comment:
   Why fully qualified `java.util.HashMap` name?



##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/StreamsMembershipManager.java:
##########
@@ -699,17 +710,42 @@ public void 
onHeartbeatSuccess(StreamsGroupHeartbeatResponse response) {
             processAssignmentReceived(
                 toTasksAssignment(activeTasks),
                 toTasksAssignment(standbyTasks),
-                toTasksAssignment(warmupTasks)
+                toTasksAssignment(warmupTasks),
+                isGroupReady
             );
-        } else {
-            if (responseData.activeTasks() != null ||
-                responseData.standbyTasks() != null ||
-                responseData.warmupTasks() != null) {
+        } else if (responseData.activeTasks() != null ||
+            responseData.standbyTasks() != null ||
+            responseData.warmupTasks() != null) {

Review Comment:
   nit: indention



##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/StreamsMembershipManager.java:
##########
@@ -699,17 +710,42 @@ public void 
onHeartbeatSuccess(StreamsGroupHeartbeatResponse response) {
             processAssignmentReceived(
                 toTasksAssignment(activeTasks),
                 toTasksAssignment(standbyTasks),
-                toTasksAssignment(warmupTasks)
+                toTasksAssignment(warmupTasks),
+                isGroupReady
             );
-        } else {
-            if (responseData.activeTasks() != null ||
-                responseData.standbyTasks() != null ||
-                responseData.warmupTasks() != null) {
+        } else if (responseData.activeTasks() != null ||
+            responseData.standbyTasks() != null ||
+            responseData.warmupTasks() != null) {
+
+            throw new IllegalStateException("Invalid response data, task 
collections must be all null or all non-null: "
+                + responseData);
+        } else if (isGroupReady != targetAssignment.isGroupReady) {
+            // If the client did not provide a new assignment, but the group 
is now ready, update the target
+            // assignment and reconcile it.
+            processAssignmentReceived(
+                targetAssignment.activeTasks,
+                targetAssignment.standbyTasks,
+                targetAssignment.warmupTasks,
+                isGroupReady
+            );
+        }
+    }
 
-                throw new IllegalStateException("Invalid response data, task 
collections must be all null or all non-null: "
-                    + responseData);
+    private boolean 
isGroupReady(List<StreamsGroupHeartbeatResponseData.Status> statuses) {
+        if (statuses != null) {
+            for (final StreamsGroupHeartbeatResponseData.Status status : 
statuses) {
+                switch 
(StreamsGroupHeartbeatResponse.Status.fromCode(status.statusCode())) {
+                    case MISSING_SOURCE_TOPICS:
+                    case MISSING_INTERNAL_TOPICS:
+                    case INCORRECTLY_PARTITIONED_TOPICS:
+                    case ASSIGNMENT_DELAYED:
+                        return false;
+                    default:
+                        // continue checking other statuses
+                }
             }
         }
+        return true;

Review Comment:
   If this correct? I thought filed would only be `null` if they did not change 
compare to previous HB? But if we report an error status back, and the error 
does not change, we could go to "ready" incorrectly? Or is my understanding of 
the logic incorrect?



##########
clients/src/main/resources/common/message/StreamsGroupHeartbeatResponse.json:
##########
@@ -94,6 +94,7 @@
       //                                       The group coordinator will 
attempt to create all missing internal topics, if any errors occur during
       //                                       topic creation, this will be 
indicated in StatusDetail.
       //  4 - SHUTDOWN_APPLICATION           - A client requested the shutdown 
of the whole application.
+      //  5 - ASSIGNMENT_DELAYED             - No assignment was provided 
because assignment computation was delayed.

Review Comment:
   Why is `255` not added?



##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/StreamsMembershipManager.java:
##########
@@ -952,11 +988,14 @@ private void leaving() {
      * @param activeTasks Target active tasks assignment received from the 
broker.
      * @param standbyTasks Target standby tasks assignment received from the 
broker.
      * @param warmupTasks Target warm-up tasks assignment received from the 
broker.
+     * @param isGroupReady True if the group is ready, false otherwise.
      */
     private void processAssignmentReceived(Map<String, SortedSet<Integer>> 
activeTasks,
                                            Map<String, SortedSet<Integer>> 
standbyTasks,
-                                           Map<String, SortedSet<Integer>> 
warmupTasks) {
-        replaceTargetAssignmentWithNewAssignment(activeTasks, standbyTasks, 
warmupTasks);
+                                           Map<String, SortedSet<Integer>> 
warmupTasks,
+                                           boolean isGroupReady
+                                           ) {

Review Comment:
   nit: why new line for this?



##########
clients/src/main/java/org/apache/kafka/common/requests/StreamsGroupHeartbeatResponse.java:
##########
@@ -81,7 +81,19 @@ public enum Status {
         MISSING_SOURCE_TOPICS((byte) 1, "One or more source topics are missing 
or a source topic regex resolves to zero topics."),
         INCORRECTLY_PARTITIONED_TOPICS((byte) 2, "One or more topics expected 
to be copartitioned are not copartitioned."),
         MISSING_INTERNAL_TOPICS((byte) 3, "One or more internal topics are 
missing."),
-        SHUTDOWN_APPLICATION((byte) 4, "A client requested the shutdown of the 
whole application.");
+        SHUTDOWN_APPLICATION((byte) 4, "A client requested the shutdown of the 
whole application."),
+        ASSIGNMENT_DELAYED((byte) 5, "The assignment was delayed by the 
coordinator."),
+        UNKNOWN_STATUS((byte) 255, "Status unrecognized.");

Review Comment:
   Are there new statuses which we need to document on the KIP?



##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/StreamsMembershipManager.java:
##########
@@ -699,17 +710,42 @@ public void 
onHeartbeatSuccess(StreamsGroupHeartbeatResponse response) {
             processAssignmentReceived(
                 toTasksAssignment(activeTasks),
                 toTasksAssignment(standbyTasks),
-                toTasksAssignment(warmupTasks)
+                toTasksAssignment(warmupTasks),
+                isGroupReady
             );
-        } else {
-            if (responseData.activeTasks() != null ||
-                responseData.standbyTasks() != null ||
-                responseData.warmupTasks() != null) {
+        } else if (responseData.activeTasks() != null ||
+            responseData.standbyTasks() != null ||
+            responseData.warmupTasks() != null) {
+
+            throw new IllegalStateException("Invalid response data, task 
collections must be all null or all non-null: "
+                + responseData);
+        } else if (isGroupReady != targetAssignment.isGroupReady) {
+            // If the client did not provide a new assignment, but the group 
is now ready, update the target
+            // assignment and reconcile it.
+            processAssignmentReceived(
+                targetAssignment.activeTasks,
+                targetAssignment.standbyTasks,
+                targetAssignment.warmupTasks,
+                isGroupReady
+            );
+        }
+    }
 
-                throw new IllegalStateException("Invalid response data, task 
collections must be all null or all non-null: "
-                    + responseData);
+    private boolean 
isGroupReady(List<StreamsGroupHeartbeatResponseData.Status> statuses) {
+        if (statuses != null) {
+            for (final StreamsGroupHeartbeatResponseData.Status status : 
statuses) {
+                switch 
(StreamsGroupHeartbeatResponse.Status.fromCode(status.statusCode())) {
+                    case MISSING_SOURCE_TOPICS:
+                    case MISSING_INTERNAL_TOPICS:
+                    case INCORRECTLY_PARTITIONED_TOPICS:
+                    case ASSIGNMENT_DELAYED:
+                        return false;

Review Comment:
   Might be good to add this logging?



-- 
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: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]

Reply via email to