Commit Graph

3905 Commits

Author SHA1 Message Date
Bill Bejeck dd82542493
KAFKA-19504: Remove unused metrics reporter initialization in KafkaAdminClient (#20166)
CI / build (push) Waiting to run Details
The `AdminClient` adds a telemetry reporter to the metrics reporters
list in the constructor.  The problem is that the reporter was already
added in the `createInternal` method.  In the `createInternal` method
call, the `clientTelemetryReporter` is added to a
`List<MetricReporters>` which is passed to the `Metrics` object, will
get closed when `Metrics.close()` is called.  But adding a reporter to
the reporters list in the constructor is not used by the `Metrics`
object and hence doesn't get closed, causing a memory leak.

All related tests pass after this change.

Reviewers: Apoorv Mittal <apoorvmittal10@apache.org>, Matthias J. Sax
 <matthias@confluent.io>, Chia-Ping Tsai <chia7712@gmail.com>,
 Jhen-Yung Hsu <jhenyunghsu@gmail.com>
2025-07-14 20:19:16 -04:00
Luke Chen e1ff387605
KAFKA-14915: Allow reading from remote storage for multiple partitions in one fetchRequest (#20045)
This PR enables reading remote storage for multiple partitions in one
fetchRequest. The main changes are:
1. In `DelayedRemoteFetch`, we accept multiple remoteFetchTasks and
other metadata now.
2. In `DelayedRemoteFetch`, we'll wait until all remoteFetch done,
either succeeded or failed.
3. In `ReplicaManager#fetchMessage`, we'll create one
`DelayedRemoteFetch` and pass multiple remoteFetch metadata to it, and
watch all of them.
4. Added tests

Reviewers: Kamal Chandraprakash<kamal.chandraprakash@gmail.com>, Federico Valeri <fedevaleri@gmail.com>, Satish Duggana <satishd@apache.org>
2025-07-14 19:42:08 +05:30
yangxuze b6fce13e3a
KAFKA-19460: Improve documentation of fetch.min.bytes and replica.fetch.min.bytes (#20111)
While walking through the source code I confirmed that the broker checks
`replica.fetch.min.bytes` exactly the same way it checks
`fetch.min.bytes`, so this patch updates the wording for both config
keys.

Co-authored-by: yangxuze <xuze_yang@163.com>

Reviewers: Luke Chen <showuon@gmail.com>
2025-07-14 12:57:15 +08:00
Shivsundar R 56a3c6dde9
KAFKA-19485: Added check before sending acknowledgements on initial epoch. (#20135)
CI / build (push) Waiting to run Details
https://issues.apache.org/jira/browse/KAFKA-19485

**Bug :**
There is a bug in `ShareConsumeRequestManager` where we are adding
acknowledgements on initial `ShareSession` epoch even after checking for
it.
Added fix to only include acknowledgements in the request if we have to,

PR also adds the check at another point in the code where we could
potentially be sending such acknowledgements.  One of the cases could be
when metadata is refreshed with empty topic IDs after a broker restart.
This means leader information would not be available on the node.

- Consumer subscribed to a partition whose leader was node-0.
- Broker restart happens and node-0 is elected leader again. Broker
starts a new `ShareSession`.
- Background thread sends a fetch request with **non-zero** epoch.
- Broker responds with `SHARE_SESSION_NOT_FOUND`.
- Client updates session epoch to 0 once it receives this error.
- Client updates metadata but receives empty metadata response. (Leader
unavailable)
- Application thread processing the previous fetch, completes and sends
acks to piggyback on next fetch.
- Next fetch will send the piggyback acknowledgements on the fetch for
previously subscribed partitions resulting in error from broker
("`Acknowledge data present on initial epoch`"). (Currently we attempt
to send even if leader is unavailable).

**Fix** :  Add a check before sending out acknowledgments if we are on
an initial epoch.
Added unit test covering the above scenario.

Reviewers: Andrew Schofield <aschofield@confluent.io>
2025-07-10 09:06:19 +01:00
Andrew Schofield 7b8a594a22
MINOR: Tidy up in AlterShareGroupOffsetsHandler (#20130)
Minor tidying up in AlterShareGroupOffsetsHandler based on review
comment
https://github.com/apache/kafka/pull/20049#discussion_r2192904850.

Reviewers: Jimmy Wang <wangzhiwang611@gmail.com>, Lan Ding
 <isDing_L@163.com>, TaiJuWu <tjwu1217@gmail.com>, Ken Huang
 <s7133700@gmail.com>, Jhen-Yung Hsu <jhenyunghsu@gmail.com>, Chia-Ping
 Tsai <chia7712@gmail.com>
2025-07-10 01:24:13 +08:00
Chang-Chi Hsu 22698493e9
MINOR: Move partitions == 0 logic from waitForTopic to waitTopicDeletion (#20108)
## Changes

- The partitions == 0 branch has been moved from **waitForTopic** to
**waitTopicDeletion**.

## Reasons

- Clarify the responsibility of each helper method makes the test code
easier to reason by moving the partitions == 0 logic from
**waitForTopic** into a dedicated method **waitTopicDeletion**.

Reviewers: Jhen-Yung Hsu <jhenyunghsu@gmail.com>, TaiJuWu
 <tjwu1217@gmail.com>, TengYao Chi <kitingiao@gmail.com>, Ken Huang
 <s7133700@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-07-10 01:10:20 +08:00
Jonah Hooper d86ba7f54a
KAFKA-18681: Created GetReplicaLogInfo RPCs (#19664)
CI / build (push) Waiting to run Details
Creates GetReplicaLogInfoRequest and GetReplicaLogInfoResponse RPCs
Information returned by these brokers will be used to aid
unclean-recovery by selecting longest logs.

Reviewers: Alyssa Huang <ahuang@confluent.io>, Calvin Liu <caliu@confluent.io>, Colin P. McCabe <cmccabe@apache.org>, TaiJuWu <tjwu1217@gmail.com>
2025-07-08 10:41:01 -07:00
Alieh Saeedi db1c6f31a3
KAFKA-18288: Fix Streams CLI describe (#20099)
CI / build (push) Waiting to run Details
This PR includes the following fixes:

- Streams CLI used to list and return the description of the first group
which is a bug. With this fix, it returns the descriptions of the groups
specified by the `--group` or `all-groups`. Integration test are added
to verify the fix.
- `timeoutOption` is missing in describe groups. This fix adds and tests
it with short timeout.
- `DescribeStreamsGroupsHandler` used to return an empty group in `DEAD`
state when the group id was not found, but with this fix, it throws
`GroupIdNotFoundException`
2025-07-08 15:28:56 +02:00
Ken Huang a399852ced
KAFKA-19042 Move PlaintextConsumerTest to client-integration-tests module (#20081)
Use Java to rewrite PlaintextConsumerTest by new test infra and  move it
to client-integration-tests module.

Reviewers: Jhen-Yung Hsu <jhenyunghsu@gmail.com>, TengYao Chi
<kitingiao@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-07-08 01:41:59 +08:00
Ismael Juma 4b607616c7
KAFKA-19444: Add back JoinGroup v0 & v1 (#20116)
This fixes librdkafka older than the recently released 2.11.0 with
Kerberos authentication and Apache Kafka 4.x.

Even though this is a bug in librdkafka, a key goal of KIP-896 is not to
break the popular client libraries listed in it. Adding back JoinGroup
v0 & v1 is a very small change and worth it from that perspective.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-07-07 08:44:24 -07:00
Bolin Lin e8ee7fc210
KAFKA-19315 Move ControllerMutationQuotaManager to server module (#19807)
CI / build (push) Has been cancelled Details
Migrate ControllerMutationQuotaManager to Java implementation and move
to server module, including ClientQuotaManager and associated files.

Reviewers: TengYao Chi <kitingiao@gmail.com>, Ken Huang
<s7133700@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-07-07 01:55:38 +08:00
Ken Huang d31885d33c
MINOR: Use <code> block instead of backtick (#20107)
CI / build (push) Waiting to run Details
When writing HTML, it's recommended to use the <code> element instead of
backticks for inline code formatting.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, TengYao Chi
<frankvicky@apache.org>
2025-07-06 14:49:51 +08:00
Omnia Ibrahim 9df616da76
KAFKA-19397: Ensure consistent metadata usage in produce request and response (#19964)
CI / build (push) Has been cancelled Details
Flaky Test Report / Flaky Test Report (push) Has been cancelled Details
- Metadata doesn't have the full view of topicNames to ids during
rebootstrap of client or when topic has been deleted/recreated. The
solution is to pass down topic id and stop trying to figure it out later
in the logic.

---------

Co-authored-by: Kirk True <kirk@kirktrue.pro>
2025-07-04 17:44:09 +01:00
Jhen-Yung Hsu 4e31e270ba
MINOR: Update the docs for Metrics (#20094)
CI / build (push) Waiting to run Details
Update the outdated Javadocs in Metrics.java. The `MetricName(String
name, String group)` constructor in MetricName.java was removed in

59b918ec2b
Minor typo fixes included.

Reviewers: Ken Huang <s7133700@gmail.com>, TengYao Chi
 <kitingiao@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-07-04 02:01:29 +08:00
Andrew Schofield 729f9ccf06
KAFKA-19440: Handle top-level errors in AlterShareGroupOffsets RPC (#20049)
While testing the code in https://github.com/apache/kafka/pull/19820, it
became clear that the error handling problems were due to the underlying
Admin API. This PR fixes the error handling for top-level errors in the
AlterShareGroupOffsets RPC.

Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>, Lan Ding
 <isDing_L@163.com>, TaiJuWu <tjwu1217@gmail.com>
2025-07-03 11:00:56 +01:00
Jhen-Yung Hsu 64aebb5621
MINOR: remove unused FlattenedIterator (#20067)
CI / build (push) Waiting to run Details
Remove FlattenedIterator. It’s no longer used anywhere after
https://github.com/apache/kafka/pull/20037.

Reviewers: TengYao Chi <kitingiao@gmail.com>, Lan Ding
 <isDing_L@163.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-06-30 20:02:08 +08:00
TaiJuWu bd14ed21b4
KAFKA-18486 Remove ReplicaManager#becomeLeaderOrFollower (#20037)
The PR do following:

1. Remove  ReplicaManager#becomeLeaderOrFollower.
2. Remove `LeaderAndIsrRequest` and `LeaderAndIsrResponse`
3. Migrate `LeaderAndIsrRequest.PartitionState` to server-common module
and change to `PartitionState`
4. Remove `ControllerEpoch` from PartitionState
5. Remove `isShuttingDown` from BrokerServer and ReplicaManager

Reviewers: Kuan-Po Tseng <brandboat@gmail.com>, Chia-Ping Tsai
 <chia7712@gmail.com>
2025-06-30 01:20:49 +08:00
Xuan-Zhang Gong 7d34cb4412
MINOR: Correct ProducerFailureHandlingTest Typo, Use waitForTopic Instead of waitForDeletion (#20054)
from https://github.com/apache/kafka/pull/19319#discussion_r2169555430

This pull request addresses a minor typo in the
ProducerFailureHandlingTest within the Apache Kafka project.
Specifically, it corrects an erroneous method call where waitForDeletion
was used instead of waitForTopic (or createTopic).

Reviewers: PoAn Yang <payang@apache.org>, TaiJuWu <tjwu1217@gmail.com>,
 Jhen-Yung Hsu <jhenyunghsu@gmail.com>, TengYao Chi
 <kitingiao@gmail.com>, Ken Huang <s7133700@gmail.com>, Chia-Ping Tsai
 <chia7712@gmail.com>
2025-06-30 00:53:36 +08:00
TaiJuWu a95522a5ba
KAFKA-19042 Rewrite ConsumerBounceTest by Java (#19822)
This PR does the following:
1) Rewrites consumerBounceTest in Java.
2) Moves the test to clients-integration-test.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-06-30 00:40:36 +08:00
Ken Huang b919836551
KAFKA-17662: config.providers configuration missing from the docs (#18930)
Ensure the config.providers configuration is documented for all
components supporting it

Reviewers: Mickael Maison <mickael.maison@gmail.com>, Greg Harris
<gharris1727@gmail.com>, Matthias J. Sax <mjsax@apache.org>
2025-06-27 14:13:55 +02:00
David Jacot f6a78c4c2b
KAFKA-19246; OffsetFetch API does not return group level errors correctly with version 1 (#19704)
The OffsetFetch API does not support top level errors in version 1.
Hence, the top level error must be returned at the partition level.

Side note: It is a tad annoying that we create error response in
multiple places (e.g. KafkaApis, Group CoordinatorService). There were a
reason for this but I cannot remember.

Reviewers: Dongnuo Lyu <dlyu@confluent.io>, Sean Quah <squah@confluent.io>, Ken Huang <s7133700@gmail.com>, TengYao Chi <frankvicky@apache.org>
2025-06-26 06:29:43 -07:00
Yu-Syuan Jheng 2d943419a3
MINOR: Remove unused DescribeTopicPartitionResponse.prepareResponse function (#20032)
The prepareResponse function is no longer used anywhere in the codebase.
Removing unused code improves maintainability and readability.

Reviewers: PoAn Yang <payang@apache.org>, Yung
<yungyung7654321@gmail.com>, TengYao Chi <frankvicky@apache.org>, Ken
Huang <s7133700@gmail.com>
2025-06-25 17:26:22 +08:00
Bolin Lin 3404f65cdb
KAFKA-19324 Make org.apache.kafka.common.test.TestUtils package-private to prevent cross-module access (#19884)
Description

* Replace `org.apache.kafka.common.test.TestUtils` with
`org.apache.kafka.test.TestUtils` in outer package modules to
standardize test utility usage
* Move `waitUntilLeaderIsElectedOrChangedWithAdmin` method from
`org.apache.kafka.test.TestUtils` to `ClusterInstance` and refactor for
better code organization
* Add `org.apache.kafka.test.TestUtils` dependency to
`transaction-coordinator` import control

Reviewers: PoAn Yang [payang@apache.org](mailto:payang@apache.org), Ken
 Huang  [s7133700@gmail.com](mailto:s7133700@gmail.com), Ken Huang
 [s7133700@gmail.com](mailto:s7133700@gmail.com), Chia-Ping Tsai
 [chia7712@gmail.com](mailto:chia7712@gmail.com)
2025-06-22 22:47:40 +08:00
Nick Guo 583acb60d6
MINOR: Restore original behavior of GroupAuthorizerIntegrationTest (#19968)
CI / build (push) Waiting to run Details
this is a follow-up for https://github.com/apache/kafka/pull/19685

The timeout issue in `AsyncConsumer#unsubscribe` was fixed by
https://github.com/apache/kafka/pull/19779. As a result, the test

`GroupAuthorizerIntegrationTest#testConsumeUnsubscribeWithoutGroupPermission`
should now retain its original behavior as expected prior to the issue.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-06-22 22:08:20 +08:00
S.Y. Wang 22bef988d4
KAFKA-18926 KafkaPrincipalBuilder should extend KafkaPrincipalSerde (#19987)
In KRaft, custom KafkaPrincipalBuilder instances must implement
KafkaPrincipalSerde to support the forward mechanism. Currently, this
requirement is not enforced and relies on the developer's attention.

With this patch, we can prevent incorrect implementations at compile
time.

Reviewers: Ken Huang <s7133700@gmail.com>, TengYao Chi
 <kitingiao@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-06-22 22:01:03 +08:00
Ken Huang fa4d8836d3
MINOR: DescribeAuthorizedOperationsTest cluster setting has been overridden (#20012)
The old approach `OFFSETS_TOPIC_REPLICATION_FACTOR_CONFIG` will override
`OFFSETS_TOPIC_PARTITIONS_CONFIG` config, this behaviour is not
expected, we should fix it.

Reviewers: TengYao Chi <frankvicky@apache.org>
2025-06-22 21:42:03 +08:00
Alieh Saeedi a797d644bc
KAFKA-19244: Add support for kafka-streams-groups.sh options (reset-offsets) [1/N] (#19646)
This PR implements all the options for `--reset-offset`, (supports
`--execute`, `--dry-run`, `--export`). Includes unit and integration
tests.

- --reset-offsets --input-topic String: topics --shift-by Long
- --reset-offsets --input-topic String: topics --to-offset  Long
- --reset-offsets --input-topic String: topics -to-latest
- --reset-offsets --input-topic String: topics -to-earliest
- --reset-offsets --all-input-topics --shift-by Long
- --reset-offsets --all-input-topics --to-offset  Long
- --reset-offsets --all-input-topics -to-latest
- --reset-offsets --all-input-topics -to-earliest
- --reset-offsets --all-input-topics -to-earliest
- --reset-offsets --from-file String:fileName
- --reset-offsets --input-topic String: topics --by-duration String:
duration
- --reset-offsets --all-input-topics --by-duration String: duration
- --reset-offsets --input-topics <String: topics> --to-datetime String:
datetime
- --reset-offsets --all-input-topics --to-datetime String: datetime
- --reset-offsets --input-topics <String: topics> -to-current
- --reset-offsets --all-input-topics -to-current

Reviewers: Lucas Brutschy <lbrutschy@confluent.io>, Bill Bejeck
 <bill@confluent.io>
2025-06-20 16:14:18 +02:00
Lan Ding 659ace836c
MINOR: Replace `singleton` with `Set.of` in the SubscriptionStateTest (#19993)
Replace `singleton` with `Set.of` in the SubscriptionStateTest.

Reviewers: Ken Huang <s7133700@gmail.com>, PoAn Yang
<payang@apache.org>, Yung <yungyung7654321@gmail.com>, TengYao Chi
<frankvicky@apache.org>, Chia-Ping Tsai <chia7712@gmail.com>
2025-06-20 15:33:41 +08:00
Juha Mynttinen d59d39a229
MINOR: fix MetadataRequest and MetadataResponse doc (#19963)
CI / build (push) Waiting to run Details
MetadataRequest and MetadataResponse version 0 is still supported.
Remove from README.md mentioning that they're not supported. It's
unclear how such a mention was ever included in the file. It has always
been there

Reviewers: Andrew Schofield <aschofield@confluent.io>
2025-06-19 22:06:02 +01:00
Lan Ding dbbc45f7a0
KAFKA-19416: Reduce running time of ShareConsumerTest (#19992)
Configure tests in ShareConsumer to run exclusively in Raft-Isolated
mode to reduce execution time.

Reviewers: Andrew Schofield <aschofield@confluent.io>, Jhen-Yung Hsu
<jhenyunghsu@gmail.com>, Ken Huang <s7133700@gmail.com>
2025-06-19 15:30:23 +01:00
Shivsundar R 55297967a9
MINOR : Fix possible flake in ShareConsumerTest:testConsumerCloseInGroupSequential (#19986)
CI / build (push) Waiting to run Details
*What*

- There was a possibility of a flake occurring in one of the tests in
`ShareConsumerTest` where we are not waiting for the assignment to be
received and synced before we start consuming on `poll()`.

- There should ideally be a `waitedPoll()` or a dedicated poll which
would trigger the reconcilation and updation of the assignment.

Scenario :

- We are doing 3 `polls` to receive 1500 records.

- On debugging, I noticed 1st poll always returned 0 records as this was
triggering reconcilation, the test passed locally as 2nd and 3rd polls
add up to give us 1500 records almost always (as 500 is a soft limit).
The test can fail if these 2 `polls` do not add up 1500. We need a
maximum
of 3 polls to receive 1500 records with `max.poll.records` set to 500.
- Now we have introduced a loop which would keep polling until we
receive the expected records and then close.

Reviewers: Andrew Schofield <aschofield@confluent.io>
2025-06-19 12:53:49 +01:00
Lianet Magrans ce996b34e9
KAFKA-19356: Prevent new consumer fetch assigned partitions not in explicit subscription (#19983)
CI / build (push) Waiting to run Details
Fix to ensure assigned partitions whose topics are not in the consumer
explicit subscription are considered not fetchable (so that no records
are returned on poll for them)

This scenario could happen in the new async consumer (using the Consumer
rebalance protocol) when the subscription changes, because the consumer
will keep its assignment until the coordinator sends a new one (broker
drives assignments).

This does not happen in the classic consumer because the assignment
logic lives on the client-side, so the consumer pro-actively updates
assignment as needed.

This PR validates assignment vs subscription on fetch for explicit
subscription  only. Regular expressions, shared subscription  remain
unchanged (regex case still under discussion, will be handled separately
if needed)

Reviewers: Andrew Schofield <aschofield@confluent.io>, TengYao Chi
 <frankvicky@apache.org>, Kirk True <ktrue@confluent.io>, Jhen-Yung Hsu
 <jhenyunghsu@gmail.com>
2025-06-18 18:31:46 -04:00
Xuan-Zhang Gong b8fc96272e
KAFKA-18627 Deprecated "org.apache.kafka.disallowed.login.modules" (#18683)
CI / build (push) Waiting to run Details
Fix https://issues.apache.org/jira/browse/KAFKA-18627 and update same
test case


[KIP-link](https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=340037077)

⚠️ This PR cannot be merged at the moment because KIPs for version 4.1
are already frozen. It will need to wait for version 4.2.

update upgrade.html


![image](https://github.com/user-attachments/assets/00bd6579-6914-42a7-a1b8-4a0b0a8cc3c3)

add configuration doc

![image](https://github.com/user-attachments/assets/e20d5c97-29c6-4fe4-a41a-58267179e4e0)

Reviewers: TaiJuWu <tjwu1217@gmail.com>, Ken Huang <s7133700@gmail.com>,
 Luke Chen <showuon@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-06-18 23:05:30 +08:00
Kirk True adcf10ca8b
KAFKA-19297: Refactor AsyncKafkaConsumer's use of Java Streams APIs in critical sections (#19917)
Profiling has shown that using the Collections Streams API approach adds
unnecessary overhead compared to a traditional for loop. Minor revisions
to the code have been made to use simpler constructs to improve
performance.

Reviewers: Lianet Magrans <lmagrans@confluent.io>, Andrew Schofield
 <aschofield@confluent.io>
2025-06-18 10:00:45 -04:00
Lucas Brutschy 788781d4bb
MINOR: Add entity types in streams group RPCs (#19979)
These can be used to implement transformations on top of the RPC
definitions. Group IDs were already marked. This PR additionally adds
the entityType for all topic names.

Reviewers: Matthias J. Sax <matthias@confluent.io>
2025-06-18 09:14:37 +02:00
José Armando García Sancio 742b327025
KAFKA-14145; Faster KRaft HWM replication (#19800)
CI / build (push) Waiting to run Details
This change compares the remote replica's HWM with the leader's HWM and
completes the FETCH request if the remote HWM is less than the leader's
HWM. When the leader's HWM is updated any pending FETCH RPC is
completed.

Reviewers: Alyssa Huang <ahuang@confluent.io>, David Arthur
 <mumrah@gmail.com>, Andrew Schofield <aschofield@confluent.io>
2025-06-17 13:00:43 -04:00
Ken Huang 91ce182ec5
KAFKA-19042 Move ProducerSendWhileDeletionTest to client-integration-tests module (#19971)
Use Java to rewrite ProducerSendWhileDeletionTest by new test infra and
move it to client-integration-tests module.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-06-18 00:47:30 +08:00
Lan Ding 183ad524dc
MINOR: replace test "catch exception" by assertThrows (#19975)
see https://github.com/apache/kafka/pull/19948#discussion_r2150617216,
replace test "catch exception" by assertThrows.

Reviewers: PoAn Yang <payang@apache.org>, Ken Huang
 <s7133700@gmail.com>, TengYao Chi <kitingiao@gmail.com>, Chia-Ping Tsai
 <chia7712@gmail.com>
2025-06-18 00:07:49 +08:00
Dave Troiano 6f946d5026
MINOR: in min.insync.replicas config doc, explicitly state that all ISR must ack when acks=all (#19973)
CI / build (push) Waiting to run Details
Clarify the interaction of `min.insync.replicas` and `ack=all`
configuration.  Prior to this change, the doc for `min.insync.replicas`
could have been  interpreted as being used to short-circuit in the
`acks=all` case as if  it would be enough if `min.inscyn.replicas`
number of brokers replicated  a message before it can be acknowledged
back to the producer.

Reviewers: Matthias J. Sax <matthias@confluent.io>

---------

Co-authored-by: Matthias J. Sax <mjsax@apache.org>
2025-06-16 18:40:54 -07:00
Abhiram98 99e1e684ef
MINOR: rename exception `e` to `swallow` where appropriate (#19948)
CI / build (push) Waiting to run Details
I noticed that in commit

[a662bc56](a662bc5634),
renamed ignored exceptions `e` to `swallow`.

Here's a small patch to make that change consistent in other files:
AsyncKafkaConsumer.java, KafkaConsumerTest.java,
AsyncKafkaConsumerTest.java

Reviewers: Kirk True <kirk@kirktrue.pro>, Lan Ding <isDing_L@163.com>,
 Chia-Ping Tsai <chia7712@gmail.com>
2025-06-13 17:48:02 +08:00
Kirk True 7d6e5edf8e
KAFKA-19153: Add OAuth integration tests (#19938)
Adds a test dependency on
[mock-oauth2-server](https://github.com/navikt/mock-oauth2-server/) for
integration tests for OAuth layer. Also includes fixes for some
regressions that were caught by the integration tests.

Reviewers: Manikumar Reddy <manikumar@confluent.io>, Lianet Magrans
 <lmagrans@confluent.io>
2025-06-12 15:48:14 -04:00
Lianet Magrans 8f2ee4d7cd
KAFKA-18117; KAFKA-18729: Use assigned topic IDs to avoid full metadata requests on broker-side regex (#19814)
This PR uses topic IDs received in assignment (under new protocol) to
ensure that only these assigned topics are included in the consumer
metadata requests performed when the user subscribes to broker-side
regex (RE2J).

For handling the edge case of consumer needing metadata for topics IDs
(from RE2J) and topic names (from transient topics), the approach is to
send a request for the transient topics needed temporarily, and once
those resolved, the request for the topic IDs needed for RE2J will
follow. (this is because the broker doesn't accept requests for names
and IDs at the same time)

With the changes we also end up fixing another issue (KAFKA-18729) aimed
at avoiding iterating the full set of assigned partitions when checking
if a topic should be retained from the metadata response when using
RE2J.

Reviewers: David Jacot <djacot@confluent.io>
2025-06-12 10:50:31 -04:00
Kirk True 78c1da1aca
KAFKA-17089: Incorrect JWT parsing in OAuthBearerUnsecuredJws (#19946)
Fixed a long-standing issue where the client JWT validation was decoding
the JWT sections using base 64 instead of URL-safe base 64.

Note: server-side validation leverages the jose4j library for parsing
JWTs, hence no fix is needed there.

Reviewers: Lianet Magrans <lmagrans@confluent.io>, Manikumar Reddy
<manikumar@confluent.io>

---------

Co-authored-by: Lianet Magrans <98415067+lianetm@users.noreply.github.com>
2025-06-12 09:53:50 -04:00
Jhen-Yung Hsu 2e968560e0
MINOR: Cleanup simplify set initialization with Set.of (#19925)
Simplify Set initialization and reduce the overhead of creating extra
collections.

The changes mostly include:
- new HashSet<>(List.of(...))
- new HashSet<>(Arrays.asList(...)) / new HashSet<>(asList(...))
- new HashSet<>(Collections.singletonList()) / new
HashSet<>(singletonList())
- new HashSet<>(Collections.emptyList())
- new HashSet<>(Set.of())

This change takes the following into account, and we will not change to
Set.of in these scenarios:
- Require `mutability` (UnsupportedOperationException).
- Allow `duplicate` elements (IllegalArgumentException).
- Allow `null` elements (NullPointerException).
- Depend on `Ordering`. `Set.of` does not guarantee order, so it could
make tests flaky or break public interfaces.

Reviewers: Ken Huang <s7133700@gmail.com>, PoAn Yang
 <payang@apache.org>, Chia-Ping Tsai <chia7712@gmail.com>
2025-06-11 18:36:14 +08:00
Gaurav Narula edd0efdebf
KAFKA-19221 Propagate IOException on LogSegment#close (#19607)
Log segment closure results in right sizing the segment on disk along
with the associated index files.

This is specially important for TimeIndexes where a failure to right
size may eventually cause log roll failures leading to under replication
and log cleaner failures.

This change uses `Utils.closeAll` which propagates exceptions, resulting
in an "unclean" shutdown. That would then cause the broker to attempt to
recover the log segment and the index on next startup, thereby avoiding
the failures described above.

Reviewers: Omnia Ibrahim <o.g.h.ibrahim@gmail.com>, Jun Rao
 <junrao@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-06-11 01:09:52 +08:00
Ken Huang d6861f3f15
MINOR: Use `pollUntilTrue` instead of `waitForCondition` (#19911)
CI / build (push) Waiting to run Details
We can use `pollUntilTrue` instead of `waitForCondition`, thus do a
little refactor to reduce the duplicate code

Reviewers: TengYao Chi <frankvicky@apache.org>, Lan Ding
 <isDing_L@163.com>, TaiJuWu <tjwu1217@gmail.com>
2025-06-09 15:33:00 +08:00
Ken Huang 8fd0d33670
KAFKA-19042 Move PlaintextConsumerSubscriptionTest to client-integration-tests module (#19827)
CI / build (push) Waiting to run Details
Use Java to rewrite PlaintextConsumerSubscriptionTest by new test infra
and move it to client-integration-tests module.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-06-08 05:06:03 +08:00
Kirk True 861eeb859d
KAFKA-19295: Remove AsyncKafkaConsumer event ID generation (#19915)
CI / build (push) Waiting to run Details
Remove the event IDs from the ApplicationEvent and BackgroundEvent as it
serves no functional purpose other than uniquely identifying events in
the logs.

Reviewers: Andrew Schofield <aschofield@confluent.io>
2025-06-07 13:08:22 +01:00
hgh1472 c4a769bc8b
MINOR: Rename ambiguous method name (#19875)
CI / build (push) Waiting to run Details
While reading through the code, I found the method name to be somewhat
ambiguous and not fully descriptive of its purpose.

So I renamed the method to make its purpose clearer and more
self-explanatory. If there was another reason for the original naming,
I’d be happy to hear about it.

Reviewers: Lianet Magrans <lmagrans@confluent.io>
2025-06-06 15:03:51 -04:00
PoAn Yang 844b0e651b
KAFKA-19369: Add group.share.assignors config and integration test (#19900)
CI / build (push) Waiting to run Details
* Add `group.share.assignors` config to `GroupCoordinatorConfig`.
* Send `rackId` in share group heartbeat request if it's not null.
* Add integration test `testShareConsumerWithRackAwareAssignor`.

Reviewers: Lan Ding <53332773+DL1231@users.noreply.github.com>, Andrew
 Schofield <aschofield@confluent.io>

---------

Signed-off-by: PoAn Yang <payang@apache.org>
2025-06-06 14:20:56 +01:00