-
Notifications
You must be signed in to change notification settings - Fork 14.1k
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
KAFKA-16770; [2/2] Coalesce records into bigger batches #16215
Conversation
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Outdated
Show resolved
Hide resolved
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/GroupCoordinatorConfig.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Outdated
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Outdated
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Outdated
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Outdated
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Outdated
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Outdated
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Outdated
Show resolved
Hide resolved
@jeffkbkim @jolshan Thanks for your comments. I addressed them. I pushed the tests and a few fixes too. |
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Outdated
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Outdated
Show resolved
Hide resolved
...dinator/src/test/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntimeTest.java
Show resolved
Hide resolved
...dinator/src/test/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntimeTest.java
Show resolved
Hide resolved
...dinator/src/test/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntimeTest.java
Show resolved
Hide resolved
...dinator/src/test/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntimeTest.java
Outdated
Show resolved
Hide resolved
...dinator/src/test/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntimeTest.java
Outdated
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...dinator/src/test/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntimeTest.java
Show resolved
Hide resolved
...dinator/src/test/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntimeTest.java
Show resolved
Hide resolved
...dinator/src/test/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntimeTest.java
Show resolved
Hide resolved
...dinator/src/test/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntimeTest.java
Outdated
Show resolved
Hide resolved
...dinator/src/test/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntimeTest.java
Show resolved
Hide resolved
Looks pretty good. I was wondering -- do we want to test the behavior of when we commit before a batch is flushed? (Is this possible) |
@jolshan Committing offsets above the last written offset is not allowed so it is probably not necessary. We already have tests to validate the logic to commit offsets up to the last written offset. |
@dajac makes sense. I guess I just didn't know if there was a case where we had one in a batch so it wasn't written to the log yet, but we tried to commit offsets. Thanks for clarifying. |
...coordinator/src/main/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntime.java
Show resolved
Hide resolved
...dinator/src/test/java/org/apache/kafka/coordinator/group/runtime/CoordinatorRuntimeTest.java
Show resolved
Hide resolved
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
thanks David!
This patch is the continuation of #15964. It introduces the records coalescing to the CoordinatorRuntime. It also introduces a new configuration `group.coordinator.append.linger.ms` which allows administrators to chose the linger time or disable it with zero. The new configuration defaults to 10ms. Reviewers: Jeff Kim <[email protected]>, Justine Olshan <[email protected]>
Merged to trunk and to 3.8. |
This patch is the continuation of apache#15964. It introduces the records coalescing to the CoordinatorRuntime. It also introduces a new configuration `group.coordinator.append.linger.ms` which allows administrators to chose the linger time or disable it with zero. The new configuration defaults to 10ms. Reviewers: Jeff Kim <[email protected]>, Justine Olshan <[email protected]>
commit 9368ef8 Author: Gantigmaa Selenge <[email protected]> Date: Wed Jun 12 16:04:24 2024 +0100 KAFKA-16865: Add IncludeTopicAuthorizedOperations option for DescribeTopicPartitionsRequest (apache#16136) Reviewers: Mickael Maison <[email protected]>, Chia-Ping Tsai <[email protected]>, Calvin Liu <[email protected]>, Andrew Schofield <[email protected]>, Apoorv Mittal <[email protected]> commit 46eb081 Author: gongxuanzhang <[email protected]> Date: Wed Jun 12 22:23:39 2024 +0800 KAFKA-10787 Apply spotless to log4j-appender, trogdor, jmh-benchmarks, examples, shell and generator (apache#16296) Reviewers: Chia-Ping Tsai <[email protected]> commit 79b9c44 Author: gongxuanzhang <[email protected]> Date: Wed Jun 12 22:19:47 2024 +0800 KAFKA-10787 Apply spotless to connect module (apache#16299) Reviewers: Chia-Ping Tsai <[email protected]> commit b5fb654 Author: Abhijeet Kumar <[email protected]> Date: Wed Jun 12 19:47:46 2024 +0530 KAFKA-15265: Dynamic broker configs for remote fetch/copy quotas (apache#16078) Reviewers: Kamal Chandraprakash<[email protected]>, Satish Duggana <[email protected]> commit faee6a4 Author: Dmitry Werner <[email protected]> Date: Wed Jun 12 15:44:11 2024 +0500 MINOR: Use predetermined dir IDs in ReplicationQuotasTest Use predetermined directory IDs instead of Uuid.randomUuid() in ReplicationQuotasTest. Reviewers: Igor Soarez <[email protected]> commit 638844f Author: David Jacot <[email protected]> Date: Wed Jun 12 08:29:50 2024 +0200 KAFKA-16770; [2/2] Coalesce records into bigger batches (apache#16215) This patch is the continuation of apache#15964. It introduces the records coalescing to the CoordinatorRuntime. It also introduces a new configuration `group.coordinator.append.linger.ms` which allows administrators to chose the linger time or disable it with zero. The new configuration defaults to 10ms. Reviewers: Jeff Kim <[email protected]>, Justine Olshan <[email protected]> commit 39ffdea Author: Bruno Cadonna <[email protected]> Date: Wed Jun 12 07:51:38 2024 +0200 KAFKA-10199: Enable state updater by default (apache#16107) We have already enabled the state updater by default once. However, we ran into issues that forced us to disable it again. We think that we fixed those issues. So we want to enable the state updater again by default. Reviewers: Lucas Brutschy <[email protected]>, Matthias J. Sax <[email protected]> commit 0782232 Author: Antoine Pourchet <[email protected]> Date: Tue Jun 11 22:31:43 2024 -0600 KAFKA-15045: (KIP-924 pt. 22) Add RackAwareOptimizationParams and other minor TaskAssignmentUtils changes (apache#16294) We now provide a way to more easily customize the rack aware optimizations that we provide by way of a configuration class called RackAwareOptimizationParams. We also simplified the APIs for the optimizeXYZ utility functions since they were mutating the inputs anyway. Reviewers: Anna Sophie Blee-Goldman <[email protected]> commit 226ac5e Author: Murali Basani <[email protected]> Date: Wed Jun 12 05:38:50 2024 +0200 KAFKA-16922 Adding unit tests for NewTopic (apache#16255) Reviewers: Chia-Ping Tsai <[email protected]> commit 23fe71d Author: Abhijeet Kumar <[email protected]> Date: Wed Jun 12 06:27:02 2024 +0530 KAFKA-15265: Integrate RLMQuotaManager for throttling copies to remote storage (apache#15820) - Added the integration of the quota manager to throttle copy requests to the remote storage. Reference KIP-956 - Added unit-tests for the copy throttling logic. Reviewers: Satish Duggana <[email protected]>, Luke Chen <[email protected]>, Kamal Chandraprakash<[email protected]> commit 2fa2c72 Author: Chris Egerton <[email protected]> Date: Tue Jun 11 23:15:07 2024 +0200 MINOR: Wait for embedded clusters to start before using them in Connect OffsetsApiIntegrationTest (apache#16286) Reviewers: Greg Harris <[email protected]>
This patch is the continuation of #15964. It introduces the records coalescing to the CoordinatorRuntime. It also introduces a new configuration
group.coordinator.append.linger.ms
which allows administrators to chose the linger time or disable it with zero. The new configuration defaults to 10ms.I am still working on the tests so I keep the PR as a draft for now. However, all the existing tests that uses no linger are all passing. I will push the new tests asap.
Committer Checklist (excluded from commit message)