Fix compile error in scala tests.
The compile error is:
```
[Error] /home/jenkins/jenkins-agent/workspace/Kafka_kafka-pr_PR-9229/core/src/test/scala/unit/kafka/server/metadata/BrokerMetadataListenerTest.scala:97: polymorphic expression cannot be instantiated to expected type;
[2021-05-29T02:34:50.308Z] found : [T]()T
[2021-05-29T02:34:50.308Z] required: kafka.server.RequestLocal
```
This error happens only in scala 2.12
Reviewers: Bruno Cadonna <cadonna@apache.org>
Use a caching `BufferSupplier` per request handler thread so that
decompression buffers are cached if supported by the underlying
`CompressionType`. This achieves a similar outcome as #9220, but
with less contention.
We introduce a `RequestLocal` class to make it easier to introduce
new request scoped stateful instances (one example we discussed
previously was an `ActionQueue` that could be used to avoid
some of the complex group coordinator locking).
This is a small win for zstd (no synchronization or soft references) and
a more significant win for lz4. In particular, it reduces allocations
significantly when the number of partitions is high. The decompression
buffer size is typically 64 KB, so a produce request with 1000 partitions
results in 64 MB of allocations even if each produce batch is small (likely,
when there are so many partitions).
I did a quick producer perf local test with 5000 partitions, 1 KB record
size,
1 broker, lz4 and ~0.5 for the producer compression rate metric:
Before this change:
> 20000000 records sent, 346314.349535 records/sec (330.27 MB/sec),
148.33 ms avg latency, 2267.00 ms max latency, 115 ms 50th, 383 ms 95th, 777 ms 99th, 1514 ms 99.9th.
After this change:
> 20000000 records sent, 431956.113259 records/sec (411.95 MB/sec),
117.79 ms avg latency, 1219.00 ms max latency, 99 ms 50th, 295 ms 95th, 440 ms 99th, 662 ms 99.9th.
That's a 25% throughput improvement and p999 latency was reduced to
under half (in this test).
Default arguments will be removed in a subsequent PR.
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
The following error happens on my mac m1 when building docker image for system tests.
Collecting pynacl
Using cached PyNaCl-1.4.0.tar.gz (3.4 MB)
Installing build dependencies ... error
ERROR: Command errored out with exit status 1:
command: /usr/bin/python3 /usr/local/lib/python3.8/dist-packages/pip install --ignore-installed --no-user --prefix /tmp/pip-build-env-k867aac0/overlay --no-warn-script-location --no-binary :none: --only-binary :none: -i https://pypi.org/simple -- 'setuptools>=40.8.0' wheel 'cffi>=1.4.1; python_implementation != '"'"'PyPy'"'"''
cwd: None
Complete output (14 lines):
Traceback (most recent call last):
File "/usr/lib/python3.8/runpy.py", line 194, in _run_module_as_main
return _run_code(code, main_globals, None,
File "/usr/lib/python3.8/runpy.py", line 87, in _run_code
exec(code, run_globals)
File "/usr/local/lib/python3.8/dist-packages/pip/__main__.py", line 23, in <module>
from pip._internal.cli.main import main as _main # isort:skip # noqa
File "/usr/local/lib/python3.8/dist-packages/pip/_internal/cli/main.py", line 5, in <module>
import locale
File "/usr/lib/python3.8/locale.py", line 16, in <module>
import re
File "/usr/lib/python3.8/re.py", line 145, in <module>
class RegexFlag(enum.IntFlag):
AttributeError: module 'enum' has no attribute 'IntFlag'
----------------------------------------
ERROR: Command errored out with exit status 1: /usr/bin/python3 /usr/local/lib/python3.8/dist-packages/pip install --ignore-installed --no-user --prefix /tmp/pip-build-env-k867aac0/overlay --no-warn-script-location --no-binary :none: --only-binary :none: -i https://pypi.org/simple -- 'setuptools>=40.8.0' wheel 'cffi>=1.4.1; python_implementation != '"'"'PyPy'"'"'' Check the logs for full command output.
There was a related issue: pypa/pip#9689 and it is already fixed by pypa/pip#9689 (included by pip 21.1.1). I test the pip 21.1.1 and it works well on mac m1.
Reviewers: Ismael Juma <ismael@juma.me.uk>
* Lay the groundwork for migrating KTable Processors to the new PAPI.
* Migrate the KTableFilter processor to prove that the groundwork works.
This is an effort to help break up #10507 into multiple PRs.
Reviewers: Boyang Chen <boyang@apache.org>
This, upgrades JDK to version 15 for the docs generation, this way we
can circumvent bug https://bugs.openjdk.java.net/browse/JDK-8215291
present in JDK11
Reviewers: Ismael Juma <ismael@juma.me.uk>
I'm migrating Ranger's kafka plugin from deprecated Authorizer (this is already removed by 976e78e) to new API (see https://issues.apache.org/jira/browse/RANGER-3231). The kafka plugin needs to take something from field resourcePattern but it does not know whether the field is nullable (or users need to add null check). I check all usages and I don't observe any null case.
Reviewers: Ismael Juma <ismael@juma.me.uk>
Recently we have noticed multiple instances where KafkaProducers have failed to constructor due to the following exception:
```
org.apache.kafka.common.KafkaException: Failed to construct kafka producer at
org.apache.kafka.clients.producer.KafkaProducer.<init>(KafkaProducer.java:440) at
org.apache.kafka.clients.producer.KafkaProducer.<init>(KafkaProducer.java:291) at
org.apache.kafka.clients.producer.KafkaProducer.<init>(KafkaProducer.java:318)
java.base/java.lang.Thread.run(Thread.java:832) Caused by: java.util.ConcurrentModificationException at
java.base/java.util.HashMap$HashIterator.nextNode(HashMap.java:1584) at
java.base/java.util.HashMap$KeyIterator.next(HashMap.java:1607) at
java.base/java.util.AbstractSet.removeAll(AbstractSet.java:171) at
org.apache.kafka.common.config.AbstractConfig.unused(AbstractConfig.java:221) at
org.apache.kafka.common.config.AbstractConfig.logUnused(AbstractConfig.java:379) at
org.apache.kafka.clients.producer.KafkaProducer.<init>(KafkaProducer.java:433) ... 9 more
exception.class:org.apache.kafka.common.KafkaException exception.message:Failed to construct kafka producer
```
This is due to the fact that `used` below is a synchronized set. `used` is being modified while removeAll is being called. This is due to the use of RecordingMap in the Sender thread (see below). Switching to a ConcurrentHashSet avoids this issue as it support concurrent iteration.
```
at org.apache.kafka.clients.producer.ProducerConfig.ignore(ProducerConfig.java:569)
at org.apache.kafka.common.config.AbstractConfig$RecordingMap.get(AbstractConfig.java:638)
at org.apache.kafka.common.network.ChannelBuilders.createPrincipalBuilder(ChannelBuilders.java:242)
at org.apache.kafka.common.network.PlaintextChannelBuilder$PlaintextAuthenticator.<init>(PlaintextChannelBuilder.java:96)
at org.apache.kafka.common.network.PlaintextChannelBuilder$PlaintextAuthenticator.<init>(PlaintextChannelBuilder.java:89)
at org.apache.kafka.common.network.PlaintextChannelBuilder.lambda$buildChannel$0(PlaintextChannelBuilder.java:66)
at org.apache.kafka.common.network.KafkaChannel.<init>(KafkaChannel.java:174)
at org.apache.kafka.common.network.KafkaChannel.<init>(KafkaChannel.java:164)
at org.apache.kafka.common.network.PlaintextChannelBuilder.buildChannel(PlaintextChannelBuilder.java:79)
at org.apache.kafka.common.network.PlaintextChannelBuilder.buildChannel(PlaintextChannelBuilder.java:67)
at org.apache.kafka.common.network.Selector.buildAndAttachKafkaChannel(Selector.java:356)
at org.apache.kafka.common.network.Selector.registerChannel(Selector.java:347)
at org.apache.kafka.common.network.Selector.connect(Selector.java:274)
at org.apache.kafka.clients.NetworkClient.initiateConnect(NetworkClient.java:1097)
at org.apache.kafka.clients.NetworkClient.access$700(NetworkClient.java:87)
at org.apache.kafka.clients.NetworkClient$DefaultMetadataUpdater.maybeUpdate(NetworkClient.java:1276)
at org.apache.kafka.clients.NetworkClient$DefaultMetadataUpdater.maybeUpdate(NetworkClient.java:1164)
at org.apache.kafka.clients.NetworkClient.poll(NetworkClient.java:637)
at org.apache.kafka.clients.producer.internals.Sender.runOnce(Sender.java:327)
at org.apache.kafka.clients.producer.internals.Sender.run(Sender.java:242)
```
Reviewers: Ismael Juma <ismael@juma.me.uk>
Log if deletion fails and don't expose log topic id for mutability outside of `assignTopicId()`.
Also remove an unnecessary parameter in `PartitionTest`.
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, Justine Olshan <jolshan@confluent.io>
2.10.x is no longer supported, so we should move to 2.12 for the 3.0
release.
ScalaObjectMapper has been deprecated and it looks like we don't
actually need it, so remove its usage.
Reviewers: David Jacot <djacot@confluent.io>
In Log.collectAbortedTransactions() I've restored a previously used logic, such that it would handle the case where the starting segment could be null. This was the case previously, but the PR #10401 accidentally changed the behavior causing the code to assume that the starting segment won't be null.
In Log.rebuildProducerState() I've removed usage of the allSegments local variable. The logic looks a bit simpler after I removed it.
I've introduced a new LogSegments.higherSegments() API. This is now used to make the logic a bit more readable in Log. collectAbortedTransactions() and Log.deletableSegments() APIs.
I've removed the unnecessary use of java.lang.Long in LogSegments class' segments map definition.
I've converted a few LogSegments API from public to private, as they need not be public.
Reviewers: Ismael Juma <ismael@juma.me.uk>, Cong Ding <cong@ccding.com>, Jun Rao <junrao@gmail.com>
Quick followup to KIP-740 to actually deprecate this constructor, and update the upgrade guide with what we changed in KIP-740. I also noticed the TaskId#parse method had been modified previously, and should be re-added to the public TaskId class. It had no tests, so now it does
Reviewers: Matthias J. Sax <mjsax@confluent.io>, Luke Chen <showuon@gmail.com>
This patch logs more information when a producer snapshot is written to the disk.
Reviewers: Ismael Juma <mlists@juma.me.uk>, Lucas Bradstreet <lucas@confluent.io>
The docs for the max.in.flight.requests.per.connection and enable.idempotence configs currently imply that setting the max in-flight request greater than 1 will break the message ordering guarantee, but that is only true if enable.idempotence is false. When using an idempotent producer, the max in-flight request can be up to 5 without re-ordering messages.
Reviewers: Matthias J. Sax <mjsax@confluent.io>, Ismael Juma <mlists@juma.me.uk>, Luke Chen <showuon@gmail.com>
When #10478 was merged, we accidentally lost the identifier/prefix string that we used to previously log to stderr from some of the functions in the Log class. In this PR, I have reinstated the identifier/prefix logging in these functions, so that the debuggability is restored.
Reviewers: Luke Chen <showuon@gmail.com>, Cong Ding <cong@ccding.com>, Jun Rao <junrao@gmail.com>
This PR tries to add the check for transaction timeout for a comparison against commit interval of streams. If transaction timeout is smaller than commit interval, stream should crash and inform user to update their commit interval to be larger or equal to the given transaction timeout, or vise versa.
Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>, A. Sophie Blee-Goldman <sophie@confluent.io>
Introduce new AllocateProducerIds RPC and IBP 3.0-IV0 as part of KIP-730.
This change adds a new AllocateProducerIds RPC which is used by the broker to request a block of
producer IDs from the controller. The new IBP added will determine if the broker should talk directly to
ZooKeeper (IBP < 3.0) or it if should use the new RPC to talk to the controller (IBP >= 3.0).
Per-broker property overrides for ClusterTests were also added (in order to test mixed IBPs in a cluster)
Reviewers: Colin P. McCabe <cmccabe@apache.org>
Code samples are now unified and correctly formatted.
Samples under Streams use consistently the prism library.
Reviewers: Bruno Cadonna <cadonna@apache.org>
CVE-2021-26291, which makes Man-In-The-Middle-Attack possible, was fixed in maven 3.8.1.
Reviewers: Luke Chen <showuon@gmail.com>, Manikumar Reddy <manikumar.reddy@gmail.com>
This patch removes the temporary shim layer we added to bridge the interface
differences between MetaLogManager and RaftClient. Instead, we now use the
RaftClient directly from the metadata module. This also means that the
metadata gradle module now depends on raft, rather than the other way around.
Finally, this PR also consolidates the handleResign and handleNewLeader APIs
into a single handleLeaderChange API.
Co-authored-by: Jason Gustafson <jason@confluent.io>
As described in KIP-740, we clean up the public TaskId class and introduce new APIs to return it from TaskMetadata
Reviewers: Guozhang Wang <guozhang@confluent.io>
This patch adds support for running the ZooKeeper-based
kafka.security.authorizer.AclAuthorizer with KRaft clusters. Set the
authorizer.class.name config as well as the zookeeper.connect config while also
setting the typical KRaft configs (node.id, process.roles, etc.), and the
cluster will use KRaft for metadata and ZooKeeper for ACL storage. A system
test that exercises the authorizer is included.
This patch also changes "Raft" to "KRaft" in several system test files. It also
fixes a bug where system test admin clients were unable to connect to a cluster
with broker credentials via the SSL security protocol when the broker was using
that for inter-broker communication and SASL for client communication.
Reviewers: Colin P. McCabe <cmccabe@apache.org>, Ismael Juma <ismael@juma.me.uk>
Correct empty meta response comment, since it is no longer related only to brokers associating with the query topic.
Reviewers: Boyang Chen <boyang@confluent.io>
This includes TASTy Reader support for Scala 3.0.0. This makes it easier
for Kafka libraries to be used in Scala 3.0 projects
Release notes: https://github.com/scala/scala/releases/tag/v2.13.6
Reviewers: Ismael Juma <ismael@juma.me.uk>
Increase session timeout to fix flaky KTableKTableForeignKeyInnerJoinMultiIntegrationTest.shouldInnerJoinMultiPartitionQueryable
Reviewers: Anna Sophie Blee-Goldman <ableegoldman@apache.org>