mjsax commented on code in PR #20147:
URL: https://github.com/apache/kafka/pull/20147#discussion_r2198869297
##########
streams/integration-tests/src/test/java/org/apache/kafka/streams/integration/KafkaStreamsTelemetryIntegrationTest.java:
##########
@@ -279,18 +281,28 @@ public void shouldPassMetrics(final String topologyType,
final boolean stateUpda
// Streams metrics should get passed to Admin and Consumer
streamsApplicationProperties = props(stateUpdaterEnabled,
groupProtocol);
final Topology topology = topologyType.equals("simple") ?
simpleTopology(false) : complexTopology();
-
+
+ final AtomicInteger runningStateCount = new AtomicInteger(0);
+ final int expectedRunningStateCount =
DEFAULT_GROUP_PROTOCOL.equals(groupProtocol) || "simple".equals(topologyType) ?
1 : 2;
Review Comment:
It's not clear to my, why "simple" topologies show different behavior,
compare to "complex" ones.
When I run the code for "simple" I get the follow log from the consumer:
```[2025-07-10 14:28:52,035] INFO [Consumer
clientId=shouldPassMetricsBi3tKRB2SZOSy_YU2ch9rw-5921224d-0525-4866-9db4-71f9f95c99cd-StreamThread-1-consumer,
groupId=shouldPassMetricsBi3tKRB2SZOSy_YU2ch9rw] Assigned tasks with local
epoch 0
Member: dL_1PoEdQ4C9BhYUVUzFUQ
Assigned active tasks: [0_0]
Owned active tasks: []
Active tasks to revoke: []
Assigned standby tasks: []
Owned standby tasks: []
Assigned warm-up tasks: []
Owned warm-up tasks: []
(org.apache.kafka.clients.consumer.internals.StreamsMembershipManager:1029)
```
As you can see, the task is assigned and we only get a single rebalance.
However, when the test executed for "complex" topology, the consumer first
logs:
```
[2025-07-10 14:29:23,384] INFO [Consumer
clientId=shouldPassMetricsx1__GHUPQvS0AkXE48xleQ-3f598d1a-1f8a-4923-9332-0d947bb38ab2-StreamThread-1-consumer,
groupId=shouldPassMetricsx1__GHUPQvS0AkXE48xleQ] Assigned tasks with local
epoch 0
Member: mcJBdaR5TF2QjDoMvLXKBw
Assigned active tasks: []
Owned active tasks: []
Active tasks to revoke: []
Assigned standby tasks: []
Owned standby tasks: []
Assigned warm-up tasks: []
Owned warm-up tasks: []
(org.apache.kafka.clients.consumer.internals.StreamsMembershipManager:1029)
```
and there is no task assignment yet. Only a second rebalance will do the
assignment:
```
[2025-07-10 14:29:28,393] INFO [Consumer
clientId=shouldPassMetricsx1__GHUPQvS0AkXE48xleQ-3f598d1a-1f8a-4923-9332-0d947bb38ab2-StreamThread-1-consumer,
groupId=shouldPassMetricsx1__GHUPQvS0AkXE48xleQ] Assigned tasks with local
epoch 1
Member: mcJBdaR5TF2QjDoMvLXKBw
Assigned active tasks: [0_0, 0_1, 1_0, 1_1]
Owned active tasks: []
Active tasks to revoke: []
Assigned standby tasks: []
Owned standby tasks: []
Assigned warm-up tasks: []
Owned warm-up tasks: []
(org.apache.kafka.clients.consumer.internals.StreamsMembershipManager:1029)
```
Why do we get the assignment for the simple case in the first rebalance
already, but not for the second case? Is it because the complex topology needs
to create a repartition topic, and thus, does not even try to compute an
assignment on the first HB?
I am worried that there might be some race condition that is not covered by
this test fix.
--
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]