Commit Graph

367 Commits

Author SHA1 Message Date
Masahiro Mori daece61a50
MINOR: Refactor LockUtils and improve comments (follow up to KAFKA-19390) (#20131)
CI / build (push) Waiting to run Details
This PR performs a refactoring of LockUtils and improves inline
comments, as a follow-up to https://github.com/apache/kafka/pull/19961.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, Jun Rao <junrao@gmail.com>
2025-07-15 10:07:01 -07:00
Lan Ding 6437135bc0
KAFKA-19451: fix flaky test RemoteIndexCacheTest.testCacheEntryIsDeletedOnRemoval() (#20085)
**Problem Description**
In the `RemoteIndexCache.cleanup()` method, the asynchronous invocation
of `index.deleteIfExists()` may cause a conflict. When the
`getIndexFileFromRemoteCacheDir()` method is executed, it utilizes
`Files.walk()` to traverse all files in the directory path. If
`index.deleteIfExists()` is triggered during this traversal, a
`NoSuchFileException` will be thrown.

**Solution**
To resolve this issue, ensure that `index.deleteIfExists()` has been
fully executed before invoking `getIndexFileFromRemoteCacheDir()`.

Reviewers: Jun Rao <junrao@gmail.com>
2025-07-14 12:01:50 -07: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
Jhen-Yung Hsu 007fe6e92a
KAFKA-19466 LogConcurrencyTest should close the log when the test completes (#20110)
- Fix testUncommittedDataNotConsumedFrequentSegmentRolls() and
testUncommittedDataNotConsumed(), which call createLog() but never close
the log when the tests complete.
- Move LogConcurrencyTest to the Storage module and rewrite it in Java.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-07-10 01:01:42 +08:00
Gaurav Narula 36b9bb94f1
KAFKA-19474 Move WARN log on log truncation below HWM (#20106)
CI / build (push) Waiting to run Details
#5608 introduced a regression where the check for `targetOffset <
log.highWatermark`
to emit a `WARN` log was made incorrectly after truncating the log.

This change moves the check for `targetOffset < log.highWatermark`  to
`UnifiedLog#truncateTo` and ensures we emit a `WARN` log on truncation
below  the replica's HWM by both the `ReplicaFetcherThread` and
`ReplicaAlterLogDirsThread`

Reviewers: Jun Rao <junrao@gmail.com>, Chia-Ping Tsai
 <chia7712@gmail.com>
2025-07-09 09:55:02 +08:00
Masahiro Mori ea7b145860
KAFKA-19390: Call safeForceUnmap() in AbstractIndex.resize() on Linux to prevent stale mmap of index files (#19961)
https://issues.apache.org/jira/browse/KAFKA-19390

The AbstractIndex.resize() method does not release the old memory map
for both index and time index files.  In some cases, Mixed GC may not
run for a long time, which can cause the broker to crash when the
vm.max_map_count limit is reached.

The root cause is that safeForceUnmap() is not being called on Linux
within resize(), so we have changed the code to unmap old mmap on all
operating systems.

The same problem was reported in
[KAFKA-7442](https://issues.apache.org/jira/browse/KAFKA-7442), but the
PR submitted at that time did not acquire all necessary locks around the
mmap accesses and was closed without fixing the issue.

Reviewers: Jun Rao <junrao@gmail.com>
2025-07-08 09:15:32 -07: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
Jhen-Yung Hsu 2e3ddb22ae
MINOR: Fix the tests in LogValidatorTest (#20093)
CI / build (push) Waiting to run Details
Fix incorrect tests introduced in the refactor

5b9cbcf886

Reviewers: TaiJuWu <tjwu1217@gmail.com>, Ken Huang <s7133700@gmail.com>,
Chia-Ping Tsai <chia7712@gmail.com>
2025-07-03 19:04:43 +08:00
stroller 14ea11dc31
KAFKA-19371: Don't create the __remote_log_metadata topic when it already exists during broker restarts (#19899)
* The CREATE_TOPIC request gets issued only when it is clear that the
topic does not exist in the cluster.
* When the request to describe the topic gets timed-out or any exception
thrown other than UnknownTopicOrPartitionException, then the same gets
re-thrown and the describe/create topic request gets retried in the next
iteration until the initializationRetryMaxTimeoutMs gets breached.

Fixes: https://issues.apache.org/jira/browse/KAFKA-19371

Reviewers: Luke Chen <showuon@gmail.com>, Kamal Chandraprakash
<kamal.chandraprakash@gmail.com>

---------

Co-authored-by: stroller.fu <stroller.fu@zoom.us>
2025-07-02 11:22:26 +05:30
Okada Haruki 959021de59
KAFKA-19407 Fix potential IllegalStateException when appending to timeIndex (#19972)
## Summary
- Fix potential race condition in
LogSegment#readMaxTimestampAndOffsetSoFar(), which may result in
non-monotonic offsets and causes replication to stop.
- See https://issues.apache.org/jira/browse/KAFKA-19407 for the details
how it happen.

Reviewers: Vincent PÉRICART <mauhiz@gmail.com>, Jun Rao
 <junrao@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-06-25 00:35:53 +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
Xuan-Zhang Gong 79d2c3c62a
KAFKA-19406 Remove BrokerTopicStats#removeOldFollowerMetrics (#19962)
BTW: whether we should rename
`ReplicaManager#updateLeaderAndFollowerMetrics`

Reviewers: Ken Huang <s7133700@gmail.com>, PoAn Yang
 <payang@apache.org>, TengYao Chi <kitingiao@gmail.com>, Lan Ding
 <isDing_L@163.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-06-19 17:57:22 +08:00
Kuan-Po Tseng 12d8a1bbf8
KAFKA-19237: Add dynamic config remote.log.manager.follower.thread.pool.size (#19809)
Deprecate the `remote.log.manager.thread.pool.size` configuration and introduce a new dynamic configuration:
`remote.log.manager.follower.thread.pool.size`.

Reviewers: Kamal Chandraprakash <kamal.chandraprakash@gmail.com>, Luke Chen <showuon@gmail.com>
2025-06-13 09:33:45 +05:30
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
Dmitry Werner f69379cf6b
MINOR: Remove unused code from storage classes (#19853)
CI / build (push) Waiting to run Details
Remove unused code from storage classes.

Reviewers: Kamal Chandraprakash <kamal.chandraprakash@gmail.com>,
 TengYao Chi <kitingiao@gmail.com>, Kuan-Po Tseng <brandboat@gmail.com>,
 Chia-Ping Tsai <chia7712@gmail.com>
2025-06-11 00:22:50 +08:00
Ritika Reddy 3479ce793b
KAFKA-18202: Add rejection for non-zero sequences in TV2 (KIP-890) (#19902)
This change handles rejecting non-zero sequences when there is an empty
producerIDState with TV2.  The scenario will be covered with the
re-triable OutOfOrderSequence error.

For Transactions V2 with empty state:   Allow only sequence 0 is allowed for
new producers or after state cleanup (new validation added)   Don't allow any
non-zero sequence is rejected with our specific error message   Don't allow any epoch
bumps still require sequence 0 (existing validation remains)

For Transactions V1 with empty state:   Allow ANY sequence number is allowed
(0, 5, 100, etc.)   Don't allow epoch bumps still require sequence 0 (existing
validation)

Reviewers: Justine Olshan <jolshan@confluent.io>, Artem Livshits
 <alivshits@confluent.io>
2025-06-06 09:23:10 -07:00
Ritika Reddy cc25d217da
KAFKA-18042: Reject the produce request with lower producer epoch early (KIP-890) (#19844)
CI / build (push) Waiting to run Details
With the transaction V2, replica manager checks whether the incoming
producer request produces to a partition belonging to a transaction.
ReplicaManager figures this out by checking the producer epoch stored in
the partition log. However, the current code does not reject the produce
request if its producer epoch is lower than the stored producer epoch.
It is an optimization to reject such requests earlier instead of sending
an AddPartitionToTxn request and getting rejected in the response.

Reviewers: Justine Olshan <jolshan@confluent.io>, Artem Livshits
 <alivshits@confluent.io>
2025-06-04 13:21:53 -07:00
Ken Huang bcda92b5b9
KAFKA-19080 The constraint on segment.ms is not enforced at topic level (#19371)
CI / build (push) Waiting to run Details
The main issue was that we forgot to set
`TopicConfig.SEGMENT_BYTES_CONFIG` to at least `1024 * 1024`, which
caused problems in tests with small segment sizes.

To address this, we introduced a new internal config:
`LogConfig.INTERNAL_SEGMENT_BYTES_CONFIG`, allowing us to set smaller
segment bytes specifically for testing purposes.

We also updated the logic so that if a user configures the topic-level
segment bytes without explicitly setting the internal config, the
internal value will no longer be returned to the user.

In addition, we removed
`MetadataLogConfig#METADATA_LOG_SEGMENT_MIN_BYTES_CONFIG` and added
three new internal configurations:
- `INTERNAL_MAX_BATCH_SIZE_IN_BYTES_CONFIG`
- `INTERNAL_MAX_FETCH_SIZE_IN_BYTES_CONFIG`
- `INTERNAL_DELETE_DELAY_MILLIS_CONFIG`

Reviewers: Jun Rao <junrao@gmail.com>, Chia-Ping Tsai
 <chia7712@gmail.com>
2025-05-25 20:57:22 +08:00
Hong-Yi Chen 69a457d8a5
KAFKA-19034 [1/N] Rewrite RemoteTopicCrudTest by ClusterTest and move it to storage module (#19681)
CI / build (push) Waiting to run Details
This PR rewrites `RemoteTopicCrudTest` in Java using the `@ClusterTest`
framework and moves it to the `storage` module.

**Note:** Two test cases have not yet been migrated

- `testClusterWideDisablementOfTieredStorageWithEnabledTieredTopic`
-

`testClusterWithoutTieredStorageStartsSuccessfullyIfTopicWithTieringDisabled`

These tests rely on modifying broker configs during the test lifecycle,
which `ClusterTest` currently does not support. They will be migrated in
a follow-up PR after
[#16808](https://github.com/apache/kafka/pull/16808) is merged, which
introduces support for config updates in `ClusterTest`.

Reviewers: Ken Huang <s7133700@gmail.com>, Chia-Ping Tsai
 <chia7712@gmail.com>
2025-05-25 14:50:16 +08:00
Yu-Syuan Jheng 1407b12e2f
KAFKA-19313 Replace LogOffsetMetadata#UNIFIED_LOG_UNKNOWN_OFFSET by UnifiedLog.UNKNOWN_OFFSET (#19767)
CI / build (push) Waiting to run Details
Replaces the UNIFIED_LOG_UNKNOWN_OFFSET constant in LogOffsetMetadata
with UnifiedLog.UNKNOWN_OFFSET.

Reviewers: PoAn Yang <payang@apache.org>, Ken Huang
<s7133700@gmail.com>, YuChia Ma <minecraftmiku831@gmail.com>, Chia-Ping
Tsai <chia7712@gmail.com>
2025-05-24 23:33:26 +08:00
Lucas Brutschy bff1602df3
KAFKA-19280: Fix NoSuchElementException in UnifiedLog (#19717)
In FETCH requests and TXN_OFFSET_COMMIT requests, on current trunk we
run into a race condition inside UnifiedLog, causing a
`NoSuchElementException` in
`UnifiedLog.fetchLastStableOffsetMetadata(UnifiedLog.java:651)`.

The cause is that the line a performing an `isPresent` check on a
volatile Optional before accessing it in `get`, leaving the door open to
a race condition when the optional changes between `isPresent` and
`get`. This change takes a copy of the volatile variable first.
2025-05-17 21:17:38 +02:00
Jhen-Yung Hsu ced56a320b
MINOR: Move logDirs config out of KafkaConfig (#19579)
CI / build (push) Waiting to run Details
Follow up https://github.com/apache/kafka/pull/19460/files#r2062664349

Reviewers: Ismael Juma <ismael@juma.me.uk>, PoAn Yang
<payang@apache.org>, TaiJuWu <tjwu1217@gmail.com>, Ken Huang
<s7133700@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-05-17 00:52:20 +08:00
Andrew Schofield 7ae9a26fc2
MINOR: Mark RemoteIndexCacheTest.testConcurrentRemoveReadForCache1 flaky (#19732)
Marking flaky test as a result of 5% failure rate.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-05-16 09:03:08 +01:00
YuChia Ma 05169aa201
MINOR: Add deprecation warning for `log.cleaner.enable` and `log.cleaner.threads` (#19674)
Add a warning message when using log.cleaner.enable to remind users that
this configuration is deprecated. Also, add a warning message for
log.cleaner.threads=0 because in version 5.0, the value must be greater
than zero.

Reviewers: Ken Huang <s7133700@gmail.com>, PoAn Yang
<payang@apache.org>, TengYao Chi <frankvicky@apache.org>, TaiJuWu
<tjwu1217@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-05-15 00:18:27 +08:00
Xuan-Zhang Gong 15331bbfc4
KAFKA-19273 Ensure the delete policy is configured when the tiered storage is enabled (#19702)
We updated the validation rule for cleanup.policy in remote storage
mode.

If remote log storage is enabled, only cleanup.policy=delete is allowed.
Any other value (e.g. compact, compact,delete) will now result in a
config validation error.

Reviewers: Luke Chen <showuon@gmail.com>, Ken Huang
 <s7133700@gmail.com>, PoAn Yang <payang@apache.org>, Chia-Ping Tsai
 <chia7712@gmail.com>
2025-05-15 00:13:55 +08:00
Kuan-Po Tseng 54fd1361e5
KAFKA-19264 Remove fallback for thread pool sizes in RemoteLogManagerConfig (#19673)
The fallback mechanism for `remote.log.manager.copier.thread.pool.size`
and `remote.log.manager.expiration.thread.pool.size` defaulting to
`remote.log.manager.thread.pool.size` was introduced in KIP-950. This
approach was abandoned in KIP-1030, where default values were changed
from -1 to 10, and a configuration validator enforcing a minimum value
of 1 was added. As a result, this commit removes the fallback mechanism
from `RemoteLogManagerConfig.java` to align with the new defaults and
validation.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-05-11 23:48:45 +08:00
Stanislav Kozlovski 0bc8d0c962
MINOR: Add documentation about KIP-405 remote reads serving just one partition per FetchRequest (#19336)
[As discussed in the mailing
list](https://lists.apache.org/thread/m03mpkm93737kk6d1nd6fbv9wdgsrhv9),
the broker only fetches remote data for ONE partition in a given
FetchRequest. In other words, if a consumer sends a FetchRequest
requesting 50 topic-partitions, and each partition's requested offset is
not stored locally - the broker will fetch and respond with just one
partition's worth of data from the remote store, and the rest will be
empty.

Given our defaults for total fetch response is 50 MiB and per partition
is 1 MiB, this can limit throughput. This patch documents the behavior
in 3 configs - `fetch.max.bytes`, `max.partition.fetch.bytes` and
`remote.fetch.max.wait.ms`

Reviewers: Luke Chen <showuon@gmail.com>, Kamal Chandraprakash
 <kamal.chandraprakash@gmail.com>, Satish Duggana <satishd@apache.org>
2025-05-10 16:48:55 +05:30
Hong-Yi Chen c4dc78746e
KAFKA-18537 Fix flaky RemoteIndexCacheTest#testCleanerThreadShutdown (#19628)
Add a wait for cleaner thread shutdown in `testCleanerThreadShutdown` to
eliminate flakiness. After calling `cache.close()`, the test now uses
`TestUtils.waitForCondition` to poll until the background
“remote-log-index-cleaner” thread has fully exited before asserting that
no cleaner threads remain. This ensures the asynchronous shutdown always
completes before the final assertions.

Reviewers: TengYao Chi <kitingiao@gmail.com>, Chia-Ping Tsai
 <chia7712@gmail.com>
2025-05-06 01:05:34 +08:00
Jhen-Yung Hsu 28ad4dd5c5
MINOR: Remove unnecessary Optional from offsetsToSnapshot (#19613)
CI / build (push) Waiting to run Details
Reviewers: PoAn Yang <payang@apache.org>, Ken Huang
 <s7133700@gmail.com>, TengYao Chi <kitingiao@gmail.com>, Chia-Ping Tsai
 <chia7712@gmail.com>
2025-05-03 12:58:08 +08: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
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
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
TaiJuWu 6e4e0df057
KAFKA-18891: Add KIP-877 support to RemoteLogMetadataManager and RemoteStorageManager (#19286)
1. Remove `RemoteLogManager#startup` and
`RemoteLogManager#onEndpointCreated`
2. Move endpoint creation to `BrokerServer`
3. Move `RemoteLogMetadataManager#configure` and
`RemoteLogStorageManager#configure` to RemoteLogManager constructor

Reviewers: Mickael Maison <mickael.maison@gmail.com>, Ken Huang
 <s7133700@gmail.com>, Jhen-Yung Hsu <jhenyunghsu@gmail.com>
2025-04-18 15:04:37 +02:00
Kamal Chandraprakash 2cd733c9b3
KAFKA-17184: Fix the error thrown while accessing the RemoteIndexCache (#19462)
For segments that are uploaded to remote, RemoteIndexCache caches the
fetched offset, timestamp, and transaction index entries on the first
invocation to remote, then the subsequent invocations are accessed from
local.

The remote indexes that are cached locally gets removed on two cases:

1. Remote segments that are deleted due to breach by retention size/time
and start-offset.
2. The number of cached indexes exceed the remote-log-index-cache size
limit of 1 GB (default).

There are two layers of locks used in the RemoteIndexCache. First-layer
lock on the RemoteIndexCache and the second-layer lock on the
RemoteIndexCache#Entry.

**Issue**

1. The first-layer of lock coordinates the remote-log reader and deleter
threads. To ensure that the reader and deleter threads are not blocked
on each other, we only take `lock.readLock()` when accessing/deleting
the cached index entries.
2. The issue happens when both the reader and deleter threads took the
readLock, then the deleter thread marked the index as
`markedForCleanup`. Now, the reader thread which holds the `indexEntry`
gets an IllegalStateException when accessing it.
3. This is a concurrency issue, where we mark the entry as
`markedForCleanup` before removing it from the cache. See
RemoteIndexCache#remove, and RemoteIndexCache#removeAll methods.
4. When an entry gets evicted from cache due to breach by maxSize of 1
GB, then the cache remove that entry before calling the evictionListener
and all the operations are performed atomically by caffeine cache.

**Solution**

1. When the deleter thread marks an Entry for deletion, then we rename
the underlying index files with ".deleted" as suffix and add a job to
the remote-log-index-cleaner thread which perform the actual cleanup.
Previously, the indexes were not accessible once it was marked for
deletion. Now, we allow to access those renamed files (from entry that
is about to be removed and held by reader thread) until those relevant
files are removed from disk.
2. Similar to local-log index/segment deletion, once the files gets
renamed with ".deleted" as suffix then the actual deletion of file
happens after `file.delete.delay.ms` delay of 1 minute. The renamed
index files gets deleted after 30 seconds.
3. During this time, if the same index entry gets fetched again from
remote, then it does not have conflict with the deleted entry as the
file names are different.

Reviewers: Satish Duggana <satishd@apache.org>
2025-04-18 16:43:37 +05:30
Mickael Maison c73d97de0c
KAFKA-14523: Move kafka.log.remote classes to storage (#19474)
Pretty much a straight forward move of these classes. I just updated
`RemoteLogManagerTest` to not use `KafkaConfig`

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-04-17 11:05:14 +02:00
Ken Huang b4e75fbab1
HOTFIX: add SuppressWarnings to TieredStorageTestUtils (#19494)
We need add SuppressWarnings annotation, because `log.cleaner.enable`
mark deprecated.

Reviewers: PoAn Yang <payang@apache.org>, Kuan-Po Tseng
<brandboat@gmail.com>, TengYao Chi <kitingiao@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-04-17 11:10:59 +08:00
TengYao Chi 73afcc9b69
KAFKA-13610: Deprecate log.cleaner.enable configuration (#19472)
JIRA: KAFKA-13610  This patch deprecates the `log.cleaner.enable`
configuration. It's part of
[KIP-1148](https://cwiki.apache.org/confluence/x/XAyWF).

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, PoAn Yang
 <payang@apache.org>, Ken Huang <s7133700@gmail.com>, Jhen-Yung Hsu
 <jhenyunghsu@gmail.com>
2025-04-16 10:27:44 +08:00
Mickael Maison 321a380d0a
KAFKA-14523: Decouple RemoteLogManager and Partition (#19391)
Remove the last dependency in the core module.

Reviewers: Luke Chen <showuon@gmail.com>, PoAn Yang <poan.yang@suse.com>
2025-04-15 09:56:27 +02:00
Mickael Maison d183cf9ac1
KAFKA-18172 Move RemoteIndexCacheTest to the storage module (#19469)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-04-15 15:53:41 +08:00
Dmitry Werner 7863b35064
KAFKA-14485: Move LogCleaner to storage module (#19387)
Move LogCleaner and related classes to storage module and rewrite in
Java.

Reviewers: Mickael Maison <mickael.maison@gmail.com>, Jun Rao <junrao@gmail.com>
2025-04-11 09:21:05 -07:00
Nick Guo e69a311068
KAFKA-19076 replace `String` by `Supplier<String>` for UnifiedLog#maybeHandleIOException (#19392)
jira: https://issues.apache.org/jira/browse/KAFKA-19076

the message is used when the function encounters error, so the error
message should be created lazy.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-04-07 00:43:44 +08:00
Xuan-Zhang Gong ab4a0f0ec1
MINOR: small optimization by judgment (#19386)
judgments can help avoid unnecessary `segments.sizeInBytes()`  loops

from https://github.com/apache/kafka/pull/18393/files#r2029925512

Reviewers: PoAn Yang <payang@apache.org>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-04-06 22:08:05 +08:00
TengYao Chi 74acbd200d
KAFKA-16758: Extend Consumer#close with an option to leave the group or not (#17614)
JIRA: [KAFKA-16758](https://issues.apache.org/jira/browse/KAFKA-16758)
This PR is aim to deliver

[KIP-1092](https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=321719077),
please refer to KIP-1092 and KAFKA-16758 for further details.

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>, Chia-Ping
Tsai <chia7712@gmail.com>, Kirk True <kirk@kirktrue.pro>
2025-04-05 22:02:45 -07:00
Mickael Maison 08a93fe12a
KAFKA-14523: Move DelayedRemoteListOffsets to the storage module (#19285)
Decouple RemoteLogManager and ReplicaManager.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-04-05 19:51:13 +08:00
Ken Huang ef73fb921b
MINOR: Remove DeleteSegmentsByRetentionTimeTest#executeTieredStorageTest flaky annotation (#19301)
This test was fixed by [this
commit](https://github.com/apache/kafka/pull/18861) and hasn't failed
for about two weeks. Therefore, we can remove the `@Flaky` annotation.

Gradle report:
https://develocity.apache.org/scans/tests?search.rootProjectNames=kafka&search.startTimeMax=1743081652853&search.startTimeMin=1741795200000&search.tags=github%2Ctrunk&search.timeZoneId=Asia%2FTaipei&tests.container=org.apache.kafka.tiered.storage.integration.DeleteSegmentsByRetentionTimeTest

Reviewers: TaiJuWu <tjwu1217@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-04-05 00:05:57 +08:00
Sanskar Jhajharia 03b1b720e9
MINOR: Cleanup Storage Module (#19072)
Given that now we support Java 17 on our brokers, this PR replace the
use of the following in storage module:

- Collections.singletonList() and Collections.emptyList() with List.of()
- Collections.singletonMap() and Collections.emptyMap() with Map.of()
- Collections.singleton() and Collections.emptySet() with Set.of()
- Arrays.asList() with List.of()

Reviewers: Ken Huang <s7133700@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-04-04 02:15:58 +08:00
TaiJuWu f1bb29b93a
MINOR: migrate BrokerCompressionTest to storage module (#19277)
There are two change for this PR.

1. Move `BrokerCompressionTest ` from core to storage
2. Rewrite `BrokerCompressionTest ` from scala to java

Reviewers: TengYao Chi <kitingiao@gmail.com>, PoAn Yang
<payang@apache.org>, Ken Huang <s7133700@gmail.com>, Chia-Ping Tsai
<chia7712@gmail.com>
2025-04-03 22:43:42 +08:00
PoAn Yang be80e3cb8a
KAFKA-18923: resource leak in RSM fetchIndex inputStream (#19111)
Fix resource leak in RSM inputStream.

Reviewers: Luke Chen <showuon@gmail.com>
2025-04-03 15:18:05 +08:00
PoAn Yang 4a5ae144ea
KAFKA-19032 Remove TestInfoUtils.TestWithParameterizedQuorumAndGroupProtocolNames (#19270)
The zookeeper mode was removed in 4.0. The test cases don't need to
specify quorum. Following variable and functions can be replaced:
- TestWithParameterizedQuorumAndGroupProtocolNames
- getTestQuorumAndGroupProtocolParametersClassicGroupProtocolOnly
- getTestQuorumAndGroupProtocolParametersConsumerGroupProtocolOnly
- getTestQuorumAndGroupProtocolParametersAll

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2025-03-30 02:11:07 +08:00