This is an automated email from the ASF dual-hosted git repository.
ijuma pushed a commit to branch 2.5
in repository https://gitbox.apache.org/repos/asf/kafka.git
The following commit(s) were added to refs/heads/2.5 by this push:
new e9b58cf7 KAFKA-10158: Fix flaky
testDescribeUnderReplicatedPartitionsWhenReassignmentIsInProgress (#9022)
e9b58cf7 is described below
commit e9b58cf7e936be8011ccef6015be7f7399881394
Author: Brian Byrne <[email protected]>
AuthorDate: Sun Jul 26 08:28:47 2020 -0700
KAFKA-10158: Fix flaky
testDescribeUnderReplicatedPartitionsWhenReassignmentIsInProgress (#9022)
Set `replica.fetch.max.bytes` to `1` and produce multiple record batches to
allow
for throttling to take place. This helps avoid a race condition where the
reassignment would complete more quickly than expected causing an
assertion to fail some times.
Reviewers: Lucas Bradstreet <[email protected]>, Jason Gustafson
<[email protected]>, Chia-Ping Tsai <[email protected]>, Ismael Juma
<[email protected]>
---
.../kafka/admin/TopicCommandWithAdminClientTest.scala | 19 +++++++++++++++++--
1 file changed, 17 insertions(+), 2 deletions(-)
diff --git
a/core/src/test/scala/unit/kafka/admin/TopicCommandWithAdminClientTest.scala
b/core/src/test/scala/unit/kafka/admin/TopicCommandWithAdminClientTest.scala
index 87750f3..4b25af7 100644
--- a/core/src/test/scala/unit/kafka/admin/TopicCommandWithAdminClientTest.scala
+++ b/core/src/test/scala/unit/kafka/admin/TopicCommandWithAdminClientTest.scala
@@ -49,14 +49,20 @@ class TopicCommandWithAdminClientTest extends
KafkaServerTestHarness with Loggin
/**
* Implementations must override this method to return a set of
KafkaConfigs. This method will be invoked for every
* test and should not reuse previous configurations unless they select
their ports randomly when servers are started.
+ *
+ * Note the replica fetch max bytes is set to `1` in order to throttle the
rate of replication for test
+ * `testDescribeUnderReplicatedPartitionsWhenReassignmentIsInProgress`.
*/
override def generateConfigs: Seq[KafkaConfig] =
TestUtils.createBrokerConfigs(
numConfigs = 6,
zkConnect = zkConnect,
rackInfo = Map(0 -> "rack1", 1 -> "rack2", 2 -> "rack2", 3 -> "rack1", 4
-> "rack3", 5 -> "rack3"),
numPartitions = numPartitions,
- defaultReplicationFactor = defaultReplicationFactor
- ).map(KafkaConfig.fromProps)
+ defaultReplicationFactor = defaultReplicationFactor,
+ ).map { props =>
+ props.put(KafkaConfig.ReplicaFetchMaxBytesProp, "1")
+ KafkaConfig.fromProps(props)
+ }
private val numPartitions = 1
private val defaultReplicationFactor = 1.toShort
@@ -655,8 +661,13 @@ class TopicCommandWithAdminClientTest extends
KafkaServerTestHarness with Loggin
adminClient.createTopics(
Collections.singletonList(new NewTopic(testTopicName, partitions,
replicationFactor).configs(configMap))).all().get()
waitForTopicCreated(testTopicName)
+
+ // Produce multiple batches.
+ TestUtils.generateAndProduceMessages(servers, testTopicName, numMessages =
10, acks = -1)
TestUtils.generateAndProduceMessages(servers, testTopicName, numMessages =
10, acks = -1)
+ // Enable throttling. Note the broker config sets the replica max fetch
bytes to `1` upon to minimize replication
+ // throughput so the reassignment doesn't complete quickly.
val brokerIds = servers.map(_.config.brokerId)
TestUtils.setReplicationThrottleForPartitions(adminClient, brokerIds,
Set(tp), throttleBytes = 1)
@@ -686,6 +697,10 @@ class TopicCommandWithAdminClientTest extends
KafkaServerTestHarness with Loggin
topicService.describeTopic(new
TopicCommandOptions(Array("--under-replicated-partitions"))))
assertEquals("--under-replicated-partitions shouldn't return anything",
"", underReplicatedOutput)
+ // Verify reassignment is still ongoing.
+ val reassignments =
adminClient.listPartitionReassignments(Collections.singleton(tp)).reassignments.get().get(tp)
+ assertFalse(Option(reassignments).forall(_.addingReplicas.isEmpty))
+
TestUtils.removeReplicationThrottleForPartitions(adminClient, brokerIds,
Set(tp))
TestUtils.waitForAllReassignmentsToComplete(adminClient)
}