Replace Docker-based Kafka with embedded KRaft broker for integration tests#17790
Merged
xiangfu0 merged 1 commit intoapache:masterfrom Mar 8, 2026
Merged
Replace Docker-based Kafka with embedded KRaft broker for integration tests#17790xiangfu0 merged 1 commit intoapache:masterfrom
xiangfu0 merged 1 commit intoapache:masterfrom
Conversation
Codecov Report✅ All modified and coverable lines are covered by tests. Additional details and impacted files@@ Coverage Diff @@
## master #17790 +/- ##
============================================
- Coverage 63.23% 63.18% -0.05%
Complexity 1456 1456
============================================
Files 3187 3188 +1
Lines 191620 191736 +116
Branches 29315 29347 +32
============================================
- Hits 121172 121156 -16
- Misses 60968 61092 +124
- Partials 9480 9488 +8
Flags with carried forward coverage won't be shown. Click here to find out more. ☔ View full report in Codecov by Sentry. 🚀 New features to boost your workflow:
|
ba33bd3 to
1fcba20
Compare
Contributor
There was a problem hiding this comment.
Pull request overview
This PR modernizes Pinot’s Kafka-backed integration testing by replacing Docker-managed Kafka startup with an in-process Kafka KRaft cluster (Kafka KafkaClusterTestKit), aiming to reduce CI flakiness and improve test startup time.
Changes:
- Introduces
EmbeddedKafkaCluster(KRaft, in-process) implementingStreamDataServerStartablefor integration tests. - Refactors Kafka lifecycle in
BaseClusterIntegrationTestand updates Kafka-based tests to use the embedded broker. - Updates Maven + GitHub Actions integration test partitioning to run
ExactlyOnceKafkaRealtimeClusterIntegrationTestin its own test set with required Kafka test dependencies.
Reviewed changes
Copilot reviewed 10 out of 10 changed files in this pull request and generated 3 comments.
Show a summary per file
| File | Description |
|---|---|
| pom.xml | Adds dependency management entries for Kafka test artifacts, Pinot Kafka test-jar, and JUnit Jupiter API needed by Kafka testkit. |
| pinot-plugins/pinot-stream-ingestion/pinot-kafka-3.0/src/test/java/org/apache/pinot/plugin/stream/kafka30/server/EmbeddedKafkaCluster.java | New embedded Kafka (KRaft) cluster wrapper for test usage. |
| pinot-plugins/pinot-stream-ingestion/pinot-kafka-3.0/src/test/java/org/apache/pinot/plugin/stream/kafka30/KafkaPartitionLevelConsumerTest.java | Switches test Kafka setup from Docker-based starter to embedded cluster. |
| pinot-plugins/pinot-stream-ingestion/pinot-kafka-3.0/pom.xml | Adds Kafka testkit + JUnit Jupiter API dependencies required to compile/run embedded broker tests. |
| pinot-integration-test-base/src/test/java/org/apache/pinot/integration/tests/BaseClusterIntegrationTest.java | Replaces Docker Kafka startup with embedded cluster and adjusts topic readiness/replication handling. |
| pinot-integration-test-base/pom.xml | Adds test dependencies on pinot-kafka-3.0 test-jar and Kafka test artifacts. |
| pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/PartialUpsertTableRebalanceIntegrationTest.java | Updates cleanup strategy (drop/recreate table + recreate topic) and adjusts lease extender executor handling after stopping servers. |
| pinot-integration-tests/pom.xml | Adds a new integration-tests-set-3 profile for ExactlyOnce test and pulls in embedded Kafka dependencies. |
| .github/workflows/scripts/pr-tests/.pinot_tests_integration.sh | Adds support for running integration test set 3. |
| .github/workflows/pinot_tests.yml | Expands integration test matrix to include test set 3. |
655816f to
d3f883b
Compare
c09cdc7 to
7ac2a59
Compare
… tests - Replace KafkaServerStartable (Docker CLI-based) with EmbeddedKafkaCluster using Kafka's KafkaClusterTestKit in KRaft mode for integration tests - Eliminate Docker dependency, improving startup speed and removing CI flakiness - Add essential embedded Kafka broker configs for transactions - Rewrite ExactlyOnce test to use single KafkaProducer for both abort and commit transactions, avoiding transaction marker race conditions - Fix PartialUpsertTableRebalanceIntegrationTest with retry logic for embedded Kafka topic metadata propagation delays Co-Authored-By: Claude Opus 4.6 <noreply@anthropic.com>
7ac2a59 to
a240775
Compare
Jackie-Jiang
approved these changes
Mar 6, 2026
This file contains hidden or bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Sign up for free
to join this conversation on GitHub.
Already have an account?
Sign in to comment
Add this suggestion to a batch that can be applied as a single commit.This suggestion is invalid because no changes were made to the code.Suggestions cannot be applied while the pull request is closed.Suggestions cannot be applied while viewing a subset of changes.Only one suggestion per line can be applied in a batch.Add this suggestion to a batch that can be applied as a single commit.Applying suggestions on deleted lines is not supported.You must change the existing code in this line in order to create a valid suggestion.Outdated suggestions cannot be applied.This suggestion has been applied or marked resolved.Suggestions cannot be applied from pending reviews.Suggestions cannot be applied on multi-line comments.Suggestions cannot be applied while the pull request is queued to merge.Suggestion cannot be applied right now. Please check back later.
Summary
KafkaServerStartable(Docker CLI-based) withEmbeddedKafkaClusterusing Kafka'sKafkaClusterTestKitin KRaft mode for integration testsBaseClusterIntegrationTestKafka lifecycle — no retry loop, no Docker container management, no port scanningChanges
EmbeddedKafkaCluster— in-process KRaft cluster wrapper implementingStreamDataServerStartable, supporting multi-broker clusters and Kafka transactionsBaseClusterIntegrationTest— rewrote Kafka start/stop to use embedded cluster; fixed replication factor calculation for multi-broker clusters; removed Docker-specific dead code (~170 lines)KafkaPartitionLevelConsumerTest— useEmbeddedKafkaClusterinstead of DockerExactlyOnceKafkaRealtimeClusterIntegrationTest— rewritten to use single KafkaProducer for both abort and commit transactions, avoiding transaction marker race conditions; added post-commit verification with System.err diagnosticsPartialUpsertTableRebalanceIntegrationTest— added retry logic for embedded Kafka topic metadata propagation delaysEmbeddedKafkaCluster— added essential broker configs for transactions (transaction.state.log.num.partitions=1,log.flush.interval.messages=1, etc.); useMetadataVersion.latestProduction()pom.xml(root, pinot-kafka-3.0, pinot-integration-test-base, pinot-integration-tests) — added kafka test-jar dependencies for embedded broker supportKafkaServerStartable(production Docker-based class for QuickStart) — not modifiedTest plan
KafkaPartitionLevelConsumerTest— 9 tests pass (single broker)LLCRealtimeClusterIntegrationTest— passes (2 brokers, non-transactional)ExactlyOnceKafkaRealtimeClusterIntegrationTest— passes (3 brokers, Kafka transactions withread_committed)PartialUpsertTableRebalanceIntegrationTest— passes with embedded Kafka🤖 Generated with Claude Code