This is an automated email from the ASF dual-hosted git repository.

cadonna pushed a commit to branch 3.1
in repository https://gitbox.apache.org/repos/asf/kafka.git


The following commit(s) were added to refs/heads/3.1 by this push:
     new 836210a  MINOR: Fix system test 
StreamsCooperativeRebalanceUpgradeTest.test_upgrade_to_cooperative_rebalance 
(#11532)
836210a is described below

commit 836210ad7a11a946f6cc16ca6f99ebece2ed419b
Author: Bruno Cadonna <[email protected]>
AuthorDate: Thu Nov 25 10:48:09 2021 +0100

    MINOR: Fix system test 
StreamsCooperativeRebalanceUpgradeTest.test_upgrade_to_cooperative_rebalance 
(#11532)
    
    Log messages were changed in the AssignorConfiguration (#11490) that are
    also used for verification in system test
    
StreamsCooperativeRebalanceUpgradeTest.test_upgrade_to_cooperative_rebalance.
    
    This commit fixes the test and adds comments to the log messages
    that point to the test that needs to be updated in case of
    changes to the log messages.
    
    Reviewers: John Roesler <[email protected]>, Luke Chen 
<[email protected]>, David Jacot <[email protected]>
---
 .../processor/internals/assignment/AssignorConfiguration.java     | 8 ++++++++
 .../tests/streams/streams_cooperative_rebalance_upgrade_test.py   | 4 ++--
 2 files changed, 10 insertions(+), 2 deletions(-)

diff --git 
a/streams/src/main/java/org/apache/kafka/streams/processor/internals/assignment/AssignorConfiguration.java
 
b/streams/src/main/java/org/apache/kafka/streams/processor/internals/assignment/AssignorConfiguration.java
index a71c2fb..c14ab70 100644
--- 
a/streams/src/main/java/org/apache/kafka/streams/processor/internals/assignment/AssignorConfiguration.java
+++ 
b/streams/src/main/java/org/apache/kafka/streams/processor/internals/assignment/AssignorConfiguration.java
@@ -105,6 +105,10 @@ public final class AssignorConfiguration {
                 case StreamsConfig.UPGRADE_FROM_21:
                 case StreamsConfig.UPGRADE_FROM_22:
                 case StreamsConfig.UPGRADE_FROM_23:
+                    // ATTENTION: The following log messages is used for 
verification in system test
+                    // 
streams/streams_cooperative_rebalance_upgrade_test.py::StreamsCooperativeRebalanceUpgradeTest.test_upgrade_to_cooperative_rebalance
+                    // If you change it, please do also change the system test 
accordingly and
+                    // verify whether the test passes.
                     log.info("Eager rebalancing protocol is enabled now for 
upgrade from {}.x", upgradeFrom);
                     log.warn("The eager rebalancing protocol is deprecated and 
will stop being supported in a future release." +
                         " Please be prepared to remove the 'upgrade.from' 
config soon.");
@@ -113,6 +117,10 @@ public final class AssignorConfiguration {
                     throw new IllegalArgumentException("Unknown configuration 
value for parameter 'upgrade.from': " + upgradeFrom);
             }
         }
+        // ATTENTION: The following log messages is used for verification in 
system test
+        // 
streams/streams_cooperative_rebalance_upgrade_test.py::StreamsCooperativeRebalanceUpgradeTest.test_upgrade_to_cooperative_rebalance
+        // If you change it, please do also change the system test accordingly 
and
+        // verify whether the test passes.
         log.info("Cooperative rebalancing protocol is enabled now");
         return RebalanceProtocol.COOPERATIVE;
     }
diff --git 
a/tests/kafkatest/tests/streams/streams_cooperative_rebalance_upgrade_test.py 
b/tests/kafkatest/tests/streams/streams_cooperative_rebalance_upgrade_test.py
index 4658a53..b412583 100644
--- 
a/tests/kafkatest/tests/streams/streams_cooperative_rebalance_upgrade_test.py
+++ 
b/tests/kafkatest/tests/streams/streams_cooperative_rebalance_upgrade_test.py
@@ -39,8 +39,8 @@ class StreamsCooperativeRebalanceUpgradeTest(Test):
     processing_message = "Processed [0-9]* records so far"
     stopped_message = "COOPERATIVE-REBALANCE-TEST-CLIENT-CLOSED"
     running_state_msg = "STREAMS in a RUNNING State"
-    cooperative_turned_off_msg = "Eager rebalancing enabled now for upgrade 
from %s"
-    cooperative_enabled_msg = "Cooperative rebalancing enabled now"
+    cooperative_turned_off_msg = "Eager rebalancing protocol is enabled now 
for upgrade from %s"
+    cooperative_enabled_msg = "Cooperative rebalancing protocol is enabled now"
     first_bounce_phase = "first_bounce_phase-"
     second_bounce_phase = "second_bounce_phase-"
 

Reply via email to