Author: Eno Thereska <eno.thereska@gmail.com>
Reviewers: Damian Guy <damian.guy@gmail.com>, Guozhang Wang <wangguoz@gmail.com>
Closes#2124 from enothereska/KAFKA-4359-intergration-tests-commit1
I think we should add a space here, otherwise the two words will join together.
And replace the host string with a constant, otherwise when I need to modify the host, I need to modify several files.
Author: 郑谦00117553 <00117553@zte.intra>
Reviewers: Guozhang Wang <wangguoz@gmail.com>
Closes#2132 from ZhengQian1/trunk
Since #1911 was merged it is hard to externally test a connector transitioning to FAILED state due to an initialization failure, which is what this test was attempting to verify.
The unit test added in #1778 already exercises exception-handling around Connector instantiation.
Author: Shikhar Bhushan <shikhar@confluent.io>
Reviewers: Jason Gustafson <jason@confluent.io>
Closes#2131 from shikhar/test_bad_connector_class
The consumer can deadlock on shutdown if a topic event fires during shutdown. The shutdown acquires the rebalance lock and then the topic-event-watcher lock. The topic event watcher acquires these in the reverse order. Shutdown should not need to acquire the topic-event-watcher’s lock - all it does is unsubscribes from topic events.
Author: Joel Koshy <jjkoshy@gmail.com>
Reviewers: Jiangjie Qin <becket.qin@gmail.com>
Closes#2129 from jjkoshy/KAFKA-4409
Author: Mickael Maison <mickael.maison@gmail.com>
Reviewers: Jiangjie Qin <becket.qin@gmail.com>, Jason Gustafson <jason@confluent.io>
Closes#1827 from mimaison/KAFKA-4081
The `StoreChangeLogger` currently keeps a cache of dirty and removed keys and will batch the changelog records such that we don't send a record for each update. However, with KIP-63 this is unnecessary as the batching and de-duping is done by the caching layer. Further, the `StoreChangeLogger` relies on `context.timestamp()` which is likely to be incorrect when caching is enabled
Author: Damian Guy <damian.guy@gmail.com>
Reviewers: Matthias J. Sax, Eno Thereska, Guozhang Wang
Closes#2103 from dguy/store-change-logger
leverage fix from KAFKA-2690 to remove secrets from task logging
Author: rnpridgeon <ryan.n.pridgeon@gmail.com>
Reviewers: Ewen Cheslack-Postava <ewen@confluent.io>
Closes#2115 from rnpridgeon/KAFKA-4364
Fix incorrect logging when unable to create an active task. The output was: Failed to create an active task %s:
It should have the taskId.
Author: Damian Guy <damian.guy@gmail.com>
Reviewers: Ismael Juma, Eno Thereska
Closes#2109 from dguy/minor-logging
Remove `keySerde`, `valSerde`, `OUTERTHIS_NAME`, `OUTEROTHER_NAME`, `LEFTTHIS_NAME`, `LEFTOTHER_NAME` from `KTableImpl` as they are all unused fields
Author: Damian Guy <damian.guy@gmail.com>
Reviewers: Guozhang Wang <wangguoz@gmail.com>
Closes#2119 from dguy/minor-ktable-unused
Given a topology like the one below. If a record arriving in `tableOne` causes a cache eviction, it will trigger the `leftJoin` that will do a `get` from `reducer-store`. If the key is not currently cached in `reducer-store`, but is in the backing store, it will be put into the cache, and it may also trigger an eviction. If it does trigger an eviction and the eldest entry is dirty it will flush the dirty keys. It is at this point that a ClassCastException is thrown. This occurs because the ProcessorContext is still set to the context of the `leftJoin` and the next child in the topology is `mapValues`.
We need to set the correct `ProcessorNode`, on the context, in the `ForwardingCacheFlushListener` prior to calling `context.forward`. We also need to remember to reset the `ProcessorNode` to the previous node once `context.forward` has completed.
```
final KTable<String, String> one = builder.table(Serdes.String(), Serdes.String(), tableOne, tableOne);
final KTable<Long, String> two = builder.table(Serdes.Long(), Serdes.String(), tableTwo, tableTwo);
final KTable<String, Long> reduce = two.groupBy(new KeyValueMapper<Long, String, KeyValue<String, Long>>() {
Override
public KeyValue<String, Long> apply(final Long key, final String value) {
return new KeyValue<>(value, key);
}
}, Serdes.String(), Serdes.Long())
.reduce(new Reducer<Long>() {..}, new Reducer<Long>() {..}, "reducer-store");
one.leftJoin(reduce, new ValueJoiner<String, Long, String>() {..})
.mapValues(new ValueMapper<String, String>() {..});
```
Author: Damian Guy <damian.guy@gmail.com>
Reviewers: Eno Thereska, Guozhang Wang
Closes#2051 from dguy/kafka-4311
Remove commented out code and System.out.println from KTableKTableJoinIntegrationTest
Author: Damian Guy <damian.guy@gmail.com>
Reviewers: Matthias J. Sax, Eno Thereska, Guozhang Wang
Closes#2092 from dguy/cleanup-comments
[KAFKA-4284](https://issues.apache.org/jira/browse/KAFKA-4284)
Even though Partitioner has a close method it is not closed when the producer is closed. Serializers, interceptors and metrics are all closed, so partitioners should be closed to.
To be able to use the same mechanism to close the partitioner as the serializers, etc. I had to make the `Partitioner` interface extend `Closeable`. Since this doesn't change the interface that feels ok and should be backwards compatible.
Looking at [KAFKA-2091](https://issues.apache.org/jira/browse/KAFKA-2091) (d6c45c70fb) that introduced the `Partitioner` interface it looks like the intention was that the producer should close the partitioner.
This contribution is my original work and I license the work to the project under the project's open source license.
Author: Theo <theo@iconara.net>
Reviewers: Ewen Cheslack-Postava <ewen@confluent.io>
Closes#2000 from iconara/kafka-4284
Kafka Connect REST API does not handle in many places connectors with slashes in their names because it expects PathParams, this PR intends to :
* Reject as bad requests API calls trying to create connectors with slashes in their names
* Add support for connector with slashes in their names in the DELETE part of the API to allow users to cleanup their connectors without dropping everything.
This PR adds as well the Unit Test needed for the creation part and was tested manually for the DELETE part.
Author: Olivier Girardot <o.girardot@lateral-thoughts.com>
Reviewers: Shikhar Bhushan <shikhar@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>
Closes#2096 from ogirardot/fix/connectors-with-slashes-cannot-be-deleted
I think the Javadoc should describe what happens if wakeup is called and no other thread is currently blocking. This may be important in some cases, e.g. trying to shut down a poll thread, followed by manually committing offsets.
Author: Stig Rohde Døssing <sdo@it-minds.dk>
Reviewers: Jason Gustafson <jason@confluent.io>
Closes#2093 from srdo/minor-expand-wakeup-javadoc
Fixes a bug that inappropriately applies backoff as interval between metadata updates even though the current one is outdated.
Author: Yuto Kawamura <kawamuray.dadada@gmail.com>
Reviewers: Ismael Juma <ismael@juma.me.uk>, Jason Gustafson <jason@confluent.io>
Closes#1707 from kawamuray/KAFKA-4024-metadata-backoff
This bug was recently introduced via 2959bc2ad.
Author: Himani Arora <1himani.arora@gmail.com>
Reviewers: Ismael Juma <ismael@juma.me.uk>
Closes#2091 from himani1/minor_fix
When storing a non-primitive type in a Connect offset, the following NullPointerException will occur:
```
07:18:23.702 [pool-3-thread-1] ERROR o.a.k.c.storage.OffsetStorageWriter - CRITICAL: Failed to serialize offset data, making it impossible to commit offsets under namespace tenant-db-bootstrap-source. This likely won't recover unless the unserializable partition or offset information is overwritten.
07:18:23.702 [pool-3-thread-1] ERROR o.a.k.c.storage.OffsetStorageWriter - Cause of serialization failure:
java.lang.NullPointerException: null
at org.apache.kafka.connect.storage.OffsetUtils.validateFormat(OffsetUtils.java:51)
at org.apache.kafka.connect.storage.OffsetStorageWriter.doFlush(OffsetStorageWriter.java:143)
at org.apache.kafka.connect.runtime.WorkerSourceTask.commitOffsets(WorkerSourceTask.java:319)
... snip ...
```
The attached patch fixes the specific case where OffsetUtils.validateFormat is attempting to provide a useful error message, but fails to because the schemaType method could return null.
This contribution is my original work and I license the work to the project under the project's open source license.
Author: Mathieu Fenniak <mathieu.fenniak@replicon.com>
Reviewers: Gwen Shapira
Closes#2087 from mfenniak/fix-npr-with-clearer-error-message
Removed a duplicate line and also cleaned up some of the language around compaction guarantees.
Author: Apurva Mehta <apurva.1618@gmail.com>
Reviewers: Gwen Shapira
Closes#2089 from apurvam/fix-documentation-of-compaction and squashes the following commits:
03c5bdd [Apurva Mehta] Fix line length to be consistent with the rest of the file
0af1a86 [Apurva Mehta] MINOR: fix duplicate line in docs for compaction.
Author: Vahid Hashemian <vahidhashemian@us.ibm.com>
Reviewers: Sriharsha Chintalapani <harsha@hortonworks.com>, Jason Gustafson <jason@confluent.io>
Closes#2075 from vahidhashemian/KAFKA-4357
- increased timeout to stabilize test
Author: Matthias J. Sax <matthias@confluent.io>
Reviewers: Eno Thereska, Guozhang Wang
Closes#2082 from mjsax/kafka-4352-hotfix
Enable user provided consumer and producer configs to override the streams default configs.
Author: Damian Guy <damian.guy@gmail.com>
Reviewers: Eno Thereska, Matthias J. Sax, Guozhang Wang
Closes#2084 from dguy/kafka-4361
This test fails locally when I run it, but somehow Jenkins builds are passed. Not clear how.
Author: Ismael Juma <ismael@juma.me.uk>
Reviewers: Jason Gustafson <jason@confluent.io>
Closes#2073 from ijuma/async-producer-test-port-config
Author: Matthias J. Sax <matthias@confluent.io>
Reviewers: Guozhang Wang, Ismael Juma, Michael G. Noll, Eno Thereska
Closes#2076 from mjsax/hotfixTSExtractor
KTableSource is always materialized since IQ:
- removed flag KTableSource#materialized
- removed MaterializedKTableSourceProcessor
Author: Matthias J. Sax <matthias@confluent.io>
Reviewers: Eno Thereska, Guozhang Wang
Closes#2065 from mjsax/kafka-4302-simplify-ktablesource
1. Create a new `ClientMetadata` to collapse `Set<String> consumerMemberIds`, `ClientState<TaskId> state`, and `HostInfo hostInfo`.
2. Stop reusing `stateChangelogTopicToTaskIds` and `internalSourceTopicToTaskIds` to access the (sub-)topology's internal repartition and changelog topics for clarity; also use the source topics num.partitions to set the num.partitions for repartition topics, and clarify to NOT have cycles since otherwise the while loop will fail.
3. `ensure-copartition` at the end to modify the number of partitions for repartition topics if necessary to be equal to other co-partition topics.
4. Refactor `ClientState` as well and update the logic of `TaskAssignor` for clarity as well.
5. Change default `clientId` from `applicationId-suffix` to `applicationId-processId` where `processId` is an UUID to avoid conflicts of clientIds that are from different JVMs, and hence conflicts in metrics.
6. Enforce `assignment` partitions to have the same size, and hence 1-1 mapping to `activeTask` taskIds.
7. Remove the `AssignmentSupplier` class by always construct the `partitionsByHostState` before assigning tasks to consumers within a client.
8. Remove all unnecessary member variables in `StreamPartitionAssignor`.
9. Some other minor fixes on unit tests, e.g. remove `test only` functions with java field reflection.
Author: Guozhang Wang <wangguoz@gmail.com>
Reviewers: Xavier Léauté, Matthias J. Sax, Eno Thereska, Jason Gustafson
Closes#2012 from guozhangwang/K4117-stream-partitionassignro-cleanup
Author: Jason Gustafson <jason@confluent.io>
Reviewers: Guozhang Wang <wangguoz@gmail.com>, Ismael Juma <ismael@juma.me.uk>
Closes#2053 from hachikuji/KAFKA-4326
The edge case where consumer group state is `PreparingRebalance` or `AwaitingSync` will be separately handled as the group assignment is not yet determined.
Author: Vahid Hashemian <vahidhashemian@us.ibm.com>
Reviewers: Jason Gustafson <jason@confluent.io>
Closes#2070 from vahidhashemian/KAFKA-4349
We had already made this change to the client created in `ZooKeeperTestHarness`.
I last saw this failure when `SaslPlaintextTopicMetadataTest.testAliveBrokerListWithNoTopics`
was executed in Jenkins.
Author: Ismael Juma <ismael@juma.me.uk>
Reviewers: Jason Gustafson <jason@confluent.io>
Closes#2066 from ijuma/increase-zk-timeout-in-kafka-server
Author: Matthias J. Sax <matthias@confluent.io>
Reviewers: Guozhang Wang <wangguoz@gmail.com>
Closes#2068 from mjsax/hotfixImproveWindowRetentionTimeJavaDoc
Also use named parameters in KafkaServer for clarity (even though it was correct previously).
Author: Matt <wangm92@163.com>
Reviewers: Guozhang Wang <wangguoz@gmail.com>, Ismael Juma <ismael@juma.me.uk>
Closes#1646 from wangzzu/wangzzu