As the PR title suggests, this PR is an attempt to perform some cleanups
in the server module. The changes are mostly around the use of Record
type for some classes, changes to use enhanced switch, etc.
Reviewers: Ken Huang <s7133700@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
# Description
`kafka-acls.sh` doesn't print message about duplicate authorization.
# Changes
Now the cli searches for existing AclBinding, prints duplicate bindings,
and removes them from the adding list.
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
This is a follow-up to #19449, which do the following things:
1. Add document to explain `schema.content` only work for sink connector
when `schemas.enable` set to true.
2. Handle the case that while jsonValue contains the `schema` and
`payload` fields, we should use the corresponding value.
Reviewers: Priyanka K U <priyanka.ku@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
**This upgrade includes:**
- Dependency configurations are now realized only when necessary, which
helps improve configuration performance and memory usage.
- The configuration cache improves build time by caching the result of
the configuration phase and reusing it for subsequent builds. This
feature can significantly improve build performance.
reference: [Gradle 8.14.3 Release
Notes](https://docs.gradle.org/8.14.3/release-notes.html#build-authoring-improvements)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
This PR removes two System.out.println(...) statements from
StreamsGraphTest. These outputs were left over from debugging and are
not needed in the test logic.
Reviewers: Ken Huang <s7133700@gmail.com>, TengYao Chi
<kitingiao@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
This PR aims at cleaning up the `raft` module further by getting rid of
some extra code which can be replaced by `record`
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
This PR aims at cleaning up the`connect:runtime` module further by
getting rid of some extra code which can be replaced by record and the
relevant changes.
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
With "merge.repartition.topic" optimization enabled, Kafka Streams tries
to push repartition topics upstream, to be able to merge multiple
repartition topics from different downstream branches together.
However, it is not safe to push a repartition topic if the parent node
is value-changing: because of potentially changing data types, the
topology might become invalid, and fail with serde error at runtime.
The optimization itself work correctly, however, processValues() is not
correctly declared as value-changing, what can lead to invalid
topologies.
Reviewers: Bill Bejeck <bill@confluent.io>, Lucas Brutschy
<lbrutschy@confluent.io>
We add the three main changes in this PR
- Disallowing null values for most LIST-type configurations makes sense,
since users cannot explicitly set a configuration to null in a
properties file. Therefore, only configurations with a default value of
null should be allowed to accept null.
- Disallowing duplicate values is reasonable, as there are currently no
known configurations in Kafka that require specifying the same value
multiple times. Allowing duplicates is both rare in practice and
potentially confusing to users.
- Disallowing empty list, even though many configurations currently
accept them. In practice, setting an empty list for several of these
configurations can lead to server startup failures or unexpected
behavior. Therefore, enforcing non-empty lists helps prevent
misconfiguration and improves system robustness.
These changes may introduce some backward incompatibility, but this
trade-off is justified by the significant improvements in safety,
consistency, and overall user experience.
Additionally, we introduce two minor adjustments:
- Reclassify some STRING-type configurations as LIST-type, particularly
those using comma-separated values to represent multiple entries. This
change reflects the actual semantics used in Kafka.
- Update the default values for some configurations to better align with
other configs.
These changes will not introduce any compatibility issues.
Reviewers: Jun Rao <junrao@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
Should be `headers.separator=<headers.separator>` instead of
`headers.separator=<line.separator>`
Reviewers: Kuan-Po Tseng <brandboat@gmail.com>, Ken Huang
<s7133700@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
Add to the consumer.close java doc to describe the error handling
behaviour.
Reviewers: Matthias J. Sax <matthias@confluent.io>, Chia-Ping Tsai
<chia7712@gmail.com>, Andrew Schofield <aschofield@confluent.io>,
TengYao Chi <frankvicky@apache.org>
This reverts commit d86ba7f54a.
Reverting since we are planning to change how KIP-966 is implemented. We
should revert this RPC until we have more clarity on how this KIP will
be executed.
Reviewers: José Armando García Sancio <jsancio@apache.org>
There’s a difference in the two consumers’ `pollForFetches()` methods in
this case: `ClassicKafkaConsumer` doesn't block waiting for data in the
fetch buffer, but `AsyncKafkaConsumer` does.
In `ClassicKafkaConsumer.pollForFetches()`, after enqueuing the `FETCH`
request, the consumer makes a call to `ConsumerNetworkClient.poll()`. In
most cases `poll()` returns almost immediately because it successfully
sent the `FETCH` request. So even when the `pollTimeout` value is, e.g.
3000, the call to `ConsumerNetworkClient.poll()` doesn't block that long
waiting for a response.
After sending out a `FETCH` request, `AsyncKafkaConsumer` then calls
`FetchBuffer.awaitNotEmpty()` and proceeds to block there for the full
length of the timeout. In some cases, the response to the `FETCH` comes
back with no results, which doesn't unblock
`FetchBuffer.awaitNotEmpty()`. So because the application thread is
still waiting for data in the buffer, it remains blocked, preventing any
more `FETCH` requests from being sent, causing the long pauses in the
console consumer.
Reviewers: Lianet Magrans <lmagrans@confluent.io>, Andrew Schofield
<aschofield@confluent.io>
1. Optimize the `equals()`, `hashCode()`, and `toString()` methods in
`OffsetAndMetadata`.
2. Add UT and IT to these modifications.
Reviewers: TengYao Chi <kitingiao@gmail.com>, Sean Quah
<squah@confluent.io>, Chia-Ping Tsai <chia7712@gmail.com>
The `kafka-share-groups.sh` tool checks whether a topic already has a
start-offset in the share group when resetting offsets. This is not
necessary. By removing the check, it is possible to set a start offset
for a topic which has not yet but will be subscribed in the future, thus
initialising the consumption point.
There is still a small piece of outstanding work to do with resetting
the offset for a non-existent group which should also create the group.
A subsequent PR will be used to address that.
Reviewers: Jimmy Wang <48462172+JimmyWang6@users.noreply.github.com>,
Lan Ding <isDing_L@163.com>, Apoorv Mittal <apoorvmittal10@gmail.com>
Ensure that metrics are retrieved and displayed (when requested) before
ShareConsumer.close() is called. This is important because metrics are
technically supposed to be removed on ShareConsumer.close(), which means
retrieving them after close() would yield an empty map.
Related to https://github.com/apache/kafka/pull/20267.
Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>
#20390 Replace the -`-producer.config` for the verifiable producer and
`--consumer.config` option by `--command-config` for the verifiable
consumer. However, for e2e tests targeting older broker versions, the
original configuration should still be used.
Fix the following tests:
`consumer_protocol_migration_test.py`、`compatibility_test_new_broker_test.py`
and `upgrade_test.py`.
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, Manikumar Reddy <manikumar.reddy@gmail.com>
Refactor help and version handling in command-line tools by replacing
duplicate code with `CommandLineUtils#maybePrintHelpOrVersion`.
Reviewers: TengYao Chi <kitingiao@gmail.com>, Ken Huang
<s7133700@gmail.com>, Jhen-Yung Hsu <jhenyunghsu@gmail.com>, Chia-Ping
Tsai <chia7712@gmail.com>
*What*
- Currently in `ShareConsumerImpl`, we were not resetting
`background-event-queue-size` metric to 0 after draining the events from
the queue.
- This PR fixes it by using `BackgroundEventHandler::drainEvents`
similar to `AsyncKafkaConsumer`.
- Added a unit test to verify the metric is reset to 0 after draining
the events.
Reviewers: Andrew Schofield <aschofield@confluent.io>, Chia-Ping Tsai
<chia7712@gmail.com>
This change adds:
- Integration test for `Admin#describeStreamsGroups` API
- Integration test for `Admin#deleteStreamsGroup` API
Reviewers: Alieh Saeedi <asaeedi@confluent.io>, Lucas Brutschy
<lucasbru@apache.org>
---------
Co-authored-by: Lucas Brutschy <lbrutschy@gmail.com>
* Log error message if `broker.heartbeat.interval.ms * 2` is large than
`broker.session.timeout.ms`.
* Add test case
`testLogBrokerHeartbeatIntervalMsShouldBeLowerThanHalfOfBrokerSessionTimeoutMs`.
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
Add tests for producer state listing with, without, and invalid
brokerId.
Reviewers: TengYao Chi <kitingiao@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
jira: https://issues.apache.org/jira/browse/KAFKA-19011 kip:
https://cwiki.apache.org/confluence/display/KAFKA/KIP-1172%3A+Improve+EndToEndLatency+tool
This PR improves the usability and maintainability of the
`kafka-e2e-latency.sh` tool:
- Replaces fixed-index argument parsing with a proper argument parser
(joptsimple)
- Adds support for configuring:
- -record-key-size: size of the message key
- -num-headers: number of headers per message
- -record-header-key-size: size of each header key
- -record-header-size: size of each header value
- Renames existing arguments to align with Kafka CLI conventions:
- broker_list → bootstrap-server
- num_messages → num-records
- message_size_bytes → record-size
- properties_file → command-config
-
Reviewers: Jhen-Yung Hsu <jhenyunghsu@gmail.com>, Ken Huang
<s7133700@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
The original implementation uses a linear search to find the least
loaded process in O(n), and we can replace this by look-ups in a heap is
O(log(n)), as described below
Active tasks: For active tasks, we can do exactly the same assignment as
in the original algorithm by first building a heap (by load) of all
processes. When we assign a task, we pick the head off the heap, assign
the task to it, update the load, and re-insert it into the heap in
O(log(n)).
Standby tasks: For standby tasks, we cannot do this optimization
directly, because of the order in which we assign tasks:
1. We first try to assign task A to a process that previously owned A.
2. If we did not find such a process, we assign A to the least loaded
node.
3. We now try to assign task B to a process that previously owned B
4. If we did not find such a process, we assign B to the least loaded
node
...
The problem is that we cannot efficiently keep a heap (by load)
throughout this process, because finding and removing process that
previously owned A (and B and…) in the heap is O(n). We therefore need
to change the order of evaluation to be able to use a heap:
1. Try to assign all tasks A, B.. to a process that previously owned the
task
2. Build a heap.
3. Assign all remaining tasks to the least-loaded process that does not
yet own the task. Since at most NumStandbyReplicas already own the task,
we can do it by removing up to NumStandbyReplicas from the top of the
heap in O(log(n)), so we get O(log(NumProcesses)*NumStandbyReplicas).
Note that the change in order changes the resulting standby assignments
(although this difference does not show up in the existing unit tests).
I would argue that the new order of assignment will actually yield
better assignments, since the assignment will be more sticky, which has
the potential to reduce the amount of store we have to restore from the
changelog topic after assingments.
In our worst-performing benchmark, this improves the runtime by ~107x.
Reviewers: Bill Bejeck<bbejeck@apache.org>
*What*
https://issues.apache.org/jira/browse/KAFKA-18220
- Currently, `AsyncConsumerMetrics` extends `KafkaConsumerMetrics`, but
is being used both by `AsyncKafkaConsumer` and `ShareConsumerImpl`.
- `ShareConsumerImpl` only needs the async consumer metrics(the metrics
associated with the new consumer threading model).
- This needs to be fixed, we are unnecessarily having
`KafkaConsumerMetrics` as a parent class for `ShareConsumer` metrics.
Fix :
- In this PR, we have removed the dependancy of `AsyncConsumerMetrics`
on `KafkaConsumerMetrics` and made it an independent class which both
`AsyncKafkaConsumer` and `ShareConsumerImpl` will use.
- The "`asyncConsumerMetrics`" field represents the metrics associated
with the new consumer threading model (like application event queue
size, background queue size, etc).
- The "`kafkaConsumerMetrics`" and "`kafkaShareConsumerMetrics`" fields
denote the actual consumer metrics for `KafkaConsumer` and
`KafkaShareConsumer` respectively.
Reviewers: Andrew Schofield <aschofield@confluent.io>
A few cleanups including Java 17 syntax, collections and assertEquals() order
Reviewers: Luke Chen <showuon@gmail.com>, Ken Huang <s7133700@gmail.com>, Jhen-Yung Hsu <jhenyunghsu@gmail.com>
Ensure that metrics are retrieved and displayed (when requested) before
`Consumer.close()` is called. This is important because metrics are
technically supposed to be removed on `Consumer.close()`, which means
retrieving them _after_ `close()` would yield an empty map.
Reviewers: Andrew Schofield <aschofield@confluent.io>
This is an attempt at improving the client configuration files. We now
have sections and comments similar to the other properties files.
Reviewers: Kirk True <ktrue@confluent.io>, Luke Chen <showuon@gmail.com>
---------
Signed-off-by: Federico Valeri <fedevaleri@gmail.com>
Add a lower bound to num.replica.fetchers.
Reviewers: PoAn Yang <payang@apache.org>, TaiJuWu <tjwu1217@gmail.com>,
Ken Huang <s7133700@gmail.com>, jimmy <wangzhiwang611@gmail.com>,
Jhen-Yung Hsu <jhenyunghsu@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
issue: [KAFKA-19616](https://issues.apache.org/jira/browse/KAFKA-19616)
**why**: validate log compaction works correctly with compressed data.
**what**: adds compression config options to `LogCompactionTester` tool
and extends test coverage to validate log compaction with different
compression types and levels.
Reviewers: TengYao Chi <kitingiao@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
In the 4.1 `upgrade-guide` describing the new KIP-1071 protocol it would
be helpful to display the configs you can set via `kafka-configs.sh`
with `streams` pre-pended to the configs, the command will fail
otherwise.
Reviewers: Andrew J Schofield<aschofield@apache.org>, Matthias J
Sax<mjsax@apache.org>, Genseric Ghiro<gghiro@confluent.io>
Cleanup default configs in
AutoTopicCreationManager#createStreamsInternalTopics. The streams
protocol would like to be consistent with the kafka streams using the
classic protocol - which would create the internal topics using
CreateTopic and therefore use the controller config.
Reviewers: Lucas Brutschy <lucasbru@apache.org>
### Summary
This PR fixes two critical issues related to producer batch splitting
that can cause infinite retry loops and stack overflow errors when batch
sizes are significantly larger than broker-configured message size
limits.
### Issues Addressed
- **KAFKA-8350**: Producers endlessly retry batch splitting when
`batch.size` is much larger than topic-level `message.max.bytes`,
leading to infinite retry loops with "MESSAGE_TOO_LARGE" errors
- **KAFKA-8202**: Stack overflow errors in
`FutureRecordMetadata.chain()` due to excessive recursive splitting
attempts
### Root Cause
The existing batch splitting logic in
`RecordAccumulator.splitAndReenqueue()` always used the configured
`batchSize` parameter for splitting, regardless of whether the batch had
already been split before. This caused:
1. **Infinite loops**: When `batch.size` (e.g., 8MB) >>
`message.max.bytes` (e.g., 1MB), splits would never succeed since the
split size was still too large
2. **Stack overflow**: Repeated splitting attempts created deep call
chains in the metadata chaining logic
### Solution
Implemented progressive batch splitting logic:
```java
int maxBatchSize = this.batchSize;
if (bigBatch.isSplitBatch()) {
maxBatchSize = Math.max(bigBatch.maxRecordSize,
bigBatch.estimatedSizeInBytes() / 2);
}
```
__Key improvements:__
- __First split__: Uses original `batchSize` (maintains backward
compatibility)
- __Subsequent splits__: Uses the larger of:
- `maxRecordSize`: Ensures we can always split down to individual
records - `estimatedSizeInBytes() / 2`: Provides geometric reduction
for faster convergence
### Testing
Added comprehensive test
`testSplitAndReenqueuePreventInfiniteRecursion()` that:
- Creates oversized batches with 100 records of 1KB each
- Verifies splitting can reduce batches to single-record size
- Ensures no infinite recursion (safety limit of 100 operations)
- Validates no data loss or duplication during splitting
- Confirms all original records are preserved with correct keys
### Backward Compatibility
- No breaking changes to public APIs
- First split attempt still uses original `batchSize` configuration
- Progressive splitting only engages for retry scenarios
Reviewers: Colin P. McCabe <cmccabe@apache.org>, Jason Gustafson
<jason@confluent.io>, Rajini Sivaram <rajinisivaram@googlemail.com>
###
---------
Co-authored-by: Michael Knox <mrknox@amazon.com>
These tests were written while finalizing approach for making inflight
state class thread safe but later approach changed and the lock is now
always required by SharePartition to change inflight state. Hence these
tests are incorrect and do not add any value.
Reviewers: Andrew Schofield <aschofield@confluent.io>
Improve the wording in the upgrade doc slightly. Also fix a tiny
annoyance in the output from the message generator.
Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>
In KAFKA-19570 we implemented offline migration between groups, that is,
the following integration test or system test should be possible:
Test A:
- Start a streams application with classic protocol, process up to a
certain offset and commit the offset and shut down. - Start the same
streams application with streams protocol (same app ID!). - Make sure
that the offsets before the one committed in the first run are not
reprocessed in the second run.
Test B:
- Start a streams application with streams protocol, process up to a
certain offset and commit the offset and shut down. - Start the same
streams application with classic protocol (same app ID!). - Make sure
that the offsets before the one committed in the first run are not
reprocessed in the second run.
We have unit tests that make sure that non-empty groups will not be
converted. This should be enough.
Reviewers: Bill Bejeck <bbejeck@apache.org>
Minor PR to update name of maxInFlightMessages to maxInFlightRecords to
maintain consistency in share partition related classes.
Reviewers: Andrew Schofield <aschofield@confluent.io>
This field was used for replica_id, but after
51c833e795,
the OffsetsForLeaderEpochRequest directly relies on the internal structs
generated by the automated protocol. Therefore, we can safely remove it.
Reviewers: Lan Ding <isDing_L@163.com>, TengYao Chi
<frankvicky@apache.org>