Commit Graph

15606 Commits

Author SHA1 Message Date
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
Alieh Saeedi 443c01ca80
MINOR: add repartitionSourceTopics to Streams group description (#19561)
This is a follow-up of this #19433  This PR aims at adding the
`repartition source topics` to the output of `--describe` for streams
groups.

Reviewers: Lucas Brutschy <lbrutschy@confluent.io>
2025-04-28 11:54:53 +02:00
Gantigmaa Selenge 416f8941ca
MINOR: Update jgit to the latest version (#19564)
Upgrade to the latest version, 7.2.0.202503040940-r to address CVE-2023-4759.

Reviewers: Luke Chen <showuon@gmail.com>
2025-04-28 08:29:46 +08: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
PoAn Yang 7293f3a90e
KAFKA-19183 Replace Pool with ConcurrentHashMap (#19535)
1. Replace `Pool.scala` with `ConcurrentHashMap`.
2. Remove `PoolTest.scala`.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-04-27 23:19:49 +08:00
Nick Guo 51ef2903f7
KAFKA-19178 Replace Vector by ArrayList for PluginClassLoader#getResources (#19529)
The vector is a synchronized collection, and in the case we don't need
to sync. Also, we can use `Collections.enumeration` to convert
collection to enumeration easily.

Reviewers: PoAn Yang <payang@apache.org>, Ken Huang
 <s7133700@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-04-26 23:39:02 +08:00
Matthias J. Sax 0cf2f0e55d
MINOR: update KafkaStreamsTelemetryIntegrationTest for Java17 (#19567)
Reviewers: Bill Bejeck <bill@confluent.io>
2025-04-25 16:43:18 -07: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
Matthias J. Sax 6462f7a0e2
MINOR: code cleanup GroupMetadataManagerTestContext (#19554)
Removes redundant calls, and updates code to Java 17.

Reviewers: Lucas Brutschy <lbrutschy@confluent.io>
2025-04-25 10:12:55 -07: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
David Jacot 5a9681651f
MINOR: Move request static validations to GroupCoordinatorService (#19556)
This patches moves the static request validations from the
`GroupMetadataManager` to the `GroupCoordinatorService`. We already had
static validation in the service for other requests so it makes sense to
consolidate all the static validations at the same place. Moreover, it
also prevents faulty requests from unnecessarily using group
coordinator's resources.

Reviewers: Lucas Brutschy <lbrutschy@confluent.io>, Andrew Schofield <aschofield@confluent.io>
2025-04-25 08:32:33 -07:00
Lucas Brutschy d087ade527
HOTFIX: Fix StreamThreadTest (#19562)
Commit 732ed06 changed the logic of handling shutdowns, but in parallel
commit 3fae785 had introduced a new unit test for checking how to shut
down, which was broken by the later commit.

Reviewers: David Jacot <djacot@confluent.io>
2025-04-25 15:03:39 +02:00
PoAn Yang 965743c35b
KAFKA-19131: Adjust remote storage reader thread maximum pool size to avoid illegal argument (#19532)
The remote storage reader thread pool use same count for both maximum
and core size. If users adjust the pool size larger than original value,
it throws `IllegalArgumentException`. Updated both value to fix the
issue.

---------

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

Reviewers: Kamal Chandraprakash <kamal.chandraprakash@gmail.com>
2025-04-25 15:36:17 +05:30
Lucas Brutschy b3161ba1d1
HOTFIX: GroupMetadataManagerTest fails to compile (#19558)
Commit 369cc56 added a new parameter to newStreamsGroupEpochRecord, but
did not update the test that was added in 732ed06, breaking compilation.

Reviewers: David Jacot <djacot@confluent.io>
2025-04-25 11:16:46 +02:00
PoAn Yang 369cc569b4
KAFKA-17747: [1/N] Add MetadataHash field to Consumer/Share/StreamGroupMetadataValue (#19504)
* Add MetadataHash field to ConsumerGroupMetadataValue,
ShareGroupMetadataValue, and StreamGroupMetadataValue.
* Add metadataHash field to
GroupCoordinatorRecordHelpers#newConsumerGroupEpochRecord,
GroupCoordinatorRecordHelpers#newShareGroupEpochRecord, and
StreamsCoordinatorRecordHelpers#newStreamsGroupEpochRecord.
* Add deprecated message to ConsumerGroupPartitionMetadataKey and
ConsumerGroupPartitionMetadataValue.
* ShareGroupPartitionMetadataKey / ShareGroupPartitionMetadataValue /
StreamGroupPartitionMetadataKey / StreamGroupPartitionMetadataValue will
be removed in next PR.

Reviewers: Lucas Brutschy <lbrutschy@confluent.io>, David Jacot <djacot@confluent.io>

---------

Signed-off-by: PoAn Yang <payang@apache.org>
2025-04-25 01:21:08 -07: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
Andrew Schofield 2ce7c44612
KAFKA-19198: Resolve NPE when topic assigned in share group is deleted (#19552)
This PR just resolves an NPE when a topic assigned in a share group is
deleted. The NPE is caused by code which uses the current metadata image
to convert from a topic ID to the topic name. For a deleted topic, there
is no longer any entry in the image. A future PR will properly handle
the topic deletion.

Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>, PoAn Yang <payang@apache.org>
2025-04-25 08:44:56 +01:00
Lucas Brutschy e79f5f0f65
KAFKA-19195: Only send the right group ID subset to each GC shard (#19548)
If a streams, share or consumer group is described, all group IDs sent
to all shards of the group coordinator. This change fixes it. It tested
in the unit tests, since it's somewhat inconvenient to test the passed
read operation lambda.

Reviewers: David Jacot <djacot@confluent.io>, Andrew Schofield
<aschofield@confluent.io>
2025-04-25 09:14:24 +02:00
Alieh Saeedi ee4debb9f0
KAFKA-19128: Kafka Streams should not get offsets when close dirty (#19450)
Kafka Streams calls `prepareCommit()` in `Taskmanager#closeTaskDirty()`.
However, the dirty task must not get committed and therefore,
prepare-commit tasks such as getting offsets should not be needed as
well. The only thing needed before closing a task dirty is flushing.
Therefore, separating `flush` and `prepareCommit` could be a good fix.

Reviewers: Bill Bejeck <bill@confluent.io>, Matthias J. Sax <matthias@confluent.io>
2025-04-24 21:23:07 -07: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
Colin Patrick McCabe 22b89b6413
KAFKA-19192; Old bootstrap checkpoint files cause problems updated servers (#19545)
Old bootstrap.metadata files cause problems with server that include
KAFKA-18601. When the server tries to read the bootstrap.checkpoint
file, it will fail if the metadata.version is older than 3.3-IV3
(feature level 7). This causes problems when these clusters are
upgraded.

This PR makes it possible to represent older MVs in BootstrapMetadata
objects without causing an exception. An exception is thrown only if we
attempt to access the BootstrapMetadata. This ensures that only the code
path in which we start with an empty metadata log checks that the
metadata version is 7 or newer.

Reviewers: José Armando García Sancio <jsancio@apache.org>, Ismael Juma
 <ismael@juma.me.uk>, PoAn Yang <payang@apache.org>, Liu Zeyu
 <zeyu.luke@gmail.com>, Alyssa Huang <ahuang@confluent.io>
2025-04-24 15:43:35 -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
snehashisp d6133f6997
KAFKA-18988: Connect Multiversion Support (Updates to status and metrics) (#17988)
Reviewers: Greg Harris <greg.harris@aiven.io>
2025-04-24 10:23:31 -07: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
Andrew Schofield 039ba5e29d
MINOR: Improve member epoch mismatch logging for share groups (#19549)
The heartbeat logic for share groups is tricky when the set of
topic-partitions eligible for assignment changes. We have observed epoch
mismatches when brokers are restarted, which should not be possible.
Improving the logging so we can see the previous member epoch and tally
this with the logged state.

Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>, Sushant Mahajan <smahajan@confluent.io>
2025-04-24 16:18:30 +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
Sushant Mahajan 6fe1598e6b
KAFKA-18170: Add scheduled job to snapshot cold share partitions. (#19443)
* There could be scenarios where share partition records in
`__share_group_state` internal topic are not updated for a while
implying these partitions are basically cold.
* In this situation, the presence of these holds back the
pruner from keeping the topic clean and of manageable size.
* To remedy the situation, we have added a periodic
`setupSnapshotColdPartitions` in `ShareCoordinatorService` which does a
writeAll operation on the associated shards in the coordinator and
forces snapshot creation for any cold partitions. In this way the pruner
can continue.
This job has been added as a timer task.
* A new internal config
`share.coordinator.cold.partition.snapshot.interval.ms` has been
introduced to set the period of the job.
* Any failures are logged and ignored.
* New tests have been added to verify the feature.

Reviewers: PoAn Yang <payang@apache.org>, Andrew Schofield <aschofield@confluent.io>
2025-04-23 11:52:28 +01:00
Chirag Wadhwa 43350274e8
KAFKA-19156: Streamlined share group configs, with usage in ShareSessionCache (#19505)
This PR removes the group.share.max.groups config. This config was used
to calculate the maximum size of share session cache. But with the new
config group.share.max.share.sessions in place with exactly this
purpose, the ShareSessionCache initialization has also been passed the
new config.

Refer: [KAFKA-19156](https://issues.apache.org/jira/browse/KAFKA-19156)

Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>, Andrew Schofield <aschofield@confluent.io>, Chia-Ping Tsai <chia7712@gmail.com>
2025-04-23 10:52:02 +01:00
Chirag Wadhwa b088ba7999
KAFKA-19181: removed assertions in test_share_multiple_partitions as a result of change in assignor algorithm (#19541)
The system test `ShareConsumerTest.test_share_multiple_partitions`
started failing because of the recent change in the SimpleAssignor
algorithm. The tests assumed that if a share group is subscribed to a
topic, then every share consumers part of the group will be assigned all
partitions of the topic. But that does not happen now, and partitions
are split between the share consumers in certain cases, in which some
partitions are only assigned to a subset of share consumers. This change
removes that assumption

Reviewers: PoAn Yang <payang@apache.org>, Andrew Schofield <aschofield@confluent.io>
2025-04-23 09:46:07 +01: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
José Armando García Sancio b97a130c08
KAFKA-16538; Enable upgrading kraft version for existing clusters (#19416)
This change implements upgrading the kraft version from 0 to 1 in existing clusters.
Previously, clusters were formatted with either version 0 or version 1, and could not
be moved between them.

The kraft version for the cluster metadata partition is recorded using the
KRaftVersion control record. If there is no KRaftVersion control record
the default kraft version is 0.

The kraft version is upgraded using the UpdateFeatures RPC. These RPCs
are handled by the QuorumController and FeatureControlManager. This
change adds special handling in the FeatureControlManager so that
upgrades to the kraft.version are directed to
RaftClient#upgradeKRaftVersion.

To allow the FeatureControlManager to call
RaftClient#upgradeKRaftVersion is a non-blocking fashion, the kraft
version upgrade uses optimistic locking. The call to
RaftClient#upgradeKRaftVersion does validations of the version change.
If the validations succeeds, it generates the necessary control records
and adds them to the BatchAccumulator.

Before the kraft version can be upgraded to version 1, all of the
brokers and controllers in the cluster need to support kraft version 1.
The check that all brokers support kraft version 1 is done by the
FeatureControlManager. The check that all of the controllers support
kraft version is done by KafkaRaftClient and LeaderState.

When the kraft version is 0, the kraft leader starts by assuming that
all voters do not support kraft version 1. The leader discovers which
voters support kraft version 1 through the UpdateRaftVoter RPC. The
KRaft leader handles UpdateRaftVoter RPCs by storing the updated
information in-memory until the kraft version is upgraded to version 1.
This state is stored in LeaderState and contains the latest directory
id, endpoints and supported kraft version for each voter.

Only when the KRaft leader has received an UpdateRaftVoter RPC from all
of the voters will it allow the upgrade from kraft.version 0 to 1.

Reviewers: Alyssa Huang <ahuang@confluent.io>, Colin P. McCabe <cmccabe@apache.org>
2025-04-22 16:02:51 -07:00
grace 5af3547c6e
KAFKA-18572: Update Kafka Streams metric documenation (#18673)
Updated the Kafka Streams documentation to include metrics for tasks,
process nodes, and threads that were missing. I was unable to find
metrics such as stream-state-metrics, client-metrics,
state-store-metrics, and record-cache-metrics in the codebase, so they
are not included in this update.

Reviewers: Bill Bejeck <bbejeck@apache.org>
2025-04-22 16:32:23 -04: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
Chirag Wadhwa 22c5794bc3
KAFKA-19159: Removed time based evictions for share sessions (#19500)
Currently the share session cache is desgined like the fetch session
cache. If the cache is full and a new share session is trying to get get
initialized, then the sessions which haven't been touched for more than
2minutes are evicted. This wouldn't be right for share sessions as the
members also hold locks on the acquired records, and session eviction
would mean theose locks will need to be dropped and the corresponding
records re-delivered. This PR removes the time based eviction logic for
share sessions.

Refer: [KAFKA-19159](https://issues.apache.org/jira/browse/KAFKA-19159)

Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-04-22 14:59:35 +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
David Arthur 11904c74e0
KAFKA-19166: Fix RC tag in release script (#19518)
The release script was pushing the RC tag off of a temporary branch that
was never merged back into the release branch. This meant that our RC
and release tags were detached from the rest of the repository.

This patch changes the release script to merge the RC tag back into the
release branch and pushes both the tag and the branch.

Reviewers: Luke Chen <showuon@gmail.com>
2025-04-22 17:04:22 +08:00
Dejan Stojadinović 070892dafc
MINOR: redundant gradle expression is removed (#19200)
Note: this is a #18018 offshoot.

See this comment made by @Goooler:
https://github.com/apache/kafka/pull/18018#discussion_r1897025851


![image](https://github.com/user-attachments/assets/c41acba6-9f2a-44a5-b47f-60a4d7078100)


Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>, David Arthur <mumrah@gmail.com>, Goooler <wangzongler@gmail.com>
2025-04-22 08:24:58 +01:00
Xuan-Zhang Gong 4410d35cdc
KAFKA-19179: remove the dot from thread_dump_url (#19525)
As the title.
Ticket: https://issues.apache.org/jira/browse/KAFKA-19179

Reviewers: PoAn Yang <payang@apache.org>, Jhen-Yung Hsu
 <jhenyunghsu@gmail.com>, TengYao Chi <frankvicky@apache.org>, Nick Guo
 <lansg0504@gmail.com>, Ken Huang <s7133700@gmail.com>, Chia-Ping Tsai
 <chia7712@gmail.com>
2025-04-22 10:16:54 +08:00
Abhinav Dixit a8f49999cc
KAFKA-19019: Add support for remote storage fetch for share groups (#19437)
This PR adds the support for remote storage fetch for share groups.

There is a limitation in remote storage fetch for consumer groups that
we can only perform remote fetch for a single topic partition in a fetch
request. Since, the logic of share fetch requests is largely based on
how consumer groups work, we are following similar logic in implementing 
remote storage fetch. However, this problem should be addressed as 
part of KAFKA-19133 which should help us perform fetch for multiple 
remote fetch topic partition in a single share fetch request.

Reviewers: Jun Rao <junrao@gmail.com>
2025-04-21 15:30:24 -07:00
Xuan-Zhang Gong 18584b11ec
MINOR: ignore null judgement in LogCleaner (#19524)
about https://github.com/apache/kafka/pull/19387/files#r2052025917

Reviewers: PoAn Yang <payang@apache.org>, Chia-Ping Tsai
 <chia7712@gmail.com>, TengYao Chi <frankvicky@apache.org>
2025-04-21 21:22:56 +08: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
Jhen-Yung Hsu a04c2fed04
KAFKA-19180 Fix the hanging testPendingTaskSize (#19526)
The check for `scheduler.pendingTaskSize()` may fail if the thread pool
is too slow to consume the runnable objects

Reviewers: Ken Huang <s7133700@gmail.com>, PoAn Yang
 <payang@apache.org>, Chia-Ping Tsai <chia7712@gmail.com>
2025-04-21 20:19:42 +08:00
Mickael Maison 7710d1c951
KAFKA-14487: Move LogManager static methods/fields to storage module (#19302)
Move the static fields/methods

Reviewers: Luke Chen <showuon@gmail.com>
2025-04-21 12:03:30 +02: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
Luke Chen ed79212fb8
MINOR: bump zstd minor version to 1.5.6-10 (#19512)
Bump zstd minor version to 1.5.6-10.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-04-21 14:13:41 +08:00