Commit Graph

3175 Commits

Author SHA1 Message Date
Matthias J. Sax faff2de6a5
KAFKA-17561: add processId tag to thread-state metric (#18581)
Part of KIP-1091.

Reviewers: Bill Bejeck <bill@confluent.io>
2025-01-21 08:09:43 -08:00
Sanskar Jhajharia bcbc72e29b
[KAFKA-16720] AdminClient Support for ListShareGroupOffsets (1/n) (#18571)
Reviewers: Apoorv Mittal <apoorvmittal10@gmail.com>, Andrew Schofield <aschofield@confluent.io>
2025-01-20 07:47:14 +00:00
Matthias J. Sax 07c860b730
MINOR: change warning suppression in RocksDBTimeOrderedKeyValueBufferTest (#18580)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, Mickael Maison <mickael.maison@gmail.com>
2025-01-16 12:59:01 -08:00
Jason Taylor 23c459286b
KAFKA-16368: Update defaults for LOG_MESSAGE_TIMESTAMP_AFTER_MAX_MS_DEFAULT and NUM_RECOVERY_THREADS_PER_DATA_DIR_CONFIG (#18106)
Reviewers: Divij Vaidya <diviv@amazon.com>
2025-01-16 15:30:00 +01:00
A. Sophie Blee-Goldman c7d3999d4f
KAFKA-18026: KIP-1112, clean up graph node grace period resolution (#18342)
Minor followup to #18195 that I split out into a separate PR since that one was getting a bit long. Should be rebased & reviewed after that one is merged.

Introduces a new class for windowed graph nodes with a grace period defined to improve (slightly) the type safety

Reviewers: Guozhang Wang <guozhang.wang.us@gmail.com>, Almog Gavra <almog@responsive.dev>
2025-01-15 18:01:19 -08:00
Matthias J. Sax 0bc770d7b6
MINOR: avoid WARN logs in KafkaStreams test (#18517)
Avoiding 'WARN Method #getTimestampedKeyValueStore() should be
used to access a TimestampedKeyValueStore.' running
KTableKTableForeignKeyJoinIntegrationTest.

Reviewers: Bill Bejeck <bill@confluent.io>
2025-01-15 14:08:55 -08:00
Matthias J. Sax 3f3bc90509
MINOR: Kafka Streams doc updates for 4.0 release (#18488)
Reviewers: Bill Bejeck <bill@confluent.io>
2025-01-15 11:58:53 -08:00
Matthias J. Sax 30f94b5320
KAFKA-18479: RocksDBTimeOrderedKeyValueBuffer not initialized correctly (#18490)
RocksDBTimeOrderedKeyValueBuffer is not initialize with serdes provides
via Joined, but always uses serdes from StreamsConfig.

Reviewers: Bill Bejeck <bill@confluent.io>
2025-01-15 10:04:13 -08:00
Alieh Saeedi 512e0c9628
MINOR: remove old comment in streams StateDirectory (#18546)
Reviewers: Bruno Cadonna <cadonna@apache.org>, Chia-Ping Tsai <chia7712@gmail.com>
2025-01-16 01:07:55 +08:00
Matthias J. Sax 1672a4bc27
KAFKA-18476: KafkaStreams should swallow TransactionAbortedException (#18487)
TransactionAbortedException is a follow up error to a previous error,
and such a previous error would already be handled when
`producer.abortTransaction()` is called. Thus, a
TransactionAbortedException can just be silently swallowed.

Reviewers: Bill Bejeck <bill@confluent.io>
2025-01-14 18:03:14 -08:00
A. Sophie Blee-Goldman faef80a2e2
KAFKA-18026: KIP-1112, clean up StatefulProcessorNode (#18195)
Final cleanup of StatefulProcessorNode after converting all stateful operators to adding state stores via implementing the #stores method.
2025-01-14 17:42:58 -08:00
Matthias J. Sax 0e831d0840
KAFKA-8116: Update Kafka Streams archetype for Java 11 (#18471)
Reviewers: Ismael Juma <ismael@juma.me.uk>, Bill Bejeck <bill@confluent.io>
2025-01-10 15:45:39 -08:00
Almog Gavra 2a0c31e768
MINOR: improve StreamThread periodic processing log (#18430)
The current log is really helpful, this PR adds a bit more information to that log to help debug some issues. In particular, it is interesting to be able to debug situations that have long intervals between polls. It also includes a reference to how long it has been since it last logged so you don't have to find the previous time it was logged to compute quick per-second ratios.

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2025-01-09 11:01:43 -08:00
Ao Li ebe7dd2b15
KAFKA-18418: Use CDL to block the thread termination to avoid flaky tests (#18418)
This PR fixed a race-condition in KafkaStreamsTest, by replacing a non-deterministic
sleep with a CountDownLatch to fully control how long a thread blocks.

Reviewers:  David Arthur <mumrah@gmail.com>, Matthias J. Sax <matthias@confluent.io>
2025-01-09 10:19:40 -08:00
Swikar Patel 8b72204bfd
KAFKA-15443: Upgrade RocksDB to 9.7.3 (#18275)
This PR upgrades RocksDB from 7.9.2 to 9.7.3 and addresses the following compatibility issues introduced by the RocksDB upgrade:

- Removal of AccessHint: The AccessHint class was completely removed in RocksDB 9.7.3. This required removing all import statements, variable declarations, method parameters, method return types, and static method calls related to AccessHint in RocksDBGenericOptionsToDbOptionsColumnFamilyOptionsAdapter.java RocksDBGenericOptionsToDbOptionsColumnFamilyOptionsAdapterTest.java Unused methods are removed in RocksDBGenericOptionsToDbOptionsColumnFamilyOptionsAdapter.java
- Removal of NO_FILE_CLOSES: The NO_FILE_CLOSES metric was also removed in RocksDB 9.7.3. The calculation for numberOfOpenFiles in RocksDBMetricsRecorder.java has been adjusted to now track the total number of file opens since the last reset. The previous calculation, which subtracted NO_FILE_CLOSES from NO_FILE_OPENS, is no longer possible. The reason RocksDB removed NO_FILE_CLOSES seems to be that it did not properly work: https://github.com/search?q=repo%3Afacebook%2Frocksdb+NO_FILE_CLOSES&type=issues
- Removal of methods related to compressed block cache configuration in BlockBasedTableConfig
- Change of the signature of org.rocksdb.Options.setLogger()

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>, Matthias J. Sax <matthias@confluent.io>, Bruno Cadonna <cadonna@apache.org>
2025-01-09 11:55:56 +01:00
Almog Gavra 26771a6d28
KAFKA-18326: fix merge iterator with cache tombstones (#18287)
See https://issues.apache.org/jira/browse/KAFKA-18326 for more information. The main bug here is that in the old implementation, deleted cache entries would be skipped so long as they didn't equal the next store key, which resulted in potentially skipping tombstones for future keys in the store.

Reviewers: Guozhang Wang <guozhang.wang.us@gmail.com>, Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2025-01-07 23:03:44 -08:00
Matthias J. Sax 738bd928f1
MINOR: cleanup JavaDocs for deprecation warnings (#18402)
Reviewers: Bill Bejeck <bbejeck@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2025-01-07 09:18:33 +00:00
Ismael Juma 409a43eff7
MINOR: Collection/Option usage simplification via methods introduced in Java 9 & 11 (#18305)
Relevant methods:
1. `List.of`, `Set.of`, `Map.of` and similar (introduced in Java 9)
2. Optional: `isEmpty` (introduced in Java 11), `stream` (introduced in Java 9).

Reviewers: Mickael Maison <mimaison@users.noreply.github.com>
2025-01-03 16:13:39 -08:00
Ismael Juma 73ab7ee4ea
MINOR: Use `Files.readString/writeString` and `String.repeat` to simplify code (#18372)
The 3 methods were introduced in Java 11.

Reviewers: Divij Vaidya <diviv@amazon.com>
2025-01-02 17:50:27 -08:00
A. Sophie Blee-Goldman 86b552835c
KAFKA-18026: KIP-1112, configure all StoreBuilder & StoreFactory layers (#18345)
Because of how we have to wrap StoreFactory and StoreBuilder layers on top of each other for various parts of the topology building process, we need to make sure both of these are capable of configuration and will check for & delegate to an underlying layer if it exists

Reviewers: Guozhang Wang <guozhang.wang.us@gmail.com>
2024-12-31 11:45:29 -08:00
Matthias J. Sax 8c21857877
KAFKA-13722: code cleanup after deprecated StateStore.init() was removed (#18249)
We remove the deprecated overload of StateStore.init() and thus do not
need to cast any longer. This PR removes all unnecessary casts, and
additionally cleans ups all related classed to reduce warnings.

Reviewers: Bill Bejeck <bill@confluent.io>
2024-12-20 14:31:44 -08:00
PoAn Yang 753a003480
KAFKA-18262 Remove DefaultPartitioner and UniformStickyPartitioner (#18204)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-12-20 15:33:51 +08:00
Matthias J. Sax f7ae4e8b71
KAFKA-12829: Remove old Processor and ProcessorSupplier interfaces (#18238)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, Bill Bejeck <bill@confluent.io>
2024-12-19 22:26:17 -08:00
Matthias J. Sax 9aa900f076
KAFKA-12829: Remove deprecated Topology#addProcessor of old Processor API (#18154)
Reviewers: John Huang <pegasashjy@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>, Bill Bejeck <bill@confluent.io>, Lucas Brutschy <lbrutschy@confluent.io>
2024-12-19 16:05:16 -08:00
Matthias J. Sax b3b40bb77b
KAFKA-13722: Refactor Kafka Streams store interfaces (#18243)
Refactor Segments and TimestampedSegments to not use old
ProcessorContext any longer.

Reviewers: Bruno Cadonna <bruno@confluent.io>
2024-12-19 11:55:57 -08:00
Matthias J. Sax fe9847dd4a
KAFKA-17131: Refactor TimeDefinitions (#18241)
Refactor TimeDefintiions to not use old ProcessorContext any longer.

Reviewers: Bruno Cadonna <bruno@confluent.io>
2024-12-19 11:40:43 -08:00
Bruno Cadonna a0291a8d50
MINOR: Fix flaky state updater test (#18253)
The tests are flaky because the tests end before the verified calls
are executed. This happens because the state updater thread executes
the verified calls, but the thread that executes the
tests with the verifications is a different thread.

This commit fixes the flaky tests by enusring that the calls were
performed by the state updater by either shutting down the state
updater or waiting for the condition.

Reviewers: Lucas Brutschy <lbrutschy@confluent.io>, Matthias J. Sax <matthias@confluent.io>
2024-12-19 09:02:38 +01:00
Matthias J. Sax e551cb7bb3
KAFKA-13722: Refactor SerdeGetter (#18242)
Reviewers: Bruno Cadonna <cadonna@apache.org>, Chia-Ping Tsai <chia7712@gmail.com>
2024-12-18 19:01:46 +08:00
Almog Gavra 4bcbf9fae7
KAFKA-18026: KIP-1112 migrate KTableSuppressProcessorSupplier (#18150)
Migrates KTableSuppressProcessorSupplier to use the the ProcessorSupplier#stores() method

Reviewers: Guozhang Wang <guozhang.wang.us@gmail.com>, Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2024-12-17 23:45:49 -08:00
Rohan 501da383fa
KAFKA-18026: transition KTable#filter impl to use processor wrapper (#18205)
This patch transitions the KTable#filter implementation to provide the materialized store via the ProcessorSupplier so that it can be wrapped by the processor wrapper if the wrapper is configured

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2024-12-17 23:32:24 -08:00
Matthias J. Sax 16f1aa84df
MINOR: improve Kafka Streams config docs (#18087)
Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2024-12-16 15:21:52 -08:00
Matthias J. Sax 85c43e49e3
MINOR: improving some internal comments (#18152)
Reviewers: Yaroslav Kutsela (@Serwios), Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2024-12-16 15:21:03 -08:00
David Jacot 84049369c1
MINOR: Bump trunk to 4.1.0-SNAPSHOT (#18213)
Reviewers: Mickael Maison <mickael.maison@gmail.com>
2024-12-16 16:47:13 +01:00
A. Sophie Blee-Goldman bac8928521
KAFKA-18026: KIP-1112, migrate foreign-key joins to use ProcesserSupplier#stores (#18194)
Convert FKJ processors to implementing the #stores method

Reviewers: Guozhang Wang <guozhang.wang.us@gmail.com>
2024-12-15 20:00:42 -08:00
Rohan ed10fc63a9
KAFKA-18026: supply stores for KTable#mapValues using ProcessorSupplier#stores (#18155)
KAFKA-18026: supply stores for KTable#mapValues using ProcessorSupplier#stores

Reviewers: Guozhang Wang <guozhang.wang.us@gmail.com>, Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2024-12-14 20:18:49 -08:00
A. Sophie Blee-Goldman 91575892d2
HOTFIX: RocksDBMetricsRecorder#init should null check taskId (#18151)
Appears to be a typo in the code, since the error message indicates this check is for taskId being null, but instead we accidentally check the streams metrics twice

Reviewers: Matthias Sax <mjsax@apache.org>, runo Cadonna <cadonna@apache.org>, Lucas Brutschy <lbrutschy@confluent.io>, Bill Bejeck <bbejeck@gmail.com>
2024-12-13 20:36:08 -08:00
Lianet Magrans 84bc0c26ee
KAFKA-18224: Explicit group protocol setting in streams resetter (#18172)
Reviewers: Matthias J. Sax <matthias@confluent.io>
2024-12-13 14:31:50 -05:00
TengYao Chi b37b89c668
KAFKA-9366 Upgrade log4j to log4j2 (#17373)
This pull request replaces Log4j with Log4j2 across the entire project, including dependencies, configurations, and code. The notable changes are listed below:

1. Introduce Log4j2 Instead of Log4j
2. Change Configuration File Format from Properties to YAML
3. Adds warnings to notify users if they are still using Log4j properties, encouraging them to transition to Log4j2 configurations

Co-authored-by: Lee Dongjin <dongjin@apache.org>

Reviewers: Luke Chen <showuon@gmail.com>, Mickael Maison <mickael.maison@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2024-12-14 01:14:31 +08:00
A. Sophie Blee-Goldman ef2e4600f3
KAFKA-18026: KIP-1112, migrate stream-stream joins to use ProcesserSupplier#stores (#18111)
Covers wrapping of processors and state stores for KStream-KStream joins.

Includes self-joins and the spurious results fix optimization

Reviewers: Guozhang Wang <guozhang.wang.us@gmail.com>
2024-12-12 14:54:58 -08:00
Almog Gavra 9b776ffc50
KAFKA-18026: KIP-1112 convert StreamToTableNode (#18149)
Covers wrapping of processors and state stores for StreamToTableSource

Reviewers: Guozhang Wang <guozhang.wang.us@gmail.com>, Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2024-12-12 14:52:21 -08:00
Lianet Magrans 7a64623e40
Set protocol for streams tests (#18160)
Reviewers: Bill Bejeck <bill@confluent.io>
2024-12-12 13:33:43 -05:00
Matthias J. Sax a0a501952b
MINOR: improve Kafka Streams metrics documentation (#17900)
Reviewers: Bruno Cadonna <bruno@confluent.io>, Guozhang Wang <guozhang.wang.us@gmail.com>
2024-12-11 18:34:43 -08:00
Almog Gavra 21563380f3
KAFKA-18026: KIP-1112, migrate table-table joins to use ProcesserSuppliers#stores (#18048)
Covers wrapping of processors and state stores for KTable-KTable joins

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>, Guozhang Wang <guozhang.wang.us@gmail.com>
2024-12-11 17:37:34 -08:00
Matthias J. Sax 6cdb8c352a
KAFKA-18015: add byDuration auto.offset.reset to Kafka Streams (#18115)
Part of KIP-1106.

Adds support for "by_duration" and "none" reset strategy
to the Kafka Streams runtime.

Reviewers: Bill Bejeck <bill@confluent.io>, Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2024-12-11 15:12:16 -08:00
Matthias J. Sax 990c8c750c
MINOR: remove old procesor API MockInternalProcessorContext (#18103)
Reviewers: Bill Bejeck <bill@confluent.io>
2024-12-11 15:09:13 -08:00
Matthias J. Sax ab2facca58
KAFKA-12829: Remove deprecated KStream.process() for old Processor API (#18088)
Reviewers: Bill Bejeck <bill@confluent.io>
2024-12-11 14:28:47 -08:00
KApolinario1120 d83f09d014
KAFKA-18015: Add support for duration based offset reset strategy to Kafka Streams (#17973)
Part of KIP-1106.

Adds the public APIs to Kafka Streams, to support the the newly added "by_duration" reset policy,
plus adds the missing "none" reset policy. Deprecates the enum `Topology.AutoOffsetReset` and
all related methods, and replaced them with new overload using the new `AutoOffsetReset` class.

Co-authored-by: Matthias J. Sax <matthias@confluent.io>

Reviewers: Matthias J. Sax <matthias@confluent.io>, Bill Bejeck <bill@confluent.io>, Manikumar Reddy <manikumar.reddy@gmail.com>
2024-12-11 10:47:25 -08:00
mingdaoy 4603f7495e
KAFKA-18030 Remove old upgrade-system-tests modules (#17843)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-12-10 11:19:14 +08:00
TengYao Chi e8837465a5
KAFKA-18067: Kafka Streams can leak Producer client under EOS (#17931)
To avoid leaking producers, we should add a 'closedflag toStreamProducer` indicating whether we should reset prouder.

Reviewers: Guozhang Wang <guozhang.wang.us@gmail.com>, Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2024-12-09 16:12:05 -08:00
Joao Pedro Fonseca Dantas d5c2029434
KAFKA-16339: [4/4 KStream#flatTransformValues] Remove Deprecated "transformer" methods and classes (#17882)
Reviewer: Matthias J. Sax <matthias@confluent.io>
2024-12-08 20:10:11 -08:00