Commit Graph

8820 Commits

Author SHA1 Message Date
Adil Houmadi cc00dffade
MINOR: Fix typo in MirrorMaker v2 documentation (#10433)
Reviewers: Mickael Maison <mickael.maison@gmail.com>
2021-04-01 16:08:28 +01:00
Ismael Juma 2342ec1d1c
KAFKA-12600: Remove deprecated config value `default` for client config `client.dns.lookup` (#10458)
The config has been deprecated since Kafka 2.6 (released ~1 year before
3.0), but it was the default before it got deprecated. As such, it's
reasonably unlikely that people would have set it explicitly.

Given the confusing `default` name even though it's _not_ the default, I
think we should remove it in 3.0.

Also remove `ClientDnsLookup.DEFAULT` (not public API), which unlocks
a number of code simplications.

Reviewers: David Jacot <djacot@confluent.io>
2021-04-01 07:59:59 -07:00
Ismael Juma ad83fd00bf
KAFKA-12952: Remove deprecated LogConfig.Compact (#10451)
Deprecated since 1.0.0 for misleading name. Not a public API technically,
but we were conservative originally.

Reviewers: David Jacot <djacot@confluent.io>
2021-04-01 05:13:39 -07:00
Kowshik Prakasam 7c5c739b7f
Initial commit (#10454)
This PR is a precursor to the recovery logic refactor work (KAFKA-12553).

I've renamed the file: core/src/test/scala/unit/kafka/log/LogUtils.scala to core/src/test/scala/unit/kafka/log/LogTestUtils.scala. Also I've renamed the underlying lass from LogUtils to LogTestUtils. This is going to help avoid a naming conflict with a new file called LogUtils.scala that I plan to introduce in core/src/main/scala/kafka/log/ as part of the recovery logic refactor. The new file will also contain a bunch of static functions.

Tests:
Relying on existing tests to catch regressions (if any) since this is a simple change.

Reviewers: Satish Duggana <satishd@apache.org>, Dhruvil Shah <dhruvil@confluent.io>, Jun Rao <junrao@gmail.com>
2021-03-31 16:48:40 -07:00
Kowshik Prakasam b6278ee79d
KAFKA-12575: Eliminate Log.isLogDirOffline boolean attribute (#10430)
This PR is a precursor to the recovery logic refactor work (KAFKA-12553).

I have made a change to eliminate Log.isLogDirOffline attribute. This boolean also comes in the way of refactoring the recovery logic. This attribute was added in #9676. But it is redundant and can be eliminated in favor of looking up LogDirFailureChannel to check if the logDir is offline. The performance/latency implication of such a ConcurrentHashMap lookup inside LogDirFailureChannel should be very low given that ConcurrentHashMap reads are usually lock free.

Tests:
Relying on existing unit/integration tests.

Reviewers: Dhruvil Shah <dhruvil@confluent.io>, Jun Rao <junrao@gmail.com>
2021-03-31 15:26:40 -07:00
David Jacot 6d7a9012dc
KAFKA-8405; Remove deprecated `kafka-preferred-replica-election` command (#10443)
The `kafka-preferred-replica-election` command was deprecated in 2.4. This path removes it for 3.0. `kafka-leader-election` can be used instead.

Reviewers: Ismael Juma <ismael@juma.me.uk>, Chia-Ping Tsai <chia7712@gmail.com>, Jason Gustafson <jason@confluent.io>
2021-03-31 20:45:18 +02:00
Bruno Cadonna 58d4e527d0
MINOR: Fix docs for end-to-end record latency metrics (#10449)
Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2021-03-31 11:02:16 -07:00
Satish Duggana 3829df103e
MINOR Replaced File with Path in LogSegmentData. (#10424)
Replaced File with Path in LogSegment Data.

This is a followup of #10173

Reviewers: Kowshik Prakasam <kprakasam@confluent.io>, Jun Rao <junrao@gmail.com>
2021-03-31 10:43:10 -07:00
Lee Dongjin b6a787a072
KAFKA-12583: Upgrade netty to 4.1.62.Final
Reviewers: Ismael Juma <ismael@juma.me.uk>
2021-03-31 08:45:30 -07:00
David Jacot 667578860a
KAFKA-12577; Remove deprecated `ConfigEntry` constructor for 3.0 (#10436)
ConfigEntry's public constructor was deprecated in 1.1.0. This patch removes it in AK 3.0.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, Ismael Juma <ismael@juma.me.uk>
2021-03-31 11:28:29 +02:00
Luke Chen 0981d25d62
KAFKA-12561: Fix flaky kafka.server.RaftClusterTest.testCreateClusterAndCreateListDeleteTopic (#10410)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2021-03-31 13:14:56 +08:00
Ismael Juma a7251ce174
KAFKA-12581: Remove deprecated `Admin.electPreferredLeaders` (#10440)
`Admin.electLeaders` is the replacement since the deprecation in Apache Kafka 2.4.0.
The methods were originally introduced in Apache Kafka 2.2.0, so they were only
non deprecated for two releases.

Reviewers: David Jacot <djacot@confluent.io>, Chia-Ping Tsai <chia7712@gmail.com>
2021-03-30 22:04:13 -07:00
Ismael Juma e8754ba7a0
KAFKA-12578: Remove deprecated security classes/methods for 3.0 (#10435)
More specifically, remove deprecated:
- Constants in SslConfigs
- Constants in SaslConfigs
- AclBinding constructor
- AclBindingFilter constructor
- PrincipalBuilder and DefaultPrincipalBuilder classes
- ResourceFilter

Also simplify tests and code that no longer have to handle the removed `PrincipalBuilder`.

These removals seem non controversial. There is a straightforward alternative. The
deprecations happened in 1.0.0 and 2.0.0.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2021-03-30 22:02:16 -07:00
dengziming 482b79dc25
MINOR: Remove unnecessary suppress (#10434)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2021-03-31 12:02:46 +08:00
A. Sophie Blee-Goldman 3eff8d39f1
HOTFIX: move rebalanceInProgress check to skip commit during handleCorrupted (#10444)
Minor followup to #10407 -- we need to extract the rebalanceInProgress check down into the commitAndFillInConsumedOffsetsAndMetadataPerTaskMap method which is invoked during handleCorrupted, otherwise we may attempt to commit during a a rebalance which will fail

Reviewers: Matthias J. Sax <mjsax@confluent.io>
2021-03-30 18:55:38 -07:00
Kowshik Prakasam b35704a843
KAFKA-12571: Eliminate LeaderEpochFileCache constructor dependency on logEndOffset (#10426)
This PR is a precursor to the recovery logic refactor work (KAFKA-12553).

Problems:
For refactoring the recovery logic (KAFKA-12553), we would like to move the logic to initialize LeaderEpochFileCache out of the Log class and into a separate static function. In the future, once we successfully initialize LeaderEpochFileCache outside Log, we will be able pass it as a dependency into both the Log recovery module and Log class constructor. However, currently the LeaderEpochFileCache constructor takes a dependency on logEndOffset (via a callback), which poses the following problems:

Blocks the instantiation of LeaderEpochFileCache outside Log class. Because, outside Log the logEndOffset is unavailable to be passed into LeaderEpochFileCache constructor. As a result, this situation blocks the recovery logic (KAFKA-12553) refactor work.
It turns out the logEndOffset dependency is used only in 1 of the LeaderEpochFileCache methods: LeaderEpochFileCache.endOffsetFor, and just for 1 particular case. Therefore, it is overkill to pass it in the constructor as a dependency. Also a callback is generally not a neat way to access dependencies and it poses code readability problems too.

Solution:
This PR modifies the code such that we only pass the logEndOffset as a parameter into LeaderEpochFileCache.endOffsetFor whenever the method is called, thus eliminating the constructor dependency. This will also unblock the recovery logic refactor work (KAFKA-12553).

Tests:
I have modified the existing tests to suit the above refactor.

Reviewers: Dhruvil Shah <dhruvil@confluent.io>, Jun Rao <junrao@gmail.com>
2021-03-30 17:31:27 -07:00
A. Sophie Blee-Goldman 0189298d86
KAFKA-12288: remove task-level filesystem locks (#10342)
The filesystem locks don't protect access between StreamThreads, only across different instances of the same Streams application. Running multiple processes in the same physical state directory is not supported, and as of PR #9978 it's explicitly guarded against), so there's no reason to continue locking the task directories with anything heavier than an in-memory map.

Reviewers: Rohan Desai <rodesai@confluent.io>, Walker Carlson <wcarlson@confluent.io>, Guozhang Wang <guozhang@confluent.io>
2021-03-30 17:02:42 -07:00
Jason Gustafson d22c49217e
MINOR: Fix newly added client side timeout tests in `KafkaAdminClientTest` (#10398)
This patch fixes a race condition between the background thread calling `ready` and the call to `MockTime.sleep` in the test. If the call to `sleep` happens first, then the test hangs. I fixed it by giving `MockClient` a way to listen to `ready` calls. This combined with a latch fixes the race. 

This patch also fixes a similar race condition in `testClientSideTimeoutAfterFailureToReceiveResponse`. After the disconnect, there is a race between the background thread sending the retry request and the foreground sleeping for the needed backoff delay.

Reviewers: Guozhang Wang <wangguoz@gmail.com>, David Arthur <mumrah@gmail.com>
2021-03-30 15:45:08 -07:00
ketulgupta1995 617ee00322
KAFKA-12509 Tighten up StateDirectory thread locking (#10418)
Modified LockAndOwner class to have Thread reference instead of just name

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2021-03-30 12:13:53 -07:00
Kowshik Prakasam d92d464b12
KAFKA-12552: Introduce LogSegments class abstracting the segments map (#10401)
This PR is a precursor to the recovery logic refactor work (KAFKA-12553).

In this PR, I've extracted the behavior surrounding segments map access within kafka.log.Log class into a new class: kafka.log.LogSegments. This class encapsulates a thread-safe navigable map of kafka.log.LogSegment instances and provides the required read and write behavior on the map. The Log class now encapsulates an instance of the LogSegments class.

Couple advantages of this PR:

Makes the Log class a bit more modular as it moves out certain private behavior thats otherwise within the Log class.
This is a precursor to refactoring the recovery logic (KAFKA-12553). In the future, the logic for recovery and loading of segments from disk (during Log) init will reside outside the Log class. Such logic would need to instantiate and access an instance of the newly added LogSegments class.
Tests:
Added a new test suite: kafka.log.LogSegmentsTest covering the APIs of the newly introduced class.

Reviewers: Satish Duggana <satishd@apache.org>, Dhruvil Shah <dhruvil@confluent.io>, Jun Rao <junrao@gmail.com>
2021-03-30 09:51:49 -07:00
John Roesler d02ad2c0e1
KAFKA-12557: Fix hanging KafkaAdminClientTest (#10404)
Fix a hanging test in KafkaAdminClientTest by forcing the admin client to shut down
whether or not there are pending requests once the test harness enters shutdown.

Reviewers: Ismael Juma <ijuma@apache.org>, Guozhang Wang <guozhang@apache.org>
2021-03-30 11:36:13 -05:00
David Jacot 0018c3c089
KAFKA-12573; Remove deprecated `Metric#value` (#10425)
Reviewers: Ismael Juma <ismael@juma.me.uk>
2021-03-30 09:28:48 +02:00
Ismael Juma 16b2d4f3a7
MINOR: Self-managed -> KRaft (Kafka Raft) (#10414)
`Self-managed` is also used in the context of Cloud vs on-prem and it can
be confusing.

`KRaft` is a cute combination of `Kafka Raft` and it's pronounced like `craft`
(as in `craftsmanship`).

Reviewers: Colin P. McCabe <cmccabe@apache.org>, Jose Sancio <jsancio@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>, Ron Dagostino <rdagostino@confluent.io>
2021-03-29 15:39:10 -07:00
Walker Carlson e971d94eb3
KAFKA-8784: remove default close for RocksDBConfigSetter (#10416)
Remove the default close implementation for RocksDBConfigSetter to avoid accidental memory leaks via C++ backed objects which are constructed but not closed by the user

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2021-03-29 14:26:39 -07:00
A. Sophie Blee-Goldman fb2eef94a4
KAFKA-12523: handle TaskCorruption and TimeoutException during handleCorruption and handleRevocation (#10407)
Need to handle TaskCorruptedException and TimeoutException that can be thrown from offset commit during handleRevocation or handleCorruption

Reviewers: Matthias J. Sax <mjsax@confluent.org>, Guozhang Wang <guozhang@confluent.io>
2021-03-29 14:06:22 -07:00
Guozhang Wang d5fd491bf7
KAFKA-7106: remove deprecated Windows APIs (#10378)
1. Remove all deprecated APIs in KIP-328.
2. Remove deprecated APIs in Windows in KIP-358.

Reviewers: John Roesler <vvcephei@apache.org>
2021-03-28 12:33:40 -07:00
Guozhang Wang b8058829bb
KAFKA-12562: Remove deprecated APIs in KafkaStreams and returned state classes (#10412)
Reviewers: A. Sophie Blee-Goldman <sophie@confluent.io>
2021-03-28 12:20:31 -07:00
Luke Chen 69baa03404
MINOR: fix the wrong/missing anchor in docs caused link error (#10413)
Reviewers: Matthias J. Sax <matthias@confluent.io>
2021-03-28 11:39:53 -07:00
Matthias J. Sax bd9a5da9d1
MINOR: update Streams upgrade notes with removed APIs (#10420)
Reviewer: Guozhang Wang <guozhang@confluent.io>
2021-03-28 11:36:42 -07:00
Walker Carlson f91d592a27
KAFKA-12537: fix application shutdown corner case with only one thread (#10387)
When in EOS the run loop terminates on that thread before the shutdown can be called. This is a problem for EOS single thread applications using the application shutdown feature.

I changed it so in all cases with a single thread, the dying thread will spin up a new thread to communicate the shutdown and terminate the dying thread. Also @ableegoldman refactored the catch blocks in runloop.

Co-authored-by: A. Sophie Blee-Goldman <ableegoldman@gmail.com>

Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>
2021-03-26 19:55:27 -07:00
Ismael Juma 704cabf10d
MINOR: Use Java 11 for generating aggregated javadoc in release.py (#10399)
Java 11 generates html5 pages with search support, which provides a better user experience.

Fixed `get_jdk` bugs found during testing and updated `release.py` blurb to indicate that
both JDK 8 and JDK 11 are required to perform a release.

Tested by running `python2 release.py stage-docs`, which triggers the `aggregateJavadoc`
path without some of the undesired (for testing) release steps.

Reviewers: John Roesler <vvcephei@apache.org>
2021-03-25 20:09:32 -07:00
Matthias J. Sax 03f13d1c41
KAFKA-12452: Remove deprecated overloads of ProcessorContext#forward (#10300)
ProcessorContext#forward was changed via KIP-251 in 2.0.0 release. This PR removes the old and deprecated overloads.

Reviewers: A. Sophie Blee-Goldman <sophie@confluent.io>
2021-03-25 19:49:18 -07:00
John Roesler 9ef52dd2db
KAFKA-12508: Disable KIP-557 (#10397)
A major issue has been raised that this implementation of
emit-on-change is vulnerable to a number of data-loss bugs
in the presence of recovery with dirty state under at-least-once
semantics. This should be fixed in the future when we implement
a way to avoid or clean up the dirty state under at-least-once,
at which point it will be safe to re-introduce KIP-557 and
complete it.

Reviewers: A. Sophie Blee-Goldman <ableegoldman@apache.org>
2021-03-25 14:42:26 -05:00
Ismael Juma e840b03a02
MINOR: Always apply the java-library gradle plugin (#10394)
As @chia7712 found, we currently apply the `java` plugin indirectly via `rat.gradle`
if the `.git` folder exists (7071ded2a6/gradle/rat.gradle (L101)).

This led to inconsistent behavior if the `.git` directory was not present. With
this change, the `java-library` plugin (which has slightly more features than
the `java` plugin) is always applied.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, John Roesler <vvcephei@apache.org>
2021-03-24 20:19:28 -07:00
Jason Gustafson 3ab9ab6b6f
KAFKA-12434; Admin support for `DescribeProducers` API (#10275)
This patch adds the new `Admin` API to describe producer state as described by KIP-664: https://cwiki.apache.org/confluence/display/KAFKA/KIP-664%3A+Provide+tooling+to+detect+and+abort+hanging+transactions.

The three new APIs added by KIP-664 require different lookup and request patterns:

- DescribeProducers: send to partition leaders
- DescribeTransactions: send to coordinators
- ListTransactions: send to all brokers

Our method of handling complex workflows such as these in `KafkaAdminClient` by chaining together `Call` instances has been clumsy and error-prone at best. I have attempted to introduce a new pattern which separates the lookup stage (e.g. finding partition leaders) from the fulfillment stage (e.g. sending `DescribeProducers`). The lookup stage is implemented by `AdminApiLookupStrategy` and the fulfillment stage is implemented by `AdminApiHandler`. There is a new class `AdminApiDriver` which manages the bookkeeping for these two stages. See the corresponding javadocs for more detail. 

This PR provides an example of usage through `DescribeProducersHandler`, which is an implementation of `AdminApiHandler`. It relies on `PartitionLeaderStrategy` which implements `AdminApiLookupStrategy`. In addition to allowing for easier reuse of lookup strategies, this approach provides a more convenient way for testing since all of the logic is not crammed into `KafkaAdminClient`. Follow-up PRs for the rest of KIP-664 will flesh out additional lookup strategies such as for coordinator APIs.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, David Jacot <djacot@confluent.io>
2021-03-24 20:15:58 -07:00
John Roesler 9bf5c57997
KAFKA-12435: Fix javadoc errors (#10392)
There were errors while generating javadoc for the streams:test-utils module
because the included TopologyTestDriver imported some excluded classes.

This fixes the errors by inlining the previously excluded packages.

Reviewers: Chia-Ping Tsai <chia7712@apache.org>, Ismael Juma <ijuma@apache.org>
2021-03-24 13:55:27 -05:00
Colin Patrick McCabe 11f0ea3a5e
KAFKA-12432: AdminClient should time out nodes that are never ready (#10281)
Previously, if we assigned one or more calls to a remote node, but it
never became available, AdminClient would block until the calls hit
their the API timeout.  This was particularly unfortunate in the case
where the calls could have been sent to a different node in the cluster.
This PR fixes this behavior by timing out pending connections to
remote nodes if they take longer than the request timeout.

There are a few other small cleanups in this PR: it removes the
unecessary Call#aborted, sets Call#curNode to null after the call has
failed to avoid confusion when debugging or logging, and adds a
"closing" boolean rather than setting newCalls to null when the client
is closed.  Also, it increases the log level of the log message that
indicates that we timed out some calls because AdminClient closed,
and simplifies the type of callsInFlight.

Reviewers: Jason Gustafson <jason@confluent.io>
2021-03-24 11:10:19 -07:00
Chia-Ping Tsai 9af81955c4
KAFKA-12173 Migrate streams:streams-scala module to JUnit 5 (#9858)
1. replace org.junit.Assert by org.junit.jupiter.api.Assertions
2. replace org.junit by org.junit.jupiter.api
3. replace Before by BeforeEach
4. replace After by AfterEach
5. remove ExternalResource from all scala modules
6. add explicit AfterClass/BeforeClass to stop/start EmbeddedKafkaCluster

Noted that this PR does not migrate stream module to junit 5 so it does not introduce callback of junit 5 to deal with beforeAll/afterAll. The next PR of migrating stream module can replace explicit beforeAll/afterAll by junit 5 extension. Or we can keep the beforeAll/afterAll if it make code more readable.

Reviewers: John Roesler <vvcephei@apache.org>
2021-03-25 01:04:39 +08:00
Guozhang Wang 7071ded2a6
KAFKA-12524: Remove deprecated segments() (#10379)
Reviewers: Boyang Chen <boyang@confluent.io>
2021-03-23 21:05:42 -07:00
Jason Gustafson 99b9b3e84f
MINOR: Remove duplicate `createKafkaMetricsContext` (#10376)
This patch removes the duplicated `createKafkaMetricsContext` from `KafkaBroker`. It is already present in `Server`. Also added some small style cleanups.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, David Jacot <djacot@confluent.io>
2021-03-23 12:44:42 -07:00
Rajini Sivaram e3f60c254c
KAFKA-12479: Batch partition offset requests in ConsumerGroupCommand (#10371)
Reviewers: David Jacot <djacot@confluent.io>, Chia-Ping Tsai <chia7712@gmail.com>, Ismael Juma <ismael@juma.me.uk>
2021-03-23 09:56:56 +00:00
Justine Olshan 3d0b4d910b
MINOR: use new method to get number of topics in DeleteTopicsRequest (#10351)
Reviewers: David Jacot <djacot@confluent.io>
2021-03-23 08:29:17 +01:00
wenbingshen e0cbd0fa66
MINOR: Remove duplicate definition about 'the' from kafka project (#10370)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2021-03-23 10:44:55 +08:00
Jason Gustafson f5f66b982d
KAFKA-12181; Loosen raft fetch offset validation of remote replicas (#10309)
Currently the Raft leader raises an exception if there is a non-monotonic update to the fetch offset of a replica. In a situation where the replica had lost it disk state, this would prevent the replica from being able to recover. In this patch, we relax the validation to address this problem. It is worth pointing out that this validation could not be relied on to protect from data loss after a voter has lost committed state.

Reviewers: José Armando García Sancio <jsancio@gmail.com>, Boyang Chen <boyang@confluent.io>
2021-03-22 16:05:07 -07:00
Boyang Chen 80f373d34f
(Cherry-pick) KAFKA-9274: handle TimeoutException on task reset (#10000) (#10372)
This PR was removed by accident in trunk and 2.8, bringing it back.

Co-authored-by: Matthias J. Sax <matthias@confluent.io>
Reviewers: Matthias J. Sax <matthias@confluent.io>
2021-03-22 13:39:29 -07:00
John Roesler 7c45976c8d MINOR: bump documentation.html to 3.0 2021-03-22 10:59:17 -05:00
David Arthur e820eb42b2
KAFKA-12383: Get RaftClusterTest.java and other KIP-500 junit tests working (#10220)
Introduce "testkit" package which includes KafkaClusterTestKit class for enabling integration tests of self-managed clusters. Also make use of this new integration test harness in the ClusterTestExtentions JUnit extension. 

Adds RaftClusterTest for basic self-managed integration test. 

Reviewers: Jason Gustafson <jason@confluent.io>, Colin P. McCabe <cmccabe@apache.org>

Co-authored-by: Colin P. McCabe <cmccabe@apache.org>
2021-03-22 11:45:56 -04:00
dengziming e6f8ca80cd
MINOR: Fix log statement whose placeholders are inconsistent with arguments (#10312)
1. When the 2nd argument is an exception we don't need a placeholder
2. Placeholders should equal to arguments.

Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
2021-03-22 13:39:05 +08:00
John Roesler 2bfe7a9d1e
KAFKA-12514: Fix NPE in SubscriptionState (#10369)
Return null for partitionLag if there is no current position.
This was the desired semantics, the lack of the check was an
oversight.

Patches: KIP-695
Patches: a92b986c85

Reviewers: Walker Carlson <wcarlson@confluent.io>, A. Sophie Blee-Goldman <ableegoldman@apache.org>
2021-03-21 20:56:16 -05:00
CHUN-HAO TANG 2e2fad747d
MINOR: Remove redundant semicolon (#10358) 2021-03-21 17:43:03 +01:00