Commit Graph

10811 Commits

Author SHA1 Message Date
Ismael Juma aad5b0a463
MINOR: Avoid unnecessary allocations in index binary search (#13024)
* MINOR: Avoid unnecessary allocations in index binary search

* Fix bug due to inverse usage of SearchType.

Also improve name clarity.
2022-12-21 05:10:44 -08:00
Lucas Brutschy 26daa8d610
MINOR: Fix various memory leaks in tests (#12959)
Various tests in the streams park were leaking native memory.

Most tests were fixed by closing the corresponding rocksdb resource.

I tested that the corresponding leak is gone by using a previous rocksdb
release with finalizers and checking if the finalizers would be called at some
point.

Reviewer: Bruno Cadonna <cadonna@apache.org>
2022-12-21 13:38:05 +01:00
vamossagar12 409794b5ae
KAFKA-14461: Move StoreQueryIntegrationTest to junit5 and fixing logic in a couple of tests for finding active streams (#13014)
StoreQueryIntegrationTest#shouldQuerySpecificActivePartitionStores and StoreQueryIntegrationTest#shouldQueryOnlyActivePartitionStoresByDefault has a logic to find active partitions by doing a modulo with 2 and comparing the remainder. This can break when a new test is added and since Junit chooses an arbitrary order to run the tests, modulo checks can fail. This PR tries to make it deterministic.
Also, this PR uses Junit5 annotations so that the cluster and input topic can be setup/destroyed once.

Reviewer: Bruno Cadonna <cadonna@apache.org>
2022-12-21 13:33:55 +01:00
Calvin Liu 53eeaad946
MINOR: Fix testPreferredReplicaAsLeaderWhenSameRackFollowerIsOutOfIsr (#12982)
Reviewers: David Jacot <djacot@confluent.io>
2022-12-21 09:38:43 +01:00
Lucas Brutschy 6a6c730241
KAFKA-14532: Correctly handle failed fetch when partitions unassigned (#13023)
The failure handling code for fetches could run into an IllegalStateException if a fetch response came back with a failure after the corresponding topic partition has already been removed from the assignment.

Reviewers: David Jacot <djacot@confluent.io>
2022-12-21 09:17:11 +01:00
Lucas Brutschy 9df069f372
KAFKA-14299: Avoid interrupted exceptions during clean shutdown (#13026)
The call to `interrupt` on the state updater thread during shutdown
could interrupt the thread while writing the checkpoint file. This
can cause a failure to write the checkpoint file and a misleading
stack trace in the logs.

Reviewer: Bruno Cadonna <cadonna@apache.org>
2022-12-21 08:48:12 +01:00
Luke Chen 2575362639
KAFKA-14498: reduce the startup nodes to avoid timeout error (#13016)
In MetadataQuorumCommandTest, we sometimes got the error:

java.util.concurrent.ExecutionException: java.lang.RuntimeException: Received a fatal error while waiting for the broker to catch up with the current cluster metadata.

Since we tried to bring up 3 broker + 3 controllers at the same time, and the config initial.broker.registration.timeout.ms (default 1 min) is sometimes not enough for them to start up. Checking the tests, it doesn't require so many nodes. Reducing the nodes number to make these tests reliable.

Reviewers: dengziming <dengziming1993@gmail.com>, Ismael Juma <ismael@juma.me.uk>
2022-12-21 11:19:22 +08:00
Ismael Juma c4f10364cd
KAFKA-14475: Move TimeIndex/LazyIndex to storage module (#13010)
For broader context on this change, please check:

* KAFKA-14470: Move log layer to storage module

 Reviewers: Jun Rao <junrao@gmail.com>, Satish Duggana <satishd@apache.org>
2022-12-20 19:08:40 -08:00
Ismael Juma 7b634c7034
KAFKA-14521: Replace BrokerCompressionCodec with BrokerCompressionType (#13011)
This is a requirement for:

* KAFKA-14477: Move LogValidator to storage module.

For broader context on this change, please check:

* KAFKA-14470: Move log layer to storage module

Reviewers: dengziming <dengziming1993@gmail.com>
2022-12-20 11:53:49 -08:00
Ismael Juma 82c9216c77
KAFKA-14474: Move OffsetIndex to storage module (#13009)
For broader context on this change, please check:

* KAFKA-14470: Move log layer to storage module

Reviewers: Jun Rao <junrao@gmail.com>, Satish Duggana <satishd@apache.org>
2022-12-20 11:45:47 -08:00
José Armando García Sancio 44b3177a08
KAFKA-14457; Controller metrics should only expose committed data (#12994)
The controller metrics in the controllers has three problems. 1) the active controller exposes uncommitted data in the metrics. 2) the active controller doesn't update the metrics when the uncommitted data gets aborted. 3) the controller doesn't update the metrics when the entire state gets reset.

We fix these issues by only updating the metrics when processing committed metadata records and reset the metrics when the metadata state is reset.

This change adds a new type `ControllerMetricsManager` which processes committed metadata records and updates the metrics accordingly. This change also removes metrics updating responsibilities from the rest of the controller managers. 

Reviewers: Ron Dagostino <rdagostino@confluent.io>
2022-12-20 10:55:14 -08:00
David Arthur 71ea16f8eb
KAFKA-14531 Fix controller snapshot interval (#13019)
This patch applies the necessary conversion of milliseconds to nanoseconds for SnapshotGenerator. 

Reviewers: Colin P. McCabe <cmccabe@apache.org>, José Armando García Sancio <jsancio@gmail.com>
2022-12-20 10:20:41 -05:00
Mickael Maison 429a2570b0
MINOR: Don't throw if MirrorMaker topics already exist (#13005)
Reviewers: Luke Chen <showuon@gmail.com>
2022-12-20 14:33:32 +01:00
Lucas Brutschy c8675d4723
KAFKA-14343: Upgrade tests for state updater (#12801)
A test that verifies the upgrade from a version of Streams with
state updater disabled to a version with state updater enabled
and vice versa, so that we can offer a save upgrade path.

 - upgrade test from a version of Streams with state updater
disabled to a version with state updater enabled
 - downgrade test from a version of Streams with state updater
 enabled to a version with state updater disabled

Reviewer: Bruno Cadonna <cadonna@apache.org>
2022-12-20 09:35:59 +01:00
Ismael Juma d521f8110e
KAFKA-14473: Move AbstractIndex to storage module (#13007)
For broader context on this change, please check:

* KAFKA-14470: Move log layer to storage module

Reviewers: Jun Rao <junrao@gmail.com>, Satish Duggana <satishd@apache.org>
2022-12-19 19:33:24 -08:00
Ismael Juma 26fcf73feb
MINOR: Use `withClassLoader` in a couple more places (#13018)
Instead of try/finally.

Reviewers: Ron Dagostino <rndgstn@gmail.com>
2022-12-19 16:30:54 -05:00
Ismael Juma e2678d57d0
KAFKA-14472: Move TransactionIndex and related to storage module (#12996)
For broader context on this change, please check:

* KAFKA-14470: Move log layer to storage module

Reviewers: Jun Rao <junrao@gmail.com>, Satish Duggana <satishd@apache.org>
2022-12-19 11:31:37 -08:00
David Arthur 802fb11d4e
KAFKA-14529 Use the MetadataVersion from ClusterConfig in ZK tests (#13020)
Reviewers: Ismael Juma <ismael@juma.me.uk>
2022-12-19 14:30:21 -05:00
Bill Bejeck ea65d74f6b
MINOR: No error with zero results state query (#13002)
This PR updates StateQueryResult.getOnlyPartitionResult() to not throw an IllegaArgumentException when there are 0 query results.

Added a test that will fail without this patch

Reviewers: John Roesler<vvcephei@apache.org>
2022-12-19 13:39:06 -05:00
Philip Nee 4548c272ae
KAFKA-14264; New logic to discover group coordinator (#12862)
[KAFKA-14264](https://issues.apache.org/jira/browse/KAFKA-14264)
In this patch, we refactored the existing FindCoordinator mechanism. In particular, we first centralize all of the network operation (send, poll) in `NetworkClientDelegate`, then we introduced a RequestManager interface that is responsible to handle the timing of different kind of requests, based on the implementation.  In this path, we implemented a `CoordinatorRequestManager` which determines when to create an `UnsentRequest` upon polling the request manager.

Reviewers: Jason Gustafson <jason@confluent.io>
2022-12-19 09:48:52 -08:00
Justine Olshan 22bb3e7a58
KAFKA-14417: Address incompatible error code returned by broker from `InitProducerId` (#12968)
Older clients can not handle the `REQUEST_TIMED_OUT` error that is returned from `InitProducerId` when the next producerId block cannot be fetched from the controller. In this patch, we return `COORDINATOR_LOAD_IN_PROGRESS` instead which is retriable.

Reviewers: Jason Gustafson <jason@confluent.io>
2022-12-19 09:33:11 -08:00
Satish Duggana e3cb2deff6
KAFKA-14466: Move ClassloaderAwareRemoteStorageManager to storage module (#13013)
Reviewers: Ismael Juma <ismael@juma.me.uk>
2022-12-19 06:37:33 -08:00
David Jacot 28f29a719d
MINOR: Serialize response in KafkaApisTest (#12865)
Every request used in KafkaApiTests is serialized before hitting KafkaApis. This ensures that backward compatibility is handled correctly, especially ignorable fields. We don't do the same for the responses though. This patch fixes this and refactor how we get back the response in tests.

Reviewers: Mickael Maison <mickael.maison@gmail.com>
2022-12-19 11:30:23 +01:00
Ismael Juma 8d09fc9935
MINOR: Fix compiler errors in `RemoteIndex*` related to `OffsetPosition` (#13008)
#11390 and #12993 were merged in relatively quick succession, which resulted in compiler errors that weren't present when each change was on top of trunk.

Reviewers: Mickael Maison <mickael.maison@gmail.com>
2022-12-17 20:25:33 +01:00
Ismael Juma c0b28fde66
MINOR: Use INFO logging for tools and trogdor tests (#13006)
`TRACE` is too noisy and makes the build slower.

Reviewers: David Jacot <djacot@confluent.io>
2022-12-17 10:22:40 -08:00
Ismael Juma 95dc9d9eed
Move IndexEntry and related to storage module (#12993)
For broader context on this change, please check:

* KAFKA-14470: Move log layer to storage module

Reviewers: dengziming <dengziming1993@gmail.com>
2022-12-17 10:07:11 -08:00
Satish Duggana 7146ac57ba
[KAFKA-13369] Follower fetch protocol changes for tiered storage. (#11390)
This PR implements the follower fetch protocol as mentioned in KIP-405.

Added a new version for ListOffsets protocol to receive local log start offset on the leader replica. This is used by follower replicas to find the local log star offset on the leader.

Added a new version for FetchRequest protocol to receive OffsetMovedToTieredStorageException error. This is part of the enhanced fetch protocol as described in KIP-405.

We introduced a new field locaLogStartOffset to maintain the log start offset in the local logs. Existing logStartOffset will continue to be the log start offset of the effective log that includes the segments in remote storage.

When a follower receives OffsetMovedToTieredStorage, then it tries to build the required state from the leader and remote storage so that it can be ready to move to fetch state.

Introduced RemoteLogManager which is responsible for

initializing RemoteStorageManager and RemoteLogMetadataManager instances.
receives any leader and follower replica events and partition stop events and act on them
also provides APIs to fetch indexes, metadata about remote log segments.
Followup PRs will add more functionality like copying segments to tiered storage, retention checks to clean local and remote log segments. This will change the local log start offset and make sure the follower fetch protocol works fine for several cases.

You can look at the detailed protocol changes in KIP: https://cwiki.apache.org/confluence/display/KAFKA/KIP-405%3A+Kafka+Tiered+Storage#KIP405:KafkaTieredStorage-FollowerReplication

Co-authors: satishd@apache.org, kamal.chandraprakash@gmail.com, yingz@uber.com

Reviewers: Kowshik Prakasam <kprakasam@confluent.io>, Cong Ding <cong@ccding.com>, Tirtha Chatterjee <tirtha.p.chatterjee@gmail.com>, Yaodong Yang <yangyaodong88@gmail.com>, Divij Vaidya <diviv@amazon.com>, Luke Chen <showuon@gmail.com>, Jun Rao <junrao@gmail.com>
2022-12-17 09:36:44 -08:00
Kirk True f247aac96a
KAFKA-14496: Wrong Base64 encoder used by OIDC OAuthBearerLoginCallbackHandler (#13000)
The OAuth code to generate the Authentication header was incorrectly
using the URL-safe base64 encoder. For client IDs and/or secrets with
dashes and/or plus signs would not be encoded correctly, leading to the
OAuth server to reject the credentials.

This change uses the correct base64 encoder, per RFC-7617.

Co-authored-by: Endre Vig <vendre@gmail.com>
2022-12-16 19:44:41 +05:30
Daniel Scanteianu e3585a4cd5
MINOR: Document Offset and Partition 0-indexing, fix typo (#12753)
Add comments to clarify that both offsets and partitions are 0-indexed, and fix a minor typo. Clarify which offset will be retrieved by poll() after seek() is used in various circumstances. Also added integration tests.

Reviewers: Luke Chen <showuon@gmail.com>
2022-12-16 17:12:40 +08:00
Colin Patrick McCabe 29c09e2ca1
MINOR: ControllerServer should use the new metadata loader and snapshot generator (#12983)
This PR introduces the new metadata loader and snapshot generator. For the time being, they are
only used by the controller, but a PR for the broker will come soon.

The new metadata loader supports adding and removing publishers dynamically. (In contrast, the old
loader only supported adding a single publisher.) It also passes along more information about each
new image that is published. This information can be found in the LogDeltaManifest and
SnapshotManifest classes.

The new snapshot generator replaces the previous logic for generating snapshots in
QuorumController.java and associated classes. The new generator is intended to be shared between
the broker and the controller, so it is decoupled from both.

There are a few small changes to the old snapshot generator in this PR. Specifically, we move the
batch processing time and batch size metrics out of BrokerMetadataListener.scala and into
BrokerServerMetrics.scala.

Finally, fix a case where we are using 'is' rather than '==' for a numeric comparison in
snapshot_test.py.

Reviewers: David Arthur <mumrah@gmail.com>
2022-12-15 16:53:07 -08:00
vamossagar12 a46d16e7ab
Removing Multicasting partitioner for IQ (#12977)
Follow up PR for KIP-837. We don't want to allow multicasting for IQ. This PR imposes that restriction.

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2022-12-15 15:09:41 -08:00
Akhilesh C 8b045dcbf6
KAFKA-14446: API forwarding support from zkBrokers to the Controller (#12961)
This PR enables brokers which are upgrading from ZK mode to KRaft mode to forward certain metadata
change requests to the controller instead of applying them directly through ZK. To faciliate this,
we now support EnvelopeRequest on zkBrokers (instead of only on KRaft nodes.)

In BrokerToControllerChannelManager, we can now reinitialize our NetworkClient. This is needed to
handle the case when we transition from forwarding requests to a ZK-based broker over the
inter-broker listener, to forwarding requests to a quorum node over the controller listener.

In MetadataCache.scala, distinguish between KRaft and ZK controller nodes with a new type,
CachedControllerId.

In LeaderAndIsrRequest, StopReplicaRequest, and UpdateMetadataRequest, switch from sending both a
zk and a KRaft controller ID to sending a single controller ID plus a boolean to express whether it
is KRaft. The previous scheme was ambiguous as to whether the system was in KRaft or ZK mode when
both IDs were -1 (although this case is unlikely to come up in practice). The new scheme avoids
this ambiguity and is simpler to understand.

Reviewers: dengziming <dengziming1993@gmail.com>, David Arthur <mumrah@gmail.com>, Colin P. McCabe <cmccabe@apache.org>
2022-12-15 14:16:41 -08:00
Chris Egerton dd76872264
KAFKA-13709: Add docs for exactly-once support in Connect (#12941)
Reviewers: Mickael Maison <mickael.maison@gmail.com>
2022-12-15 11:20:49 -05:00
Greg Harris dcc02346c5
KAFKA-13881: Add Clients package description javadocs (#12895)
Reviewers: Mickael Maison <mickael.maison@gmail.com>

, Tom Bentley <tbentley@redhat.com>
2022-12-15 10:03:44 +01:00
David Jacot f8556fe791
KAFKA-14367; Add `DeleteGroups` to the new `GroupCoordinator` interface (#12858)
This patch adds `deleteGroups` to the new `GroupCoordinator` interface and updates `KafkaApis` to use it.

Reviewers: Omnia G H Ibrahim <o.g.h.ibrahim@gmail.com>, Justine Olshan <jolshan@confluent.io>, Jason Gustafson <jason@confluent.io>
2022-12-15 09:29:56 +01:00
Matthias J. Sax de088a2e97 MINOR: update Streams upgrade guide for 3.1 release (#12926)
Reviewers: John Roesler <john@confluent.io>, Bill Bejeck <bill@confluent.io>
2022-12-14 14:35:18 -05:00
Hao Li 9b23d9305d
KAFKA-14395: add config to configure client supplier (#12944)
Implements KIP-884.

Reviewers: Matthias J. Sax <matthias@confluent.io>
2022-12-14 09:17:27 -08:00
David Arthur 87b9c572c6
MINOR: Change KRaft ZK controller registration algorithm (#12973)
Increment the value in "/controller_epoch" when registering a KRaft controller as the active controller. Use the "kraftControllerEpoch" stored under "/controller" to ensure we are registering a newer KRaft controller.

Reviewers: Colin P. McCabe <cmccabe@apache.org>
2022-12-14 08:37:06 -05:00
Artem Livshits 7d1e37ea7e
MINOR: Fix FetchFromFollowerIntegrationTest.testFetchFromFollowerWithRoll (#12987)
The test was added with a fix to KAFKA-14379, the problem was that the replication factor for the offset topic was 1 and consumer group coordinator got unavailable when one of the brokers got shut down.

Reviewers: David Jacot <djacot@confluent.io>
2022-12-14 08:30:15 +01:00
David Arthur 67c72596af
KAFKA-14448 Let ZK brokers register with KRaft controller (#12965)
Prior to starting a KIP-866 migration, the ZK brokers must register themselves with the active
KRaft controller. The controller waits for all brokers to register in order to verify that all the
brokers can

A) Communicate with the quorum
B) Have the migration config enabled
C) Have the proper IBP set

This patch uses the new isMigratingZkBroker field in BrokerRegistrationRequest and
RegisterBrokerRecord. The type was changed from int8 to bool for BrokerRegistrationRequest (a
mistake from #12860). The ZK brokers use the existing BrokerLifecycleManager class to register and
heartbeat with the controllers.

Reviewers: Mickael Maison <mickael.maison@gmail.com>, Colin P. McCabe <cmccabe@apache.org>
2022-12-13 13:15:21 -08:00
Jason Gustafson 26a4d42072
MINOR: Pass snapshot ID directly in `RaftClient.createSnapshot` (#12981)
Let `RaftClient.createSnapshot` take the snapshotId directly instead of the committed offset/epoch (which may not exist). 

Reviewers: José Armando García Sancio <jsancio@apache.org>
2022-12-13 10:44:56 -08:00
vamossagar12 73ea6986df
KAFKA-13602: Remove unwanted logging in RecordCollectorImpl.java (#12985)
There is unwanted logging introduced by #12803 as pointed out in this comment: #12803 (comment). This PR removes it.

Reviewers: Lucas Brutschy <lbrutschy@confluent.io>, Bruno Cadonna <cadonna@apache.org>
2022-12-13 16:36:00 +01:00
Greg Harris 526af63cfe
KAFKA-13881: Add Connect package infos (#12937)
Reviewers: Chris Egerton <chrise@aiven.io>
2022-12-13 09:39:10 -05:00
Junyang Liu 7ff779f3f4
KAFKA-14285: Delete quota node in zookeeper when configs are empty (#12729)
The PR resolve issue KAFKA-14285. After doing changeConfigs, check and clean quota nodes if configs are empty, to avoid infinite increasement of quota nodes in zookeeper

Reviewers: Luke Chen <showuon@gmail.com>, Igor Soarez <soarez@apple.com>, dengziming <dengziming1993@gmail.com>
2022-12-13 17:04:11 +08:00
David Jacot 4a9c0fa4a4
KAFKA-14367; Add `DescribeGroups` to the new `GroupCoordinator` interface (#12855)
This patch adds `describeGroups` to the new `GroupCoordinator` interface and updates `KafkaApis` to use it.

Reviewers: Justine Olshan <jolshan@confluent.io>, Jason Gustafson <jason@confluent.io>
2022-12-13 09:19:21 +01:00
David Jacot 2935a52073
MINOR: Small refactor in KafkaApis.handleHeartbeatRequest (#12978)
This is a small follow-up to https://github.com/apache/kafka/pull/12848. 

Reviewers: Justine Olshan <jolshan@confluent.io>, Jason Gustafson <jason@confluent.io>
2022-12-13 09:18:05 +01:00
José Armando García Sancio 3541d5ab18
MINOR; Improve high watermark log messages (#12975)
While debugging KRaft and the metadata state machines it is helpful to always log the first time the replica discovers the high watermark. All other updates to the high watermark are logged at trace because they are more frequent and less useful.

Reviewers: Luke Chen <showuon@gmail.com>
2022-12-12 16:32:16 -08:00
Colin Patrick McCabe b2dea17041
MINOR: Introduce MetadataProvenance and ImageReWriter (#12964)
Introduce MetadataProvenance to encapsulate the three-tuple of (offset, epoch, timestamp) that is
associated with each MetadataImage, as well as each on-disk snapshot. Also introduce a builder
for MetadataDelta.

Remove offset and epoch tracking from MetadataDelta. We do not really need to know this information
until we are creating the final MetadataImage object. Therefore, this bookkeeping should be done by
the metadata loading code, not inside the delta code, like the other bookkeeping. This simplifies a
lot of tests, as well as simplifying RecordTestUtils.  It also makes more sense for snapshots, where
the offset and epoch are the same for every record.

Add ImageReWriter, an ImageWriter that applies records to a MetadataDelta. This is useful when you
need to create a MetadataDelta object that holds the contents of a MetadataImage. This will be
used in the new image loader code (coming soon).

Add ImageWriterOptionsTest to test ImageWriterOptions.

Reviewers: David Arthur <mumrah@gmail.com>
2022-12-12 09:52:06 -08:00
Artem Livshits 43f39c2e60
KAFKA-14379: Consumer should refresh preferred read replica on update metadata (#12956)
The consumer (fetcher) used to refresh the preferred read replica on
three conditions:
    
1. the consumer receives an OFFSET_OUT_OF_RANGE error
2. the follower does not exist in the client's metadata (i.e., offline)
3. after metadata.max.age.ms (5 min default)
    
For other errors, it will continue to reach to the possibly unavailable
follower and only after 5 minutes will it refresh the preferred read
replica and go back to the leader.
    
Another problem is that the client might have stale metadata and not
send fetches to preferred replica, even after the leader redirects to
the preferred replica.
    
A specific example is when a partition is reassigned. the consumer will
get NOT_LEADER_OR_FOLLOWER which triggers a metadata update but the
preferred read replica will not be refreshed as the follower is still
online. it will continue to reach out to the old follower until the
preferred read replica expires.
    
The consumer can instead refresh its preferred read replica whenever it
makes a metadata update request, so when the consumer receives i.e.
NOT_LEADER_OR_FOLLOWER it can find the new preferred read replica without
waiting for the expiration.
    
Generally, we will rely on the leader to choose the correct preferred
read replica and have the consumer fail fast (clear preferred read replica
cache) on errors and reach out to the leader.

Co-authored-by: Jeff Kim <jeff.kim@confluent.io>

Reviewers: David Jacot <djacot@confluent.io>, Jason Gustafson <jason@confluent.io>
2022-12-12 09:55:22 +01:00
vamossagar12 2fa1879247
KAFKA-14454: Making unique StreamsConfig for tests (#12971)
Newly added test KTableKTableForeignKeyInnerJoinCustomPartitionerIntegrationTest#shouldThrowIllegalArgumentExceptionWhenCustomPartionerReturnsMultiplePartitions as part of KIP-837 passes when run individually but fails when is part of IT class and hence is marked as Ignored.

That seemed to have been because of the way StreamsConfig was being initialised so any new test would have used the same names. Because of which the second test never got to the desired state. With this PR, every test gets a unique app name which seems to have fixed the issue. Also, a couple of cosmetic changes

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2022-12-09 17:51:42 -08:00