Commit Graph

615 Commits

Author SHA1 Message Date
nicolasguyomar f8173c2df5
MINOR: Update Connect error message to point to the correct config validation REST endpoint (#7991)
When incorrect connector configuration is detected, the returned exception message suggests to check the connector's configuration against the `{connectorType}/config/validate` endpoint. 

Changing the error message to refer to the exact REST endpoint which is `/connector-plugins/{connectorType}/config/validate` 

This aligns the exception message with the documentation at: https://kafka.apache.org/documentation/#connect_rest 

Reviewers: Konstantine Karantasis <konstantine@confluent.io>
2020-03-22 12:42:45 -07:00
Chia-Ping Tsai 4f6907947a
MINOR: fix linking errors in javadoc (#8198)
This improvement fixes several linking errors to classes and methods from within javadocs. 

Related to #8291

Reviewers: Konstantine Karantasis <konstantine@confluent.io>
2020-03-22 10:15:18 -07:00
Konstantine Karantasis 406635bcc9
MINOR: Use Exit.exit instead of System.exit in MM2 (#8321)
Exit.exit needs to be used in code instead of System.exit.

Particularly in integration tests using System.exit is disrupting because it exits the jvm process and does not just fail the test correctly. Integration tests override procedures in Exit to protect against such cases.

Reviewers: Ryanne Dolan <ryannedolan@gmail.com>, Ismael Juma <ismael@juma.me.uk>, Randall Hauch <rhauch@gmail.com>
2020-03-20 16:39:17 -07:00
Colin Patrick McCabe 56051e7639
KAFKA-8820: kafka-reassign-partitions.sh should support the KIP-455 API (#8244)
Rewrite ReassignPartitionsCommand to use the KIP-455 API when possible, rather
than direct communication with ZooKeeper.  Direct ZK access is still supported,
but deprecated, as described in KIP-455.

As specified in KIP-455, the tool has several new flags.  --cancel stops
an assignment which is in progress.  --preserve-throttle causes the
--verify and --cancel commands to leave the throttles alone.
--additional allows users to execute another partition assignment even
if there is already one in progress.  Finally, --show displays all of
the current partition reassignments.

Reorganize the reassignment code and tests somewhat to rely more on unit
testing using the MockAdminClient and less on integration testing.  Each
integration test where we bring up a cluster seems to take about 5 seconds, so
it's good when we can get similar coverage from unit tests.  To enable this,
MockAdminClient now supports incrementalAlterConfigs, alterReplicaLogDirs,
describeReplicaLogDirs, and some other APIs.  MockAdminClient is also now
thread-safe, to match the real AdminClient implementation.

In DeleteTopicTest, use the KIP-455 API rather than invoking the reassignment
command.
2020-03-19 20:44:34 -07:00
Matthias J. Sax dffc7f8c30
MINOR: Fix build and JavaDoc warnings (#8291)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>, @SoontaekLim, Bill Bejeck <bill@confluent.io>
2020-03-16 18:23:21 -07:00
Nigel Liang 569cf994b0
KAFKA-9712: Catch and handle exception thrown by reflections scanner (#8289)
This commit works around a bug in version v0.9.12 of the upstream `reflections` library by catching and handling the exception thrown.

The reflections issue is tracked by:
https://github.com/ronmamo/reflections/issues/273

New unit tests were introduced to test the behavior.

* KAFKA-9712: Catch and handle exception thrown by reflections scanner

* Update connect/runtime/src/main/java/org/apache/kafka/connect/runtime/isolation/DelegatingClassLoader.java

Co-Authored-By: Konstantine Karantasis <konstantine@confluent.io>

* Move result initialization back to right before it is used

* Use `java.io.File` in tests

* Fix checkstyle

Co-authored-by: Konstantine Karantasis <konstantine@confluent.io>

Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Chia-Ping Tsai <chia7712@gmail.com>
2020-03-16 14:43:10 -07:00
Chris Egerton b9e3fa2322
KAFKA-9601: Stop logging raw connector config values (#8165)
Author: Chris Egerton <chrise@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2020-02-26 15:52:15 -06:00
Konstantine Karantasis 16ee326755
KAFKA-9556; Fix two issues with KIP-558 and expand testing coverage (#8085)
Correct the Connect worker logic to properly disable the new topic status (KIP-558) feature when `topic.tracking.enable=false`, and fix automatic topic status reset after a connector is deleted.

Also adds new `ConnectorTopicsIntegrationTest` and expanded unit tests.

Reviewers: Randall Hauch <rhauch@gmail.com>
2020-02-14 14:34:34 -08:00
Xavier Léauté 7e1c39f75a
KAFKA-9106 make metrics exposed via jmx configurable (#7674)
Reviewers: Colin P. McCabe <cmccabe@apache.org>, Rajini Sivaram <rajinisivaram@googlemail.com>, Manikumar Reddy <manikumar.reddy@gmail.com>
2020-02-13 10:21:14 -08:00
Konstantine Karantasis 97d2c726f1
MINOR: Small Connect integration test fixes (#8100)
Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2020-02-12 17:40:37 -06:00
Lev Zemlyanov f51e1e6548
allow ReplaceField SMT to handle tombstone records (#7731)
Signed-off-by: Lev Zemlyanov <lev@confluent.io>
2020-02-12 16:36:06 -06:00
Lev Zemlyanov c8f1ee9cd9
KAFKA-9192: fix NPE when for converting optional json schema in structs (#7733)
Author: Lev Zemlyanov <lev@confluent.io>
Reviewers: Greg Harris <gregh@confluent.io>, Randall Hauch <rhauch@gmail.com>
2020-02-12 15:44:23 -06:00
Konstantine Karantasis 0f14aef8cb
MINOR: Start using Response and replace IOException in EmbeddedConnectCluster for failures (#8055)
Changed `EmbeddedConnectCluster` to add utility methods that return `Response`, throw `ConnectException` instead of `IOException` for failures, and deprecate the old methods that returned primitive types rather than `Response`. 

Also introduce common assertions for embedded clusters under `EmbeddedConnectClusterAssertions`.

Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>
2020-02-11 14:49:26 -06:00
Gunnar Morling 5727b24509
KAFKA-7052 Avoiding NPE in ExtractField SMT in case of non-existent fields (#8059)
Author: Gunnar Morling <gunnar.morling@googlemail.com>
Reviewer: Randall Hauch <rhauch@gmail.com>
2020-02-11 13:48:22 -06:00
Sanjana Kaundinya 4d86b191c0
KAFKA-9509; Fixing flakiness of MirrorConnectorsIntegrationTest.testReplication (#8048)
The test case `org.apache.kafka.connect.mirror.MirrorConnectorsIntegrationTest.testReplication` has shown to be increasingly flaky recently. This PR aims to make this test more deterministic. Specifically, the flakiness was due to a timing issue between the tasks not starting up in time for the test to start running. This PR remediates that by introducing a status check after every connector is started up. These status checks include that the connector is found on the connect cluster as well as there are tasks created and up and running for that connector. These checks are introduced before the test starts running so that there is a confidence that the connectors and tasks are started up correctly before the test runs.

Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Jason Gustafson <jason@confluent.io>
2020-02-07 13:58:22 -08:00
Randall Hauch dd7a314591
KAFKA-9074: Correct Connect’s `Values.parseString` to properly parse a time and timestamp literal (#7568)
* KAFKA-9074: Correct Connect’s `Values.parseString` to properly parse a time and timestamp literal

Time and timestamp literal strings contain a `:` character, but the internal parser used in the `Values.parseString(String)` method tokenizes on the colon character to tokenize and parse map entries. The colon could be escaped, but then the backslash character used to escape the colon is not removed and the parser fails to match the literal as a time or timestamp value.

This fix corrects the parsing logic to properly parse timestamp and time literal strings whose colon characters are either escaped or unescaped. Additional unit tests were added to first verify the incorrect behavior and then to validate the correction.

Author: Randall Hauch <rhauch@gmail.com>
Reviewers: Chris Egerton <chrise@confluent.io>, Nigel Liang <nigel@nigelliang.com>, Jason Gustafson <jason@confluent.io>
2020-02-04 11:15:04 -06:00
Karan Kumar c8d97c6d51
KAFKA-9375: Add names to all Connect threads (#7901)
Reviewers: Mickael Maison <mickael.maison@gmail.com>, Ryanne Dolan <ryannedolan@gmail.com>, gcsaba2
2020-01-31 18:21:21 +00:00
ning2008wisc f217752bed
KAFKA-9360: Allow disabling MM2 heartbeat and checkpoint emissions (#7887)
`emit.heartbeats.enabled` and `emit.checkpoints.enabled` are supposed to
be the knobs to control if the heartbeat message or checkpoint message
will be sent or not to the topics respectively. In our experiments,
setting them to false will not suspend the activity in their SourceTasks,
e.g. MirrorHeartbeatTask, MirrorCheckpointTask.

The observations are, when setting those knobs to false, huge volume of
`SourceRecord` are being sent without interval, causing significantly high
CPU usage and GC time  of MirrorMaker 2 instance and congesting the single
partition of the heartbeat topic and checkpoint topic.

The proposed fix in the following PR is to (1) explicitly check if `interval`
is set to negative (e.g. -1), when the `emit.heartbeats.enabled` or
`emit.checkpoints.enabled` is off. (2) if `interval` is indeed set to negative,
no task is created.

Reviewers: Mickael Maison <mickael.maison@gmail.com>, Ryanne Dolan <ryannedolan@gmail.com>
2020-01-30 10:43:34 +00:00
Konstantine Karantasis 7746301c26
KAFKA-9422: Track the set of topics a connector is using (KIP-558) (#8017)
This feature corresponds to KIP-558 and extends how the internal status topic (set via `status.storage.topic` distributed worker config) is used to include information that allows Kafka Connect to keep track which topics a connector is using.

The set of topics a connector is actively using, is exposed via a new endpoint that is added to the REST API of Connect workers.
* A `GET /connectors/{name}/topics` request will return the set of topics that have been recorded as active since a connector started or since the set of topics was reset for this connector.

An additional endpoints allows users to reset the set of active topics for a connector via the second endpoint that this feature is adding:
* A `PUT /connectors/{name}/topics/reset` request clears the set of active topics. An operator may enable or disable this feature by setting `topic.tracking.enable` (true by default).

The `topic.tracking.enable` worker config property (true by default) allows an operator to enable/disable the entire feature. Or if the feature is enabled, the `topic.tracking.allow.reset` worker config property (true by default) allows an operator to control whether reset requests submitted to the Connect REST API are allowed.

Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2020-01-29 23:54:21 -06:00
belugabehr aecd3936a3
KAFKA-9405: Use Map.computeIfAbsent where applicable (#7937)
Reviewers: Ismael Juma <ismael@juma.me.uk>
2020-01-29 13:00:22 -08:00
Ron Dagostino a3509c0870 MINOR: MiniKdc JVM shutdown hook fix (#7946)
Also made all shutdown hooks consistent and added tests

Reviewers: Ismael Juma <ismael@juma.me.uk>, Rajini Sivaram <rajinisivaram@googlemail.com>
2020-01-24 22:21:12 +00:00
Ted Yu 8e5faca963 Correct exception message in DistributedHerder (#7995)
Author: Ted Yu <yuzhihong@gmail.com>
Reviewer: Randall Hauch <rhauch@gmail.com>
2020-01-24 13:56:08 -06:00
Gunnar Morling b399a731a3 KAFKA-7273 Clarification on mutability of headers passed to Converter#fromConnectData() (#7489)
Author: Gunnar Morling <gunnar.morling@googlemail.com>
Reviewer: Randall Hauch <rhauch@gmail.com>
2020-01-22 10:46:53 -06:00
Ivan Yurchenko 6594114cbd KAFKA-9143: Log task reconfiguration error only when it happened (#7648)
This commit makes `DistributedHerder` log that some error has happened during task reconfiguration only when it actually has happened.

Author: Ivan Yurchenko <ivan0yurchenko@gmail.com>
Reviewer: Randall Hauch <rhauch@gmail.com>
2020-01-21 14:52:40 -06:00
Chia-Ping Tsai a14f25546a MINOR: Change the log level from ERROR to DEBUG when failing to get plugin loader for connector (#7964)
Reviewers: Jason Gustafson <jason@confluent.io>
2020-01-21 12:39:35 -08:00
Nigel Liang 28f013708f KAFKA-9024: Better error message when field specified does not exist (#7819)
Author: Nigel Liang <nigel@nigelliang.com>
Reviewer: Randall Hauch <rhauch@gmail.com>
2020-01-21 14:25:35 -06:00
Chris Egerton 58a0b99e92 KAFKA-9083: Various fixes/improvements for Connect's Values class (#7593)
Author: Chris Egerton <chrise@confluent.io>
Reviewers: Greg Harris <gregh@confluent.io>, Randall Hauch <rhauch@gmail.com>
2020-01-21 13:33:50 -06:00
Narek Karapetian d755ca14a0 MINOR: log error message from Connect sink exception (#7555)
Author: Narek Karapetian <narek_karapetian@epam.com>
Reviewer: Randall Hauch <rhauch@gmail.com>
2020-01-21 13:21:16 -06:00
Chris Egerton eed81c66e9 MINOR: Fix typo in connect integration test class name (#7976)
Reviewers: Randall Hauch <rhauch@gmail.com>, Jason Gustafson <jason@confluent.io>
2020-01-17 08:23:34 -08:00
Edoardo Comar a88703255b KAFKA-9218: MirrorMaker 2 can fail to create topics (#7745)
When the scheduled refreshTopicPartitions runs, check existing topics in
both source and target clusters in order to compute topic partitions to
be created on target.

If a temporary failure to create the target topic is encountered (e.g.
insufficient number of brokers), on the next refresh the target topic
creation will be re-attempted.

Co-authored-by: Edoardo Comar <ecomar@uk.ibm.com>
Co-authored-by: Mickael Maison <mickael.maison@uk.ibm.com>
Reviewers: Ryanne Dolan <ryannedolan@gmail.com>, Manikumar Reddy <manikumar.reddy@gmail.com>
2020-01-17 13:44:18 +00:00
Sönke Liebau 5c00191ea9 MINOR: Removed accidental double negation in error message. (#7834) 2020-01-15 12:25:43 -08:00
Mickael Maison 3953204d35 MINOR: Fix connect:mirror checkstyle (#7951)
Reviewers: Guozhang Wang <wangguoz@gmail.com>, Jason Gustafson <jason@confluent.io>
2020-01-13 15:25:24 -08:00
ning2008wisc 4fe324b5a6 KAFKA-9352: Use 'roundrobin' to assign topic-partitions to mirroring tasks (#7880)
Reviewers: Mickael Maison <mickael.maison@gmail.com>, Ryanne Dolan <ryannedolan@gmail.com>
2020-01-13 18:06:29 +00:00
Seungha Lee 655447140e MINOR: Cleanup generic & throw syntax in connect (#7892)
Reviewers: Jason Gustafson <jason@confluent.io>
2020-01-11 10:11:29 -08:00
belugabehr 5731caf9c7 KAFKA-9396; Use JDK `emptyIterator` in `ConnectHeaders` Class (#7922)
Reviewers: Ron Dagostino <rndgstn@gmail.com>, Jason Gustafson <jason@confluent.io>
2020-01-11 10:09:46 -08:00
Jason Gustafson 2ac78ff621
MINOR: Propagate LogContext to channel builders and SASL authenticator (#7867)
The log context is useful when debugging applications which have multiple clients. This patch propagates the context to the channel builders and the SASL authenticator.

Reviewers: Ron Dagostino <rndgstn@gmail.com>, Manikumar Reddy <manikumar.reddy@gmail.com>
2020-01-06 14:27:30 -08:00
Ismael Juma 6dc6f6a60d
KAFKA-9324: Drop support for Scala 2.11 (KIP-531) (#7859)
* Adjust build and documentation.
* Use lambda syntax for SAM types in `core`, `streams-scala` and
`connect-runtime` modules.
* Remove `runnable` and `newThread` from `CoreUtils` as lambda
syntax for SAM types make them unnecessary.
* Remove stale comment in `FunctionsCompatConversions`,
`KGroupedStream`, `KGroupedTable' and `KStream` about Scala 2.11,
the conversions are needed for Scala 2.12 too.
* Deprecate `org.apache.kafka.streams.scala.kstream.Suppressed`
and use `org.apache.kafka.streams.kstream.Suppressed` instead.
* Use `Admin.create` instead of `AdminClient.create`. Static methods
in Java interfaces can be invoked since Scala 2.12. I noticed that
MirrorMaker 2 uses `AdminClient.create`, but I did not change them
as Connectors have restrictions on newer client APIs.
* Improve efficiency in a few `Gauge` implementations by avoiding
unnecessary intermediate collections.
* Remove pointless `Option.apply` in `ZookeeperClient`
`SessionState` metric.
* Fix unused import/variable and other compiler warnings.
* Reduce visibility of some vals/defs.

Reviewers: Manikumar Reddy <manikumar@confluent.io>, Guozhang Wang <wangguoz@gmail.com>, Gwen Shapira <gwen@confluent.io>
2020-01-06 19:51:01 +01:00
Lee Dongjin 8c64aa080a MINOR: trivial cleanups
- Reformat header: `CustomDeserializerTest`, `ReplicaVerificationToolTest`
- Remove unused constructor: `ConsumerGroupDescription`
- Remove unused variables in `TimeOrderedKeyValueBufferTest#shouldRestoreV2Format`
- Remove deprecated `Number` consturctor calls; use `Number#valueOf` instread.

Author: Lee Dongjin <dongjin@apache.org>

Reviewers: Stanislav Kozlovski <stanislav_kozlovski@outlook.com>, Manikumar Reddy <manikumar.reddy@gmail.com>

Closes #7202 from dongjinleekr/cleanup/201908
2019-12-18 21:09:16 +05:30
Konstantine Karantasis 0e57a3924e KAFKA-9184: Redundant task creation and periodic rebalances after zombie Connect worker rejoins the group (#7771)
Check connectivity with broker coordinator in intervals and stop tasks if coordinator is unreachable by setting `assignmentSnapshot` to null and resetting rebalance delay when there are no lost tasks. And, because we're now sometimes setting `assignmentSnapshot` to null and reading it from other methods and thread, made this member volatile and used local references to ensure consistent reads.

Adapted existing unit tests to verify additional debug calls, added more specific log messages to `DistributedHerder`, and added a new integration test that verifies the behavior when the brokers are stopped and restarted only after the workers lose their heartbeats with the broker coordinator.

Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewers: Greg Harris <gregh@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-12-04 17:27:51 -06:00
Jason Gustafson 5b6de9f2d0
KAFKA-8933; Fix NPE in DefaultMetadataUpdater after authentication failure (#7682)
This patch fixes an NPE in `DefaultMetadataUpdater` due to an inconsistency in event expectations. Whenever there is an authentication failure, we were treating it as a failed update even if was from a separate connection from an inflight metadata request. This patch fixes the problem by making the `MetadataUpdater` api clearer in terms of the events that are handled.

Reviewers: Stanislav Kozlovski <stanislav_kozlovski@outlook.com>, Rajini Sivaram <rajinisivaram@googlemail.com>
2019-12-04 13:12:17 -08:00
Cyrus Vafadari d3131a1020 KAFKA-9258 Check Connect Metrics non-null in task stop (#7768)
Remove nullcheck, and add integration tests for restarting a failed task.

Authors: Cyrus Vafadari <cyrus@confluent.io>, Chris Egerton <chrise@confluent.io>
Reviewers: Arjun Satish <arjun@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-12-03 21:13:52 -06:00
ning2008wisc 88448f6255 KAFKA-9219: prevent NullPointerException in Kafka Connect metrics (#7652)
`assignmentSnapshot` may take a moment to get initialized in some cases, especially when
Kafka Connect is started from scratch. While `assignmentSnapshot` is not initialized, return 0 in both `measure()` methods.

Reviewers: Mickael Maison <mickael.maison@gmail.com>, Edoardo Comar <ecomar@uk.ibm.com>
2019-11-27 11:46:49 +00:00
Chris Egerton da4337271e KAFKA-9051: Prematurely complete source offset read requests for stopped tasks (#7532)
Prematurely complete source offset read requests for stopped tasks, and added unit tests.

Author: Chris Egerton <chrise@confluent.io>
Reviewers: Arjun Satish <arjun@confluent.io>, Nigel Liang <nigel@nigelliang.com>, Jinxin Liu <liukrimhim@gmail.com>, Randall Hauch <rhauch@gmail.com>
2019-11-19 21:18:21 -08:00
Chris Egerton 38d243b022 KAFKA-9046: Use top-level worker configs for connector admin clients
[Jira](https://issues.apache.org/jira/browse/KAFKA-9046)

The changes here are meant to find a healthy compromise between the pre- and post-KIP-458 functionality of Connect workers when configuring admin clients for use with DLQs. Before KIP-458, admin clients were configured using the top-level worker configs; after KIP-458, they are configured using worker configs with a prefix of `admin.` and then optionally overridden by connector configs with a prefix of `admin.override.`. The behavior proposed here is to use, in ascending order of precedence, the top-level worker configs, worker configs prefixed with `admin.`, and connector configs prefixed with `admin.override.`; essentially, use the pre-KIP-458 behavior by default but allow it to be overridden by the post-KIP-458 behavior.

Author: Chris Egerton <chrise@confluent.io>

Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Randall Hauch <rhauch@gmail.com>, Nigel Liang <nigel@nigelliang.com>

Closes #7525 from C0urante/kafka-9046
2019-11-14 14:09:04 +05:30
Mickael Maison 7027d92927
KAFKA-9175: Update MirrorMaker 2 topic/partition metrics (#7688)
Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>, Ryanne Dolan <ryannedolan@gmail.com>

Co-authored-by: Mickael Maison <mickael.maison@gmail.com>
Co-authored-by: Edoardo Comar <ecomar@uk.ibm.com>
2019-11-13 20:36:25 +00:00
Greg Harris ff68b60429 KAFKA-8340, KAFKA-8819: Use PluginClassLoader while statically initializing plugins (#7315)
Added plugin isolation unit tests for various scenarios, with a `TestPlugins` class that compiles and builds multiple test plugins without them being on the classpath and verifies that the Plugins and DelegatingClassLoader behave properly. These initially failed for several cases, but now pass since the issues have been fixed.

KAFKA-8340 and KAFKA-8819 are closely related, and this fix corrects the problems reported in both issues.

Author: Greg Harris <gregh@confluent.io>
Reviewers: Chris Egerton <chrise@confluent.io>, Magesh Nandakumar <mageshn@confluent.io>, Konstantine Karantasis <konstantine@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-10-16 20:43:00 -05:00
Chris Egerton bcdc6336fc KAFKA-8945/KAFKA-8947: Fix bugs in Connect REST extension API (#7392)
Fix bug in Connect REST extension API caused by invalid constructor parameter validation, and update integration test to play nicely with Jenkins

Fix instantiation of TaskState objects by Connect framework.

Author: Chris Egerton <chrise@confluent.io>
Reviewers: Magesh Nandakumar <mageshn@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-10-15 17:27:26 -05:00
Konstantine Karantasis 5e231f1738 KAFKA-9014: Fix AssertionError when SourceTask.poll returns an empty list (#7491)
Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-10-15 16:08:31 -05:00
Guozhang Wang fb9b0dfde5
MINOR: Augment log4j to add generation number in performAssign (#7451)
Since generation is private in AbstractCoordinator, I need to modify the generation() to let it return the object directly.

Reviewers: A. Sophie Blee-Goldman <sophie@confluent.io>, Bill Bejeck <bill@confluent.io>
2019-10-09 10:34:19 -07:00
Ryanne Dolan 4ac892ca78 KAFKA-7500: MirrorMaker 2.0 (KIP-382)
Implementation of [KIP-382 "MirrorMaker 2.0"](https://cwiki.apache.org/confluence/display/KAFKA/KIP-382%3A+MirrorMaker+2.0)

Author: Ryanne Dolan <ryannedolan@gmail.com>
Author: Arun Mathew <arunmathew88@gmail.com>
Author: In Park <inpark@cloudera.com>
Author: Andre Price <obsoleted@users.noreply.github.com>
Author: christian.hagel@rio.cloud <christian.hagel@rio.cloud>

Reviewers: Eno Thereska <eno.thereska@gmail.com>, William Hammond <william.t.hammond@gmail.com>, Viktor Somogyi <viktorsomogyi@gmail.com>, Jakub Korzeniowski, Tim Carey-Smith, Kamal Chandraprakash <kamal.chandraprakash@gmail.com>, Arun Mathew, Jeremy-l-ford, vpernin, Oleg Kasian <oleg.kasian@gmail.com>, Mickael Maison <mickael.maison@gmail.com>, Qihong Chen, Sriharsha Chintalapani <sriharsha@apache.org>, Jun Rao <junrao@gmail.com>, Randall Hauch <rhauch@gmail.com>, Manikumar Reddy <manikumar.reddy@gmail.com>, Ismael Juma <ismael@juma.me.uk>

Closes #6295 from ryannedolan/KIP-382
2019-10-07 13:57:54 +05:30
Magesh Nandakumar 586c587b3d KAFKA-8974: Trim whitespaces in topic names in sink connector configs (#7442)
Trim whitespaces in topic names specified in sink connector configs before subscribing to the consumer. Topic names don't allow whitespace characters, so trimming only will eliminate potential problems and will not place additional limits on topics specified in sink connectors.

Author: Magesh Nandakumar <magesh.n.kumar@gmail.com>
Reviewers: Arjun Satish <arjun@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-10-04 13:12:32 -05:00
Nigel Liang ded1fb8c4d KAFKA-6290: Support casting from logical types in cast transform (#7371)
Adds support for the Connect Cast transforms to cast from Connect logical types, such as DATE, TIME, TIMESTAMP, and DECIMAL. Casting to numeric types will produce the underlying numeric value represented in the desired type. For logical types represented by underlying Java Date class, this means the milliseconds since EPOCH. For Decimal, this means the underlying value. If the value does not fit in the desired target type, it may overflow.

Casting to String from Date, Time, and Timestamp types will produce their ISO 8601 representation. Casting to String from Decimal will result in the value represented as a string. e.g. 1234 -> "1234".

Author: Nigel Liang <nigel@nigelliang.com>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-10-03 14:55:52 -05:00
Gunnar Morling e76770343c KAFKA-8523 Enabling InsertField transform to be used with tombstone events (#6914)
* KAFKA-8523 Avoiding raw type usage

* KAFKA-8523 Gracefully handling tombstone events in InsertField SMT
2019-10-03 13:51:00 -05:00
Cyrus Vafadari 5b829f2b61 KAFKA-8447: New Metric to Measure Number of Tasks on a Connector (#6843)
Implemented KIP-475 to add new metrics for each worker to expose the number of current tasks per connector and per status.

Author: Cyrus Vafadari <cyrus@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>, Boyang Chen <Boyang Chen <boyang@confluent.io>
2019-10-02 19:42:21 -05:00
Chris Egerton 791d0d61bf KAFKA-8804: Secure internal Connect REST endpoints (#7310)
Implemented KIP-507 to secure the internal Connect REST endpoints that are only for intra-cluster communication. A new V2 of the Connect subprotocol enables this feature, where the leader generates a new session key, shares it with the other workers via the configuration topic, and workers send and validate requests to these internal endpoints using the shared key.

Currently the internal `POST /connectors/<connector>/tasks` endpoint is the only one that is secured.

This change adds unit tests and makes some small alterations to system tests to target the new `sessioned` Connect subprotocol. A new integration test ensures that the endpoint is actually secured (i.e., requests with missing/invalid signatures are rejected with a 400 BAD RESPONSE status).

Author: Chris Egerton <chrise@confluent.io>
Reviewed: Konstantine Karantasis <konstantine@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-10-02 17:06:57 -05:00
Arjun Satish 1c831c22e1 KAFKA-7772: Dynamically Adjust Log Levels in Connect (#7403)
Implemented KIP-495 to expose a new `admin/loggers` endpoint for the Connect REST API that lists the current log levels and allows the caller to change log levels. 

Author: Arjun Satish <arjun@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-10-02 17:00:37 -05:00
Almog Gavra bfcc17f211 KAFKA-8595: Support deserialization of JSON decimals encoded in NUMERIC (#7354)
Implemented KIP-481 by adding support for deserializing Connect DECIMAL values encoded in JSON as numbers, in addition to raw byte array (base64) format used previously.

Author: Almog Gavra <almog@confluent.io>
Reviewers: Chris Egerton <chrise@confluent.io>, Konstantine Karantasis <konstantine@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-10-01 15:33:06 -05:00
Tu V. Tran f6f24c4700 KAFKA-8729, pt 2: Add error_records and error_message to PartitionResponse (#7150)
As noted in the KIP-467, the updated ProduceResponse is

```
Produce Response (Version: 8) => [responses] throttle_time_ms
  responses => topic [partition_responses]
    topic => STRING
    partition_responses => partition error_code base_offset log_append_time log_start_offset
      partition => INT32
      error_code => INT16
      base_offset => INT64
      log_append_time => INT64
      log_start_offset => INT64
      error_records => [INT32]         // new field, encodes the relative offset of the records that caused error
      error_message => STRING          // new field, encodes the error message that client can use to log itself
    throttle_time_ms => INT32
with a new error code:
```

INVALID_RECORD(86, "Some record has failed the validation on broker and hence be rejected.", InvalidRecordException::new);

Reviewers: Jason Gustafson <jason@confluent.io>, Magnus Edenhill <magnus@edenhill.se>, Guozhang Wang <wangguoz@gmail.com>
2019-09-30 19:29:36 -07:00
Yaroslav Tkachenko 70d1bb40d9 KAFKA-7273: Extend Connect Converter to support headers (#6362)
Implemented KIP-440 to allow Connect converters to use record headers when serializing or deserializing keys and values. This change is backward compatible in that the new methods default to calling the older existing methods, so existing Converter implementations need not be changed. This changes the WorkerSinkTask and WorkerSourceTask to use the new converter methods, but Connect's existing Converter implementations and the use of converters for internal topics are intentionally not modified. Added unit tests.

Author: Yaroslav Tkachenko <sapiensy@gmail.com>
Reviewers: Ryanne Dolan <ryannedolan@gmail.com>, Ewen Cheslack-Postava <me@ewencp.org>, Randall Hauch <rhauch@gmail.com>
2019-09-25 11:23:03 -05:00
Mickael Maison ac385c4c3a KAFKA-8474; Use HTML lists for config layout (#6870)
Replace the `<table>` elements by `<ul>` so the full page width can be used for the configuration descriptions instead of only a very narrow column. I moved the other fields (Type, Default Value, etc) below each entry.

Reviewers: Boyang Chen <boyang@confluent.io>, Jason Gustafson <jason@confluent.io>
2019-09-13 15:48:36 -07:00
Konstantine Karantasis 74fc3323a5 MINOR: Add unit test for KAFKA-8676 to guard against unrequired task restarts (#7287)
Added unit test for recent fix of `KafkaConfigBackingStore`.

Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-09-03 15:48:19 -05:00
LuyingLiu 31a5f92b9f Changed for updatedTasks, avoids stopping and starting of unnecessary tasks (#7097)
Corrected the `KafkaConfigBackingStore` logic to notify of only the changed tasks, rather than all tasks. This was not noticed before because Connect always stopped and restarted all tasks during a rebalanced, but since 2.3 the incremental rebalance logic exposed this bug.

Author: Luying Liu <lyliu@lyliu-mac.freewheelmedia.net>
Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-09-03 15:47:17 -05:00
Konstantine Karantasis d7f8ec8628 MINOR: Fix the doc of scheduled.rebalance.max.delay.ms config property (#7242)
Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-08-25 17:35:03 -05:00
Randall Hauch 5b4e749dc6 KAFKA-8391; Temporarily ignore flaky Connect rebalance integration tests
I've spent quite a bit of time on trying to discover the root cause, with no luck so far. I have been able to reproduce it locally by running the following 100 times:
```
./gradlew connect:runtime:clean connect:runtime:test --tests org.apache.kafka.connect.integration.RebalanceSourceConnectorsIntegrationTest
```
The `testReconfigConnector` test failed 28% of the time and the others failed 0%. This issue and KAFKA-8661 suggest that `testDeleteConnector` and `testStartTwoConnectors` are also flaky, though I've not seen those tests fail locally.

Because this flakiness is causing issues for the rest of the project, I'm going to temporarily ignore several of the flaky ITs while I continue to investigate:
* `RebalanceSourceConnectorsIntegrationTest.testReconfigConnector`
* `RebalanceSourceConnectorsIntegrationTest.testDeleteConnector`
* `RebalanceSourceConnectorsIntegrationTest.testStartTwoConnectors`

**This should be backported to the `2.3` branch, which is when these integration tests were first added.**

Author: Randall Hauch <rhauch@gmail.com>

Reviewers: Ismael Juma

Closes #7237 from rhauch/kafka-8391-temporary
2019-08-25 14:30:23 -07:00
Chris Egerton 237e83dea0 KAFKA-8586: Fail source tasks when producers fail to send records (#6993)
Changed Connect's `WorkerSourceTask` to capture non-retriable exceptions from the `producer.send(...)` (e.g., authentication or authorization errors) and to fail the connector task when such an error is encountered. Modified the existing unit tests to verify this functionality.

Note that most producer errors are retriable, and Connect will (by default) set up each producer with 1 max in-flight message and infinite retries. This change only affects non-retriable errors.
2019-08-25 15:54:00 -05:00
Randall Hauch e5657bc910 KAFKA-8391; Improved the Connect integration tests to make them less flaky
Added the ability for the connector handles and task handles, which are used by the monitorable source and sink connectors used to verify the functionality of the Connect framework, to record the number of times the connector and tasks have each been started, and to allow a test to obtain a `RestartLatch` that can be used to block until the connectors and/or tasks have been restarted a specified number of types.

Typically, a test will get the `ConnectorHandle` for a connector, and call the `ConnectorHandle.expectedRestarts(int)` method with the expected number of times that the connector and/or tasks will be restarted, and will hold onto the resulting `RestartLatch`. The test will then change the connector (or otherwise cause the connector to restart) one or more times as desired, and then call `RestartLatch.await(long, TimeUnit)` to block the test up to a specified duration for the connector and all tasks to be started the specified number of times.

This commit also increases several of the maximum wait times used in other integration tests. It doesn’t hurt to potentially wait longer, since most test runs will not need to wait the maximum amount of time anyway. However, in the rare cases that do need that extra time, waiting a bit more is fine if we can reduce the flakiness and minimize test failures that happened to time out too early.

Unit tests were added for the new `RestartLatch` and `StopAndStartCounter` utility classes. This PR only affects the tests and does not affect any runtime code or API.

**This should be merged on `trunk` and backported to the `2.3.x` branch.**

Author: Randall Hauch <rhauch@gmail.com>

Reviewers: Konstantine Karantasis, Arjun Satish

Closes #7019 from rhauch/kafka-8391
2019-08-13 11:14:41 -07:00
Paul e2c8612d01 KAFKA-7941: Catch TimeoutException in KafkaBasedLog worker thread (#6283)
When calling readLogToEnd(), the KafkaBasedLog worker thread should catch TimeoutException and log a warning, which can occur if brokers are unavailable, otherwise the worker thread terminates.

Includes an enhancement to MockConsumer that allows simulating exceptions not just when polling but also when querying for offsets, which is necessary for testing the fix.

Author: Paul Whalen <pgwhalen@gmail.com>
Reviewers: Randall Hauch <rhauch@gmail.com>, Arjun Satish <arjun@confluent.io>, Ryanne Dolan <ryannedolan@gmail.com>
2019-08-13 10:16:54 -05:00
Arjun Satish 794637232c KAFKA-8774: Regex can be found anywhere in config value (#7197)
Corrected the AbstractHerder to correctly identify task configs that contain variables for externalized secrets. The original method incorrectly used `matcher.matches()` instead of `matcher.find()`. The former method expects the entire string to match the regex, whereas the second one can find a pattern anywhere within the input string (which fits this use case more correctly).

Added unit tests to cover various cases of a config with externalized secrets, and updated system tests to cover case where config value contains additional characters besides secret that requires regex pattern to be found anywhere in the string (as opposed to complete match).

Author: Arjun Satish <arjun@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-08-13 09:40:12 -05:00
John Roesler a8aedc85eb KAFKA-8696: clean up Sum/Count/Total metrics (#7057)
* Clean up one redundant and one misplaced metric
* Clarify the relationship among these metrics to avoid future confusion

Reviewers: Matthias J. Sax <matthias@confluent.io>, Bill Bejeck <bill@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2019-07-23 16:54:20 -07:00
Andy Coates 2a133ba656 KAFKA-8454; Add Java AdminClient Interface (KIP-476) (#7087)
Adds an `Admin` interface as specified in [KIP-476](https://cwiki.apache.org/confluence/display/KAFKA/KIP-476%3A+Add+Java+AdminClient+Interface).

Reviewers: Ismael Juma <ismael@juma.me.uk>, Jason Gustafson <jason@confluent.io>
2019-07-22 15:47:34 -07:00
Jason Gustafson f65c71cf6e
MINOR: Increase `awaitCommits` timeout in ExampleConnectIntegrationTest (#7061)
The transient failures are usually caused by a timeout in `awaitCommits`. This patch increases the timeout from 15s to 30s.

Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Matthias J. Sax <mjsax@apache.org>
2019-07-16 12:00:49 -07:00
Jason Gustafson 1d873a9de9
MINOR: Use dynamic port in `RestServerTest` (#7079)
We have seen some failures recently in `RestServerTest`. It's the usual problem with reliance on static ports. 
```
Caused by: java.io.IOException: Failed to bind to 0.0.0.0/0.0.0.0:8083
	at org.eclipse.jetty.server.ServerConnector.openAcceptChannel(ServerConnector.java:346)
	at org.eclipse.jetty.server.ServerConnector.open(ServerConnector.java:308)
	at org.eclipse.jetty.server.AbstractNetworkConnector.doStart(AbstractNetworkConnector.java:80)
	at org.eclipse.jetty.server.ServerConnector.doStart(ServerConnector.java:236)
	at org.eclipse.jetty.util.component.AbstractLifeCycle.start(AbstractLifeCycle.java:68)
	at org.eclipse.jetty.server.Server.doStart(Server.java:396)
	at org.eclipse.jetty.util.component.AbstractLifeCycle.start(AbstractLifeCycle.java:68)
	at org.apache.kafka.connect.runtime.rest.RestServer.initializeServer(RestServer.java:178)
	... 56 more
Caused by: java.net.BindException: Address already in use
```
This patch makes the chosen port dynamic.

Reviewers: Ismael Juma <ismael@juma.me.uk>
2019-07-12 15:29:10 -07:00
Robert Yokota fa042bc491 KAFKA-7157: Fix handling of nulls in TimestampConverter (#7070)
Fix handling of nulls in TimestampConverter.

Authors: Valeria Vasylieva <valeria.vasylieva@gmail.com>, Robert Yokota <rayokota@gmail.com>
Reviewers: Arjun Satish <arjun@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-07-12 12:12:20 -05:00
Michał Borowiecki fc4fea6761 KAFKA-6605: Fix NPE in Flatten when optional Struct is null (#5705)
Correct the Flatten SMT to properly handle null key or value `Struct` instances.

Author: Michal Borowiecki <michal.borowiecki@openbet.com>
Reviewers: Arjun Satish <arjun@confluent.io>, Robert Yokota <rayokota@gmail.com>, Randall Hauch <rhauch@gmail.com>
2019-07-12 10:27:33 -05:00
Nacho Muñoz Gómez 289ac09292 KAFKA-8591; WorkerConfigTransformer NPE on connector configuration reloading (#6991)
A bug in `WorkerConfigTransformer` prevents the connector configuration reload when the ConfigData TTL expires. 

The issue boils down to the fact that `worker.herder().restartConnector` is receiving a null callback. 

```
[2019-06-17 14:34:12,320] INFO Scheduling a restart of connector workshop-incremental in 60000 ms (org.apache.kafka.connect.runtime.WorkerConfigTransformer:88)
[2019-06-17 14:34:12,321] ERROR Uncaught exception in herder work thread, exiting:  (org.apache.kafka.connect.runtime.distributed.DistributedHerder:227)
java.lang.NullPointerException
        at org.apache.kafka.connect.runtime.distributed.DistributedHerder$19.onCompletion(DistributedHerder.java:1187)
        at org.apache.kafka.connect.runtime.distributed.DistributedHerder$19.onCompletion(DistributedHerder.java:1183)
        at org.apache.kafka.connect.runtime.distributed.DistributedHerder.tick(DistributedHerder.java:273)
        at org.apache.kafka.connect.runtime.distributed.DistributedHerder.run(DistributedHerder.java:219)
```
This patch adds a callback which just logs the error.

Reviewers: Robert Yokota <rayokota@gmail.com>, Jason Gustafson <jason@confluent.io>
2019-07-08 23:07:10 -07:00
Cheng Pan 0b40d27647 MINOR: Remove redundant placeholder in log message (#7016)
Reviewers: Jason Gustafson <jason@confluent.io>
2019-07-03 11:23:23 -07:00
Jason Gustafson b0935e548b
MINOR: Embedded connect cluster should mask exit procedures by default (#7028)
`EmbeddedConnectCluster` has the ability to mask system exits to avoid killing the jvm. It appears that the default was intended to be `true`, but is actually `false`. The `maskExitProcedures` method on `EmbeddedConnectCluster.Builder` documents the parameter as:

```
* @param mask if false, exit and halt procedures remain unchanged; true is the default.
```
Because this is not enabled by default as intended, we are seeing some build failures which exit abruptly:
```
17:29:11 Execution failed for task ':connect:runtime:integrationTest'.
17:29:11 > Process 'Gradle Test Executor 25' finished with non-zero exit value 1
```
The culprit often appears to be `ExampleConnectIntegrationTest`, which indeed does not override the default value of `maskExitProcedures`.

Reviewers: Ewen Cheslack-Postava <me@ewencp.org>
2019-07-02 22:15:22 -07:00
Konstantine Karantasis 11b25a13ee MINOR: Fix DistributedHerderTest after adding reason to maybeLeaveGroup (#6982)
Mocking of WorkerCoordinator was not precise after adding an argument (reason) to AbstractCoordinator#maybeLeaveGroup in KAFKA-8569:

Unit test case for DistributedHerderTest is now precise with respect to the expected argument and succeeds

Reviewers: Boyang Chen <boyang@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2019-06-21 12:42:34 -07:00
Boyang Chen 03d61ebfb9 KAFKA-8569: integrate warning message under static membership (#6972)
Static members never leave the group, so potentially we could log a flooding number of warning messages in the hb thread. The solution is to only log as warning when we are on dynamic membership.

Reviewers: Guozhang Wang <wangguoz@gmail.com>
2019-06-20 13:56:00 -07:00
Boyang Chen 1b9e107388 KAFKA-7853: Refactor coordinator config (#6854)
An attempt to refactor current coordinator logic.

Reviewers: Stanislav Kozlovski <stanislav_kozlovski@outlook.com>, Konstantine Karantasis <konstantine@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2019-06-17 10:58:43 -07:00
Boyang Chen cca05cace4 KAFKA-8331: stream static membership system test (#6877)
As title suggested, we boost 3 stream instances stream job with one minute session timeout, and once the group is stable, doing couple of rolling bounces for the entire cluster. Every rejoin based on restart should have no generation bump on the client side.

Reviewers: Guozhang Wang <wangguoz@gmail.com>,  Bill Bejeck <bbejeck@gmail.com>
2019-06-07 16:52:12 -04:00
Hai-Dang Dam 1a3fe9aa52 KAFKA-8404: Add HttpHeader to RestClient HTTP Request and Connector REST API (#6791)
When Connect forwards a REST request from one worker to another, the Authorization header was not forwarded. This commit changes the Connect framework to add include the authorization header when forwarding requests to other workers.

Author: Hai-Dang Dam <damquanghaidang@gmail.com>
Reviewers: Robert Yokota <rayokota@gmail.com>, Randall Hauch <rhauch@gmail.com>
2019-06-03 21:06:00 -05:00
Konstantine Karantasis 3c7c988e39 KAFKA-8449: Restart tasks on reconfiguration under incremental cooperative rebalancing (#6850)
Restart task on reconfiguration under incremental cooperative rebalancing, and keep execution paths separate for config updates between eager and cooperative. Include the group generation in the log message when the worker receives its assignment.

Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-06-03 11:13:39 -05:00
Konstantine Karantasis 17345b3be5 KAFKA-8463: Fix redundant reassignment of tasks when leader worker leaves (#6859)
Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-06-02 12:19:19 -05:00
Chris Egerton 5351efe48a KAFKA-8407: Fix validation of class and list configs in connector client overrides (#6789)
Because of how config values are converted into strings in the `AbstractHerder.validateClientOverrides()` method after being validated by the client override policy, an exception is thrown if the value returned by the policy isn't already parsed as the type expected by the client `ConfigDef`. The fix here involves parsing client override properties before passing them to the override policy.

A unit test is added to ensure that several different types of configs are validated properly by the herder.

Author: Chris Egerton <chrise@confluent.io>
Reviewers: Magesh Nandakumar <magesh.n.kumar@gmail.com>, Randall Hauch <rhauch@gmail.com>
2019-05-23 14:21:19 -05:00
Konstantine Karantasis 68a3eb373b KAFKA-8415: Interface ConnectorClientConfigOverridePolicy needs to be excluded from class loading isolation (#6796)
Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-05-23 14:14:09 -05:00
sdreynolds 89f331eac3 KAFKA-8229; Reset WorkerSinkTask offset commit interval after task commit (#6579)
Prior to this change, the next commit time advances
_each_ time a commit happens -- including when a commit happens
because it was requested by the `Task`. When a `Task` requests a
commit several times, the clock advances far into the future
which prevents expected periodic commits from happening.

This commit changes the behavior, we reset `nextCommit` relative
to the time of the commit.

Reviewers: Jason Gustafson <jason@confluent.io>
2019-05-22 23:50:48 -07:00
Boyang Chen cafdc1e7df KAFKA-8399: bring back internal.leave.group.on.close config for KStream (#6779)
As title states. We plan to merge this to both trunk and 2.3 if it could fix the stream system tests globally.
Reference implementation: #6673

Reviewers: Guozhang Wang <wangguoz@gmail.com>, Matthias J. Sax <mjsax@apache.org>
2019-05-22 11:03:00 -04:00
Magesh Nandakumar 126230dad0 KAFKA-8265: Fix override config name to match KIP-458. (#6776)
Author: Magesh Nandakumar <magesh.n.kumar@gmail.com>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-05-21 11:28:28 -05:00
Magesh Nandakumar 7d70133b75 KAFKA-8265: Fix config name to match KIP-458. (#6755)
Return a copy of the ConfigDef in Client Configs. Related to KIP-458.

Author: Magesh Nandakumar <magesh.n.kumar@gmail.com
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-05-20 14:02:22 -05:00
Lee Dongjin b43f5446ac KAFKA-8316; Remove deprecated usage of Slf4jRequestLog, SslContextFactory (#6668)
* Remove deprecated class Slf4jRequestLog: use Slf4jRequestLogWriter, CustomRequestLog instread.

1. Remove '@SuppressWarnings("deprecation")' from RestServer#initializeResources, JsonRestServer#start.
2. Remove unused JsonRestServer#httpRequest.

* Fix deprecated class usage: SslContextFactory -> SslContextFactory.[Server, Client]

1. Split SSLUtils#createSslContextFactory into SSLUtils#create[Server, Client]SideSslContextFactory: each method instantiates SslContextFactory.[Server, Client], respectively.
2. SSLUtils#configureSslContextFactoryAuthentication is called from SSLUtils#createServerSideSslContextFactory only.
3. Update SSLUtilsTest following splittion; for client-side SSL Context Factory, SslContextFactory#get[Need, Want]ClientAuth is always false. (SSLUtilsTest#testCreateClientSideSslContextFactory)

Reviewers: Ismael Juma <ismael@juma.me.uk>, Jason Gustafson <jason@confluent.io>
2019-05-20 10:15:15 -07:00
Magesh Nandakumar 2e91a310d7 KAFKA-8265: Initial implementation for ConnectorClientConfigPolicy to enable overrides (KIP-458) (#6624)
Implementation to enable policy for Connector Client config overrides. This is
implemented per the KIP-458.

Reviewers: Randall Hauch <rhauch@gmail.com>
2019-05-17 01:37:32 -07:00
Konstantine Karantasis ce584a01ff KAFKA-5505: Incremental cooperative rebalancing in Connect (KIP-415) (#6363)
Added the incremental cooperative rebalancing in Connect to avoid global rebalances on all connectors and tasks with each new/changed/removed connector. This new protocol is backward compatible and will work with heterogeneous clusters that exist during a rolling upgrade, but once the clusters consist of new workers only some affected connectors and tasks will be rebalanced: connectors and tasks on existing nodes still in the cluster and not added/changed/removed will continue running while the affected connectors and tasks are rebalanced.

This commit attempted to minimize the changes to the existing V0 protocol logic, though that was not entirely possible.

This commit adds extensive unit and integration tests for both the old V0 protocol and the new v1 protocol. Soak testing has been performed multiple times to verify behavior while connectors and added, changed, and removed and while workers are added and removed from the cluster.

Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewers: Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <me@ewencp.org>, Robert Yokota <rayokota@gmail.com>, David Arthur <mumrah@gmail.com>, Ryanne Dolan <ryannedolan@gmail.com>
2019-05-16 22:46:03 -05:00
dan norwood 5a95c2e1cd Add '?expand' query param for additional info on '/connectors'. (#6658)
Per KIP-465, kept existing behavior of `/connectors` resource in the Connect's REST API, but added the ability to specify `?expand` query parameter to get list of connectors with status details on each connector. Added unit tests, and verified passing existing system tests (which use the older list form).

See https://cwiki.apache.org/confluence/display/KAFKA/KIP-465%3A+Add+Consolidated+Connector+Endpoint+to+Connect+REST+API.

Author: Dan Norwood <norwood@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-05-16 16:29:29 -05:00
Randall Hauch b395ef4182 KAFKA-3816: Add MDC logging to Connect runtime (#5743)
See https://cwiki.apache.org/confluence/display/KAFKA/KIP-449%3A+Add+connector+contexts+to+Connect+worker+logs

Added LoggingContext as a simple mechanism to set and unset Mapped Diagnostic Contexts (MDC) in the loggers to provide for each thread useful parameters that can be used within the logging configuration. MDC avoids having to modify lots of log statements, since the parameters are available to all log statements issued by the thread, no matter what class makes those calls.

The design intentionally minimizes the number of changes to any existing classes, and does not use Java 8 features so it can be easily backported if desired, although per this KIP it will be applied initially only in AK 2.3 and later and must be enabled via the Log4J configuration.

Reviewers: Jason Gustafson <jason@conflent.io>, Guozhang Wang <wangguoz@gmail.com>
2019-05-16 12:35:01 +01:00
Konstantine Karantasis 2327b35558 MINOR: Enable console logs in Connect tests (#6745)
Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-05-15 22:05:52 -05:00
Magesh Nandakumar 5928ffd0dc KAFKA-8320 : fix retriable exception package for source connectors (#6675)
WorkerSourceTask is catching the exception from wrong package org.apache.kafka.common.errors. It is not clear from the API standpoint as to which package the connect framework supports - the one from common or connect. The safest thing would be to support both the packages even though it's less desirable.

Author: Magesh Nandakumar <magesh.n.kumar@gmail.com>
Reviewers: Arjun Satish <arjun@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-05-15 17:20:20 -05:00
Boyang Chen 2208f9966d KAFKA-8354; Replace Sync group request/response with automated protocol (#6729)
Update SyncGroup API to use the generated protocol classes.

Reviewers: Jason Gustafson <jason@confluent.io>
2019-05-15 15:09:00 -07:00
Paul Davidson fbb09952ac KAFKA-5061 - Make default Worker Task client IDs distinct (#6097)
Use the task ID to make the default client IDs used by Worker Tasks distinct and stable. This is avoids name conflicts on JMX MBeans and enables useful monitoring.

This implements https://cwiki.apache.org/confluence/display/KAFKA/KIP-411%3A+Make+default+Kafka+Connect+worker+task+client+IDs+distinct.

See: https://issues.apache.org/jira/browse/KAFKA-5061

Author: Paul Davidson <>
Reviewer: Cyrus Vafadari <cyrusv@alum.mit.edu>, Arjun Satish <arjun@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-05-15 17:02:35 -05:00
Arabelle Hou 4c85171a1f KAFKA-7633: Allow Kafka Connect to access internal topics without cluster ACLs (#5918)
When Kafka Connect does not have cluster ACLs to create topics,
it fails to even access its internal topics which already exist.
This was originally fixed in KAFKA-6250 by ignoring the cluster
authorization error, but now Kafka 2.0 returns a different response
code that corresponds to a different error. Add a patch to ignore this
new error as well.

Reviewers: Jason Gustafson <jason@confluent.io>
2019-05-11 11:05:37 -07:00
Chris Egerton 7a4618a793 MINOR: Remove header and key/value converter config value logging (#6660)
The debug log lines in the `Plugins` class that log header and key/value converter configurations should be altered as the configurations for these converters may contain secrets that should not be logged in plaintext. Instead, only the keys for these configs are safe to expose.

Author: Chris Egerton <cegerton@oberlin.edu>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-05-09 20:35:08 -05:00
Chris Egerton 36a5aba4ec KAFKA-8231: Expansion of ConnectClusterState interface (#6584)
Expand ConnectClusterState interface and implementation with methods that provide the immutable cluster details and the connector configuration. This includes unit tests for the new methods.

Author: Chris Egerton <cegerton@oberlin.edu>
Reviews: Arjun Satish <arjun@confluent.io>, Konstantine Karantasis <konstantine@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-05-09 20:27:59 -05:00
Bob Barrett a97e55b838 KAFKA-8332: Refactor ImplicitLinkedHashSet to avoid losing ordering when converting to Scala
Because of how conversions between Java collections and Scala collections work, ImplicitLinkedHashMultiSet objects were being treated as unordered in some contexts where they shouldn't be.  This broke JOIN_GROUP handling.  

This patch renames ImplicitLinkedHashMultiSet to ImplicitLinkedHashMultCollection.  The order of Collection objects will be preserved when converting to scala.  Adding Set and List "views" to the Collection gives us a more elegant way of accessing that functionality when needed.

Reviewers: Colin P. McCabe <cmccabe@apache.org>
2019-05-09 11:08:22 -07:00
Ismael Juma c09e25fac2
MINOR: Fix bug in Struct.equals and use Objects.equals/Long.hashCode (#6680)
* Fixed bug in Struct.equals where we returned prematurely and added tests
* Update RequestResponseTest to check that `equals` and `hashCode` of
the struct is the same after serialization/deserialization only when possible.
* Use `Objects.equals` and `Long.hashCode` to simplify code
* Removed deprecated usages of `JUnitTestSuite`

Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>
2019-05-09 06:21:24 -07:00
Boyang Chen b0e82a68b3 KAFKA-8284: enable static membership on KStream (#6673)
Part of KIP-345 effort. The strategy is to extract user passed in group.instance.id config and pass it in with given thread-id (because consumer is currently per-thread level).

Reviewers: Guozhang Wang <wangguoz@gmail.com>
2019-05-07 17:43:13 -07:00
Chris Egerton cc097e909c KAFKA-8304: Fix registration of Connect REST extensions (#6651)
Fix registration of Connect REST extensions to prevent deadlocks when extensions get the list of connectors before the herder is available. Added integration test to check the behavior.

Author: Chris Egerton <cegerton@oberlin.edu>
Reviewers: Arjun Satish <arjun@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-05-07 17:20:51 -05:00
Mickael Maison 407bcdf78e KAFKA-8056; Use automatic RPC generation for FindCoordinator (#6408)
Reviewers: Jason Gustafson <jason@confluent.io>
2019-05-06 14:26:22 -07:00
Ismael Juma a37282415e
MINOR: Upgrade dependencies for Kafka 2.3 (#6665)
Many patch and minor updates.

Scalatest and Jetty deprecated classes that we
use. I removed usages for the former and filed KAFKA-8316 for the latter (I
suppressed the relevant deprecation warnings until the JIRA is fixed). As
part of the scalatest fixes, I also removed `TestUtils.fail` since it duplicates
`Assertions.fail`.

I also fixed a few compiler warnings that have crept in since my last sweep.

Updates of note:
- Jetty: 9.4.14 -> 9.4.18
  * https://github.com/eclipse/jetty.project/releases/tag/jetty-9.4.15.v20190215
  * https://github.com/eclipse/jetty.project/releases/tag/jetty-9.4.16.v20190411
  * https://github.com/eclipse/jetty.project/releases/tag/jetty-9.4.17.v20190418
  * https://github.com/eclipse/jetty.project/releases/tag/jetty-9.4.17.v20190418
  * https://github.com/eclipse/jetty.project/releases/tag/jetty-9.4.18.v20190429
- zstd: 1.3.8-1 -> 1.4.0-1
  * https://github.com/facebook/zstd/releases/tag/v1.4.0
  * zstd's fastest strategy, 6-8% faster in most scenarios
- zookeeper: 3.4.13 -> 3.4.14
  * https://zookeeper.apache.org/doc/r3.4.14/releasenotes.html

### Committer Checklist (excluded from commit message)
- [ ] Verify design and implementation 
- [ ] Verify test coverage and CI build status
- [ ] Verify documentation (including upgrade notes)

Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>
2019-05-03 10:35:07 -07:00
Zhanxiang (Patrick) Huang 6ca899e56d KAFKA-8066; Always close the sensors in Selector.close() (#6402)
When shutting down the ReplicaFetcher thread, we may fail to unregister sensors in selector.close(). When that happened, we will fail to start up the ReplicaFetcherThread with the same fetch id again because of the IllegalArgumentException in sensor registration. This issue will cause constant URPs in the cluster because the ReplicaFetchterThread is gone.

This patch addresses this issue by introducing a try-finally block in selector.close() so that we will always unregister the sensors in shutting down ReplicaFetcherThreads.

Reviewers: Rajini Sivaram <rajinisivaram@googlemail.com>, Jason Gustafson <jason@confluent.io>
2019-05-01 12:40:48 -07:00
Stanislav Kozlovski 191f2faae0 KAFKA-7992: Introduce start-time-ms metric (#6318)
Reviewers: Colin P. McCabe <cmccabe@apache.org>, Ismael Juma <ismael@juma.me.uk>
2019-05-01 08:58:02 -07:00
Boyang Chen 0f995ba6be KAFKA-7862 & KIP-345 part-one: Add static membership logic to JoinGroup protocol (#6177)
This is the first diff for the implementation of JoinGroup logic for static membership. The goal of this diff contains:

* Add group.instance.id to be unique identifier for consumer instances, provided by end user;
Modify group coordinator to accept JoinGroupRequest with/without static membership, refactor the logic for readability and code reusability.
* Add client side support for incorporating static membership changes, including new config for group.instance.id, apply stream thread client id by default, and new join group exception handling.
* Increase max session timeout to 30 min for more user flexibility if they are inclined to tolerate partial unavailability than burdening rebalance.
* Unit tests for each module changes, especially on the group coordinator logic. Crossing the possibilities like:
6.1 Dynamic/Static member
6.2 Known/Unknown member id
6.3 Group stable/unstable
6.4 Leader/Follower

The rest of the 345 change will be broken down to 4 separate diffs:

* Avoid kicking out members through rebalance.timeout, only do the kick out through session timeout.
* Changes around LeaveGroup logic, including version bumping, broker logic, client logic, etc.
* Admin client changes to add ability to batch remove static members
* Deprecate group.initial.rebalance.delay

Reviewers: Liquan Pei <liquanpei@gmail.com>, Stanislav Kozlovski <familyguyuser192@windowslive.com>, Jason Gustafson <jason@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2019-04-26 11:44:38 -07:00
Randall Hauch 9ff490509f MINOR: Correct RestServerTest formatting 2019-04-22 22:47:30 -05:00
Cyrus Vafadari d801c4fc69 MINOR: Add support for Standalone Connect configs in Rest Server extensions (#6622)
Add support for Standalone Connect configs in Rest Server extensions

A bug was introduced in 7a42750d that was caught in system tests:
The rest extensions fail if a Standalone worker config is passed,
since it does not have a definition for rebalance timeout.
A new method was introduced on WorkerConfig that by default returns
null for the rebalance timeout, and DistributedConfig overloads this
to return the configured value.

Author: Cyrus Vafadari <cyrus@confluent.io>
Reviewers: Arjun Satish <arjunconfluent.io>, Randall Hauch <rhauch@gmail.com>
2019-04-22 22:29:12 -05:00
Sebastián Ortega 3000fda8b9 KAFKA-8277: Fix NPEs in several methods of ConnectHeaders (#6550)
Replace `headers.isEmpty()` by calls to `isEmpty()` as the latter does a null check on heathers (that is lazily created).

Author: Sebastián Ortega <sebastian.ortega@letgo.com>
Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Arjun Satish <arjunconfluent.io>, Randall Hauch <rhauch@gmail.com>
2019-04-22 14:19:58 -07:00
Chris Egerton 71e721f135 KAFKA-8058: Fix ConnectClusterStateImpl.connectors() method (#6384)
Fixed the ConnectClusterStateImpl.connectors() method and throw an exception on timeout. Added unit test.

Author: Chris Egerton <chrise@confluent.io>

Reviewers: Magesh Nandakumar <magesh.n.kumar@gmail.com>, Robert Yokota <rayokota@gmail.com>, Arjun Satish <wicknicks@users.noreply.github.com>, Konstantine Karantasis <konstantine@confluent.io>, Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #6384 from C0urante:kafka-8058
2019-04-07 09:43:09 -05:00
Mickael Maison 825fa3fa09 MINOR: Fixed a few warning in core and connects (#6545)
- var -> val
- unused imports
- Javadoc fix

Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>
2019-04-05 23:30:48 +05:30
Doroszlai, Attila d3316bc6a7 KAFKA-8126: Flaky Test org.apache.kafka.connect.runtime.WorkerTest.testAddRemoveTask (#6475)
Changed the WorkerTest to use a mock Executor.

Author: Attila Doroszlai <adoroszlai@apache.org>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-04-03 15:00:05 -05:00
Konstantine Karantasis e4cad35312 KAFKA-8014: Extend Connect integration tests to add and remove workers dynamically (#6342)
Extend Connect's integration test framework to add or remove workers to EmbeddedConnectCluster, and choosing whether to fail the test on ungraceful service shutdown. Also added more JavaDoc and other minor improvements. 

Author: Konstantine Karantasis <konstantine@confluent.io>

Reviewers: Arjun Satish <arjun@confluent.io>, Randall Hauch <rhauch@gmail.com>

Closes #6342 from kkonstantine/KAFKA-8014
2019-03-25 09:29:33 -05:00
Boyang Chen 8406f3624d KAFKA-7858: Automatically generate JoinGroup request/response
Reviewers: Colin P. McCabe <cmccabe@apache.org>
2019-03-18 13:26:09 -07:00
Jason Gustafson 460e46c3bb
KAFKA-7831; Do not modify subscription state from background thread (#6221)
Metadata may be updated from the background thread, so we need to protect access to SubscriptionState. This patch restructures the metadata handling so that we only check pattern subscriptions in the foreground. Additionally, it improves the following:

1. SubscriptionState is now the source of truth for the topics that will be fetched. We had a lot of messy logic previously to try and keep the the topic set in Metadata consistent with the subscription, so this simplifies the logic.
2. The metadata needs for the producer and consumer are quite different, so it made sense to separate the custom logic into separate extensions of Metadata. For example, only the producer requires topic expiration.
3. We've always had an edge case in which a metadata change with an inflight request may cause us to effectively miss an expected update. This patch implements a separate version inside Metadata which is bumped when the needed topics changes.
4. This patch removes the MetadataListener, which was the cause of https://issues.apache.org/jira/browse/KAFKA-7764. 

Reviewers: David Arthur <mumrah@gmail.com>, Rajini Sivaram <rajinisivaram@googlemail.com>
2019-03-07 16:29:19 -08:00
Lysss 20f701cc50 KAFKA-7880:Naming worker thread by task id (#6275)
KAFKA-7880: Name worker thread to include task id

Change Connect's `WorkerTask` to name the thread using the `task-thread-<connectorTaskId>` pattern.

Reviewers: Randall Hauch <rhauch@gmail.com>
2019-03-04 18:04:34 -06:00
lzh3636 d930e5d3d8 improve some logging statements (#6078)
Reviewers: Matthias J. Sax <matthias@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-03-04 17:57:45 -06:00
Arjun Satish 2627a1be2c MINOR: Increase produce timeout to 120 seconds (#6326)
MINOR: Increase produce timeout for EmbeddedKafkaCluster to 120 seconds

Previous value was 500ms. This change gives more room to pass tests on systems with low resources running many parallel tests.

Reviewers: Randall Hauch <randall@confluent.io>
2019-02-25 23:26:59 -06:00
Chia-Ping Tsai 35a0de32ee KAFKA-6161 Add default implementation to close() and configure() for Serdes (#5348)
Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2019-02-21 09:05:13 -08:00
Alex Diachenko ec42e0378e KAFKA-7799; Use httpcomponents-client in RestServerTest.
The test `org.apache.kafka.connect.runtime.rest.RestServerTest#testCORSEnabled` assumes Jersey client can send restricted HTTP headers(`Origin`).

Jersey client uses `sun.net.www.protocol.http.HttpURLConnection`.
`sun.net.www.protocol.http.HttpURLConnection` drops restricted headers(`Host`, `Keep-Alive`, `Origin`, etc) based on static property `allowRestrictedHeaders`.
This property is initialized in a static block by reading Java system property `sun.net.http.allowRestrictedHeaders`.

So, if classloader loads `HttpURLConnection` before we set `sun.net.http.allowRestrictedHeaders=true`, then all subsequent changes of this system property won't take any effect(which happens if `org.apache.kafka.connect.integration.ExampleConnectIntegrationTest` is executed before `RestServerTest`).
To prevent this, we have to either make sure we set `sun.net.http.allowRestrictedHeaders=true` as early as possible or do not rely on this system property at all.

This PR adds test dependency on `httpcomponents-client` which doesn't depend on `sun.net.http.allowRestrictedHeaders` system property. Thus none of existing tests should interfere with `RestServerTest`.

Author: Alex Diachenko <sansanichfb@gmail.com>

Reviewers: Randall Hauch, Konstantine Karantasis, Gwen Shapira

Closes #6236 from avocader/KAFKA-7799
2019-02-12 12:03:08 -08:00
Ismael Juma c7f99bc2bd
MINOR: Update JUnit to 4.13 and annotate log cleaner integration test (#6248)
JUnit 4.13 fixes the issue where `Category` and `Parameterized` annotations
could not be used together. It also deprecates `ExpectedException` and
`assertThat`. Given this, we:

- Replace `ExpectedException` with the newly introduced `assertThrows`.
- Replace `Assert.assertThat` with `MatcherAssert.assertThat`.
- Annotate `AbstractLogCleanerIntegrationTest` with `IntegrationTest` category.

Reviewers: Ewen Cheslack-Postava <ewen@confluent.io>, David Arthur <mumrah@gmail.com>
2019-02-11 22:06:14 -08:00
Colin Patrick McCabe e2e8bdbd8c
KAFKA-7832: Use automatic RPC generation in CreateTopics (#5972)
Reviewers: Jun Rao <junrao@gmail.com>, Tom Bentley <tbentley@redhat.com>, Boyang Chen <bchen11@outlook.com>
2019-02-04 10:39:43 -08:00
Randall Hauch 176ea0d0f9 KAFKA-7873; Always seek to beginning in KafkaBasedLog (#6203)
Explicitly seek KafkaBasedLog’s consumer to the beginning of the topic partitions, rather than potentially use committed offsets (which would be unexpected) if group.id is set or rely upon `auto.offset.reset=earliest` if the group.id is null.

This should not change existing behavior but should remove some potential issues introduced with KIP-287 if `group.id` is not set in the consumer configurations. Note that even if `group.id` is set, we still always want to consume from the beginning.

Reviewers: Jason Gustafson <jason@confluent.io>
2019-02-01 14:17:52 -08:00
Chris Egerton 743607af5a KAFKA-5117: Stop resolving externalized configs in Connect REST API
[KIP-297](https://cwiki.apache.org/confluence/display/KAFKA/KIP-297%3A+Externalizing+Secrets+for+Connect+Configurations#KIP-297:ExternalizingSecretsforConnectConfigurations-PublicInterfaces) introduced the `ConfigProvider` mechanism, which was primarily intended for externalizing secrets provided in connector configurations. However, when querying the Connect REST API for the configuration of a connector or its tasks, those secrets are still exposed. The changes here prevent the Connect REST API from ever exposing resolved configurations in order to address that. rhauch has given a more thorough writeup of the thinking behind this in [KAFKA-5117](https://issues.apache.org/jira/browse/KAFKA-5117)

Tested and verified manually. If these changes are approved unit tests can be added to prevent a regression.

Author: Chris Egerton <chrise@confluent.io>

Reviewers: Robert Yokota <rayokota@gmail.com>, Randall Hauch <rhauch@gmail.com, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #6129 from C0urante/hide-provided-connect-configs
2019-01-23 11:00:23 -08:00
Arjun Satish dc935c4beb MINOR: Handle case where connector status endpoints returns 404 (#6176)
Reviewers: Randall Hauch <randall@confluent.io>, Matthias J. Sax <matthias@confluent.io>
2019-01-20 19:31:20 -08:00
Magesh Nandakumar dec68c9350 MINOR: Start Connect REST server in standalone mode to match distributed mode (KAFKA-7503 follow-up)
Start the Rest server in the standalone mode similar to how it's done for distributed mode.

Author: Magesh Nandakumar <magesh.n.kumar@gmail.com>

Reviewers: Arjun Satish <arjun@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #6148 from mageshn/KAFKA-7826
2019-01-16 22:58:30 -08:00
Chia-Ping Tsai af634a4a98 KAFKA-7391; Introduce close(Duration) to Producer and AdminClient instead of close(long, TimeUnit) (#5667)
See KIP-367: https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=89070496.

Reviewers: Viktor Somogyi <viktorsomogyi@gmail.com>, Manikumar Reddy <manikumar.reddy@gmail.com>, Jason Gustafson <jason@confluent.io>
2019-01-15 08:48:32 -08:00
Andrew Schofield aca52b6d2c KAFKA-7461: Add tests for logical types
Added testing of logical types for Kafka Connect in support of KIP-145 features.
Added tests for Boolean, Time, Date and Timestamp, including the valid conversions.

The area of ISO8601 strings is a bit of a mess because the tokenizer is not compatible with
that format, and a subsequent JIRA will be needed to fix that.

A few small fixes as well as creating test cases, but they're clearly just corrections such as
using 0 to mean January (java.util.Calendar uses zero-based month numbers).

Author: Andrew Schofield <andrew_schofield@uk.ibm.com>

Reviewers: Mickael Maison <mimaison@users.noreply.github.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #6077 from AndrewJSchofield/KAFKA-7461-ConverterValuesLogicalTypesTest
2019-01-14 15:41:23 -08:00
Arjun Satish 69d8d2ea11 KAFKA-7503: Connect integration test harness
Expose a programmatic way to bring up a Kafka and Zk cluster through Java API to facilitate integration tests for framework level changes in Kafka Connect. The Kafka classes would be similar to KafkaEmbedded in streams. The new classes would reuse the kafka.server.KafkaServer classes from :core, and provide a simple interface to bring up brokers in integration tests.

Signed-off-by: Arjun Satish <arjunconfluent.io>

Author: Arjun Satish <arjun@confluent.io>
Author: Arjun Satish <wicknicks@users.noreply.github.com>

Reviewers: Randall Hauch <rhauch@gmail.com>, Konstantine Karantasis <konstantine@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5516 from wicknicks/connect-integration-test
2019-01-14 13:50:23 -08:00
Jason Gustafson 73a9fcaa4d
KAFKA-7799; Fix flaky test RestServerTest.testCORSEnabled (#6106)
The test always fails if testOptionsDoesNotIncludeWadlOutput is executed before testCORSEnabled. It seems the problem is the use of the system property. Perhaps there is some static caching somewhere.

Reviewers: Randall Hauch <rhauch@gmail.com>, Guozhang Wang <wangguoz@gmail.com>
2019-01-08 23:47:28 -08:00
Chia-Ping Tsai 9601315420 KAFKA-7253; The returned connector type is always null when creating connector (#5470)
The null map returned from the current snapshot causes the null type in response. The connector class name can be taken from the config of request instead since we require the config should contain the connector class name.

Reviewers: Jason Gustafson <jason@confluent.io>
2019-01-08 08:52:42 -08:00
Chia-Ping Tsai b6d1450012 MINOR: Show the specified value in logging the deprecation warnings of internal converter (#5212)
Reviewers: Jason Gustafson <jason@confluent.io>
2019-01-05 02:56:15 -08:00
Renato Mefi 964e2c57c2 KAFKA-3832; Kafka Connect's JSON Converter never outputs a null value (#6027)
When using the Connect `JsonConverter`, it's impossible to produce tombstone messages, thus impacting the compaction of the topic. This patch allows the converter with and without schemas to output a NULL byte value in order to have a proper tombstone message. When it's regarding to get this data into a connect record, the approach is the same as when the payload looks like `"{ "schema": null, "payload": null }"`, this way the sink connectors can maintain their functionality and reduces the BCC.

Reviewers: Gunnar Morling <gunnar.morling@googlemail.com>, Randall Hauch <rhauch@gmail.com>, Jason Gustafson <jason@confluent.io>
2018-12-28 09:39:52 -08:00
Alex Diachenko ee370d3893 KAFKA-7759; Disable WADL output in the Connect REST API (#6051)
This patch disables support for WADL output in the Connect REST API since it was never intended to be exposed. 

Reviewers: Randall Hauch <rhauch@gmail.com>, Jason Gustafson <jason@confluent.io>
2018-12-20 14:24:05 -08:00
Cyrus Vafadari 9f954ac614 MINOR: Safe string conversion to avoid NPEs
Should be ported back to 2.0

Author: Cyrus Vafadari <cyrus@confluent.io>

Reviewers: Ewen Cheslack-Postava <ewen@confluent.io>

Closes #6004 from cyrusv/cyrus-npe
2018-12-05 13:23:52 -08:00
Magesh Nandakumar ace4dd0056 KAFKA-7551: Refactor to create producer & consumer in the worker
This is minor refactoring that brings in the creation of producer and consumer to the Worker. Currently, the consumer is created in the WorkerSinkTask. This should not affect any functionality and it just makes the code structure easier to understand.

Author: Magesh Nandakumar <magesh.n.kumar@gmail.com>

Reviewers: Ryanne Dolan <ryannedolan@gmail.com>, Randall Hauch <rhauch@gmail.com>, Robert Yokota <rayokota@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5842 from mageshn/KAFKA-7551
2018-11-29 23:38:50 -08:00
Cyrus Vafadari 4712a36416 MINOR: Add logging to Connect SMTs
Includes Update to ConnectRecord string representation to give
visibility into schemas, useful in SMT tracing

Author: Cyrus Vafadari <cyrus@confluent.io>

Reviewers: Randall Hauch <rhauch@gmail.com>, Konstantine Karantasis <konstantine@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5860 from cyrusv/cyrus-logging
2018-11-29 22:29:50 -08:00
Robert Yokota a2e87feb8b KAFKA-7620: Fix restart logic for TTLs in WorkerConfigTransformer
The restart logic for TTLs in `WorkerConfigTransformer` was broken when trying to make it toggle-able.   Accessing the toggle through the `Herder` causes the same code to be called recursively.  This fix just accesses the toggle by simply looking in the properties map that is passed to `WorkerConfigTransformer`.

Author: Robert Yokota <rayokota@gmail.com>

Reviewers: Magesh Nandakumar <magesh.n.kumar@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5914 from rayokota/KAFKA-7620
2018-11-27 22:01:21 -08:00
Mickael Maison 4e90af34c6 MINOR: Various javadoc improvement in clients and connect (#5878)
Fixed formatting issues and added links in a few classes
2018-11-23 10:50:30 +05:30
Srinivas Reddy ad26914de6 KAFKA-7418: Add the missing '--help' option to Kafka commands (KIP-374)
Changes made as part of this [KIP-374](https://cwiki.apache.org/confluence/x/FgSQBQ) and [KAFKA-7418](https://issues.apache.org/jira/browse/KAFKA-7418)
 - Checking for empty args or help option in command file to print Usage
 - Added new class to enforce help option to all commands
 - Refactored few lines (ex `PreferredReplicaLeaderElectionCommand`) to
   make use of `CommandDefaultOptions` attributes.
 - Made the changes in help text wordings

Run the unit tests in local(Windows) few Linux friendly tests are failing but
not any functionality, verified `--help` and no option response by running
Scala classes, since those all are having `main` method.

Author: Srinivas Reddy <srinivas96alluri@gmail.com>
Author: Srinivas Reddy <mrsrinivas@users.noreply.github.com>
Author: Srinivas <srinivas96alluri@gmail.com>

Reviewers: Colin Patrick McCabe <colin@cmccabe.xyz>, Manikumar Reddy <manikumar.reddy@gmail.com>, Guozhang Wang <wangguoz@gmail.com>, Mickael Maison <mickael.maison@gmail.com>

Closes #5910 from mrsrinivas/KIP-374
2018-11-22 17:12:34 +05:30
Stanislav Kozlovski 068ab9cefa KAFKA-7528: Standardize on Min/Avg/Max Kafka metrics' default value - NaN (#5908)
While metrics like Min, Avg and Max make sense to respective use Double.MAX_VALUE, 0.0 and Double.MIN_VALUE as default values to ease computation logic, exposing those values makes reading them a bit misleading. For instance, how would you differentiate whether your -avg metric has a value of 0 because it was given samples of 0 or no samples were fed to it?

It makes sense to standardize on the output of these metrics with something that clearly denotes that no values have been recorded.

Reviewers: Bill Bejeck <bill@confluent.io>, John Roesler <john@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2018-11-20 15:54:24 -08:00
Yishun Guan 9646602d68 KAFKA-7402: Implement KIP-376 AutoCloseable additions 2018-11-16 15:58:47 -08:00
Benedict Jin b21e933592 * MINOR: Catching null pointer exception for empty leader URL when assignment is null (#4798)
Catch null pointer exception for empty leader URL when assignment is null.

Reviewers: Randall Hauch <rhauch@gmail.com>, Konstantine Karantasis <konstantine@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2018-11-16 15:28:47 -08:00
Randall Hauch 545d40d83c MINOR: Avoid logging connector configuration in Connect framework (#5868)
Some connector configs may be sensitive, so we should avoid logging them.

Reviewers: Alex Diachenko, Dustin Cote <dustin@confluent.io>, Jason Gustafson <jason@confluent.io>
2018-11-13 13:30:34 -08:00
Ismael Juma 12f310d50e
KAFKA-7612: Fix javac warnings and enable warnings as errors (#5900)
- Use Xlint:all with 3 exclusions (filed KAFKA-7613 to remove the exclusions)
- Use the same javac options when compiling tests (seems accidental that
we didn't do this before)
- Replaced several deprecated method calls with non-deprecated ones:
  - `KafkaConsumer.poll(long)` and `KafkaConsumer.close(long)`
  - `Class.newInstance` and `new Integer/Long` (deprecated since Java 9)
  - `scala.Console` (deprecated in Scala 2.11)
  - `PartitionData` taking a timestamp (one of them seemingly a bug)
  - `JsonMappingException` single parameter constructor
- Fix unnecessary usage of raw types in several places.
- Add @SuppressWarnings for deprecations, unchecked and switch fallthrough in
several places.
- Scala clean-ups (var -> val, ETA expansion warnings, avoid reflective calls)
- Use lambdas to simplify code in a few places
- Add @SafeVarargs, fix varargs usage and remove unnecessary `Utils.mkList` method

Reviewers: Matthias J. Sax <mjsax@apache.org>, Manikumar Reddy <manikumar.reddy@gmail.com>, Randall Hauch <rhauch@gmail.com>, Bill Bejeck <bill@confluent.io>, Stanislav Kozlovski <stanislav_kozlovski@outlook.com>
2018-11-12 22:18:59 -08:00
Srinivas Reddy 8abbf33b59 KAFKA-7431: Clean up connect unit tests
[KAFKA-7431](https://issues.apache.org/jira/browse/KAFKA-7431)

Changes made to improve the code readability:
 - Removed `throws Exception` from the place where there won't be an
 exception
 - Removed type arguments where those can be inferred explicitly by compiler
 - Rewritten Anonymous classes to Java 8 with lambdas

Author: Srinivas Reddy <mrsrinivas@users.noreply.github.com>
Author: Srinivas Reddy <srinivas96alluri@gmail.com>

Reviewers: Randall Hauch <rhauch@gmail.com>, Ismael Juma <ismael@juma.me.uk>, Ryanne Dolan <ryannedolan@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5681 from mrsrinivas/cleanup-connect-uts
2018-11-07 08:23:19 -08:00
Jason Gustafson d71cb54672
KAFKA-7567; Clean up internal metadata usage for consistency and extensibility (#5813)
This patch makes two improvements to internal metadata handling logic and testing:

1. It reduce dependence on the public object `Cluster` for internal metadata propagation since it is not easy to evolve. As an example, we need to propagate leader epochs from the metadata response to `Metadata`, but it is not straightforward to do this without exposing it in `PartitionInfo` since that is what `Cluster` uses internally. By doing this change, we are able to remove some redundant `Cluster` building logic. 
2. We want to make the metadata handling in `MockClient` simpler and more consistent. Currently we have mix of metadata update mechanisms which are internally inconsistent with each other and do not match the implementation in `NetworkClient`.

Reviewers: Rajini Sivaram <rajinisivaram@googlemail.com>
2018-10-30 13:20:13 -07:00
Ron Dagostino e8a3bc7425 KAFKA-7352; Allow SASL Connections to Periodically Re-Authenticate (KIP-368) (#5582)
KIP-368 implementation to enable periodic re-authentication of SASL clients. Also adds a broker configuration option to terminate client connections that do not re-authenticate within the configured interval.
2018-10-26 23:18:15 +01:00
jonathanskrzypek a947fe8da8 KAFKA-6195: Resolve DNS aliases in bootstrap.server (KIP-235) (#4485)
Adds `client.dns.lookup=resolve_canonical_bootstrap_servers_only` option to perform full dns resolution of bootstrap addresses

Reviewers: Colin Patrick McCabe <colin@cmccabe.xyz>, Sriharsha Chintalapani <sriharsha@apache.org>, Edoardo Comar <ecomar@uk.ibm.com>, Mickael Maison <mickael.maison@gmail.com>, Manikumar Reddy <manikumar.reddy@gmail.com>, Rajini Sivaram <rajinisivaram@googlemail.com>
2018-10-13 21:39:35 +01:00
Edoardo Comar f393b2f7dd KAFKA-6863 Kafka clients should try to use multiple DNS resolved IP (#4987)
Implementation of KIP-302: Based on the new client configuration `client.dns.lookup`, a NetworkClient can use InetAddress.getAllByName to find all IPs and iterate over them when they fail to connect. Only uses either IPv4 or IPv6 addresses similar to the default mode.

Co-authored-by: Edoardo Comar <ecomar@uk.ibm.com>
Co-authored-by: Mickael Maison <mickael.maison@gmail.com>

Reviewers: Rajini Sivaram <rajinisivaram@googlemail.com>
2018-10-11 18:14:17 +01:00
Ismael Juma adb3a950ee
MINOR: Fix remaining core, connect and clients tests to pass with Java 11 (#5771)
- SslFactoryTest should use SslFactory to create SSLEngine
- Use Mockito instead of EasyMock in `ConsoleConsumerTest` as one of
the tests mocks a standard library class and the latest released EasyMock
version can't do that when Java 11 is used.
- Avoid mocking `ConcurrentMap` in `SourceTaskOffsetCommitterTest`
for similar reasons. As it happens, mocking is not actually needed here.

Reviewers: Rajini Sivaram <rajinisivaram@googlemail.com>
2018-10-10 13:31:06 -07:00
Konstantine Karantasis 6a03d3b42c KAFKA-6914; Set parent classloader of DelegatingClassLoader same as the worker's (#5720)
The parent classloader of the DelegatingClassLoader and therefore the classloading scheme used by Connect does not have to be fixed to the System classloader.

Setting it the same as the one that was used to load the DelegatingClassLoader class itself is more flexible and, while in most cases will result in the System classloader to be used, it will also work in othr managed environments that control classloading differently (OSGi, and others).

The fix is minimal and the mainstream use is tested via system tests.

Reviewers: Randall Hauch <rhauch@gmail.com>, Jason Gustafson <jason@confluent.io>
2018-10-05 12:46:13 -07:00
Robert Yokota 3edd8e7333 KAFKA-7476: Fix Date-based types in SchemaProjector
Various converters (AvroConverter and JsonConverter) produce a
SchemaAndValue consisting of a logical schema type and a java.util.Date.
This is a fix for SchemaProjector to properly handle the Date.

Author: Robert Yokota <rayokota@gmail.com>

Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5736 from rayokota/KAFKA-7476
2018-10-04 20:34:50 -07:00
Kevin Lu a5335e7cbd KAFKA-6123: Give client MetricsReporter auto-generated client.id (#5383) 2018-10-03 09:56:22 -07:00
Amit Sela fd44dc7fb2 KAFKA-6684: Support casting Connect values with bytes schema to string
Allow to cast LogicalType to string by calling the serialized (Java) object's toString().

Added tests for `BigDecimal` and `Date` as whole record and as fields.

Author: Amit Sela <amitsela33@gmail.com>

Reviewers: Randall Hauch <rhauch@gmail.com>, Robert Yokota <rayokota@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #4820 from amitsela/cast-transform-bytes
2018-09-30 22:24:09 -07:00
Amit Sela c1457be995 KAFKA-7460: Fix Connect Values converter date format pattern
Switches to normal year format instead of week date years and day of month instead of day of year.

This is directly from #4820, but separated into a different JIRA/PR to keep the fixes independent. Original authorship should be maintained in the commit.

Author: Amit Sela <amitsela33@gmail.com>

Reviewers: Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5718 from ewencp/fix-header-converter-date-format
2018-09-30 19:51:59 -07:00
Michał Borowiecki 22f1724123 KAFKA-7434: Fix NPE in DeadLetterQueueReporter
*More detailed description of your change,
if necessary. The PR title and PR message become
the squashed commit message, so use a separate
comment to ping reviewers.*

*Summary of testing strategy (including rationale)
for the feature or bug fix. Unit and/or integration
tests are expected for any behaviour change and
system tests should be considered for larger changes.*

Author: Michał Borowiecki <mbor81@gmail.com>

Reviewers: Arjun Satish <arjun@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5700 from mihbor/KAFKA-7434
2018-09-29 10:19:10 -07:00
Randall Hauch f0282498e7 KAFKA-6926: Simplified some logic to eliminate some suppressions of NPath complexity checks (#5051)
Modified several classes' `equals` methods and simplified a complex method to
reduce the NPath complexity so they could be removed from the checkstyle
suppressions that were required with the recent move to Java 8 and upgrade
of Checkstyle: https://github.com/apache/kafka/pull/5046.

Reviewers: Robert Yokota <rayokota@gmail.com>, Arjun Satish <arjun@confluent.io>, Ismael Juma <ismael@juma.me.uk>
2018-09-12 21:21:46 -07:00
Ismael Juma f123d2f18c KAFKA-5887; Replace findBugs with spotBugs and upgrade to Gradle 4.10
findBugs is abandoned, it doesn't work with Java 9 and the Gradle plugin will be deprecated in
Gradle 5.0: https://github.com/gradle/gradle/pull/6664

spotBugs is actively maintained and it supports Java 8, 9 and 10. Java 11 is not supported yet,
but it's likely to happen soon.

Also fixed a file leak in Connect identified by spotbugs.

Manually tested spotBugsMain, jarAll and importing kafka in IntelliJ and running
a build in the IDE.

Author: Ismael Juma <ismael@juma.me.uk>

Reviewers: Colin Patrick McCabe <colin@cmccabe.xyz>, Dong Lin <lindong28@gmail.com>

Closes #5625 from ijuma/kafka-5887-spotbugs
2018-09-10 13:14:00 -07:00
Kevin Lafferty 847780e5a5 KAFKA-7353: Connect logs 'this' for anonymous inner classes
Replace 'this' reference in anonymous inner class logs to out class's 'this'

Author: Kevin Lafferty <kevin.lafferty@gmail.com>

Reviewers: Randall Hauch <rhauch@gmail.com>, Arjun Satish <arjun@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5583 from kevin-laff/connect_logging
2018-09-05 20:15:25 -07:00
Robert Yokota fd5acd73e6 KAFKA-7242: Reverse xform configs before saving (KIP-297)
During actions such as a reconfiguration, the task configs are obtained
via `Worker.connectorTaskConfigs` and then subsequently saved into an
instance of `ClusterConfigState`.  The values of the properties that are saved
are post-transformation (of variable references) when they should be
pre-transformation.  This is to avoid secrets appearing in plaintext in
the `connect-configs` topic, for example.

The fix is to change the 2 clients of `Worker.connectorTaskConfigs` to
perform a reverse transformation (values converted back into variable
references) before saving them into an instance of `ClusterConfigState`.
The 2 places where the save is performed are
`DistributedHerder.reconfigureConnector` and
`StandaloneHerder.updateConnectorTasks`.

The way that the reverse transformation works is by using the
"raw" connector config (with variable references still intact) from
`ClusterConfigState` to convert config values back into variable
references for those keys that are common between the task config
and the connector config.

There are 2 additional small changes that only affect `StandaloneHerder`:

1) `ClusterConfigState.allTasksConfigs` has been changed to perform a
transformation (resolution) on all variable references.  This is
necessary because the result of this method is compared directly to
`Worker.connectorTaskConfigs`, which also has variable references
resolved.

2) `StandaloneHerder.startConnector` has been changed to match
`DistributedHerder.startConnector`.  This is to fix an issue where
during `StandaloneHerder.restartConnector`, the post-transformed
connector config would be saved back into `ClusterConfigState`.

I also performed an analysis of all other code paths where configs are
saved back into `ClusterConfigState` and did not find any other
issues.

Author: Robert Yokota <rayokota@gmail.com>

Reviewers: Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5475 from rayokota/KAFKA-7242-reverse-xform-props
2018-08-28 12:59:08 -07:00
Robert Yokota d31762a0c2 MINOR: Additional testing of logical type handling in Cast transform
Reviewers: Randall Hauch <rhauch@gmail.com>, Jason Gustafson <jason@confluent.io>
2018-08-21 08:21:46 -07:00
Maciej Bryński 504824b7b3 KAFKA-5891; Proper handling of LogicalTypes in Cast (#4633)
Currently logical types are dropped during Cast Transformation.
This patch fixes this behaviour.

Reviewers: Randall Hauch <rhauch@gmail.com>, Jason Gustafson <jason@confluent.io>
2018-08-20 16:43:11 -07:00
Koen De Groote 9e0e29ac5c MINOR: Use statically compiled regular expressions for efficiency (#5168)
Reviewers: Andras Beni <andrasbeni@cloudera.com>, Sriharsha Chintalapani <sriharsha@apache.org>, Jason Gustafson <jason@confluent.io>
2018-08-13 17:31:29 -07:00
Viktor Somogyi 8a78d76466 KAFKA-7140; Remove deprecated poll usages (#5319)
Reviewers: Matthias J. Sax <mjsax@apache.org>, Jason Gustafson <jason@confluent.io>
2018-08-10 22:51:17 -07:00
Arjun Satish e876c921b0 MINOR: Add connector configs to site-docs
In AK's documentation, the config props for connectors are not listed (https://kafka.apache.org/documentation/#connectconfigs). This PR adds these sink and source connector configs to the html site-docs.

Signed-off-by: Arjun Satish <arjunconfluent.io>

Author: Arjun Satish <arjun@confluent.io>

Reviewers: Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5469 from wicknicks/add-connector-configs-to-docs
2018-08-07 14:34:27 -07:00
Robert Yokota 36a8fec0ab KAFKA-7225: Pretransform validated props
If a property requires validation, it should be pretransformed if it is a variable reference, in order to have a value that will properly pass the validation.

Author: Robert Yokota <rayokota@gmail.com>

Reviewers: Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5445 from rayokota/KAFKA-7225-pretransform-validated-props
2018-08-07 13:18:16 -07:00
Jason Gustafson fc5f6b0e46
MINOR: Add Timer to simplify timeout bookkeeping and use it in the consumer (#5087)
We currently do a lot of bookkeeping for timeouts which is both error-prone and distracting. This patch adds a new `Timer` class to simplify this logic and control unnecessary calls to system time. In particular, this helps with nested timeout operations. The consumer has been updated to use the new class.

Reviewers: Ismael Juma <ismael@juma.me.uk>, Guozhang Wang <wangguoz@gmail.com>
2018-08-03 17:25:07 -07:00
Arjun Satish 70d882861e KAFKA-7228: Set errorHandlingMetrics for dead letter queue
DLQ reporter does not get a `errorHandlingMetrics` object when created by the worker. This results in an NPE.

Signed-off-by: Arjun Satish <arjunconfluent.io>

*More detailed description of your change,
if necessary. The PR title and PR message become
the squashed commit message, so use a separate
comment to ping reviewers.*

*Summary of testing strategy (including rationale)
for the feature or bug fix. Unit and/or integration
tests are expected for any behaviour change and
system tests should be considered for larger changes.*

Author: Arjun Satish <arjun@confluent.io>

Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5440 from wicknicks/KAFKA-7228
2018-08-02 14:36:02 -07:00
Jason Gustafson c3e7c0bcb2
MINOR: Producers should set delivery timeout instead of retries (#5425)
Use delivery timeout instead of retries when possible and remove various TODOs associated with completion of KIP-91.

Reviewers: Ismael Juma <ismael@juma.me.uk>, Guozhang Wang <wangguoz@gmail.com>
2018-08-01 11:04:17 -07:00
Yu Yang 7fc7136ffd KAFKA-5886; Introduce delivery.timeout.ms producer config (KIP-91) (#5270)
Co-authored-by: Sumant Tambe <sutambe@yahoo.com>
Co-authored-by: Yu Yang <yuyang@pinterest.com>

Reviewers: Ted Yu <yuzhihong@gmail.com>, Apurva Mehta <apurva@confluent.io>, Jason Gustafson <jason@confluent.io>
2018-07-26 09:13:50 -07:00
Konstantine Karantasis 5d2bf6328e MINOR: FileStreamSinkTask should create file if it doesn't exist (#5406)
A recent change from `new FileOutputStream` to `Files.newOutputStream` missed the `CREATE` flag (which is necessary in addition to `APPEND`).

Reviewers: Ismael Juma <ismael@juma.me.uk>
2018-07-20 21:01:10 -07:00
Ismael Juma 7a74ec62d2
MINOR: Avoid FileInputStream/FileOutputStream (#5281)
They rely on finalizers (before Java 11), which create
unnecessary GC load. The alternatives are as easy to
use and don't have this issue.

Also use FileChannel directly instead of retrieving
it from RandomAccessFile whenever possible
since the indirection is unnecessary.

Finally, add a few try/finally blocks.

Reviewers: Colin Patrick McCabe <colin@cmccabe.xyz>, Rajini Sivaram <rajinisivaram@googlemail.com>
2018-06-27 01:00:05 -07:00
Chia-Ping Tsai 6810617179 KAFKA-7048 NPE when creating connector (#5202)
Reviewers: Robert Yokota <rayokota@gmail.com>, Konstantine Karantasis <konstantine@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2018-06-17 16:49:01 -07:00
Gunnar Morling be846d833c KAFKA-7058: Comparing schema default values using Objects#deepEquals()
https://issues.apache.org/jira/browse/KAFKA-7058
* Summary of testing strategy: Added new unit test

Author: Gunnar Morling <gunnar.morling@googlemail.com>

Reviewers: Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5225 from gunnarmorling/KAFKA-7058
2018-06-16 23:04:31 -07:00
Randall Hauch f0282cb3de KAFKA-7047: Added SimpleHeaderConverter to plugin isolation whitelist
This was originally missed when headers were added as part of KIP-145 in AK 1.1. An additional unit test was added in line with the StringConverter.

This should be backported to the AK `1.1` branch so that it is included in the next bugfix release. The `SimpleHeaderConverter` class that we're referencing was first added in the 1.1.0 release, so there's no reason to backport earlier.

Author: Randall Hauch <rhauch@gmail.com>

Reviewers: Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5204 from rhauch/kafka-7047
2018-06-16 22:23:20 -07:00
Magesh Nandakumar 239dd0fb9b KAFKA-7039: Create an instance of the plugin only it's a Versioned Plugin
Create an instance of the plugin only it's a Versioned Plugin. Prior to KIP-285, this was done for only for Connector and this PR will continue to have the same behavior.

Author: Magesh Nandakumar <magesh.n.kumar@gmail.com>

Reviewers: Randall Hauch <rhauch@gmail.com>, Konstantine Karantasis <konstantine@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5191 from mageshn/KAFKA-7039
2018-06-16 22:18:20 -07:00
Randall Hauch fab8b7e676 KAFKA-7056: Moved Connect’s new numeric converters to runtime (KIP-305)
KIP-305 added numeric converters to Connect, but these were added in Connect’s API module in the same package as the `StringConverter`. This commit moves them into the Runtime module and into the `converters` package where the `ByteArrayConverter` already lives. These numeric converters have not yet been included in a release, and so they can be moved without concern.

All of Connect’s converters must be referenced in worker / connector configurations and are therefore part of the API, but otherwise do not need to be in the “api” module as they do not need to be instantiated or directly used by extensions. This change makes them more similar to and aligned with the `ByteArrayConverter`.

It also gives us the opportunity to move them into the “api” module in the future (keeping the same package name), should we ever want or need to do so. However, if we were to start out with them in the “api” module, we would never be able to move them out into the “runtime” module, even if we kept the same package name. Therefore, moving them to “runtime” now gives us a bit more flexibility.

This PR moves the unit tests for the numeric converters accordingly, and updates the `PluginsUtil` and `PluginUtilsTest` as well.

Author: Randall Hauch <rhauch@gmail.com>

Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5222 from rhauch/kafka-7056
2018-06-15 14:52:28 -07:00
Randall Hauch 22356d55ef KAFKA-7043: Modified plugin isolation whitelist with recently added converters (KIP-305)
Several recently-added converters are included in the plugin isolation whitelist, similarly to the `StringConverter`. This is a change in the implementation, and does not affect the approved KIP. Several unit tests were added to verify they are being loaded in isolation, again similarly to `StringConverter`.

These changes should be applied only to `trunk` and `2.0`, since these converters were added as part of KIP-305 for AK 2.0.

Author: Randall Hauch <rhauch@gmail.com>

Reviewers: Magesh Nandakumar <magesh.n.kumar@gmail.com>, Konstantine Karantasis <konstantine@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5198 from rhauch/kafka-7043
2018-06-12 21:35:45 -07:00
Chia-Ping Tsai 70e13f032e MINOR: Remove the unused field in DelegatingClassLoader
After [3173](https://github.com/apache/kafka/commit/e0150a25e8), the field "activePaths" is not used anymore.

Author: Chia-Ping Tsai <chia7712@gmail.com>

Reviewers: Ewen Cheslack-Postava <ewen@confluent.io>

Closes #4984 from chia7712/remove_unused_field_from_DelegatingClassLoader
2018-06-12 21:03:19 -07:00
Robert Yokota 16190e9bfd MINOR: Move FileConfigProvider to provider subpackage (#5194)
This moves FileConfigProvider to the org.apache.common.config.provider package to more easily isolate provider implementations going forward.

Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Randall Hauch <rhauch@gmail.com>, Ismael Juma <ismael@juma.me.uk>, Jason Gustafson <jason@confluent.io>
2018-06-12 12:52:51 -07:00
Arjun Satish 3face7fce2 KAFKA-7003: Set error context in message headers (KIP-298)
If the property `errors.deadletterqueue.context.headers.enable` is set to true, add a set of headers to the message describing the context under which the error took place.

A unit test is added to check the correctness of header creation.

Signed-off-by: Arjun Satish <arjunconfluent.io>

Author: Arjun Satish <arjun@confluent.io>

Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5159 from wicknicks/KAFKA-7003
2018-06-11 15:16:46 -07:00
Arjun Satish 22612be46d KAFKA-7002: Add a config property for DLQ topic's replication factor (KIP-298)
Currently, the replication factor is hardcoded to a value of 3. This means that we cannot use a DLQ in any cluster setup with less than three brokers. It is better to have the user specify this value if the default value does meet the requirements.

Testing: A unit test is added.

Signed-off-by: Arjun Satish <arjunconfluent.io>

Author: Arjun Satish <arjun@confluent.io>

Reviewers: Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5145 from wicknicks/KAFKA-7002
2018-06-07 15:49:57 -07:00
Arjun Satish aa014b2709 KAFKA-7001: Rename errors.allowed.max property in Connect to errors.tolerance (KIP-298)
Signed-off-by: Arjun Satish <arjunconfluent.io>

Author: Arjun Satish <arjun@confluent.io>

Reviewers: Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5146 from wicknicks/KAFKA-7001
2018-06-07 15:27:13 -07:00
Magesh Nandakumar 642a09168c KAFKA-6991: Fix ServiceLoader issue with PluginClassLoader (KIP-285)
Fix ServiceLoader issue with PluginClassLoader and add basic-auth-extension packaging & classpath

*More detailed description of your change,
if necessary. The PR title and PR message become
the squashed commit message, so use a separate
comment to ping reviewers.*

*Summary of testing strategy (including rationale)
for the feature or bug fix. Unit and/or integration
tests are expected for any behaviour change and
system tests should be considered for larger changes.*

Author: Magesh Nandakumar <magesh.n.kumar@gmail.com>

Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5135 from mageshn/KAFKA-6991
2018-06-06 21:09:16 -07:00
Robert Yokota 8264492dee MINOR: Use service loader for ConfigProvider impls (KIP-297)
This is a small change to use the Java ServiceLoader to load ConfigProvider plugins.  It uses code added by mageshn for Connect Rest Extensions.

Author: Robert Yokota <rayokota@gmail.com>

Reviewers: Magesh Nandakumar <magesh.n.kumar@gmail.com>, Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5141 from rayokota/service-loader-for-config-plugins
2018-06-06 08:55:57 -07:00
Arjun Satish faa15b8b75 KAFKA-6981: Move the error handling configuration properties into the ConnectorConfig and SinkConnectorConfig classes (KIP-298)
Move the error handling configuration properties into the ConnectorConfig and SinkConnectorConfig classes, and refactor the tests and classes to use these new properties.

Testing: Unit tests and running the connect-standalone script with a file sink connector.

Author: Arjun Satish <arjun@confluent.io>
Author: Randall Hauch <rhauch@gmail.com>

Reviewers: Konstantine Karantasis <konstantine@confluent.io>, Magesh Nandakumar <magesh.n.kumar@gmail.com>, Robert Yokota <rayokota@gmail.com>, Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5125 from wicknicks/KAFKA-6981
2018-06-05 13:59:15 -07:00
Robert Yokota 08e8facdc9 KAFKA-6886: Externalize secrets from Connect configs (KIP-297)
This commit allows secrets in Connect configs to be externalized and replaced with variable references of the form `${provider:[path:]key}`, where the "path" is optional.

There are 2 main additions to `org.apache.kafka.common.config`: a `ConfigProvider` and a `ConfigTransformer`.  The `ConfigProvider` is an interface that allows key-value pairs to be provided by an external source for a given "path".  An a TTL can be associated with the key-value pairs returned from the path.  The `ConfigTransformer` will use instances of `ConfigProvider` to replace variable references in a set of configuration values.

In the Connect framework, `ConfigProvider` classes can be specified in the worker config, and then variable references can be used in the connector config.  In addition, the herder can be configured to restart connectors (or not) based on the TTL returned from a `ConfigProvider`.  The main class that performs restarts and transformations is `WorkerConfigTransformer`.

Finally, a `configs()` method has been added to both `SourceTaskContext` and `SinkTaskContext`.  This allows connectors to get configs with variables replaced by the latest values from instances of `ConfigProvider`.

Most of the other changes in the Connect framework are threading various objects through classes to enable the above functionality.

Author: Robert Yokota <rayokota@gmail.com>
Author: Ewen Cheslack-Postava <me@ewencp.org>

Reviewers: Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5068 from rayokota/KAFKA-6886-connect-secrets
2018-05-30 14:43:11 -07:00
Arjun Satish f8dfbb067c KAFKA-6738: Implement error handling for source and sink tasks (KIP-298)
This PR implements the features described in this KIP: https://cwiki.apache.org/confluence/display/KAFKA/KIP-298%3A+Error+Handling+in+Connect

This PR changes the Connect framework to allow it to automatically deal with errors encountered while processing records in a Connector. The following behavior changes are introduced here:

**Retry on Failure**: Retry the failed operation a configurable number of times, with backoff between each retry.
**Task Tolerance Limits**: Tolerate a configurable number of failures in a task.

We also add the following ways to report errors, along with sufficient context to simplify the debugging process:

**Log Error Context**: The error information along with processing context is logged along with standard application logs.
**Dead Letter Queue**: Produce the original message into a Kafka topic (applicable only to sink connectors).

New **metrics** which will monitor the number of failures, and the behavior of the response handler are added.

The changes proposed here **are backward compatible**. The current behavior in Connect is to kill the task on the first error in any stage. This will remain the default behavior if the connector does not override any of the new configurations which are provided as part of this feature.

Testing: added multiple unit tests to test the retry and tolerance logic.

Author: Arjun Satish <arjun@confluent.io>
Author: Ewen Cheslack-Postava <me@ewencp.org>

Reviewers: Magesh Nandakumar <magesh.n.kumar@gmail.com>, Randall Hauch <rhauch@gmail.com>, Konstantine Karantasis <konstantine@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5065 from wicknicks/KAFKA-6378
2018-05-30 11:39:45 -07:00
Magesh Nandakumar 98094954a2 KAFKA-6776: ConnectRestExtension Interfaces & Rest integration (KIP-285)
This PR provides the implementation for KIP-285 and also a reference implementation for authenticating BasicAuth credentials using JAAS LoginModule

Author: Magesh Nandakumar <magesh.n.kumar@gmail.com>

Reviewers: Randall Hauch <rhauch@gmail.com>, Arjun Satish <wicknicks@users.noreply.github.com>, Konstantine Karantasis <konstantine@confluent.io>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #4931 from mageshn/KIP-285
2018-05-29 21:35:22 -07:00
Randall Hauch fffb9c5b5c KAFKA-6913: Add Connect converters and header converters for short, integer, long, float, and double (KIP-305)
*[KIP-305](https://cwiki.apache.org/confluence/display/KAFKA/KIP-305%3A+Add+Connect+primitive+number+converters) has been approved.*

Added converters and header converters for the primitive number types for which Kafka already had serializers and deserializers. All extend a common base class, `NumberConverter`, that encapsulates most of the shared functionality. Unit tests were added to check the basic functionality.

These classes are not used by any other Connect code, and must be explicitly used in Connect workers and connectors.

Author: Randall Hauch <rhauch@gmail.com>

Reviewers: Arjun Satish <wicknicks@users.noreply.github.com>, Magesh Nandakumar <magesh.n.kumar@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5034 from rhauch/kafka-6913
2018-05-29 16:37:33 -07:00
Chris Egerton a64ab91238 KAFKA-5540: Deprecate internal converter configs (KIP-174)
Implementation of [KIP-174](https://cwiki.apache.org/confluence/display/KAFKA/KIP-174+-+Deprecate+and+remove+internal+converter+configs+in+WorkerConfig)

Configuration properties 'internal.key.converter' and 'internal.value.converter'
are deprecated, and default to org.apache.kafka.connect.json.JsonConverter.

Warnings are logged if values are specified for either, or if properties that
appear to configure instances of internal converters (i.e., ones prefixed with
either 'internal.key.converter.' or 'internal.value.converter.') are given.

The property 'schemas.enable' is also defaulted to false for internal
JsonConverter instances (both for keys and values) if it isn't specified.

Documentation and code have also been updated with deprecation notices and
annotations, respectively.

Unit tests have been updated in `PluginsTest` to account for the new defaults for `schemas.enable` for internal key/value converters, and to ensure that (for the time being), internal key/value converters are still configurable despite being deprecated.

Author: Chris Egerton <chrise@confluent.io>
Author: Ewen Cheslack-Postava <me@ewencp.org>

Reviewers: Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #4693 from C0urante/kafka-5540
2018-05-29 16:22:47 -07:00
John Roesler c470ff70d3 KAFKA-5697; Implement new consumer poll API from KIP-266 (#4855)
Add the new stricter-timeout version of `poll` proposed in KIP-266.

The pre-existing variant `poll(long timeout)` would block indefinitely for metadata
updates if they were needed, then it would issue a fetch and poll for `timeout` ms 
for new records. The initial indefinite metadata block caused applications to become
stuck when the brokers became unavailable. The existence of the timeout parameter
made the indefinite block especially unintuitive.

This PR adds `poll(Duration timeout)` with the semantics:
1. iff a metadata update is needed:
    1. send (asynchronous) metadata requests
    2. poll for metadata responses (counts against timeout)
        - if no response within timeout, **return an empty collection immediately**
2. if there is fetch data available, **return it immediately**
3. if there is no fetch request in flight, send fetch requests
4. poll for fetch responses (counts against timeout)
    - if no response within timeout, **return an empty collection** (leaving async fetch request for the next poll)
    - if we get a response, **return the response**

The old method, `poll(long timeout)` is deprecated, but we do not change its semantics, so it remains:
1. iff a metadata update is needed:
    1. send (asynchronous) metadata requests
    2. poll for metadata responses *indefinitely until we get it*
2. if there is fetch data available, **return it immediately**
3. if there is no fetch request in flight, send fetch requests
4. poll for fetch responses (counts against timeout)
    - if no response within timeout, **return an empty collection** (leaving async fetch request for the next poll)
    - if we get a response, **return the response**

One notable usage is prohibited by the new `poll`: previously, you could call `poll(0)` to block for metadata updates, for example to initialize the client, supposedly without fetching records. Note, though, that this behavior is not according to any contract, and there is no guarantee that `poll(0)` won't return records the first time it's called. Therefore, it has always been unsafe to ignore the response.
2018-05-26 11:50:51 -07:00
Jeremy Custenborder 7ecfbab92f KAFKA-5807 - Check Connector.config() and Transformation.config() returns a valid ConfigDef
Little back story on this. Was helping a user over email. This could be much easier to debug if we assume that the connector developer might not return valid configs. For example Intellij will generate a stub that returns a null. This was the case that inspired this JIRA.

Author: Jeremy Custenborder <jcustenborder@gmail.com>

Reviewers: Jason Gustafson <jason@confluent.io>, Randall Hauch <rhauch@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #3762 from jcustenborder/KAFKA-5807
2018-05-22 13:53:15 -07:00
Ismael Juma a30ecc6755 MINOR: Remove o.a.kafka.common.utils.Base64 and IS_JAVA8_COMPATIBLE
We no longer need them since we now require Java 8.

Author: Ismael Juma <ismael@juma.me.uk>

Reviewers: Andras Beni <andrasbeni@cloudera.com>, Manikumar Reddy O <manikumar.reddy@gmail.com>, Dong Lin <lindong28@gmail.com>

Closes #5049 from ijuma/remove-base64
2018-05-22 09:57:39 -07:00