Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

Refactor integration and system tests to remove Zookeeper, add group protocol parameter #17581

Closed
Closed
Show file tree
Hide file tree
Changes from 6 commits
Commits
Show all changes
28 commits
Select commit Hold shift + click to select a range
965ff03
Upgraded ConsumerWithLegacyMessageFormatIntegrationTest
kirktrue Oct 22, 2024
753626a
Update LogAppendTimeTest.scala
kirktrue Oct 22, 2024
270fde8
Update FetchFromFollowerIntegrationTest.scala
kirktrue Oct 22, 2024
4277b23
Update RemoteTopicCrudTest.scala
kirktrue Oct 22, 2024
61fec39
Update ListOffsetsIntegrationTest.scala
kirktrue Oct 22, 2024
cb06a79
Updating BaseProducerSendTest and its subclass PlaintextProducerSendTest
kirktrue Oct 22, 2024
9a46023
Merge branch 'trunk' into kirktrue-remove-zookeeper-tests-part-1
kirktrue Oct 23, 2024
18d46e8
Updates to reduce boilerplate
kirktrue Oct 23, 2024
1f6b564
Removed more uses of "zk" as a quorum test parameter
kirktrue Oct 23, 2024
c1df673
More clean up to remove ZK tests
kirktrue Oct 23, 2024
322aeb9
Updates for more tests
kirktrue Oct 24, 2024
b52488c
Merge branch 'trunk' into kirktrue-remove-zookeeper-tests-part-1
kirktrue Oct 24, 2024
eabc67c
Merge branch 'trunk' into kirktrue-remove-zookeeper-tests-part-1
kirktrue Oct 24, 2024
c789dfb
The Great Migration
kirktrue Oct 25, 2024
36535c6
Fixed whitespace change
kirktrue Oct 25, 2024
6400957
Reverted removal of @nowarn
kirktrue Oct 25, 2024
7f0e4a1
Temporarily added getTestQuorumAndGroupProtocolParametersClassicGroup…
kirktrue Oct 25, 2024
e3e2c8c
Merge branch 'trunk' into kirktrue-remove-zookeeper-tests-part-1
kirktrue Oct 25, 2024
a7dd66c
Merge branch 'trunk' into kirktrue-remove-zookeeper-tests-part-1
kirktrue Oct 28, 2024
999c528
Support for changing the group.protocol from the default and removing…
kirktrue Oct 29, 2024
1252f08
Merge branch 'trunk' into kirktrue-remove-zookeeper-tests-part-1
kirktrue Oct 29, 2024
28248e9
WIP system test updates
kirktrue Oct 29, 2024
c02835e
Whitespace and other minor-ish changes
kirktrue Oct 29, 2024
e0900f5
Reverting tests to allow them to fail
kirktrue Oct 29, 2024
8d80b63
Hopefully that's obvious enough :)
kirktrue Oct 30, 2024
a113f6b
Merge branch 'trunk' into kirktrue-remove-zookeeper-tests-part-1
kirktrue Oct 31, 2024
46c6f4a
Merge branch 'trunk' into kirktrue-remove-zookeeper-tests-part-1
kirktrue Nov 1, 2024
f03f28c
Merge branch 'trunk' into kirktrue-remove-zookeeper-tests-part-1
kirktrue Nov 2, 2024
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -66,7 +66,7 @@ class ListOffsetsIntegrationTest extends KafkaServerTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testListMaxTimestampWithEmptyLog(quorum: String): Unit = {
Copy link
Collaborator

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Isn't the proper change here to remove the quorum parameter entirely?

Copy link
Collaborator Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I'd wondered that and wanted someone to raise the question.

I think it makes sense to remove it.

Copy link
Collaborator

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Happy to oblige :)

Copy link
Collaborator Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I have a question out to @cmccabe regarding his preference.

Copy link
Collaborator Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

TestInfoUtils uses the parameters to determine if it isKraft(), isZkMigrationTest(), etc. Usages of those methods would need to be updated too.

Copy link
Collaborator

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yes, I'm sure, but wouldn't AK 4.0 have all of the ZK migration tests removed, and wouldn't all tests be ZK?

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

There are some shenanigans done in the base test classes regarding the quorum param. Until we remove the ZK bits from that test infra, I think we actually need the quorum param :-/

Copy link
Collaborator Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I approve of the use of "shenanigans" to describe this 👍

val maxTimestampOffset = runFetchOffsets(adminClient, OffsetSpec.maxTimestamp(), topicName)
assertEquals(ListOffsetsResponse.UNKNOWN_OFFSET, maxTimestampOffset.offset())
Expand All @@ -88,7 +88,7 @@ class ListOffsetsIntegrationTest extends KafkaServerTestHarness {


@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testThreeCompressedRecordsInOneBatch(quorum: String): Unit = {
produceMessagesInOneBatch("gzip")
verifyListOffsets()
Expand All @@ -102,7 +102,7 @@ class ListOffsetsIntegrationTest extends KafkaServerTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testThreeNonCompressedRecordsInOneBatch(quorum: String): Unit = {
produceMessagesInOneBatch()
verifyListOffsets()
Expand All @@ -117,7 +117,7 @@ class ListOffsetsIntegrationTest extends KafkaServerTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testThreeNonCompressedRecordsInSeparateBatch(quorum: String): Unit = {
produceMessagesInSeparateBatch()
verifyListOffsets()
Expand Down Expand Up @@ -162,7 +162,7 @@ class ListOffsetsIntegrationTest extends KafkaServerTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testThreeRecordsInOneBatchHavingDifferentCompressionTypeWithServer(quorum: String): Unit = {
val props: Properties = new Properties()
props.setProperty(TopicConfig.COMPRESSION_TYPE_CONFIG, "lz4")
Expand All @@ -172,7 +172,7 @@ class ListOffsetsIntegrationTest extends KafkaServerTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testThreeRecordsInSeparateBatchHavingDifferentCompressionTypeWithServer(quorum: String): Unit = {
val props: Properties = new Properties()
props.setProperty(TopicConfig.COMPRESSION_TYPE_CONFIG, "lz4")
Expand All @@ -182,7 +182,7 @@ class ListOffsetsIntegrationTest extends KafkaServerTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testThreeCompressedRecordsInSeparateBatch(quorum: String): Unit = {
produceMessagesInSeparateBatch("gzip")
verifyListOffsets()
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -74,7 +74,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testCreateRemoteTopicWithValidRetentionTime(quorum: String): Unit = {
val topicConfig = new Properties()
topicConfig.put(TopicConfig.REMOTE_LOG_STORAGE_ENABLE_CONFIG, "true")
Expand All @@ -86,7 +86,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testCreateRemoteTopicWithValidRetentionSize(quorum: String): Unit = {
val topicConfig = new Properties()
topicConfig.put(TopicConfig.REMOTE_LOG_STORAGE_ENABLE_CONFIG, "true")
Expand All @@ -98,7 +98,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testCreateRemoteTopicWithInheritedLocalRetentionTime(quorum: String): Unit = {
// inherited local retention ms is 1000
val topicConfig = new Properties()
Expand All @@ -110,7 +110,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testCreateRemoteTopicWithInheritedLocalRetentionSize(quorum: String): Unit = {
// inherited local retention bytes is 1024
val topicConfig = new Properties()
Expand All @@ -122,7 +122,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testCreateRemoteTopicWithInvalidRetentionTime(quorum: String): Unit = {
// inherited local retention ms is 1000
val topicConfig = new Properties()
Expand All @@ -134,7 +134,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testCreateRemoteTopicWithInvalidRetentionSize(quorum: String): Unit = {
// inherited local retention bytes is 1024
val topicConfig = new Properties()
Expand All @@ -146,7 +146,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testCreateCompactedRemoteStorage(quorum: String): Unit = {
val topicConfig = new Properties()
topicConfig.put(TopicConfig.REMOTE_LOG_STORAGE_ENABLE_CONFIG, "true")
Expand Down Expand Up @@ -301,7 +301,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testEnableRemoteLogOnExistingTopicTest(quorum: String): Unit = {
val admin = createAdminClient()
val topicConfig = new Properties()
Expand All @@ -319,7 +319,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testEnableRemoteLogWhenSystemRemoteStorageIsDisabled(quorum: String): Unit = {
val admin = createAdminClient()

Expand All @@ -343,7 +343,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testUpdateTopicConfigWithValidRetentionTimeTest(quorum: String): Unit = {
val admin = createAdminClient()
val topicConfig = new Properties()
Expand All @@ -364,7 +364,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testUpdateTopicConfigWithValidRetentionSizeTest(quorum: String): Unit = {
val admin = createAdminClient()
val topicConfig = new Properties()
Expand All @@ -385,7 +385,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testUpdateTopicConfigWithInheritedLocalRetentionTime(quorum: String): Unit = {
val admin = createAdminClient()
val topicConfig = new Properties()
Expand All @@ -405,7 +405,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testUpdateTopicConfigWithInheritedLocalRetentionSize(quorum: String): Unit = {
val admin = createAdminClient()
val topicConfig = new Properties()
Expand Down Expand Up @@ -474,7 +474,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testTopicDeletion(quorum: String): Unit = {
MyRemoteStorageManager.deleteSegmentEventCounter.set(0)
val numPartitions = 2
Expand All @@ -493,7 +493,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testClusterWideDisablementOfTieredStorageWithEnabledTieredTopic(quorum: String): Unit = {
val topicConfig = new Properties()
topicConfig.setProperty(TopicConfig.REMOTE_LOG_STORAGE_ENABLE_CONFIG, "true")
Expand All @@ -515,7 +515,7 @@ class RemoteTopicCrudTest extends IntegrationTestHarness {
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
@ValueSource(strings = Array("kraft"))
def testClusterWithoutTieredStorageStartsSuccessfullyIfTopicWithTieringDisabled(quorum: String): Unit = {
val topicConfig = new Properties()
topicConfig.setProperty(TopicConfig.REMOTE_LOG_STORAGE_ENABLE_CONFIG, false.toString)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,12 +19,12 @@ package kafka.api

import java.time.Duration
import java.nio.charset.StandardCharsets
import java.util.{Collections, Properties}
import java.util.{Collections, Properties, stream}
import java.util.concurrent.TimeUnit
import kafka.integration.KafkaServerTestHarness
import kafka.security.JaasTestUtils
import kafka.server.KafkaConfig
import kafka.utils.TestUtils
import kafka.utils.{TestInfoUtils, TestUtils}
import org.apache.kafka.clients.admin.{Admin, NewPartitions}
import org.apache.kafka.clients.consumer.Consumer
import org.apache.kafka.clients.producer._
Expand All @@ -38,7 +38,7 @@ import org.apache.kafka.server.config.ServerLogConfigs
import org.junit.jupiter.api.Assertions._
import org.junit.jupiter.api.{AfterEach, BeforeEach, TestInfo}
import org.junit.jupiter.params.ParameterizedTest
import org.junit.jupiter.params.provider.ValueSource
import org.junit.jupiter.params.provider.{Arguments, MethodSource}

import scala.collection.mutable
import scala.concurrent.ExecutionException
Expand Down Expand Up @@ -126,9 +126,9 @@ abstract class BaseProducerSendTest extends KafkaServerTestHarness {
* 1. Send with null key/value/partition-id should be accepted; send with null topic should be rejected.
* 2. Last message of the non-blocking send should return the correct offset metadata
*/
@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
def testSendOffset(quorum: String): Unit = {
@ParameterizedTest(name = TestInfoUtils.TestWithParameterizedQuorumAndGroupProtocolNames)
@MethodSource(Array("getTestQuorumAndGroupProtocolParametersAll"))
def testSendOffset(quorum: String, groupProtocol: String): Unit = {
val producer = createProducer()
val partition = 0

Expand Down Expand Up @@ -188,19 +188,19 @@ abstract class BaseProducerSendTest extends KafkaServerTestHarness {
}
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
def testSendCompressedMessageWithCreateTime(quorum: String): Unit = {
@ParameterizedTest(name = TestInfoUtils.TestWithParameterizedQuorumAndGroupProtocolNames)
@MethodSource(Array("getTestQuorumAndGroupProtocolParametersAll"))
def testSendCompressedMessageWithCreateTime(quorum: String, groupProtocol: String): Unit = {
val producer = createProducer(
compressionType = "gzip",
lingerMs = Int.MaxValue,
deliveryTimeoutMs = Int.MaxValue)
sendAndVerifyTimestamp(producer, TimestampType.CREATE_TIME)
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
def testSendNonCompressedMessageWithCreateTime(quorum: String): Unit = {
@ParameterizedTest(name = TestInfoUtils.TestWithParameterizedQuorumAndGroupProtocolNames)
@MethodSource(Array("getTestQuorumAndGroupProtocolParametersAll"))
def testSendNonCompressedMessageWithCreateTime(quorum: String, groupProtocol: String): Unit = {
val producer = createProducer(lingerMs = Int.MaxValue, deliveryTimeoutMs = Int.MaxValue)
sendAndVerifyTimestamp(producer, TimestampType.CREATE_TIME)
}
Expand Down Expand Up @@ -291,9 +291,9 @@ abstract class BaseProducerSendTest extends KafkaServerTestHarness {
*
* After close() returns, all messages should be sent with correct returned offset metadata
*/
@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
def testClose(quorum: String): Unit = {
@ParameterizedTest(name = TestInfoUtils.TestWithParameterizedQuorumAndGroupProtocolNames)
@MethodSource(Array("getTestQuorumAndGroupProtocolParametersAll"))
def testClose(quorum: String, groupProtocol: String): Unit = {
val producer = createProducer()

try {
Expand Down Expand Up @@ -325,9 +325,9 @@ abstract class BaseProducerSendTest extends KafkaServerTestHarness {
*
* The specified partition-id should be respected
*/
@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
def testSendToPartition(quorum: String): Unit = {
@ParameterizedTest(name = TestInfoUtils.TestWithParameterizedQuorumAndGroupProtocolNames)
@MethodSource(Array("getTestQuorumAndGroupProtocolParametersAll"))
def testSendToPartition(quorum: String, groupProtocol: String): Unit = {
val producer = createProducer()

try {
Expand Down Expand Up @@ -365,9 +365,9 @@ abstract class BaseProducerSendTest extends KafkaServerTestHarness {
}
}

@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
def testSendToPartitionWithFollowerShutdownShouldNotTimeout(quorum: String): Unit = {
@ParameterizedTest(name = TestInfoUtils.TestWithParameterizedQuorumAndGroupProtocolNames)
@MethodSource(Array("getTestQuorumAndGroupProtocolParametersAll"))
def testSendToPartitionWithFollowerShutdownShouldNotTimeout(quorum: String, groupProtocol: String): Unit = {
// This test produces to a leader that has follower that is shutting down. It shows that
// the produce request succeed, do not timeout and do not need to be retried.
val producer = createProducer()
Expand Down Expand Up @@ -418,9 +418,9 @@ abstract class BaseProducerSendTest extends KafkaServerTestHarness {
* Producer will attempt to send messages to the partition specified in each record, and should
* succeed as long as the partition is included in the metadata.
*/
@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
def testSendBeforeAndAfterPartitionExpansion(quorum: String): Unit = {
@ParameterizedTest(name = TestInfoUtils.TestWithParameterizedQuorumAndGroupProtocolNames)
@MethodSource(Array("getTestQuorumAndGroupProtocolParametersAll"))
def testSendBeforeAndAfterPartitionExpansion(quorum: String, groupProtocol: String): Unit = {
val producer = createProducer(maxBlockMs = 5 * 1000L)

// create topic
Expand Down Expand Up @@ -476,9 +476,9 @@ abstract class BaseProducerSendTest extends KafkaServerTestHarness {
/**
* Test that flush immediately sends all accumulated requests.
*/
@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
def testFlush(quorum: String): Unit = {
@ParameterizedTest(name = TestInfoUtils.TestWithParameterizedQuorumAndGroupProtocolNames)
@MethodSource(Array("getTestQuorumAndGroupProtocolParametersAll"))
def testFlush(quorum: String, groupProtocol: String): Unit = {
val producer = createProducer(lingerMs = Int.MaxValue, deliveryTimeoutMs = Int.MaxValue)
try {
TestUtils.createTopicWithAdmin(admin, topic, brokers, controllerServers, 2, 2)
Expand All @@ -498,9 +498,9 @@ abstract class BaseProducerSendTest extends KafkaServerTestHarness {
/**
* Test close with zero timeout from caller thread
*/
@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
def testCloseWithZeroTimeoutFromCallerThread(quorum: String): Unit = {
@ParameterizedTest(name = TestInfoUtils.TestWithParameterizedQuorumAndGroupProtocolNames)
@MethodSource(Array("getTestQuorumAndGroupProtocolParametersAll"))
def testCloseWithZeroTimeoutFromCallerThread(quorum: String, groupProtocol: String): Unit = {
TestUtils.createTopicWithAdmin(admin, topic, brokers, controllerServers, 2, 2)
val partition = 0
consumer.assign(List(new TopicPartition(topic, partition)).asJava)
Expand All @@ -524,9 +524,9 @@ abstract class BaseProducerSendTest extends KafkaServerTestHarness {
/**
* Test close with zero and non-zero timeout from sender thread
*/
@ParameterizedTest
@ValueSource(strings = Array("zk", "kraft"))
def testCloseWithZeroTimeoutFromSenderThread(quorum: String): Unit = {
@ParameterizedTest(name = TestInfoUtils.TestWithParameterizedQuorumAndGroupProtocolNames)
@MethodSource(Array("getTestQuorumAndGroupProtocolParametersAll"))
def testCloseWithZeroTimeoutFromSenderThread(quorum: String, groupProtocol: String): Unit = {
TestUtils.createTopicWithAdmin(admin, topic, brokers, controllerServers, 1, 2)
val partition = 0
consumer.assign(List(new TopicPartition(topic, partition)).asJava)
Expand Down Expand Up @@ -565,3 +565,15 @@ abstract class BaseProducerSendTest extends KafkaServerTestHarness {
}

}

object BaseProducerSendTest {
// We want to test the following combinations:
// * KRaft and the classic group protocol
// * KRaft and the consumer group protocol
def getTestQuorumAndGroupProtocolParametersAll() : java.util.stream.Stream[Arguments] = {
stream.Stream.of(
Arguments.of("kraft", "classic"),
Arguments.of("kraft", "consumer")
)
}
}
Loading