Commit Graph

3198 Commits

Author SHA1 Message Date
Colin P. McCabe 234ea4bceb address review feedback 2024-07-01 15:38:58 -07:00
Colin P. McCabe ab1b566759 Address review comments 2024-06-27 15:24:58 -07:00
Colin P. McCabe 3e07a3b7e0 Add BrokerRegistrationRequest v4 2024-06-27 15:21:11 -07:00
Colin P. McCabe c0e3c7ca13 Merge branch 'trunk' into KAFKA-17011 2024-06-27 14:11:43 -07:00
Colin Patrick McCabe ebaa108967
KAFKA-16968: Introduce 3.8-IV0, 3.9-IV0, 3.9-IV1
Create 3 new metadata versions:

- 3.8-IV0, for the upcoming 3.8 release.
- 3.9-IV0, to add support for KIP-1005.
- 3.9-IV1, as the new release vehicle for KIP-966.

Create ListOffsetRequest v9, which will be used in 3.9-IV0 to support KIP-1005. v9 is currently an unstable API version.

Reviewers: Jun Rao <junrao@gmail.com>, Justine Olshan <jolshan@confluent.io>
2024-06-27 14:03:03 -07:00
Ken Huang 9b4f13efbc
KAFKA-15623 Remove junit 4 from stream module (#16447)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-27 15:11:32 +08:00
José Armando García Sancio adee6f0cc1
KAFKA-16527; Implement request handling for updated KRaft RPCs (#16235)
Implement request handling for the updated versions of the KRaft RPCs (Fetch, FetchSnapshot, Vote,
BeginQuorumEpoch and EndQuorumEpoch). This doesn't add support for KRaft replicas to send the new
version of the KRaft RPCs. That will be implemented in KAFKA-16529.

All of the RPCs responses were extended to include the leader's endpoint for the listener of the
channel used in the request. EpochState was extended to include the leader's endpoint information
but only the FollowerState and LeaderState know the leader id and its endpoint(s).

For the Fetch request, the replica directory id was added. The leader now tracks the follower's log
end offset using both the replica id and replica directory id.

For the FetchSnapshot request, the replica directory id was added. This is not used by the KRaft
leader and it is there for consistency with Fetch and for help debugging.

For the Vote request, the replica key for both the voter (destination) and the candidate (source)
were added. The voter key is checked for consistency. The candidate key is persisted when the vote
is granted.

For the BeginQuorumEpoch request, all of the leader's endpoints are included. This is needed so
that the voters can return the leader's endpoint for all of the supported listeners.

For the EndQuorumEpoch request, all of the leader's endpoints are included. This is needed so that
the voters can return the leader's endpoint for all of the supported listeners. The successor list
has been extended to include the directory id. Receiving voters can use the entire replica key when
searching their position in the successor list.

Updated the existing test in KafkaRaftClientTest and KafkaRaftClientSnapshotTest to execute using
both the old version and new version of the RPCs.

Reviewers: Luke Chen <showuon@gmail.com>, Colin P. McCabe <cmccabe@apache.org>
2024-06-25 13:45:15 -07:00
Chia Chuan Yu 5b0e96d785
KAFKA-17034 Tweak some descriptions in FeatureUpdate (#16448)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-26 03:05:34 +08:00
Colin P. McCabe 1dc5f990b3 Update NetworkClientTest 2024-06-25 10:56:39 -07:00
Andrew Schofield 63304fb6e5
KAFKA-17028: FindCoordinator v6 initial implementation (#16440)
KIP-932 introduces FindCoordinator v6 for finding share coordinators. The initial implementation:

Checks that share coordinators are only requested with v6 or above.
Share coordinator requests are authorized as cluster actions (this is for inter-broker use only)
Responds with COORDINATOR_NOT_AVAILABLE because share coordinators are not yet available.
When the share coordinator code is delivered, the request handling will be gated by configurations which enable share groups and the share coordinator specifically. If these are not enabled, COORDINATOR_NOT_AVAILABLE is the response.

Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>
2024-06-25 21:13:16 +05:30
Colin P. McCabe 6235178031 pass through the ApiVersionsResponse version to ChannelBuilders 2024-06-24 17:04:36 -07:00
Alieh Saeedi f4cbf71ea6
KAFKA-16965: Throw cause of TimeoutException (#16344)
Add the cause of TimeoutException for Producer send() errors.

Reviewers: Matthias J. Sax <matthias@confluent.io>,  Lianet Magrans <lianetmr@gmail.com>, Artem Livshits <alivshits@confluent.io>, Justine Olshan <jolshan@confluent.io>
2024-06-24 14:51:27 -07:00
TingIāu "Ting" Kì d78ff06476
KAFKA-16967 NioEchoServer fails to register connection and causes flaky failure. (#16384)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-24 21:39:40 +08:00
Ken Huang 8a109d87d1
KAFKA-17009 Add unit test to query nonexistent replica by describeReplicaLogDirs (#16423)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-23 16:25:57 +08:00
Colin P. McCabe 1c1e556528 KAFKA-17011: Fix a bug preventing features from supporting v0
As part of KIP-584, brokers expose a range of supported versions for each feature. For example,
metadata.version might be supported from 1 to 21. (Note that feature level ranges are always
inclusive, so this would include both level 1 and 21.)

These supported ranges are supposed to be able to include 0. For example, it should be possible for
a broker to support a kraft.version between 0 and 1. However, in older software versions, there is
an assertion in org.apache.kafka.common.feature.SupportedVersionRange that prevents this. This
causes problems when the older software attempts to deserialize an ApiVersionsResponse containing
such a range.

In order to resolve this dilemma, this PR bumps the version of ApiVersionsRequest from 3 to 4.
Clients which send v4 promise to be able to handle ranges including 0. Clients which send v3 will
not be exposed to these ranges -- the feature will simply be omitted from the response. This work
is part of KIP-1022.
2024-06-21 16:23:54 -07:00
gongxuanzhang e294ea433c
KAFKA-17012 Enable CONSUMER protocol for some tests of KafkaConsumerTest (#16415)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-21 21:58:24 +08:00
gongxuanzhang 80f31224aa
KAFKA-10787 Apply spotless to `clients` module (#16393)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-20 17:43:25 +08:00
Chia Chuan Yu 4ff83dc733
KAFKA-16957 Enable KafkaConsumerTest#configurableObjectsShouldSeeGeneratedClientId to work with CLASSIC and CONSUMER (#16370)
Reviewers: Kirk True <kirk@kirktrue.pro>, Chia-Ping Tsai <chia7712@gmail.com>
2024-06-19 09:35:10 +08:00
Lianet Magrans 8199290500
MINOR: consumer log fixes (#16345)
Reviewers: Kirk True <kirk@kirktrue.pro>, Chia-Ping Tsai <chia7712@gmail.com>
2024-06-18 22:51:33 +08:00
dujian0068 823d6f7555
KAFKA-16958 add STRICT_STUBS to EndToEndLatencyTest, OffsetCommitCallbackInvokerTest, ProducerPerformanceTest, and TopologyTest (#16348)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-18 18:51:43 +08:00
Lianet Magrans 6c4e777079
KAFKA-16954: fix consumer close to release assignment in background (#16343)
This PR fixes consumer close to avoid updating the subscription state object in the app thread. Now the close simply triggers an UnsubscribeEvent that is handled in the background to trigger callbacks, clear assignment, and send leave heartbeat. Note that after triggering the event, the unsubscribe will continuously process background events until the event completes, to ensure that it allows for callbacks to run in the app thread.
The logic around what happens if the unsubscribe fails remain unchanged: close will log, keep the first exception and carry on.

It also removes the redundant LeaveOnClose event (it used to do the the exact same thing as the UnsubscribeEvent, both calling membershipMgr.leaveGroup).

Reviewers: Lucas Brutschy <lbrutschy@confluent.io>
2024-06-17 21:27:33 +02:00
Dongnuo Lyu 21d60eabab
KAFKA-16673; Simplify `GroupMetadataManager#toTopicPartitions` by using `ConsumerProtocolSubscription` instead of `ConsumerPartitionAssignor.Subscription` (#16309)
In `GroupMetadataManager#toTopicPartitions`, we generate a list of `ConsumerGroupHeartbeatRequestData.TopicPartitions` from the input deserialized subscription. Currently the input subscription is `ConsumerPartitionAssignor.Subscription`, where the topic partitions are stored as (topic-partition) pairs, whereas in `ConsumerGroupHeartbeatRequestData.TopicPartitions`, we need the topic partitions to be stored as (topic-partition list) pairs.

`ConsumerProtocolSubscription` is an intermediate data structure in the deserialization where the topic partitions are stored as (topic-partition list) pairs. This pr uses `ConsumerProtocolSubscription` instead as the input subscription to make `toTopicPartitions` more efficient. 

Reviewers: David Jacot <djacot@confluent.io>
2024-06-17 02:47:52 -07:00
Chia Chuan Yu 768e90f667
KAFKA-16669 Remove extra collection copy when generating DescribeAclsResource (#15924)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-17 14:47:44 +08:00
Gaurav Narula 4a37c2e18f
KAFKA-16219 set SO_TIMEOUT in EchoServer (#16354)
We observed some runs of the test suite caused CI pipelines to stall.

A thread dump revealed that the test runner was blocked trying to read from a
socket, while attempting to close the socket [[0]]. It turns out this is
due to a bug in JDK which is very similar to JDK-8274524, but it affects
the else branch of `SSLSocketImpl::bruteForceCloseInput` [[1]] which wasn't
fixed in JDK-8274524.

Since the blocking happens in a native call, the test runner's timeouts have
no effect as the blocked test runner thread doesn't seem to respond to
interrupts.

As a mitigation in Kafka's test suite, this change adds `SO_TIMEOUT` of
30 seconds to all the TLS sockets handled by `EchoServer`. The timeout is
reasonably high for tests and a finite upper bound avoids infinite
blocking of the test suite.

[0]: https://issues.apache.org/jira/secure/attachment/13066427/timeout.log
[1]: 890adb6410/src/java.base/share/classes/sun/security/ssl/SSLSocketImpl.java (L808)

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-17 14:11:37 +08:00
Andrew Schofield fecbfb8133
KAFKA-16950: Define Persister interfaces and RPCs (#16335)
Define the interfaces and RPCs for share-group persistence. (KIP-932). This PR is just RPCs and interfaces to allow building of the broker components which depend upon them. The implementation will follow in subsequent PRs.

Reviewers:  Manikumar Reddy <manikumar.reddy@gmail.com>, Apoorv Mittal <apoorvmittal10@gmail.com>
2024-06-15 20:52:49 +05:30
gongxuanzhang 3a9d877686
MINOR: refactor BuiltInPartitioner to remove mockRandom from production code (#16277)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-15 12:18:42 +08:00
Kirk True 8f86b9c4ec
KAFKA-16637 AsyncKafkaConsumer removes offset fetch responses from cache too aggressively (#16310)
Allow the committed offsets fetch to run for as long as needed. This handles the case where a user invokes Consumer.poll() with a very small timeout (including zero).

Reviewers: Andrew Schofield <aschofield@confluent.io>, Lianet Magrans <lianetmr@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2024-06-15 08:48:53 +08:00
TingIāu "Ting" Kì 09bc5be63e
KAFKA-16946: Utils.getHost/getPort cannot parse SASL_PLAINTEXT://host:port (#16319)
In previous PR(#16048), I mistakenly excluded the underscore (_) from the set of valid characters for the protocol,
resulting in the inability to correctly parse the connection string for SASL_PLAINTEXT. This bug fix addresses the
issue and includes corresponding tests.

Reviewers: Andrew Schofield <aschofield@confluent.io>, Luke Chen <showuon@gmail.com>
2024-06-14 13:07:11 -07:00
TingIāu "Ting" Kì 4e2f26bfc6
KAFKA-16917 DescribeTopicsResult should use mutable map in order to keep compatibility (#16250)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-14 23:48:35 +08:00
Omnia Ibrahim e99da2446c
KAFKA-15853: Move KafkaConfig.configDef out of core (#16116)
Reviewers: Mickael Maison <mickael.maison@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2024-06-14 17:26:00 +02:00
Lianet Magrans 46714dbaed
KAFKA-16933: New consumer unsubscribe close commit fixes (#16272)
Fixes for the leave group flow (unsubscribe/close):

Fix to send Heartbeat to leave group on close even if the callbacks fail
fix to ensure that if a member gets fenced while blocked on callbacks (ex. on unsubscribe), it will clear its epoch to not include it in commit requests
fix to avoid race on the subscription state object on unsubscribe, updating it only on the background thread when the callbacks to leave complete (success or failure).
Also improving logging in this area.

Reviewers: Lucas Brutschy <lbrutschy@confluent.io>, Philip Nee <pnee@confluent.io>
2024-06-14 13:03:58 +02:00
Kuan-Po (Cooper) Tseng 888a177603
KAFKA-12708 Rewrite org.apache.kafka.test.Microbenchmarks by JMH (#16231)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-14 16:47:34 +08:00
dujian0068 133f2b0f31
KAFKA-16879 SystemTime should use singleton mode (#16266)
Reviewers: Greg Harris <gharris1727@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2024-06-14 08:49:19 +08:00
brenden20 a0b716ec9f
KAFKA-16001: Migrated ConsumerNetworkThreadTest away from ConsumerTestBuilder (#16140)
Completely migrates ConsumerNetworkThreadTest away from ConsumerTestBuilder and removes all usages of spy objects and replaced with mocks. Removes testEnsureMetadataUpdateOnPoll() since it was doing integration testing. Also I adds new tests to get more complete test coverage of ConsumerNetworkThread.

Reviewers: Kirk True <kirk@kirktrue.pro>, Lianet Magrans <lianetmr@gmail.com>, Philip Nee <pnee@confluent.io>, Matthias J. Sax <matthias@confluent.io>
2024-06-13 13:35:36 -07:00
gongxuanzhang 596b945072
KAFKA-16643 Add ModifierOrder checkstyle rule (#15890)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-13 15:39:32 +08:00
brenden20 e59c887bfd
KAFKA-16557 Implemented OffsetFetchRequestState toStringBase and added a test for it (#16291)
Reviewers: Lianet Magrans <lianetmr@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2024-06-13 15:30:05 +08:00
TingIāu "Ting" Kì dd6fcc650e
KAFKA-16901 Add unit tests for ConsumerRecords#records(String) (#16227)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-13 14:35:33 +08:00
Lianet Magrans fe98888960
MINOR: Improving log for outstanding requests on close and cleanup (#16304)
Reviewers: Andrew Schofield <aschofield@confluent.io>, Chia-Ping Tsai <chia7712@gmail.com>
2024-06-13 14:31:16 +08:00
Gantigmaa Selenge 6d1f8f8727
MINOR: Clean up for KafkaAdminClientTest (#16285)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-13 09:42:39 +08:00
Ivan Yurchenko dd755b7ea9
KAFKA-8206: KIP-899: Allow client to rebootstrap (#13277)
This commit implements KIP-899: Allow producer and consumer clients to rebootstrap. It introduces the new setting `metadata.recovery.strategy`, applicable to all the types of clients.

Reviewers: Greg Harris <gharris1727@gmail.com>, Rajini Sivaram <rajinisivaram@googlemail.com>
2024-06-12 20:48:32 +01:00
Edoardo Comar 615e6e705c
KAFKA-16570 FenceProducers API returns "unexpected error" when succes… (#16229)
KAFKA-16570 FenceProducers API returns "unexpected error" when successful

* Client handling of ConcurrentTransactionsException as retriable
* Unit test
* Integration test

Reviewers: Chris Egerton <chrise@aiven.io>, Justine Olshan <jolshan@confluent.io>
2024-06-12 17:07:33 +01:00
Gantigmaa Selenge 9368ef81b5
KAFKA-16865: Add IncludeTopicAuthorizedOperations option for DescribeTopicPartitionsRequest (#16136)
Reviewers: Mickael Maison <mickael.maison@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>, Calvin Liu <caliu@confluent.io>, Andrew Schofield <andrew_schofield@live.com>, Apoorv Mittal <amittal@confluent.io>
2024-06-12 17:04:24 +02:00
David Jacot 638844f833
KAFKA-16770; [2/2] Coalesce records into bigger batches (#16215)
This patch is the continuation of https://github.com/apache/kafka/pull/15964. It introduces the records coalescing to the CoordinatorRuntime. It also introduces a new configuration `group.coordinator.append.linger.ms` which allows administrators to chose the linger time or disable it with zero. The new configuration defaults to 10ms.

Reviewers: Jeff Kim <jeff.kim@confluent.io>, Justine Olshan <jolshan@confluent.io>
2024-06-11 23:29:50 -07:00
Murali Basani 226ac5e8fc
KAFKA-16922 Adding unit tests for NewTopic (#16255)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2024-06-12 11:38:50 +08:00
Nikolay aecaf44475
KAFKA-16520: Support KIP-853 in DescribeQuorum (#16106)
Add support for KIP-953 KRaft Quorum reconfiguration in the DescribeQuorum request and response.
Also add support to AdminClient.describeQuorum, so that users will be able to find the current set of
quorum nodes, as well as their directories, via these RPCs.

Reviewers: Luke Chen <showuon@gmail.com>, Colin P. McCabe <cmccabe@apache.org>, Andrew Schofield <aschofield@confluent.io>
2024-06-11 10:01:35 -07:00
Gyeongwon, Do 1426e8e920
KAFKA-16764: New consumer should throw InvalidTopicException on poll when invalid topic in metadata. (#16043)
Propagate metadata error from background thread to application thread.
So, this fix ensures that metadata errors are thrown to the user on consumer.poll()

Reviewers: Lucas Brutschy <lbrutschy@confluent.io>, Philip Nee <pnee@confluent.io>, Lianet Magrans <lianetmr@gmail.com>
2024-06-10 18:30:29 +02:00
Apoorv Mittal 96036aee85
KAFKA-16916: Fixing error in completing future (#16249)
Fix to complete Future which was stuck due the exception.getCause() throws an error.

The fix in the #16217 unblocked blocking thread but exception in catch block from blocking get call was wrapped in ExecutionException which is not the case when moved to async workflow hence getCause is not needed.

Reviewers: Luke Chen <showuon@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2024-06-08 14:25:54 +08:00
Okada Haruki d13a693ea7
KAFKA-16916: Disable blocking test temporarily (#16248)
The test runs forever. We disable the test temporarily to unblock CI

Reviewers: Luke Chen <showuon@gmail.com>
2024-06-08 09:09:00 +08:00
NICOLAS GUYOMAR eb352828ed
MINOR: Remove AK 1.0.0 reference from NetworkClient.java (#16223)
Reviewers: Justine Olshan <jolshan@confluent.io>, Chia-Ping Tsai <chia7712@gmail.com>
2024-06-08 06:54:09 +08:00
Phuc-Hong-Tran 5cd6944eaa
KAFKA-16493: Avoid unneeded subscription regex check if metadata version unchanged (#15869)
This PR includes changes for AsyncKafkaConsumer to avoid evaluating the subscription regex on every poll if metadata hasn't changed. The metadataVersionSnapshot was introduced to identify whether metadata has changed or not, if yes then the current subscription regex will be evaluated.

This is the same mechanism used by the LegacyConsumer.

Reviewers: Lianet Magrans <lianetmr@gmail.com>, Matthias J. Sax <matthias@confluent.io>
2024-06-07 10:30:07 -07:00