Commit Graph

10528 Commits

Author SHA1 Message Date
Greg Harris 31c69ae932
KAFKA-14346: Remove hard-to-mock javax.crypto calls (#12866)
Reviewers: Chris Egerton <chrise@aiven.io>
2022-11-17 18:10:17 -05:00
Greg Harris fca5bfe13c
KAFKA-14346: Remove hard-to-mock RestClient calls (#12828)
Reviewers: Chris Egerton <chrise@aiven.io>
2022-11-17 17:51:54 -05:00
Igor Soarez 5bd556a49b
KAFKA-14303 Producer.send without record key and batch.size=0 goes into infinite loop (#12752)
This fixes an bug which causes a call to producer.send(record) with a record without a key and configured with batch.size=0 never to return.

Without specifying a key or a custom partitioner the new BuiltInPartitioner, as described in KIP-749 kicks in.

BuiltInPartitioner seems to have been designed with the reasonable assumption that the batch size will never be lower than one.

However, documentation for producer configuration states batch.size=0 as a valid value, and even recommends its use directly. [1]

[1] clients/src/main/java/org/apache/kafka/clients/producer/ProducerConfig.java:87

Reviewers: Artem Livshits <alivshits@confluent.io>, Luke Chen <showuon@gmail.com>, Ismael Juma <ismael@juma.me.uk>, Jun Rao <junrao@gmail.com>
2022-11-17 14:27:02 -08:00
Proven Provenzano 529a44d5cf
KAFKA-14375: Remove use of "authorizer-properties" from EndToEndAuthorizerTest (#12843)
- This removes use of a deprecated feature and instead has all ACL
calls going through the brokers. This work is preliminary work
needed before I can make them run in KRAFT mode.

Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>,  Igor Soarez <soarez@apple.com>
2022-11-17 21:49:00 +05:30
Greg Harris 7a93d5c833
KAFKA-14346: Replace static mocking of WorkerConfig::lookupKafkaClusterId (#12839)
Reviewers: Mickael Maison <mickael.maison@gmail.com>
2022-11-17 10:32:39 +01:00
Bill Bejeck 3012332e3d
KAFKA-14388 - Fixes the NPE when using the new Processor API with the DSL (#12861)
With the addition of the new Processor API the newly added FixedKeyProcessorNodeFactory extends the ProcessorNodeFactory class. The ProcessorNodeFactory had a private field Set<String> stateStoreNames initialized to an empty see. The FixedKeyProcessorNodeFactory also had a private field Set<String> stateStoreNames.

When executing InternalTopologyBuilder.build executing the buildProcessorNode method passed any node factory as ProcessorNodeFactory and the method references the stateStoreNames field, it's pointing to the superclass field, which is empty so the corresponding StoreBuilder(s) are never added - causing NPE in the topology.

This PR makes the field protected on the ProcessorNodeFactory class so FixedKeyProcessorNodeFactory inherits it.

The added test fails without this change.

Reviewers: Matthias J. Sax <mjsax@apache.org>,  Sophie Blee-Goldman <sophie@confluent.io>, Jorge Esteban Quilcate Otoya <quilcate.jorge@gmail.com>
2022-11-16 17:06:15 -05:00
David Jacot c2fc36f331
MINOR: Handle JoinGroupResponseData.protocolName backward compatibility in JoinGroupResponse (#12864)
This is a small refactor extracted from https://github.com/apache/kafka/pull/12845. It basically moves the logic to handle the backward compatibility of `JoinGroupResponseData.protocolName` from `KafkaApis` to `JoinGroupResponse`.

The patch adds a new unit test for `JoinGroupResponse` and relies on existing tests as well.

Reviewers: Justine Olshan <jolshan@confluent.io>, Jason Gustafson <jason@confluent.io>
2022-11-16 12:43:00 -08:00
Hao Li 76214bfb85
KAFKA-13785: Add JavaDocs for emit final (#12867)
Reviewers: Matthias J. Sax <matthias@confluent.io>
2022-11-16 11:30:24 -08:00
Jeff Kim dbf5826cd5
KAFKA-14334: Complete delayed purgatory after replication (#12783)
When a consumer makes a fetch request to a follower (KIP-392), the fetch request will sit in the purgatory until `fetch.max.wait.ms` is reached because the purgatory is not completed after replication. This patch aims to complete the delayed fetch purgatory after successfully replicating from the leader.

Reviewers: Artem Livshits <alivshits@confluent.io>, Luke Chen <showuon@gmail.com>, David Jacot <djacot@confluent.io>
2022-11-16 14:54:42 +01:00
Shay Lin 039cccca24
KAFKA-14360: Fix links in documentation (#12857)
Reviewers: Matthias J. Sax <matthias@confluent.io>
2022-11-15 13:31:35 -08:00
Christo Lolov 1894856d0e
KAFKA-13414: Replace PowerMock/EasyMock with Mockito in connect.storage.KafkaOffsetBackingStoreTest (#12418)
Reviewers: Chris Egerton <chrise@aiven.io>
2022-11-15 16:30:52 -05:00
vamossagar12 09da44ed80
KAFKA-12495: Exponential backoff retry to prevent rebalance storms when worker joins after revoking rebalance (#12561)
Reviewers: Yash Mayya <yash.mayya@gmail.com>, Luke Chen <showuon@gmail.com>, Chris Egerton <chrise@aiven.io>
2022-11-15 16:26:21 -05:00
Omnia G H Ibrahim 46bee5bcf3
KAFKA-13401: KIP-787 - MM2 manage Kafka resources with custom Admin implementation. (#12577)
Reviewers: Mickael Maison <mickael.maison@gmail.com>, Tom Bentley <tombentley@users.noreply.github.com>

Co-authored-by: Tom Bentley <tombentley@users.noreply.github.com>
Co-authored-by: oibrahim3 <omnia@apple.com>
2022-11-15 11:21:24 +01:00
Lucas Brutschy a55071a99f
KAFKA-14299: Initialize tasks in state updater (#12795)
The state updater code path puts tasks into an
"initialization queue", with created, but not initialized tasks.
These are later, during the event-loop, initialized and added
to the state updater. This might lead to losing track of those 
tasks - in particular it is possible to create
tasks twice, if we do not go once around `runLoop` to initialize
the task. This leads to `IllegalStateExceptions`. 

By handing the task to the state updater immediately and let the
state updater initialize the task, we can fulfil our promise to 
preserve the invariant "every task is owned by either the task 
registry or the state updater".

Reviewer: Bruno Cadonna <cadonna@apache.org>
2022-11-14 10:00:29 +01:00
A. Sophie Blee-Goldman e422a67d3f
KAFKA-14294: check whether a transaction is in flight before skipping a commit (#12835)
Add a new #transactionInFlight API to the StreamsProducer to expose the flag of the same name, then check whether there is an open transaction when we determine whether or not to perform a commit in TaskExecutor. This is to avoid unnecessarily dropping out of the group on transaction timeout in the case a transaction was begun outside of regular processing, eg when a punctuator forwards records but there are no newly consumer records and thus no new offsets to commit.

Reviewer: Bruno Cadonna <cadonna@apache.org>
2022-11-14 09:43:46 +01:00
A. Sophie Blee-Goldman 51b7eb7937
KAFKA-14282: stop tracking Produced sensors by processor node id (#12836)
Users have been seeing a large number of these error messages being logged by the RecordCollectorImpl:

Unable to records bytes produced to topic XXX by sink node YYY as the node is not recognized.
It seems like we try to save all known sink nodes when the record collector is constructed, by we do so by going through the known sink topics which means we could miss some nodes, for example if dynamic topic routing is used. Previously we were logging an error and would skip recording the metric if we tried to send a record from a sink node it didn't recognize, but there's not really any reason to have been tracking the sensors by node in the first place -- we can just track the actual sink topics themselves.

Reviewers: John Roesler <vvcephei@apache.org>, Christo Lolov <lolovc@amazon.com>
2022-11-11 17:58:08 -08:00
Christo Lolov 876c338a60
[KAFKA-14324] Upgrade RocksDB to 7.1.2 (#12809)
Reviewers: Bruno Cadonna <cadonna@confluent.io>, Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2022-11-11 17:48:38 -08:00
Lucas Brutschy ce5faa222b
MINOR: Fix flaky RestoreIntegrationTest (#12841)
RestoreIntegrationTest used polling to determine if a rebalance
happens on one client, but if the rebalance would happen too quickly,
the polling would not pick it up and the check would time out.

Reviewer: Bruno Cadonna <cadonna@apache.org>
2022-11-10 14:11:03 +01:00
David Jacot 7387a11625
KAFKA-14363; Add new `group-coordinator` module (KIP-848) (#12827)
Introduce new group-coordinator module that will host the future new group coordinator as part of KIP-848.

Reviewers: Ismael Juma <ismael@juma.me.uk>
2022-11-09 08:49:57 +01:00
Lucas Brutschy c034388a0a
KAFKA-14299: Avoid allocation & synchronization overhead in StreamThread loop (#12808)
The state updater code path introduced allocation and synchronization
overhead by performing relatively heavy operations in every iteration of
the StreamThread loop. This includes various allocations and acquiring
locks for handling `removedTasks` and `failedTasks`, even if the
corresponding queues are empty.

This change introduces `hasRemovedTasks` and
`hasExceptionsAndFailedTasks` in the `StateUpdater` interface that
can be used to skip over any allocation or synchronization. The new
methods do not require synchronization or memory allocation.

This change increases throughput by ~15% in one benchmark.

We extend existing unit tests to cover the slightly modified
behavior.

Reviewer: Bruno Cadonna <cadonna@apache.org>
2022-11-08 17:55:37 +01:00
Colin Patrick McCabe ac3a3687a0
MINOR: Avoid highestSupportedVersion outside tests (#12829)
We should avoid using Message.highestSupportedVersion to generate metadata records. Instead, we
need to pick the correct record version based on the current metadata version which is in effect.
In cases where there is only one record version that we know how to generate, we can hard code
that version, but it should just be a simple constant zero.

Reviewers: Jason Gustafson <jason@confluent.io>
2022-11-08 08:38:47 -08:00
Chris Egerton bb84476215
KAFKA-14098: Add meaningful client IDs for Connect workers (#12544)
Reviewers: Greg Harris <greg.harris@aiven.io>, Mickael Maison <mickael.maison@gmail.com>
2022-11-08 10:22:30 -05:00
Lucas Brutschy 4560978ed7
KAFKA-14309: FK join upgrades not tested with DEV_VERSION (#12760)
The streams upgrade system inserted FK join code for every version of the
the StreamsUpgradeTest except for the latest. Also, the original code
never switched on the `test.run_fk_join` flag for the target version of
the upgrade.

The effect was that FK join upgrades were not tested at all, since
no FK join code was executed after the bounce in the system test.

We introduce `extra_properties` in the system tests, that can be used
to pass any property to the upgrade driver, which is supposed to be
reused by system tests for switching on and off flags (e.g. for the
state restoration code).

Reviewers: Alex Sorokoumov <asorokoumov@confluent.io>, Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2022-11-07 15:46:51 -08:00
Jason Gustafson 150a0758cb
MINOR: Change system test console consumer default log level (#12819)
For tests which use the console consumer service, we are currently enabling TRACE logging by default. I have seen some system tests where this produces GBs of logging. A better default is probably DEBUG.

Reviewers: José Armando García Sancio <jsancio@apache.org>
2022-11-07 13:42:36 -08:00
Christo Lolov 4af5dae05c
KAFKA-14132: Replace PowerMock and EasyMock with Mockito in connect tests (#12823)
Reviewers: Chris Egerton <chrise@aiven.io>
2022-11-07 11:34:55 -05:00
Lucas Brutschy cd5f6c60b5
KAFKA-14299: Avoid busy polling in state updater (#12772)
The state updater can enter a busy polling loop if it
only updates standby tasks. We need to use the user-provided
poll-time to update always when using the state updater, since
the only other place where the state update blocks
(inside `waitIfAllChangelogsCompletelyRead`) is also
not blocking if there is at least one standby task.

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>, Bruno Cadonna <cadonna@apache.org>
2022-11-07 16:46:25 +01:00
Greg Harris 98ed31c95b
KAFKA-14346: Remove difficult to mock Plugins.compareAndSwapLoader usages (#12817)
Reviewers: Chris Egerton <chrise@aiven.io>
2022-11-07 09:37:24 -05:00
Ashmeet Lamba a971448f3f
KAFKA-14254: Format timestamps as dates in logs (#12684)
Improves logs withing Streams by replacing timestamps to date instances to improve readability.

Approach - Adds a function within common.utils.Utils to convert a given long timestamp to a date-time string with the same format used by Kafka's logger.

Reviewers: Divij Vaidya <diviv@amazon.com>, Bruno Cadonna <cadonna@apache.org>
2022-11-07 13:42:39 +01:00
Lucas Brutschy 37a3645e7e
KAFKA-14299: Return emptied ChangelogReader to ACTIVE_RESTORING (#12773)
The ChangelogReader starts of in `ACTIVE_RESTORING` state, and
then goes to `STANDBY_RESTORING` when changelogs from standby
tasks are added. When the last standby changelogs are removed,
it remained in `STANDBY_RESTORING`, which means that an empty
ChangelogReader could be in either `ACTIVE_RESTORING` or
`STANDBY_RESTORING` depending on the exact sequence of
add/remove operations. This could lead the state updater into
an illegal state. Instead of changing the state updater,
I propose to stengthen the state invariant of the
`ChangelogReader` slightly: it should always be in
`ACTIVE_RESTORING` state, when empty.

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>, Bruno Cadonna <cadonna@apache.org>
2022-11-07 13:35:18 +01:00
Omnia G H Ibrahim 9608f3a2e7
KAFKA-14344: Build EmbeddedKafkaCluster with common configs used for all clients (#12804)
Reviewers: Mickael Maison <mickael.maison@gmail.com>

, Tom Bentley <tbentley@redhat.com>
2022-11-07 11:11:16 +01:00
vamossagar12 7fd6a9b3e2
Kafka 12960: Follow up Commit to filter expired records from Windowed/Session Stores (#12756)
KAFKA-12960: Enforcing strict retention time for WindowStore and SessionStore

Reviewers: Luke Chen <showuon@gmail.com>, Vicky Papavasileiou
2022-11-07 11:53:34 +08:00
runom b8754c074a
KAFKA-14355: Fix integer overflow in ProducerPerformance (#12822)
Change types from int to long to avoid overflow

Reviewers: Luke Chen <showuon@gmail.com>,  Igor Soarez <soarez@apple.com>
2022-11-05 20:19:08 +08:00
Shawn fcab5fb888
Revert "KAFKA-13891: reset generation when syncgroup failed with REBALANCE_IN_PROGRESS (#12140)" (#12794)
This reverts commit c23d60d56c.

Reviewers: Luke Chen <showuon@gmail.com>
2022-11-05 14:05:48 +08:00
Justine Olshan 9467fc3305
MINOR: modified ProducerIdExpirationTest to hopefully be less flaky (#12824)
ProducerIdExpirationTest has failed on a few PR runs. Using a waitUntil instead.

Reviewers: Luke Chen <showuon@gmail.com>
2022-11-05 14:04:35 +08:00
David Jacot bc780c7c32
MINOR: Move timeline data structures from metadata to server-common (#12811)
This path moves the timeline data structures from metadata module to server-common module as those will be used in the new group coordinator.

Reviewers: José Armando García Sancio <jsancio@users.noreply.github.com>, Colin Patrick McCabe <cmccabe@apache.org>
2022-11-04 08:52:32 +01:00
Chris Egerton 14c36c7539
KAFKA-14058: Migrate ExactlyOnceWorkerSourceTaskTest from EasyMock and Powermock to Mockito (#12409)
Reviewers: Mickael Maison <mickael.maison@gmail.com>
2022-11-03 12:42:08 -04:00
Philip Nee dc18dd921c
MINOR: Remove outdated comment about metadata request (#12810)
This is an outdated comment about the metadata request from the old implementation. The name of the RPC was changed from GroupMetadata to FindCoordinator, but the comment was not updated.

Reviewers: Jason Gustafson <jason@confluent.io>
2022-11-01 17:01:19 -07:00
Lucas Brutschy e7c1e4a0a1
KAFKA-14299: Handle TaskCorruptedException during initialization (#12771)
State stores are initialized from the StreamThread even when
the state updater thread is enabled. However, we were missing
the corresponding handling of exceptions when thrown directly
during the initialization. In particular, TaskCorruptedException
would directly fall through to runLoop, and the task
would fall out of book-keeping, since the exception is thrown
when neither the StreamThread nor the StateUpdater is owning
the task.

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>, Bruno Cadonna <cadonna@apache.org>
2022-11-01 18:25:19 +01:00
Matthew de Detrich 9ab140d541
KAFKA-14132; Replace EasyMock with Mockito ConnectorsResourceTest (#12725)
Reviewers: Chris Egerton <chrise@aiven.io>
2022-10-31 20:08:41 -04:00
Greg Harris 98bf375b6f
KAFKA-13989: Errors while evaluating connector type should return UNKNOWN (#12789)
Reviewers: Igor Soarez <i@soarez.me>, Chris Egerton <chrise@aiven.io>
2022-10-31 16:54:32 -04:00
Philip Nee fa10e213bf
KAFKA-14247: add handler impl to the prototype (#12775)
Minor revision for KAFKA-14247. Added how the handler is called and constructed to the prototype code path.

Reviewers: John Roesler <vvcephei@apache.org>, Kirk True <kirk@mustardgrain.com>
2022-10-31 14:01:22 -05:00
Jason Gustafson c710ecd071
MINOR: Reduce tries in RecordsIteratorTest to improve build time (#12798)
`RecordsIteratorTest` takes the longest times in recent builds (even including integration tests). The default of 1000 tries from jqwik is probably overkill and causes the test to take 10 minutes locally. Decreasing to 50 tries reduces that to less than 30s. 

Reviewers: David Jacot <djacot@confluent.io>
2022-10-31 09:29:19 -07:00
runom fdd60cc9a7
MINOR: Fix producer.id.expiration.check.interval.ms definition (#12799)
Reviewers: David Jacot <djacot@confluent.io>
2022-10-31 09:35:44 +01:00
dengziming 56d588d55a
MINOR: Fix SubscriptionInfoData name in exception message (#12076)
Reviewers: Andrew Choi <andrewchoi5@users.noreply.github.com>, Luke Chen <showuon@gmail.com>
2022-10-31 10:48:16 +08:00
Greg Harris f7304db6d2
KAFKA-14338: Use MockTime in RetryUtilTest to eliminate flakiness (#12791)
Reviewers: Chris Egerton <chrise@aiven.io>
2022-10-29 06:38:24 -04:00
Jason Gustafson cbe50d95a9
MINOR: Add test case for topic recreation with collision chars (#12796)
This patch adds a unit test for topic recreation with colliding characters (such as `.`). This was broken up until https://github.com/apache/kafka/pull/12790. 

Reviewers: José Armando García Sancio <jsancio@users.noreply.github.com>
2022-10-28 18:50:08 -07:00
Luke Chen c1bb307a36
KAFKA-14337; Correctly remove topicsWithCollisionChars after topic deletion (#12790)
In https://github.com/apache/kafka/pull/11910 , we added a feature to prevent topics with conflicting metrics names from being created. We added a map to store the normalized topic name to the topic names, but we didn't remove it correctly while deleting topics. This PR fixes this bug and add a test.

Reviewers: Igor Soarez <i@soarez.me>, dengziming <dengziming1993@gmail.com>, Jason Gustafson <jason@confluent.io>
2022-10-28 10:08:53 -07:00
Orsák Maroš a0e37b79aa
MINOR: Add test cases to the Raft module (#12692)
Reviewers: Mickael Maison <mickael.maison@gmail.com>
, Divij Vaidya <diviv@amazon.com>, Ismael Juma <mlists@juma.me.uk>
2022-10-28 17:54:34 +02:00
(+ (* 1 2 3 4) 5 6 7) 61f48a9f63
KAFKA-14314: Add check for null upstreamTopic (#12769)
Reviewers: Mickael Maison <mickael.maison@gmail.com>
Co-authored-by: John Krupka <john.krupka@united.com>
2022-10-28 17:28:26 +02:00
Divij Vaidya 5e399fe6f3
Move to mockito (#12465)
This PR build on top of #11017. I have added the previous author's comment in this PR for attribution. I have also addressed the pending comments from @chia7712 in this PR.

Notes to help the reviewer:

Mockito has mockStatic method which is equivalent to PowerMock's method.
When we run the tests using @RunWith(MockitoJUnitRunner.StrictStubs.class) Mockito performs a verify() for all stubs that are mentioned, hence, there is no need to explicitly verify the stubs (unless you want to verify the number of times etc.). Note that this does not work for static mocks.

Reviewers: Bruno Cadonna <cadonna@apache.org>, Walker Carlson <wcarlson@confluent.io>, Bill Bejeck <bbejeck@apache.org>
2022-10-27 14:08:44 -04:00