Commit Graph

3779 Commits

Author SHA1 Message Date
yunchi bff5ba4ad9
MINOR: replace .stream().forEach() with .forEach() (#19626)
CI / build (push) Waiting to run Details
replace all applicable `.stream().forEach()` in codebase with just
`.forEach()`.

Reviewers: TengYao Chi <kitingiao@gmail.com>, Ken Huang
<s7133700@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-05-04 20:39:55 +08:00
Ken Huang c85e09f7a5
KAFKA-19060 Documented null edge cases in the Clients API JavaDoc (#19393)
Some client APIs may return `null` values in the map, but this  behavior
isn’t documented in the JavaDoc. We should update the JavaDoc to include
these edge cases.

Reviewers: Kirk True <kirk@kirktrue.pro>, Jhen-Yung Hsu
<jhenyunghsu@gmail.com>, PoAn Yang <payang@apache.org>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-05-04 20:35:02 +08:00
xijiu b5cceb43e5
KAFKA-19205: inconsistent result of beginningOffsets/endoffset between classic and async consumer with 0 timeout (#19578)
CI / build (push) Waiting to run Details
In the return results of the methods beginningOffsets and endOffset, if
timeout == 0, then an empty Map should be returned uniformly instead of
in the form of <TopicPartition, null>

Reviewers: Ken Huang <s7133700@gmail.com>, PoAn Yang
 <payang@apache.org>, Chia-Ping Tsai <chia7712@gmail.com>, Lianet
 Magrans <lmagrans@confluent.io>
2025-05-03 13:12:20 -04:00
TengYao Chi 93e65c4539
KAFKA-18267 Add unit tests for CloseOptions (#19571)
There is some redundant code that could be removed in `CloseOptions`.
This patch also adds unit tests for CloseOptions.

Reviewers: Ken Huang <s7133700@gmail.com>, PoAn Yang
<payang@apache.org>, Chia-Ping Tsai <chia7712@gmail.com>
2025-05-03 22:36:43 +08:00
Matthias J. Sax 44025d8116
MINOR: fix bug in MockConsumer (#19627)
The setter of `maxPollRecords` wrongly checks the field instead of the argument.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, TengYao Chi
 <frankvicky@apache.org>
2025-05-03 14:08:18 +08:00
Sushant Mahajan e68781414e
KAFKA-19204: Allow persister retry of initializing topics. (#19603)
CI / build (push) Waiting to run Details
* Currently in the share group heartbeat flow, if we see a TP subscribed
for the first time, we move that TP to initializing state in GC and let
the GC send a persister request to share group initialize the
aforementioned TP.
* However, if the coordinator runtime request for share group heartbeat
times out (maybe due to restarting/bad broker), the future completes
exceptionally resulting in persiter request to not be sent.
* Now, we are in a bad state since the TP is in initializing state in GC
but not persister initialized. Future heartbeats for the same share
partitions will also not help since we do not allow retrying persister
request for initializing TPs.
* This PR remedies the situation by allowing the same.
* A temporary fix to increase offset commit timeouts in system tests was
added to fix the issue. In this PR, we revert that change as well.

Reviewers: Andrew Schofield <aschofield@confluent.io>
2025-05-02 14:25:29 +01:00
Matthias J. Sax f69337b37c
MINOR: use `isEmpty()` to avoid compiler warning (#19616)
Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2025-05-01 23:51:36 -07:00
Calvin Liu 0c1fbf3aeb
KAFKA-19073 add transactional ID pattern filter to ListTransactions (#19355)
Propose adding a new filter TransactionalIdPattern. This transaction ID pattern filter works as AND with the other transaction filters. Also, it is empowered with Re2j.

KIP: https://cwiki.apache.org/confluence/x/4gm9F

Reviewers: Justine Olshan <jolshan@confluent.io>, Ken Huang
<s7133700@gmail.com>, Kuan-Po Tseng <brandboat@gmail.com>, Chia-Ping
Tsai <chia7712@gmail.com>
2025-05-02 00:52:21 +08:00
Lan Ding 8dbf56e4b5
KAFKA-17541:[1/2] Improve handling of delivery count (#19430)
For records which are automatically released as a result of closing a
share session normally, the delivery count should not be incremented.
These records were fetched but they were not actually delivered to the
client since the disposition of the delivery records is carried in the
ShareAcknowledge which closes the share session. Any remaining records
were not delivered, only fetched.
This PR releases the delivery count for records when closing a share
session normally.

Co-authored-by: d00791190 <dinglan6@huawei.com>

Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>, Andrew Schofield <aschofield@confluent.io>
2025-05-01 14:40:03 +01:00
Chirag Wadhwa 800612e4a7
KAFKA-19015: Remove share session from cache on share consumer connection drop (#19329)
Up till now, the share sessions in the broker were only attempted to
evict when the share session cache was full and a new session was trying
to get registered. With the changes in this PR, whenever a share
consumer gets disconnected from the broker, the corresponding share
session would be evicted from the cache.

Note - `connectAndReceiveWithoutClosingSocket` has been introduced in
`GroupCoordinatorBaseRequestTest`. This method creates a socket
connection, sends the request, receives a response but does not close
the connection. Instead, these sockets are stored in a ListBuffer
`openSockets`, which are closed in tearDown method after each test is
run. Also, all the `connectAndReceive` calls in
`ShareFetchAcknowledgeRequestTest` have been replaced by
`connectAndReceiveWithoutClosingSocket`, because these tests depends
upon the persistence of the share sessions on the broker once
registered. But, with the new code introduced, as soon as the socket
connection is closed, a connection drop is assumed by the broker,
leading to session eviction.

Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>, Andrew Schofield <aschofield@confluent.io>
2025-05-01 14:36:18 +01:00
Lianet Magrans 1059af4eac
MINOR: Improve docs for client group configs (#19605)
CI / build (push) Waiting to run Details
Improve java docs for session and HB interval client configs & fix
max.poll.interval description

Reviewers: David Jacot <djacot@confluent.io>
2025-04-30 14:04:16 -04:00
Andrew Schofield ce97b1d5e7
KAFKA-16894: Exploit share feature [3/N] (#19542)
This PR uses the v1 of the ShareVersion feature to enable share groups
for KIP-932.

Previously, there were two potential configs which could be used -
`group.share.enable=true` and including "share" in
`group.coordinator.rebalance.protocols`. After this PR, the first of
these is retained, but the second is not. Instead, the preferred switch
is the ShareVersion feature.

The `group.share.enable` config is temporarily retained for testing and
situations in which it is inconvenient to set the feature, but it should
really not be necessary, especially when we get to AK 4.2. The aim is to
remove this internal config at that point.

No tests should be setting `group.share.enable` any more, because they
can use the feature (which is enabled in test environments by default
because that's how features work). For tests which need to disable share
groups, they now set the share feature to v0. The majority of the code
changes were related to correct initialisation of the metadata cache in
tests now that a feature is used.

Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>
2025-04-30 13:27:01 +01:00
PoAn Yang 81881dee83
KAFKA-18760: Deprecate Optional<String> and return String from public Endpoint#listener (#19191)
* Deprecate org.apache.kafka.common.Endpoint#listenerName.
* Add org.apache.kafka.common.Endpoint#listener to replace
org.apache.kafka.common.Endpoint#listenerName.
* Replace org.apache.kafka.network.EndPoint with
org.apache.kafka.common.Endpoint.
* Deprecate org.apache.kafka.clients.admin.RaftVoterEndpoint#name
* Add org.apache.kafka.clients.admin.RaftVoterEndpoint#listener to
replace org.apache.kafka.clients.admin.RaftVoterEndpoint#name

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, TaiJuWu
 <tjwu1217@gmail.com>, Jhen-Yung Hsu <jhenyunghsu@gmail.com>, TengYao
 Chi <frankvicky@apache.org>, Ken Huang <s7133700@gmail.com>, Bagda
 Parth  , Kuan-Po Tseng <brandboat@gmail.com>

---------

Signed-off-by: PoAn Yang <payang@apache.org>
2025-04-30 12:15:33 +08:00
Ken Huang 676e0f2ad6
KAFKA-19139 Plugin#wrapInstance should use LinkedHashMap instead of Map (#19519)
CI / build (push) Waiting to run Details
There will be an update to the PluginMetrics#metricName method: the type
of the tags parameter will be changed
from Map to LinkedHashMap.
This change is necessary because the order of metric tags is important
1. If the tag order is inconsistent, identical metrics may be treated as
distinct ones by the metrics backend
2. KAFKA-18390 is updating metric naming to use LinkedHashMap. For
consistency, we should follow the same approach here.

Reviewers: TengYao Chi <frankvicky@apache.org>, Jhen-Yung Hsu
 <jhenyunghsu@gmail.com>, lllilllilllilili
2025-04-30 10:43:01 +08:00
Bill Bejeck 431cffc93f
KAFKA-19135 Migrate initial IQ support for KIP-1071 from feature branch to trunk (#19588)
This PR is a migration of the initial IQ support for KIP-1071 from the
feature branch to trunk.  It includes a parameterized integration test
that expects the same results whether using either the classic or new
streams group protocol.

Note that this PR will deliver IQ information in each heartbeat
response.  A follow-up PR will change that to be only sending IQ
information when assignments change.

Reviewers Lucas Brutschy <lucasbru@apache.org>
2025-04-29 20:08:49 -04:00
Matthias J. Sax 3bb15c5dee
MINOR: improve JavaDocs for consumer CloseOptions (#19546)
Reviewers: TengYao Chi <frankvicky@apache.org>, PoAn Yang <payang@apache.org>, Lianet Magrans <lmagrans@confluent.io>, Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2025-04-29 16:38:16 -07:00
Omnia Ibrahim 6f783f8536
KAFKA-10551: Add topic id support to produce request and response (#15968)
- Add support topicId in `ProduceRequest`/`ProduceResponse`. Topic name
and Topic Id will become `ignorable` following the footstep of
`FetchRequest`/`FetchResponse`
- ReplicaManager still look for `HostedPartition` using `TopicPartition`
and doesn't check topic id. This is an **[OPEN QUESTION]** if we should
address this in this pr or wait for
[KAFKA-16212](https://issues.apache.org/jira/browse/KAFKA-16212) as this
will update `ReplicaManager::getPartition` to use `TopicIdParittion`
once we update the cache. Other option is that we compare provided
`topicId` with `Partition` topic id and return `UNKNOW_TOPIC_ID` or
`UNKNOW_TOPIC_PARTITION` if we can't find partition with matched topic
id.

Reviewers: Jun Rao <jun@confluent.io>, Justine Olshan
 <jolshan@confluent.io>
2025-04-29 15:37:28 -07:00
Ritika Reddy 2fdb687029
KAFKA-19082: [2/4] Add preparedTxnState class to Kafka Producer (KIP-939) (#19470)
CI / build (push) Waiting to run Details
This is part of the client side changes required to enable 2PC for
KIP-939

New KafkaProducer.PreparedTxnState class is going to be defined as
following:  ```  static public class PreparedTxnState {    public String
toString();    public PreparedTxnState(String serializedState); public
PreparedTxnState();  }  ```  The objects of this class can serialize to
/ deserialize from a string value and can be written to / read from a
database.  The implementation is going to store producerId and epoch in
the format **producerId:epoch**

Reviewers: Artem Livshits <alivshits@confluent.io>, Justine Olshan
 <jolshan@confluent.io>
2025-04-29 11:52:02 -07:00
David Jacot 6d67d82d5b
MINOR: Cleanup OffsetFetchRequest/Response in MessageTest (#19576)
CI / build (push) Waiting to run Details
The tests related of OffsetFetch request/response in MessageTest are
incomprehensible. This patch rewrites them in a simpler way.

Reviewers: TengYao Chi <frankvicky@apache.org>
2025-04-28 13:14:23 +02:00
David Jacot be194f5dba
MINOR: Simplify OffsetFetchRequest (#19572)
While working on https://github.com/apache/kafka/pull/19515, I came to
the conclusion that the OffsetFetchRequest is quite messy and overall
too complicated. This patch rationalize the constructors.
OffsetFetchRequest has a single constructor accepting the
OffsetFetchRequestData. This will also simplify adding the topic ids.
All the changes are mechanical, replacing data structures by others.

Reviewers: PoAn Yang <payang@apache.org>, TengYao Chi <frankvicky@apache.org>, Lianet Magran <lmagrans@confluent.io>, Chia-Ping Tsai <chia7712@gmail.com>
2025-04-27 18:58:30 +02:00
Chirag Wadhwa 2f9c2dd828
KAFKA-16718-3/n: Added the ShareGroupStatePartitionMetadata record during deletion of share group offsets (#19478)
This is a follow up PR for implementation of DeleteShareGroupOffsets
RPC. This PR adds the ShareGroupStatePartitionMetadata record to
__consumer__offsets topic to make sure the topic is removed from the
initializedTopics list. This PR also removes partitions from the request
and response schemas for DeleteShareGroupState RPC

Reviewers: Sushant Mahajan <smahajan@confluent.io>, Andrew Schofield <aschofield@confluent.io>
2025-04-25 22:01:48 +01:00
Ken Huang b4b80731c1
KAFKA-19042 Move PlaintextConsumerFetchTest to client-integration-tests module (#19520)
Use Java to rewrite `PlaintextConsumerFetchTest` by new test infra and
move it to client-integration-tests module.

Reviewers: PoAn Yang <payang@apache.org>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-04-26 00:09:23 +08:00
Lucas Brutschy 732ed0696b
KAFKA-19190: Handle shutdown application correctly (#19544)
If the streams rebalance protocol is enabled in
StreamsUncaughtExceptionHandlerIntegrationTest, the streams application
does not shut down correctly upon error.

There are two causes for this. First, sometimes, the SHUTDOWN_APPLICATION
code only sent with the leave heartbeat, but that is not handled broker
side. Second, the SHUTDOWN_APPLICATION code wasn't properly handled
client-side at all.

Reviewers: Bruno Cadonna <cadonna@apache.org>, Bill Bejeck
 <bill@confluent.io>, PoAn Yang <payang@apache.org>
2025-04-25 09:56:09 +02:00
PoAn Yang 36d2498fb3
MINOR: Use meaningful name in AsyncKafkaConsumerTest (#19550)
Replace names like a, b, c, ... with meaningful names in
AsyncKafkaConsumerTest.

Follow-up:
https://github.com/apache/kafka/pull/19457#discussion_r2056254087

Signed-off-by: PoAn Yang <payang@apache.org>

Reviewers: Bill Bejeck <bbejeck@apache.org>, Ken Huang <s7133700@gmail.com>
2025-04-24 17:17:33 -04:00
David Jacot a948537704
MINOR: Small refactor in group coordinator (#19551)
This patch does a few code changes:
* It cleans up the GroupCoordinatorService;
* It moves the helper methods to validate request to Utils;
* It moves the helper methods to create the assignment for the
ConsumerGroupHeartbeatResponse and the ShareGroupHeartbeatResponse from
the GroupMetadataManager to the respective classes.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, Jeff Kim <jeff.kim@confluent.io>
2025-04-24 20:57:23 +02:00
Ritika Reddy 62fe528f4b
KAFKA-19082: [1/4] Add client config for enable2PC and overloaded initProducerId (KIP-939) (#19429)
This is part of the client side changes required to enable 2PC for
KIP-939

**Producer Config:**
transaction.two.phase.commit.enable The default would be ‘false’.  If
set to ‘true’, the broker is informed that the client is participating
in two phase commit protocol and transactions that this client starts
never expire.

**Overloaded InitProducerId method**
If the value is 'true' then the corresponding field is set in the
InitProducerIdRequest

Reviewers: Justine Olshan <jolshan@confluent.io>, Artem Livshits
 <alivshits@confluent.io>
2025-04-24 09:41:06 -07:00
Apoorv Mittal 3c05dfdf0e
KAFKA-18889: Make records in ShareFetchResponse non-nullable (#19536)
This PR marks the records as non-nullable for ShareFetch.

This PR is as per the changes for Fetch:
https://github.com/apache/kafka/pull/18726 and some work for ShareFetch
was done here: https://github.com/apache/kafka/pull/19167. I tested with
marking `records` as non-nullable in ShareFetch, which required
additional handling. The same has been fixed in current PR.

Reviewers: Andrew Schofield <aschofield@confluent.io>, Chia-Ping Tsai
 <chia7712@gmail.com>, TengYao Chi <frankvicky@apache.org>, PoAn Yang
 <payang@apache.org>
2025-04-24 16:32:08 +01:00
Vikas Singh f4ab3a2275
MINOR: Use readable interface to parse response (#19353)
The generated response data classes take Readable as input to parse the
Response. However, the associated response objects take ByteBuffer as
input and thus convert them to Readable using `new ByteBufferAccessor`
call.

This PR changes the parse method of all the response classes to take the
Readable interface instead so that no such conversion is needed.

To support parsing the ApiVersionsResponse twice for different version
this change adds the "slice" method to the Readable interface.

Reviewers: José Armando García Sancio <jsancio@apache.org>, Truc Nguyen
<[trnguyen@confluent.io](mailto:trnguyen@confluent.io)>, Aadithya
Chandra <[aadithya.c@gmail.com](mailto:aadithya.c@gmail.com)>
2025-04-24 11:09:06 -04:00
Andrew Schofield f0f5571dbb
MINOR: Change KIP-932 log messages from early access to preview (#19547)
Change the log messages which used to warn that KIP-932 was an Early
Access feature to say that it is now a Preview feature. This will make
the broker logs far less noisy when share groups are enabled.

Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>
2025-04-24 11:22:17 +01:00
PoAn Yang 3fae785ea0
KAFKA-19110: Add missing unit test for Streams-consumer integration (#19457)
- Construct `AsyncKafkaConsumer` constructor and verify that the
`RequestManagers.supplier()` contains Streams-specific data structures.
- Verify that `RequestManagers` constructs the Streams request managers
correctly
- Test `StreamsGroupHeartbeatManager#resetPollTimer()`
- Test `StreamsOnTasksRevokedCallbackCompletedEvent`,
`StreamsOnTasksAssignedCallbackCompletedEvent`, and
`StreamsOnAllTasksLostCallbackCompletedEvent` in
`ApplicationEventProcessor`
- Test `DefaultStreamsRebalanceListener`
- Test `StreamThread`.
  - Test `handleStreamsRebalanceData`.
  - Test `StreamsRebalanceData`.

Reviewers: Lucas Brutschy <lbrutschy@confluent.io>, Bill Bejeck <bill@confluent.io>
Signed-off-by: PoAn Yang <payang@apache.org>
2025-04-24 10:38:22 +02:00
Kirk True 8b4560e3f0
KAFKA-15767 Refactor TransactionManager to avoid use of ThreadLocal (#19440)
Introduces a concrete subclass of `KafkaThread` named `SenderThread`.
The poisoning of the TransactionManager on invalid state changes is
determined by looking at the type of the current thread.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-04-24 00:31:30 +08:00
Bruno Cadonna efd785274e
KAFKA-19124: Follow up on code improvements (#19453)
Improves a variable name and handling of an Optional.

Reviewers: Bill Bejeck <bill@confluent.io>, Chia-Ping Tsai <chia7712@gmail.com>, PoAn Yang <payang@apache.org>
2025-04-23 14:24:33 +02:00
David Jacot 71d08780d1
KAFKA-14690; Add TopicId to OffsetCommit API (#19461)
This patch extends the OffsetCommit API to support topic ids. From
version 10 of the API, topic ids must be used. Originally, we wanted to
support both using topic ids and topic names from version 10 but it
turns out that it makes everything more complicated. Hence we propose to
only support topic ids from version 10. Clients which only support using
topic names can either lookup the topic ids using the Metadata API or
stay on using an earlier version.

The patch only contains the server side changes and it keeps the version
10 as unstable for now. We will mark the version as stable when the
client side changes are merged in.

Reviewers: Lianet Magrans <lmagrans@confluent.io>, PoAn Yang <payang@apache.org>
2025-04-23 08:22:09 +02:00
Andrew Schofield e78e106221
MINOR: Improve javadoc for share consumer (#19533)
Small improvements to share consumer javadoc.

Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>
2025-04-22 15:54:05 +01:00
Andrew Schofield 66147d5de7
KAFKA-19057: Stabilize KIP-932 RPCs for AK 4.1 (#19378)
This PR removes the unstable API flag for the KIP-932 RPCs.

The 4 RPCs which were exposed for the early access release in AK 4.0 are
stabilised at v1. This is because the RPCs have evolved over time and AK
4.0 clients are not compatible with AK 4.1 brokers. By stabilising at
v1, the API version checks prevent incompatible communication and
server-side exceptions when trying to parse the requests from the older
clients.

Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>
2025-04-22 11:43:32 +01:00
Rich Chen ae771d73d1
KAFKA-8830 make Record Headers available in onAcknowledgement (#17099)
Two sets of tests are added:
1. KafkaProducerTest
- when send success, both record.headers() and onAcknowledgement headers
are read only
- when send failure, record.headers() is writable as before and
onAcknowledgement headers is read only
2. ProducerInterceptorsTest
- make both old and new onAcknowledgement method are called successfully

Reviewers: Lianet Magrans <lmagrans@confluent.io>, Omnia Ibrahim
<o.g.h.ibrahim@gmail.com>, Matthias J. Sax <matthias@confluent.io>,
Andrew Schofield <aschofield@confluent.io>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-04-21 21:01:55 +08:00
Hong-Yi Chen 8fa0d9723f
MINOR: Fix typo in ApiKeyVersionsProvider exception message (#19521)
This patch addresses issue #19516 and corrects a typo in
`ApiKeyVersionsProvider`: when `toVersion` exceeds  `latestVersion`, the
`IllegalArgumentException` message was erroneously formatted with
`fromVersion`. The format argument has been updated to use `toVersion`
so that the error message reports the correct value.

Reviewers: Ken Huang <s7133700@gmail.com>, PoAn Yang
 <payang@apache.org>, Jhen-Yung Hsu <jhenyunghsu@gmail.com>, Chia-Ping
 Tsai <chia7712@gmail.com>
2025-04-21 15:35:47 +08:00
David Jacot b94c7f9167
MINOR: Extend @ApiKeyVersionsSource annotation (#19516)
This patch extends the `@ApiKeyVersionsSource` annotation to allow
specifying the `fromVersion` and the `toVersion`. This is pretty handy
when we only want to test a subset of the versions.

Reviewers: Kuan-Po Tseng <brandboat@gmail.com>, TengYao Chi
<kitingiao@gmail.com>
2025-04-20 12:25:27 +08:00
Matthias J. Sax 810beef50e
MINOR: improve (De)Serializer JavaDocs (#19467)
Reviewers: Kirk True <ktrue@confluent.io>, Lianet Magrans <lmagrans@confluent.io>
2025-04-17 11:23:15 -07:00
Logan Zhu c6496e0c57
MINOR: Cleanup 0.10.x legacy references in ClusterResourceListener and TopicConfig (clients module) (#19388)
This PR is a minor follow-up to [PR
#19320](https://github.com/apache/kafka/pull/19320), which cleaned up
0.10.x legacy information from the clients module.

It addresses remaining reviewer suggestions that were not included in
the original PR:

- `ClusterResourceListener`: Removed "Note the minimum supported broker
version is 2.1." per review suggestion to avoid repeating
version-specific details across multiple classes.
- `TopicConfig`: Simplified `MAX_MESSAGE_BYTES_DOC` by removing obsolete
notes about behavior in versions prior to 0.10.2.

These changes help reduce outdated version information in client
documentation and improve clarity.

Reviewers: PoAn Yang <payang@apache.org>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-04-17 23:17:42 +08:00
Andrew Schofield 8d66481a83
KAFKA-17897 Deprecate Admin.listConsumerGroups (#19477)
The final part of KIP-1043 is to deprecate Admin.listConsumerGroups() in
favour of Admin.listGroups() which works for all group types.

Reviewers: PoAn Yang <payang@apache.org>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-04-17 23:00:57 +08:00
Lucas Brutschy 5f80de3923
KAFKA-19162: Topology metadata contains non-deterministically ordered topic configs (#19491)
Topology description sent to broker in KIP-1071 contains
non-deterministically ordered topic configs.  Since the topology is
compared to the groups topology upon joining we may run into
`INVALID_REQUEST: Topology updates are not supported yet` failures if
the topology sent by the  application does not match the group topology
due to different topic config order.

This PR ensures that topic configs are ordered, to avoid an
`INVALID_REQUEST` error.

Reviewers: Matthias J. Sax <matthias@confluent.io>
2025-04-16 21:12:17 -07:00
yunchi effbad9e80
KAFKA-19151 docs: clarify that flush.ms requires log.flush.scheduler.interval.ms config (#19479)
Enhanced docs of `flush.ms` to remind users the flush is triggered by
`log.flush.scheduler.interval.ms`.

Reviewers: PoAn Yang <payang@apache.org>, Ken Huang
<s7133700@gmail.com>, TengYao Chi <kitingiao@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-04-17 11:19:44 +08:00
TaiJuWu 23e7158665
KAFKA-19002 Rewrite ListOffsetsIntegrationTest and move it to clients-integration-test (#19460)
the following tasks should be addressed in this ticket  rewrite it by
1. new test infra
2. use java
3. move it to clients-integration-test

Reviewers: TengYao Chi <kitingiao@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-04-17 02:26:23 +08:00
Andrew Schofield 6a4207f12a
KAFKA-19158: Add SHARE_SESSION_LIMIT_REACHED error code (#19492)
Add the new `SHARE_SESSION_LIMIT_REACHED` error code which is used when
an attempt is made to open a new share session when the share session
limit of the broker has already been reached. Support in the client and
broker will follow in subsequent PRs.

Reviewers: Lianet Magrans <lmagrans@confluent.io>
2025-04-16 18:00:07 +01:00
David Jacot 6e26ec06bb
MINOR: Update GroupCoordinator interface to use AuthorizableRequestContext instead of RequestContext (#19485)
This patch updates the `GroupCoordinator` interface to use
`AuthorizableRequestContext` instead of using `RequestContext`. It makes
the interface more generic. The only downside is that the request
version in `AuthorizableRequestContext` is an `int` instead of a `short`
so we had to adapt it in a few places. We opted for using `int` directly
wherever possible.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, Rajini Sivaram <rajinisivaram@googlemail.com>
2025-04-16 09:12:11 -07:00
Ken Huang ae608c1cb2
KAFKA-19042 Move PlaintextConsumerCallbackTest to client-integration-tests module (#19298)
Use Java to rewrite `PlaintextConsumerCallbackTest` by new test infra
and move it to client-integration-tests module.

Reviewers: TengYao Chi <kitingiao@gmail.com>, Chia-Ping Tsai
 <chia7712@gmail.com>
2025-04-16 11:57:14 +08:00
Mickael Maison fb2ce76b49
KAFKA-18888: Add KIP-877 support to Authorizer (#19050)
This also adds metrics to StandardAuthorizer

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, Ken Huang
 <s7133700@gmail.com>, Jhen-Yung Hsu <jhenyunghsu@gmail.com>, TaiJuWu
 <tjwu1217@gmail.com>
2025-04-15 19:40:24 +02:00
Ritika Reddy 598eb13d07
KAFKA-15370: ACL changes to support 2PC (KIP-939) (#19364)
This patch adds ACL support for 2PC as a part of KIP-939

A new value will be added to the enum AclOperation: TWO_PHASE_COMMIT
((byte) 15 .  When InitProducerId comes with enable2Pc=true, it would
have to have both WRITE and TWO_PHASE_COMMIT operation enabled on the
transactional id resource.

The kafka-acls.sh tool is going to support a new --operation
TwoPhaseCommit.

Reviewers: Artem Livshits <alivshits@confluent.io>, PoAn Yang
 <poan.yang@suse.com>, Justine Olshan <jolshan@confluent.io>
2025-04-15 08:39:46 -07:00
Shivsundar R f737ef31d9
KAFKA-18900: Implement share.acknowledgement.mode to choose acknowledgement mode (#19417)
Choose the acknowledgement mode based on the config
(`share.acknowledgement.mode`) and not on the basis of how the user
designs the application.
- The default value of the config is `IMPLICIT`, so if any
empty/null/invalid value is configured, then the mode defaults to
`IMPLICIT`.
- Removed AcknowledgementModes `UNKNOWN` and `PENDING` as they are no
longer required.
- Added code to ensure if the application has any unacknowledged records
in a batch in "`explicit`" mode, then it will throw an
`IllegalStateException`. The expectation is if the mode is "explicit",
all the records received in that `poll()` would be acknowledged before
the next call to `poll()`.
- Modified the `ConsoleShareConsumer` to configure the mode to
"explicit" as it was using the explicit mode of acknowledging records.

Reviewers: Andrew Schofield <aschofield@confluent.io>
2025-04-15 16:38:33 +01:00