Implement the KIP-554 API to create, describe, and alter SCRAM user configurations via the AdminClient. Add ducktape tests, and modify JUnit tests to test and use the new API where appropriate.
Reviewers: Colin P. McCabe <cmccabe@apache.org>, Rajini Sivaram <rajinisivaram@googlemail.com>
Make sure that we set the isNew field in LeaderAndIsrRequest correctly for brokers
that gets added to the replica set on reassignment.
This is tested by creating a variant of ControllerIntergationTest.testPartitionReassignment()
that makes one of the log directories on the target broker offline before initiating the
reassignment. Without the change to the way isNew is set, this fails after a timeout. With
the change, it succeeds.
To facilitate calling causeLogDirFailure() both from ControllerIntegrationTest and
LogDirFailureTest, the method was moved to TestUtils along with the other helper
methods that deals with interacting with KafkaServer instances for test cases.
Reviewers: Mickael Maison <mickael.maison@gmail.com>
Record the pollSensor after every invocation to poll, rather than just when we get records back so that we can accurately gauge how often we're invoking Consumer#poll.
Reviewers: Bruno Cadonna <bruno@confluent.io>, Guozhang Wang <guozhang@apache.org>, Matthias J. Sax <mjsax@apache.org>
Before this commit, Kafka Streams would gracefully shut down the whole application when a source topic is deleted. The graceful shutdown does not give the user the possibility to react on the deletion of the source topic in the uncaught exception handler.
This commit changes this behavior and throws an error when a source topic is deleted.
Reviewers: A. Sophie Blee-Goldman <sophie@confluent.io>, Guozhang Wang <guozhang@apache.org>, John Roesler <vvcephei@apache.org>
This commit adds the remaining property-based RocksDB metrics as described in KIP-607, except for num-entries-active-mem-table, which was added in PR #9177.
Reviewers: Guozhang Wang <wangguoz@gmail.com>
Implements KIP-617 on WindowStore that depends on #9137.
Testing strategy: extend existing tests to validate reverse operations are supported.
Reviewers: A. Sophie Blee-Goldman <sophie@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
Fixes a problem in which the Serdes class in the same package as
the tests (the old one) overshadows the one we explicitly imported
(the new one), but only in Scala 2.12. Since users (hopefully) don't
put their classes in our packages, they won't face the same problem.
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, David Arthur <mumrah@gmail.com>, John Roesler <vvcephei@apache.org>
Wildcard import of the old org.apache.kafka.streams.scala.Serdes leads
to a name clash because some of implicits has the same names as types
from the scala's std lib. The new oak.streams.scala.serialization.Serdes is
the same as the old Serdes, but without name clashes.
The old one is marked as deprecated.
Also, add missing serdes for UUID, ByteBuffer and Short types in
the new Serdes.
Implements: KIP-616
Reviewers: John Roesler <vvcephei@apache.org>
Add SlidingWindows API, implementation, and tests.
An edge case and an optimization are left to follow-on work.
Implements: KIP-450
Reviewers: A. Sophie Blee-Goldman <sophie@confluent.io>, Matthias J. Sax <mjsax@apache.org>, John Roesler <vvcephei@apache.org>
* Add the first RocksDB metric that exposes a RocksDB property: num-entries-active-mem-table.
* Add code StreamsMetricsImpl in support of exposing RocksDB properties
* unit tests and intergration tests
This commit only contains one metric to keep the PR at a reasonable size.
All other RocksDB metrics described in KIP-607 will be added in other PRs.
Implements: KIP-607
Reviewers: Guozhang Wang <guozhang@apache.org>, John Roesler <vvcephei@apache.org>
For the generated message code, put the JSON conversion functionality
in a separate JsonConverter class.
Make MessageDataGenerator simply another generator class, alongside the
new JsonConverterGenerator class. Move some of the utility functions
from MessageDataGenerator into FieldSpec and other places, so that they
can be used by other generator classes.
Use argparse4j to support a better command-line for the generator.
Reviewers: David Arthur <mumrah@gmail.com>
Adds avg, min, and max e2e latency metrics at the new TRACE level. Also adds the missing avg task-level metric at the INFO level.
I think where we left off with the KIP, the TRACE-level metrics were still defined to be "stateful-processor-level". I realized this doesn't really make sense and would be pretty much impossible to define given the DFS processing approach of Streams, and felt that store-level metrics made more sense to begin with. I haven't updated the KIP yet so I could get some initial feedback on this
Reviewers: Bruno Cadonna <bruno@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
MetadataCache#getPartitionMetadata returns an error when the topic's leader Id
is present at MetadataCache but listener endpoint is not present for this leader.
For older versions, LEADER_NOT_AVAILABLE is returned while LISTENER_NOT_FOUND is
returned for new metadata versions.
The problem is that getPartitionMetadata was looking up MetadataCache's host brokerId rather
than the topic's leader id while determining what error to return. This
could result in the call returning LISTENER_NOT_FOUND when it should
have returned LEADER_NOT_AVAILABLE. This commit corrects this behavior.
Unit tests were already present to test out the error codes returned
under different situations but they were giving out a false positive.
The test was using same broker id for both the MetadataCache's host as
well as for the topic's leader. Error manifests when the MetadataCache's
host id is changed. Improved the test.
This commit also consolidated couple of related tests to reduce code
duplication.
Reviewers: Jason Gustafson <jason@confluent.io>
Add new methods to KeyValueStore interfaces to support reverse iteration.
Reviewers: A. Sophie Blee-Goldman <sophie@confluent.io>, John Roesler <vvcephei@apache.org>
Only check if positions need validation if there is new metadata.
Also fix some inefficient java.util.stream code in the hot path of SubscriptionState.
From KIP-478, implement the new StreamBuilder#addGlobalStore() overload
that takes a stateUpdateSupplier fully typed Processor<KIn, VIn, Void, Void>.
Where necessary, use the adapters to make the old APIs defer to the new ones,
as well as limiting the scope of this change set.
Reviewers: Boyang Chen <boyang@apache.org>
* KAFKA-10407: Have KafkaLog4jAppender `batch.size` and `linger.ms`
https://issues.apache.org/jira/browse/KAFKA-10407
Currently, KafkaLog4jAppender does not support `batch.size` or `linger.ms` which would otherwise be beneficial in some situations.
ducktape diff: https://github.com/confluentinc/ducktape/compare/v0.7.8...v0.7.9
- bcrypt (a dependency of ducktape) dropped Python2.7 support.
ducktape-0.7.9 now pins bcrypt to a Python2.7-supported version.
Author: Andrew Egelhofer <aegelhofer@confluent.io>
Reviewers: Dhruvil Shah <dhruvil@confluent.io>, Manikumar Reddy <manikumar.reddy@gmail.com>
Closes#9192 from andrewegel/trunk
Implements the part of KIP-612 that adds broker configurations for broker-wide and per-listener connection creation rate limits and enforces these limits.
Reviewers: David Jacot <djacot@confluent.io>, Rajini Sivaram <rajinisivaram@googlemail.com>
Addition of configs for custom topic creation with KIP-158 created a regression when transformation configs are also included in the configuration of a source connector.
To experience the issue, just enabling topic creation at the worker is not sufficient. A user needs to supply a source connector configuration that contains both transformations and custom topic creation properties.
The issue is that the enrichment of configs in `SourceConnectorConfig` happens on top of an `AbstractConfig` rather than a `ConnectorConfig`. Inheriting from the latter allows enrichment to be composable for both topic creation and transformations.
Unit tests and integration tests are written to test these combinations.
Reviewers: Randall Hauch <rhauch@gmail.com>
The main goal is to remove usage of embedded broker (EmbeddedKafkaCluster) in AbstractJoinIntegrationTest and its subclasses.
This is because the tests under this class are no longer using the embedded broker, except for two.
testShouldAutoShutdownOnIncompleteMetadata is one of such tests.
Furthermore, this test does not actually perfom stream-table join; it is testing an edge case of joining with a non-existent topic, so it should be in a separate test.
Testing strategy: run existing unit and integration test
Reviewers: Boyang Chen <boyang@confluent.io>, Bill Bejeck <bbejeck@apache.org>
Refactor the RocksDB store and the metrics infrastructure in Streams
in preparation of the recordings of the RocksDB properties specified in KIP-607.
The refactoring includes:
* wrapper around BlockedBasedTableConfig to make the cache accessible to the
RocksDB metrics recorder
* RocksDB metrics recorder now takes also the DB instance and the cache in addition
to the statistics
* The value providers for the metrics are added to the RockDB metrics recorder also if
the recording level is INFO.
* The creation of the RocksDB metrics recording trigger is moved to StreamsMetricsImpl
Reviewers: Guozhang Wang <wangguoz@gmail.com>, John Roesler <vvcephei@apache.org>
This patch fixes the generated serde logic for the 'records' type so that it uses the compact byte array representation consistently when flexible versions are enabled.
Reviewers: David Arthur <mumrah@gmail.com>
In order to do this, I also removed the optimization such that once enforced checkpoint is set to true, we always checkpoint unless the state stores are not initialized at all (i.e. the snapshot is null).
Reviewers: Boyang Chen <boyang@confluent.io>, A. Sophie Blee-Goldman <ableegoldman@gmail.com>
Add a separate error code as PRODUCER_FENCED to differentiate INVALID_PRODUCER_EPOCH. On broker side, replace INVALID_PRODUCER_EPOCH with PRODUCER_FENCED when the request version is the latest, while still returning INVALID_PRODUCER_EPOCH to older clients. On client side, simply handling INVALID_PRODUCER_EPOCH the same as PRODUCER_FENCED if from txn coordinator APIs.
Reviewers: Guozhang Wang <wangguoz@gmail.com>
The message generator was missing conversion logic for tagged structures. This led to casting errors when either `fromStruct` or `toStruct` were invoked. This patch also adds missing null checks in the serialization of tagged byte arrays, which was found from improved test coverage.
Reviewers: Colin P. McCabe <cmccabe@apache.org>