kafka.git
2 days agoKAFKA-2334; Guard against non-monotonic offsets in the client (#5991) trunk
David Arthur [Fri, 14 Dec 2018 21:53:03 +0000 (16:53 -0500)] 
KAFKA-2334; Guard against non-monotonic offsets in the client (#5991)

After a recent leader election, the leaders high-water mark might lag behind the offset at the beginning of the new epoch (as well as the previous leader's HW). This can lead to offsets going backwards from a client perspective, which is confusing and leads to strange behavior in some clients.

This change causes Partition#fetchOffsetForTimestamp to throw an exception to indicate the offsets are not yet available from the leader. For new clients, a new OFFSET_NOT_AVAILABLE error is added. For existing clients, a LEADER_NOT_AVAILABLE is thrown.

This is an implementation of [KIP-207](https://cwiki.apache.org/confluence/display/KAFKA/KIP-207%3A+Offsets+returned+by+ListOffsetsResponse+should+be+monotonically+increasing+even+during+a+partition+leader+change).

Reviewers: Colin Patrick McCabe <colin@cmccabe.xyz>, Dhruvil Shah <dhruvil@confluent.io>, Jason Gustafson <jason@confluent.io>

2 days agoMINOR: Replace tbd with the actual link for out-of-ordering data (#6035)
Guozhang Wang [Fri, 14 Dec 2018 17:37:43 +0000 (09:37 -0800)] 
MINOR: Replace tbd with the actual link for out-of-ordering data (#6035)

Reviewers: Jason Gustafson <jason@confluent.io>

2 days agoMINOR: Update documentation for internal changelog when using table(). (#6021)
cwildman [Fri, 14 Dec 2018 04:33:25 +0000 (05:33 +0100)] 
MINOR: Update documentation for internal changelog when using table(). (#6021)

Updating the documentation for table operation because I believe it is incorrect.

In PR #5163 the table operation stopped disabling the changelog topic by default and instead moved that optimization to a configuration that is not enabled by default. This PR updates the documentation to reflect the change in behavior and point to the new configuration for optimization.

Reviewers: Bill Bejeck <bbejeck@gmail.com>, Guozhang Wang <wangguoz@gmail.com>

2 days agoMINOR:Start processor inside verify message (#6029)
Bill Bejeck [Fri, 14 Dec 2018 04:30:04 +0000 (23:30 -0500)] 
MINOR:Start processor inside verify message (#6029)

This PR fixes a flaky system test.

I ran six runs of branch builder, and each run was parameterized to repeat the test 25 times for a total of 150 runs. All test runs passed.

https://jenkins.confluent.io/job/system-test-kafka-branch-builder/2122/
https://jenkins.confluent.io/job/system-test-kafka-branch-builder/2123/
https://jenkins.confluent.io/job/system-test-kafka-branch-builder/2124/
https://jenkins.confluent.io/job/system-test-kafka-branch-builder/2128/
https://jenkins.confluent.io/job/system-test-kafka-branch-builder/2129/
https://jenkins.confluent.io/job/system-test-kafka-branch-builder/2130/

Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <guozhang@confluent.io>, John Roesler <vvcephei@users.noreply.github.com>

2 days agoFix the missing ApiUtils tests in streams module. (#6003)
Srinivas Reddy [Fri, 14 Dec 2018 03:31:58 +0000 (11:31 +0800)] 
Fix the missing ApiUtils tests in streams module. (#6003)

Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <guozhang@confluent.io>, Bill Bejeck <bill@confluent.io>

3 days agoKAFKA-7655 Metadata spamming requests from Kafka Streams under some circumstances...
Pasquale Vazzana [Thu, 13 Dec 2018 15:40:39 +0000 (15:40 +0000)] 
KAFKA-7655 Metadata spamming requests from Kafka Streams under some circumstances, potential DOS (#5929)

Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <guozhang@confluent.io>

4 days agoKAFKA-7223: document suppression buffer metrics (#6024)
John Roesler [Thu, 13 Dec 2018 02:22:54 +0000 (20:22 -0600)] 
KAFKA-7223: document suppression buffer metrics (#6024)

Document the new metrics added in #5795

Reviewers: Bill Bejeck <bill@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

4 days agoKAFKA-7712; Remove channel from Selector before propagating exception (#6023)
Rajini Sivaram [Wed, 12 Dec 2018 18:50:56 +0000 (18:50 +0000)] 
KAFKA-7712; Remove channel from Selector before propagating exception (#6023)

Ensure that channel and selection keys are removed from `Selector` collections before propagating connect exceptions. They are currently cleared on the next `poll()`, but we can't ensure that callers (NetworkClient for example) wont try to connect again before the next `poll` and hence we should clear the collections before re-throwing exceptions from `connect()`.

Reviewers: Jason Gustafson <jason@confluent.io>

4 days agoKAFKA-7705; Fix and simplify producer config in javadoc example (#6000)
hackerwin7 [Wed, 12 Dec 2018 16:37:09 +0000 (00:37 +0800)] 
KAFKA-7705; Fix and simplify producer config in javadoc example (#6000)

The example in the producer's javadoc contained an inconsistent value for `delivery.timeout.ms`. This patch removes the inconsistent config and several unnecessary overrides in order to simplify the example.

Reviewers: huxi <huxi_2b@hotmail.com>, Ismael Juma <ismael@juma.me.uk>, Jason Gustafson <jason@confluent.io>

4 days agoMINOR: Add unit for max latency in ProducerPerformance output (#6014)
Arjun Satish [Wed, 12 Dec 2018 16:28:58 +0000 (08:28 -0800)] 
MINOR: Add unit for max latency in ProducerPerformance output (#6014)

Reviewers: Jason Gustafson <jason@confluent.io>

5 days agoMINOR: improve Streams checkstyle and code cleanup (#5954)
Matthias J. Sax [Tue, 11 Dec 2018 09:54:41 +0000 (01:54 -0800)] 
MINOR: improve Streams checkstyle and code cleanup (#5954)

Reviewers: Guozhang Wang <guozhang@confluent.io>, Nikolay Izhikov <nIzhikov@gmail.com>, Ismael Juma <ismael@confluent.io>, Bill Bejeck <bill@confluent.io>

5 days agoKAFKA-6970: All standard state stores guarded with read only wrapper (#6016)
Nikolay [Tue, 11 Dec 2018 09:44:18 +0000 (12:44 +0300)] 
KAFKA-6970: All standard state stores guarded with read only wrapper (#6016)

Reviewer: Matthias J. Sax <matthias@confluent.io>, John Roesler <john@confluent.io>

5 days agoKAFKA-7443: OffsetOutOfRangeException in restoring state store from changelog topic...
linyli001 [Tue, 11 Dec 2018 08:40:18 +0000 (16:40 +0800)] 
KAFKA-7443: OffsetOutOfRangeException in restoring state store from changelog topic when start offset of local checkpoint is smaller than that of changelog topic (#5946)

Reviewer: Matthias J. Sax <matthias@confluent.io>, John Roesler <john@confluent.io>

6 days agoKAFKA-7610; Proactively timeout new group members if rebalance is delayed (#5962)
Jason Gustafson [Mon, 10 Dec 2018 22:32:29 +0000 (14:32 -0800)] 
KAFKA-7610; Proactively timeout new group members if rebalance is delayed (#5962)

When a consumer first joins a group, it doesn't have an assigned memberId. If the rebalance is delayed for some reason, the client may disconnect after a request timeout and retry. Since the client had not received its memberId, then we do not have a way to detect the retry and expire the previously generated member id. This can lead to unbounded growth in the size of the group until the rebalance has completed.

This patch fixes the problem by proactively completing all JoinGroup requests for new members after a timeout of 5 minutes. If the client is still around, we expect it to retry.

Reviewers: Stanislav Kozlovski <stanislav_kozlovski@outlook.com>, Boyang Chen <bchen11@outlook.com>, Guozhang Wang <wangguoz@gmail.com>

6 days agoKAFKA-7549; Old ProduceRequest with zstd compression does not return error to client...
Lee Dongjin [Mon, 10 Dec 2018 17:45:18 +0000 (02:45 +0900)] 
KAFKA-7549; Old ProduceRequest with zstd compression does not return error to client (#5925)

Older versions of the Produce API should return an error if zstd is used. This validation existed, but it was done during request parsing, which means that instead of returning an error code, the broker disconnected. This patch fixes the issue by moving the validation outside of the parsing logic. It also fixes several other record validations which had the same problem.

Reviewers: Jason Gustafson <jason@confluent.io>

7 days agoKAFKA-6036: Follow-up; cleanup sendOldValues logic ForwardingCacheFlushListener ...
Guozhang Wang [Sun, 9 Dec 2018 23:33:17 +0000 (15:33 -0800)] 
KAFKA-6036: Follow-up; cleanup sendOldValues logic ForwardingCacheFlushListener  (#6017)

This is a follow-up PR from the previous PR #5779, where KTabeSource always get old values from the store even if sendOldValues. It gets me to make a pass over all the KTable/KStreamXXX processor to push the sendOldValues at the callers in order to avoid unnecessary store reads.

More details: ForwardingCacheFlushListener and TupleForwarder both need sendOldValues as parameters.
a. For ForwardingCacheFlushListener it is not needed at all, since its callers XXXCachedStore already use the sendOldValues values passed from TupleForwarder to avoid getting old values from underlying stores.
b. For TupleForwarder, it actually only need to pass the boolean flag to the cached store; and then it does not need to keep it as its own variable since the cached store already respects the boolean to pass null or the actual value..

The only other minor bug I found from the pass in on KTableJoinMerge, where we always pass old values and ignores sendOldValues.

Reviewers: Matthias J. Sax <mjsax@apache.org>

7 days agoKAFKA-6036: Local Materialization for Source KTable (#5779)
Guozhang Wang [Sun, 9 Dec 2018 06:49:48 +0000 (22:49 -0800)] 
KAFKA-6036: Local Materialization for Source KTable (#5779)

Refactor the materialization for source KTables in the way that:

If Materialized.as(queryableName) is specified, materialize;
If the downstream operator requires to fetch from this KTable via ValueGetters, materialize;
If the downstream operator requires to send old values, materialize.
Otherwise do not materialize the KTable. E.g. builder.table("topic").filter().toStream().to("topic") would not create any state stores.

There's a couple of minor changes along with PR as well:

KTableImpl's queryableStoreName and isQueryable are merged into queryableStoreName only, and if it is null it means not queryable. As long as it is not null, it should be queryable (i.e. internally generated names will not be used any more).
To achieve this, splitted MaterializedInternal.storeName() and MaterializedInternal.queryableName(). The former can be internally generated and will not be exposed to users. QueryableName can be modified to set to the internal store name if we decide to materialize it during the DSL parsing / physical topology generation phase. And only if queryableName is specified the corresponding KTable is determined to be materialized.

Found some overlapping unit tests among KTableImplTest, and KTableXXTest, removed them.

There are a few typing bugs found along the way, fixed them as well.

-----------------------

This PR is an illustration of experimenting a poc towards logical materializations.

Today we've logically materialized the KTable for filter / mapValues / transformValues if queryableName is not specified via Materialized, but whenever users specify queryableName we will still always materialize. My original goal is to also consider logically materialize for queryable stores, but when implementing it via a wrapped store to apply the transformations on the fly I realized it is tougher than I thought, because we not only need to support fetch or get, but also needs to support range queries, approximateNumEntries, and isOpen etc as well, which are not efficient to support. So in the end I'd suggest we still stick with the rule of always materializing if queryableName is specified, and only consider logical materialization otherwise.

Reviewers: Bill Bejeck <bill@confluent.io>, Matthias J. Sax <mjsax@apache.org>

7 days agoMINOR: fix checkpoint write failure warning log (#6008)
John Roesler [Sun, 9 Dec 2018 03:00:57 +0000 (21:00 -0600)] 
MINOR: fix checkpoint write failure warning log (#6008)

We saw a log statement in which the cause of the failure to write a checkpoint was not properly logged.
This change logs the exception properly and also verifies the log message.

Reviewers: Bill Bejeck <bill@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

8 days agoMINOR: Catch NoRecordsException in testCommaSeparatedRegex() test (#5944)
Stanislav Kozlovski [Sat, 8 Dec 2018 17:43:43 +0000 (17:43 +0000)] 
MINOR: Catch NoRecordsException in testCommaSeparatedRegex() test (#5944)

This test sometimes fails with

```
kafka.tools.MirrorMaker$NoRecordsException
at kafka.tools.MirrorMaker$ConsumerWrapper.receive(MirrorMaker.scala:483)
at kafka.tools.MirrorMakerIntegrationTest$$anonfun$testCommaSeparatedRegex$1.apply$mcZ$sp(MirrorMakerIntegrationTest.scala:92)
at kafka.utils.TestUtils$.waitUntilTrue(TestUtils.scala:738)
```

The test should catch `NoRecordsException` instead of `TimeoutException`.

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

8 days agoMINOR: Specify character encoding in NetworkTestUtils (#5965)
Gardner Vickers [Sat, 8 Dec 2018 17:36:14 +0000 (12:36 -0500)] 
MINOR: Specify character encoding in NetworkTestUtils (#5965)

This attempts to address the flaky test `SaslAuthenticatorTest.testCannotReauthenticateWithDifferentPrincipal()`

I was not able to reproduce locally even after 150 test runs in a loop, but given the error message:

```
org.junit.ComparisonFailure: expected:
<[6QBJiMZ6o5AqbNAjDTDjWtQSa4alfuUWsYKIy2tt7dz5heDaWZlz21yr8Gl4uEJkQABQXeEL0UebdpufDb5k8SvReSK6wYwQ9huP-9]> but was:<[????����????OAUTHBEARER]>
```

`????����????` seems to mean invalid UTF-8.

We now specify the charset when writing out and reading in bytes.

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

8 days agoMINOR: Fix failing ConsumeBenchTest:test_multiple_consumers_specified_group_partition...
Stanislav Kozlovski [Sat, 8 Dec 2018 17:27:59 +0000 (17:27 +0000)] 
MINOR: Fix failing ConsumeBenchTest:test_multiple_consumers_specified_group_partitions_should_raise (#6015)

This is the error message we're after:

"You may not specify an explicit partition assignment when using multiple consumers in the same group."

We apparently changed it midway through #5810 and forgot to update the test.

Reviewers: Dhruvil Shah <dhruvil@confluent.io>, Ismael Juma <ismael@juma.me.uk>

10 days agoKAFKA-7709: Fix ConcurrentModificationException when retrieving expired inflight...
Mark Cho [Fri, 7 Dec 2018 00:55:01 +0000 (16:55 -0800)] 
KAFKA-7709: Fix ConcurrentModificationException when retrieving expired inflight batches on multiple partitions. (#6005)

Reviewers: Dhruvil Shah <dhruvil@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

10 days agoKAFKA-7704: MaxLag.Replica metric is reported incorrectly (#5998)
huxi [Thu, 6 Dec 2018 13:46:35 +0000 (21:46 +0800)] 
KAFKA-7704: MaxLag.Replica metric is reported incorrectly (#5998)

On the follower side, for the empty `LogAppendInfo` retrieved from the leader, fetcherLagStats set the wrong lag for fetcherLagStats due to `nextOffset` is zero.

11 days ago KAFKA-7673: Upgrade rocksdb to 5.15.10 (#5985)
Guozhang Wang [Thu, 6 Dec 2018 00:52:25 +0000 (16:52 -0800)] 
 KAFKA-7673: Upgrade rocksdb to 5.15.10 (#5985)

Reviewers: Matthias J. Sax <mjsax@apache.org>

11 days agoKAFKA-6388; Recover from rolling an empty segment that already exists (#5986)
Anna Povzner [Wed, 5 Dec 2018 22:49:19 +0000 (14:49 -0800)] 
KAFKA-6388; Recover from rolling an empty segment that already exists (#5986)

There were several reported incidents where the log is rolled to a new segment with the same base offset as an active segment, causing KafkaException: Trying to roll a new log segment for topic partition X-N with start offset M while it already exists. In the cases we have seen, this happens to an empty log segment where there is long idle time before the next append and somehow we get to a state where offsetIndex.isFull() returns true due to _maxEntries == 0. This PR recovers from this state by deleting and recreating the segment and all of its associated index files.

Reviewers: Jason Gustafson <jason@confluent.io>

11 days agoMINOR: Safe string conversion to avoid NPEs
Cyrus Vafadari [Wed, 5 Dec 2018 21:23:52 +0000 (13:23 -0800)] 
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

11 days agoKAFKA-7678: Avoid NPE when closing the RecordCollector (#5993)
Jonathan Santilli [Wed, 5 Dec 2018 19:48:39 +0000 (20:48 +0100)] 
KAFKA-7678: Avoid NPE when closing the RecordCollector (#5993)

Reviewers: Matthias J. Sax <matthias@confluent.io>, Bill Bejeck <bill@confluent.io>

11 days agoKAFKA-7420: Global store surrounded by read only implementation (#5865)
Nikolay [Wed, 5 Dec 2018 19:25:52 +0000 (22:25 +0300)] 
KAFKA-7420: Global store surrounded by read only implementation (#5865)

Reviewers: Matthias J. Sax <matthias@confluent.io>, Kamal Chandraprakash (@kamalcph), Bill Bejeck <bill@confluent.io>

11 days agoKAFKA-7697: Process DelayedFetch without holding leaderIsrUpdateLock (#5999)
Rajini Sivaram [Wed, 5 Dec 2018 09:05:26 +0000 (09:05 +0000)] 
KAFKA-7697: Process DelayedFetch without holding leaderIsrUpdateLock (#5999)

Delayed fetch operations acquire leaderIsrUpdate read lock of one or more Partitions from the fetch request when attempting to complete the fetch operation. While appending new records, complete fetch requests after releasing leaderIsrUpdate of the Partition to which records were appended to avoid deadlocks in request handler threads.

Reviewers: Jason Gustafson <jason@confluent.io>, Jun Rao <junrao@gmail.com>

12 days agoMINOR: Improve GlobalKTable docs (#5996)
Matthias J. Sax [Wed, 5 Dec 2018 00:21:18 +0000 (16:21 -0800)] 
MINOR: Improve GlobalKTable docs (#5996)

Reviewers: Jim Galasyn, Michael G. Noll, John Roesler, Bill Bejeck, Guozhang Wang

12 days agoKAFKA-7687; Print batch level information in DumpLogSegments when deep iterating...
huxi [Tue, 4 Dec 2018 17:04:39 +0000 (01:04 +0800)] 
KAFKA-7687; Print batch level information in DumpLogSegments when deep iterating (#5976)

DumpLogSegments should print batch level information when deep-iteration is specified.

Reviewers: Jason Gustafson <jason@confluent.io>

12 days agoKAFKA-7702: Fix matching of prefixed ACLs to match single char prefix (#5994)
Rajini Sivaram [Tue, 4 Dec 2018 09:44:11 +0000 (09:44 +0000)] 
KAFKA-7702: Fix matching of prefixed ACLs to match single char prefix (#5994)

Reviewers: Jun Rao <junrao@gmail.com>

12 days agoKAFKA-7446: Fix the duration and instant validation messages. (#5930)
Srinivas Reddy [Tue, 4 Dec 2018 06:59:54 +0000 (14:59 +0800)] 
KAFKA-7446: Fix the duration and instant validation messages. (#5930)

Changes made as part of this commit.
 - Improved error message for better readability at millis validation utility
 - Corrected java documentation on `AdvanceInterval` check.
 - Added caller specific prefix text to make error message more clear to developers/users.

Reviewers: Matthias J. Sax <matthias@confluent.io>, Bill Bejeck <bill@confluent.io>, Jacek Laskowski <jacek@japila.pl>

12 days agoMINOR: Update command line options in Authorization and ACLs documentation chapter...
Jakub Scholz [Tue, 4 Dec 2018 04:11:08 +0000 (05:11 +0100)] 
MINOR: Update command line options in Authorization and ACLs documentation chapter (#5995)

13 days agoMINOR: Adding system test for named repartition topics (#5913)
Bill Bejeck [Mon, 3 Dec 2018 20:37:31 +0000 (15:37 -0500)] 
MINOR: Adding system test for named repartition topics (#5913)

This is a system test for doing a rolling upgrade of a topology with a named repartition topic.

1. An initial Kafka Streams application is started on 3 nodes. The topology has one operation forcing a repartition and the repartition topic is explicitly named.
2. Each node is started and processing of data is validated
3. Then one node is stopped (full stop is verified)
4. A property is set signaling the node to add operations to the topology before the repartition node which forces a renumbering of all operators (except repartition node)
5. Restart the node and confirm processing records
6. Repeat the steps for the other 2 nodes completing the rolling upgrade

I ran two runs of the system test with 25 repeats in each run for a total of 50 test runs.
All test runs passed

Reviewers: John Roesler <john@confluent.io>, Matthias J. Sax <matthias@confluent.io>

13 days agoMINOR: improve QueryableStateIntegrationTest (#5987)
Matthias J. Sax [Mon, 3 Dec 2018 06:11:04 +0000 (22:11 -0800)] 
MINOR: improve QueryableStateIntegrationTest (#5987)

Fix test Comparators plus Java8 cleanup

Reviewers: Guozhang Wang <wangguoz@gmail.com>

13 days agoKAFKA-7235: Detect outdated control requests and bounced brokers using broker generat...
Zhanxiang (Patrick) Huang [Mon, 3 Dec 2018 06:00:51 +0000 (22:00 -0800)] 
KAFKA-7235: Detect outdated control requests and bounced brokers using broker generation (#5821)

* KAFKA-7235: Detect outdated control requests and bounced brokers using broker generation

* Add broker_epoch in controlled shutdown request

* Move broker epoch check into controller for ControlledShutdownRequest

* Refactor schema definition for controler requests/responses

* Address comments

* Address comments

* Address comments

* Send back STALE_BROKER_EPOCH error in ControlledShutdown response

* Fix build issue

* Address comments

* Address comments

* Address comments

* Address comments

* Fix tests after rebase

* Address comments

* Address comments

13 days agoKAFKA-4544: Add system tests for delegation token based authentication
Attila Sasvari [Mon, 3 Dec 2018 05:58:36 +0000 (11:28 +0530)] 
KAFKA-4544: Add system tests for delegation token based authentication

This change adds some basic system tests for delegation token based authentication:
- basic delegation token creation
- producing with a delegation token
- consuming with a delegation token
- expiring a delegation token
- producing with an expired delegation token

New files:
- delegation_tokens.py: a wrapper around kafka-delegation-tokens.sh  - executed in container where a secure Broker is running (taking advantage of automatic cleanup)
- delegation_tokens_test.py: basic test to validate the lifecycle of a delegation token

Changes were made in the following file to extend their functionality:
- config_property was updated to be able to configure Kafka brokers with delegation token related settings
- jaas.conf template because a broker needs to support multiple login modules when delegation tokens are used
- consule-consumer and verifiable_producer to override KAFKA_OPTS (to specify custom jaas.conf) and the client properties (to authenticate with delegation token).

Author: Attila Sasvari <asasvari@apache.org>

Reviewers: Reviewers: Viktor Somogyi <viktorsomogyi@gmail.com>, Andras Katona <41361962+akatona84@users.noreply.github.com>, Manikumar Reddy <manikumar.reddy@gmail.com>

Closes #5660 from asasvari/KAFKA-4544

13 days agoMINOR: hygene cleanup in TransactionManagerTest (#5951)
Viktor Somogyi [Mon, 3 Dec 2018 05:32:05 +0000 (06:32 +0100)] 
MINOR: hygene cleanup in TransactionManagerTest (#5951)

Reviewers: Andras Katona <41361962+akatona84@users.noreply.github.com>, Manikumar Reddy <manikumar.reddy@gmail.com>

2 weeks agoMINOR: improve state directory test (#5961)
Matthias J. Sax [Fri, 30 Nov 2018 18:31:07 +0000 (10:31 -0800)] 
MINOR: improve state directory test (#5961)

Reviewers: Bill Bejeck <bill@confluent.io>, Kamal Chandraprakash (@kamalcph), Guozhang Wang <guozhang@confluent.io>

2 weeks agoKAFKA-7617: Add authorization primitives to security page
Viktor Somogyi-Vass [Fri, 30 Nov 2018 09:12:34 +0000 (14:42 +0530)] 
KAFKA-7617: Add authorization primitives to security page

This is a security page improvement that adds documentation about Kafka authorization primitives to the security page.

Author: Viktor Somogyi-Vass <viktorsomogyi@gmail.com>
Author: Viktor Somogyi <viktorsomogyi@gmail.com>
Author: Manikumar Reddy <manikumar.reddy@gmail.com>

Reviewers: Stanislav Kozlovski <stanislav_kozlovski@outlook.com>, Srinivas <srinivas96alluri@gmail.com>, Manikumar Reddy <manikumar.reddy@gmail.com>, Mickael Maison <mickael.maison@gmail.com>

Closes #5906 from viktorsomogyi/security-page-improvement

2 weeks agoKAFKA-7551: Refactor to create producer & consumer in the worker
Magesh Nandakumar [Fri, 30 Nov 2018 07:38:50 +0000 (23:38 -0800)] 
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

2 weeks agoMINOR: Add logging to Connect SMTs
Cyrus Vafadari [Fri, 30 Nov 2018 06:29:50 +0000 (22:29 -0800)] 
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

2 weeks agoMINOR: Fix handling of dummy record in EndToEndLatency tool
Anna Povzner [Fri, 30 Nov 2018 06:21:20 +0000 (22:21 -0800)] 
MINOR: Fix handling of dummy record in EndToEndLatency tool

EndToEndLatency tool produces a dummy record in case the topic does not exist. This behavior was introduced in this PR https://github.com/apache/kafka/pull/5319  as part of updating the tool to use latest consumer API. However, if we run the tool with producer acks == 1, the high watermark may not be updated before we reset consumer offsets to latest. In rare cases when this happens, the tool will throw an exception in the for loop where the consumer will unexpectedly consume the dummy record. As a result, we occasionally see Benchmark.test_end_to_end_latency system test failures.

This PR checks if topic exists, and creates the topic using AdminClient if it does not exist.

Author: Anna Povzner <anna@confluent.io>

Reviewers: Ismael Juma <ismael@juma.me.uk>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5950 from apovzner/fix-EndToEndLatency

2 weeks agoKAFKA-7259; Remove deprecated ZKUtils usage from ZkSecurityMigrator
Manikumar Reddy [Fri, 30 Nov 2018 05:41:25 +0000 (11:11 +0530)] 
KAFKA-7259; Remove deprecated ZKUtils usage from ZkSecurityMigrator

- Remove ZKUtils usage from various tests

Author: Manikumar Reddy <manikumar.reddy@gmail.com>

Reviewers: Sriharsha Chintalapani <sriharsha@apache.org>, Ismael Juma <ismael@juma.me.uk>, Satish Duggana <satishd@apache.org>, Jun Rao <junrao@gmail.com>, Ryanne Dolan <ryannedolan@gmail.com>

Closes #5480 from omkreddy/zkutils

2 weeks agoKAFKA-7660: Fix child sensor memory leak (#5974)
John Roesler [Fri, 30 Nov 2018 04:10:24 +0000 (22:10 -0600)] 
KAFKA-7660: Fix child sensor memory leak (#5974)

A heap dump provided by Patrik Kleindl in https://issues.apache.org/jira/browse/KAFKA-7660 identifies the childrenSensors map in Metrics as keeping references to sensors alive after they have been removed.

This PR fixes it and adds a test to be sure.

Reviewers: Jason Gustafson <jason@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

2 weeks agoMINOR: Bump Gradle version to 5.0 (#5964)
Colin Hicks [Fri, 30 Nov 2018 00:04:39 +0000 (19:04 -0500)] 
MINOR: Bump Gradle version to 5.0 (#5964)

Gradle 5.0 was released on 26 November. See the release notes for enhancements and fixes: https://docs.gradle.org/5.0/release-notes.html.

A notable bugfix ensures that Javadoc artifacts are cleaned between builds.

The upgraded wrapper was spot-checked against the commands in the README and the
README was updated not to use removed system property `-Dtest.single`.

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

2 weeks agoKAFKA-7660: fix parentSensors memory leak (#5953)
John Roesler [Thu, 29 Nov 2018 23:09:50 +0000 (17:09 -0600)] 
KAFKA-7660: fix parentSensors memory leak (#5953)

In StreamsMetricsImpl, the parentSensors map was keeping references to Sensors after the sensors themselves had been removed.

Reviewers: Matthias J. Sax <matthias@confluent.io>, Bill Bejeck <bill@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

2 weeks agoMINOR: Fix ProducerPerformance bug when numRecords > Integer.MAX (#5956)
seayoun [Thu, 29 Nov 2018 06:48:56 +0000 (14:48 +0800)] 
MINOR: Fix ProducerPerformance bug when numRecords > Integer.MAX (#5956)

Current code will fall into non-stop loop and send more message to broker, and Stat in PerfCallback method record will throw ArrayIndexOutOfBoundsException

2 weeks agoForward topic from console consumer to deserializer (#5704)
Mathieu Chataigner [Thu, 29 Nov 2018 02:23:30 +0000 (03:23 +0100)] 
Forward topic from console consumer to deserializer (#5704)

Some deserializer needs the topic name to be able to correctly deserialize the payload of the message.
Console consumer works great with Deserializer<String> however it calls deserializer with topic set as null.
This breaks the API and the topic information is available in the ConsumerRecord.

Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>, Chia-Ping Tsai <chia7712@gmail.com>, Gardner Vickers <gardner@vickers.me>, Jun Rao <junrao@gmail.com>

2 weeks agoKAFKA-7671: Stream-Global Table join should not reset repartition flag (#5959)
Bill Bejeck [Thu, 29 Nov 2018 02:15:26 +0000 (21:15 -0500)] 
KAFKA-7671: Stream-Global Table join should not reset repartition flag (#5959)

This PR fixes an issue reported from a user. When we join a KStream with a GlobalKTable we should not reset the repartition flag as the stream may have previously changed its key, and the resulting stream could be used in an aggregation operation or join with another stream which may require a repartition for correct results.

I've added a test which fails without the fix.

Reviewers: John Roesler <john@confluent.io>, Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

2 weeks agoMINOR: Support long maxMessages in Trogdor consume/produce bench workers (#5957)
Stanislav Kozlovski [Wed, 28 Nov 2018 17:13:21 +0000 (17:13 +0000)] 
MINOR: Support long maxMessages in Trogdor consume/produce bench workers (#5957)

Reivewers: Colin McCabe <cmccabe@apache.org>

2 weeks agoKAFKA-7037: Improve the topic command description of `--topic` option
Vahid Hashemian [Wed, 28 Nov 2018 10:46:45 +0000 (16:16 +0530)] 
KAFKA-7037: Improve the topic command description of `--topic` option

Author: Vahid Hashemian <vahidhashemian@us.ibm.com>

Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>

Closes #5193 from vahidhashemian/KAFKA-7037

2 weeks agoMINOR: Improve docs by adding ToC links to Monitoring
lu.kevin@berkeley.edu [Wed, 28 Nov 2018 06:43:26 +0000 (12:13 +0530)] 
MINOR: Improve docs by adding ToC links to Monitoring

My top 2 reasons for visiting the Kafka docs are to:
- View configurations
- View metrics

This PR aims to improve the user experience for viewing metrics:
- Add href links to the `Monitoring` section of the Table of Contents so users do not need to scroll or Ctrl/Cmd-F to find specific metric details (Monitoring section has grown large as more component & metrics are added)

Author: lu.kevin@berkeley.edu <kelu@paypal.com>

Reviewers: Viktor Somogyi <viktorsomogyi@gmail.com>, Manikumar Reddy <manikumar.reddy@gmail.com>

Closes #5511 from KevinLiLu/feature/minor-improve-docs

2 weeks agoKAFKA-7620: Fix restart logic for TTLs in WorkerConfigTransformer
Robert Yokota [Wed, 28 Nov 2018 06:01:21 +0000 (22:01 -0800)] 
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

2 weeks agoKAFKA-7389: Enable spotBugs with Java 11 and disable false positive warnings (#5943)
Ismael Juma [Tue, 27 Nov 2018 22:40:17 +0000 (14:40 -0800)] 
KAFKA-7389: Enable spotBugs with Java 11 and disable false positive warnings (#5943)

See https://github.com/spotbugs/spotbugs/issues/756 for details on
the false positives affecting try with resources. An example is:

> RCN | Nullcheck of fc at line 629 of value previously dereferenced in
> org.apache.kafka.common.utils.Utils.readFileAsString(String, Charset)

Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>

2 weeks agoKAFKA-7367: Streams should not create state store directories unless they are needed...
Kamal Chandraprakash [Tue, 27 Nov 2018 21:09:44 +0000 (02:39 +0530)] 
KAFKA-7367: Streams should not create state store directories unless they are needed (#5696)

* KAFKA-7367: Ensure stateless topologies don't require disk access

* KAFKA-7367: Streams should not create state store directories unless they are needed.

* Addressed the review comments.

* Addressed the review-2 comments.

* Fixed FileAlreadyExistsException

* Addressed the review-3 comments.

* Resolved the conflicts.

2 weeks agoMINOR: Add system test for optimization upgrades (#5912)
Bill Bejeck [Tue, 27 Nov 2018 21:07:34 +0000 (16:07 -0500)] 
MINOR: Add system test for optimization upgrades (#5912)

This is a new system test testing for optimizing an existing topology. This test takes the following steps

1. Start a Kafka Streams application that uses a selectKey then performs 3 groupByKey() operations and 1 join creating four repartition topics
2. Verify all instances start and process data
3. Stop all instances and verify stopped
4. For each stopped instance update the config for TOPOLOGY_OPTIMIZATION to all then restart the instance and verify the instance has started successfully also verifying Kafka Streams reduced the number of repartition topics from 4 to 1
5. Verify that each instance is processing data from the aggregation, reduce, and join operation
Stop all instances and verify the shut down is complete.
6. For testing I ran two passes of the system test with 25 repeats for a total of 50 test runs.

All test runs passed

Reviewers: Matthias J. Sax <matthias@confluent.io>, Bill Bejeck <bill@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

2 weeks agoKAFKA-7223: Suppression Buffer Metrics (#5795)
John Roesler [Tue, 27 Nov 2018 20:57:04 +0000 (14:57 -0600)] 
KAFKA-7223: Suppression Buffer Metrics (#5795)

Add the final batch of metrics from KIP-328

Reviewers: Matthias J. Sax <matthias@confluent.io>, Bill Bejeck <bill@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

2 weeks agoKAFKA-7597: Add transaction support to ProduceBenchWorker (#5885)
Stanislav Kozlovski [Tue, 27 Nov 2018 20:49:53 +0000 (20:49 +0000)] 
KAFKA-7597: Add transaction support to ProduceBenchWorker (#5885)

KAFKA-7597: Add configurable transaction support to ProduceBenchWorker.  In order to get support for serializing Optional<> types to JSON, add a new library: jackson-datatype-jdk8. Once Jackson 3 comes out, this library will not be needed.

Reviewers: Colin McCabe <cmccabe@apache.org>, Ismael Juma <ismael@juma.me.uk>

2 weeks agoTrogdor: Add Task State filter to /coordinator/tasks endpoint (#5907)
Stanislav Kozlovski [Tue, 27 Nov 2018 00:07:15 +0000 (16:07 -0800)] 
Trogdor: Add Task State filter to /coordinator/tasks endpoint (#5907)

Reviewers: Colin McCabe <cmccabe@apache.org>

3 weeks agoMINOR: Refactor code for restoring tasks (#5768)
Matthias J. Sax [Fri, 23 Nov 2018 20:32:37 +0000 (12:32 -0800)] 
MINOR: Refactor code for restoring tasks (#5768)

Reviewers: Guozhang Wang <guozhang@confluent.io>, Bill Bejeck <bill@confluent.io>

3 weeks agoMINOR: Various javadoc improvement in clients and connect (#5878)
Mickael Maison [Fri, 23 Nov 2018 05:20:30 +0000 (05:20 +0000)] 
MINOR: Various javadoc improvement in clients and connect (#5878)

Fixed formatting issues and added links in a few classes

3 weeks agoMINOR: fix bootstrap-server typo in ReassignPartitionsCommand (#5941)
Viktor Somogyi [Thu, 22 Nov 2018 16:57:57 +0000 (17:57 +0100)] 
MINOR: fix bootstrap-server typo in ReassignPartitionsCommand (#5941)

3 weeks agoKAFKA-7418: Add the missing '--help' option to Kafka commands (KIP-374)
Srinivas Reddy [Thu, 22 Nov 2018 11:42:34 +0000 (17:12 +0530)] 
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

3 weeks agoMINOR: increase system test kafka start timeout (#5934)
John Roesler [Wed, 21 Nov 2018 19:49:36 +0000 (13:49 -0600)] 
MINOR: increase system test kafka start timeout (#5934)

The Kafka Streams system tests fail with some regularity due to a timeout starting the broker.

The initial start is quite quick, but many of our tests involve stopping and restarting nodes with data already loaded, and also while processing is ongoing.

Under these conditions, it seems to be normal for the broker to take about 25 seconds to start, which makes the 30 second timeout pretty close for comfort.
I have seen many test failures in which the broker successfully started within a couple of seconds after the tests timed out and already initiated the failure/shut-down sequence.

Reviewers: Guozhang Wang <wangguoz@gmail.com>

3 weeks agoMINOR: Update docs with out-dated context.schedule(...) examples (#5924)
cadonna [Wed, 21 Nov 2018 17:48:01 +0000 (18:48 +0100)] 
MINOR: Update docs with out-dated context.schedule(...) examples (#5924)

Reviewers: Matthias J. Sax <matthias@confluent.io>, Bill Bejeck <bill@confluent.io>

3 weeks agoKAFKA-7528: Standardize on Min/Avg/Max Kafka metrics' default value - NaN (#5908)
Stanislav Kozlovski [Tue, 20 Nov 2018 23:54:24 +0000 (23:54 +0000)] 
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>

3 weeks agoKAFKA-7536: Initialize TopologyTestDriver with non-null topic (#5923)
Guozhang Wang [Tue, 20 Nov 2018 22:39:12 +0000 (14:39 -0800)] 
KAFKA-7536: Initialize TopologyTestDriver with non-null topic (#5923)

In TopologyTestDriver constructor set non-null topic; and in unit test intentionally turn on caching to verify this case.

Reviewers: Bill Bejeck <bill@confluent.io>, Matthias J. Sax <matthias@confluent.io>

3 weeks agoKAFKA-7616; Make MockConsumer only add entries to the partition map returned by poll...
Stig Rohde Døssing [Tue, 20 Nov 2018 17:46:21 +0000 (23:16 +0530)] 
KAFKA-7616; Make MockConsumer only add entries to the partition map returned by poll() if there are any records to return

…eturned by poll() if there are any records to return

The MockConsumer behaves unlike the real consumer in that it can return a non-empty ConsumerRecords from poll, that also has a count of 0. This change makes the MockConsumer only add partitions to the ConsumerRecords if there are records to return for those partitions.

A unit test in MockConsumerTest demonstrates the issue.

Author: Stig Rohde Døssing <stigdoessing@gmail.com>

Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>

Closes #5901 from srdo/KAFKA-7616

3 weeks agoMINOR: Improve maven artifactory url in release.py (#5931)
Dong Lin [Tue, 20 Nov 2018 06:47:02 +0000 (22:47 -0800)] 
MINOR: Improve maven artifactory url in release.py (#5931)

3 weeks agoKAFKA-6567: Remove KStreamWindowReducer (#5922)
Samuel Hawker [Tue, 20 Nov 2018 05:28:13 +0000 (05:28 +0000)] 
KAFKA-6567: Remove KStreamWindowReducer (#5922)

This pull request removes the final reference to KStreamWindowReducer and replaces it with KStreamWindowAggregate

Signed-off-by: Samuel Hawker sam.b.hawker@gmail.com
contribution is my original work and that I license the work to the project under the project's open source license.

Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

3 weeks agoMINOR: fix failing Streams system tests (#5928)
Matthias J. Sax [Tue, 20 Nov 2018 02:44:45 +0000 (18:44 -0800)] 
MINOR: fix failing Streams system tests (#5928)

Reviewers: Guozhang Wang <guozhang@confluent.io>, Bill Bejeck <bill@confluent.io>, John Roesler <john@confluent.io>

4 weeks agoKAFKA-7402: Implement KIP-376 AutoCloseable additions
Yishun Guan [Fri, 16 Nov 2018 23:58:47 +0000 (15:58 -0800)] 
KAFKA-7402: Implement KIP-376 AutoCloseable additions

4 weeks ago* MINOR: Catching null pointer exception for empty leader URL when assignment is...
Benedict Jin [Fri, 16 Nov 2018 23:28:47 +0000 (07:28 +0800)] 
* 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>

4 weeks agoKAFKA-6774; Improve the default group id behavior in KafkaConsumer (KIP-289) (#5877)
Vahid Hashemian [Fri, 16 Nov 2018 08:58:56 +0000 (00:58 -0800)] 
KAFKA-6774; Improve the default group id behavior in KafkaConsumer (KIP-289) (#5877)

Improve the default group id behavior by:
* changing the default consumer group to null, where no offset commit or fetch, or group management operations are allowed
* deprecating the use of empty (`""`) consumer group on the client

Reviewers: Jason Gustafson <jason@confluent.io>

4 weeks agoKAFKA-7576; Fix shutdown of replica fetcher threads (#5875)
Rajini Sivaram [Fri, 16 Nov 2018 01:27:29 +0000 (01:27 +0000)] 
KAFKA-7576; Fix shutdown of replica fetcher threads (#5875)

ReplicaFetcherThread.shutdown attempts to close the fetcher's Selector while the thread is running. This in unsafe and can result in `Selector.close()` failing with an exception. The exception is caught and logged at debug level, but this can lead to socket leak if the shutdown is due to dynamic config update rather than broker shutdown. This PR changes the shutdown logic to close Selector after the replica fetcher thread is shutdown, with a wakeup() and flag to terminate blocking sends first.

Reviewers: Ismael Juma <ismael@juma.me.uk>, Jason Gustafson <jason@confluent.io>

4 weeks agoHOTFIX: remove deprecated calls
Guozhang Wang [Thu, 15 Nov 2018 22:28:13 +0000 (14:28 -0800)] 
HOTFIX: remove deprecated calls

4 weeks agoMINOR: Remove redundant SuppressIntegrationTests (#5896)
John Roesler [Thu, 15 Nov 2018 21:39:18 +0000 (15:39 -0600)] 
MINOR: Remove redundant SuppressIntegrationTests (#5896)

The removed tests have counterparts covered by SuppressScenarioTest using the TopologyTestDriver.

This will speed up the build and improve stability in the CPU-constrained Jenkins environment.

Reviewers: Bill Bejeck <bill@confluent.io>, Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

4 weeks agoMINOR: improve Puncutation JavaDocs and add runtime argument check (#5895)
Matthias J. Sax [Thu, 15 Nov 2018 21:29:25 +0000 (13:29 -0800)] 
MINOR: improve Puncutation JavaDocs and add runtime argument check (#5895)

Reviewers: Bill Bejeck <bill@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

4 weeks agoKAFKA-7584: StreamsConfig throws ClassCastException if max.in.flight.request.per...
Matthias J. Sax [Thu, 15 Nov 2018 21:26:58 +0000 (13:26 -0800)] 
KAFKA-7584: StreamsConfig throws ClassCastException if max.in.flight.request.per.connect is specified as String (#5874)

Reviewers: Bill Bejeck <bill@confluent.io>, John Roesler <john@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

4 weeks agoMINOR: Code cleanup in StreamsResetter (#5891)
Matthias J. Sax [Thu, 15 Nov 2018 21:23:39 +0000 (13:23 -0800)] 
MINOR: Code cleanup in StreamsResetter (#5891)

Reviewers: Bill Bejeck <bill@confluent.io>, John Roesler <john@confluent.io>, Guozhang Wang <wangguoz@gmail.com>

4 weeks agoMINOR: Bump documentation version to 2.1
Vahid Hashemian [Thu, 15 Nov 2018 19:22:12 +0000 (11:22 -0800)] 
MINOR: Bump documentation version to 2.1

The documentation version of 2.1.0 RC1 is still at 2.0. Updated it to 2.1.

Author: Vahid Hashemian <vahidhashemian@us.ibm.com>

Reviewers: Dong Lin <lindong28@gmail.com>

Closes #5916 from vahidhashemian/minor/update_version_in_documentation_for_2.1.0

4 weeks agoMINOR: Remove deprecated callers (#5911)
Guozhang Wang [Thu, 15 Nov 2018 01:29:19 +0000 (17:29 -0800)] 
MINOR: Remove deprecated callers (#5911)

Callers of 1) Windows#until, 2) Windows#of, 3) Serialized are replaced when possible with the new APIs.

Reviewers: Matthias J. Sax <mjsax@apache.org>, Bill Bejeck <bill@confluent.io>

4 weeks agoMINOR: Avoid logging connector configuration in Connect framework (#5868)
Randall Hauch [Tue, 13 Nov 2018 21:30:34 +0000 (15:30 -0600)] 
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>

4 weeks agoMINOR: Remove unused abstract function in test class (#5888)
Cyrus Vafadari [Tue, 13 Nov 2018 17:43:29 +0000 (09:43 -0800)] 
MINOR: Remove unused abstract function in test class (#5888)

The function `setup_producer_and_consumer` is unused in the system test
framework, which incorrectly suggests subclasses should implement
it. It is not required or even referenced by the framework, so
the requirement should be removed.

Reviewers: Viktor Somogyi <viktorsomogyi@gmail.com>, Jason Gustafson <jason@confluent.io>

4 weeks agoKAFKA-7514: Add threads to ConsumeBenchWorker (#5864)
Stanislav Kozlovski [Tue, 13 Nov 2018 16:38:42 +0000 (16:38 +0000)] 
KAFKA-7514: Add threads to ConsumeBenchWorker (#5864)

Add threads with separate consumers to ConsumeBenchWorker.  Update the Trogdor test scripts and documentation with the new functionality.

Reviewers: Colin McCabe <cmccabe@apache.org>

4 weeks agoTrogdor: Fix /coordinator/tasks parameters to accept long values (#5905)
Stanislav Kozlovski [Tue, 13 Nov 2018 16:35:03 +0000 (16:35 +0000)] 
Trogdor: Fix /coordinator/tasks parameters to accept long values (#5905)

Reviewers: Colin McCabe <cmccabe@apache.org>

4 weeks agoKAFKA-7605; Retry async commit failures in integration test cases to fix flaky tests...
Jason Gustafson [Tue, 13 Nov 2018 06:21:21 +0000 (22:21 -0800)] 
KAFKA-7605; Retry async commit failures in integration test cases to fix flaky tests (#5890)

We are seeing some timeouts in tests which depend on the awaitCommitCallback (e.g.
SaslMultiMechanismConsumerTest.testCoordinatorFailover). After some investigation,
we found that it is caused by a disconnect when attempting the async commit.
To fix the problem, we have added simple retry logic to the test utility.

Reviewers: Stanislav Kozlovski <stanislav_kozlovski@outlook.com>, Ismael Juma <ismael@juma.me.uk>

4 weeks agoKAFKA-7612: Fix javac warnings and enable warnings as errors (#5900)
Ismael Juma [Tue, 13 Nov 2018 06:18:59 +0000 (22:18 -0800)] 
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>

4 weeks agoMINOR: Remove unused IteratorTemplate (#5903)
Jason Gustafson [Mon, 12 Nov 2018 21:42:29 +0000 (13:42 -0800)] 
MINOR: Remove unused IteratorTemplate (#5903)

There seems to be no reason to keep this around since it is not used outside
of testing and AbstractIterator is basically the same thing.

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

4 weeks agoKAFKA-7518: Fix FutureRecordMetadata.get when TimeUnit is not ms (#5815)
Andras Katona [Mon, 12 Nov 2018 19:17:55 +0000 (20:17 +0100)] 
KAFKA-7518: Fix FutureRecordMetadata.get when TimeUnit is not ms (#5815)

Also check for timeout before calling `nextRecordMetadata.get`. Added unit test
validating the fix.

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

4 weeks agoKAFKA-7557: optimize LogManager.truncateFullyAndStartAt() (#5848)
huxi [Mon, 12 Nov 2018 17:02:44 +0000 (01:02 +0800)] 
KAFKA-7557: optimize LogManager.truncateFullyAndStartAt() (#5848)

Instead of calling deleteSnapshotsAfterRecoveryPointCheckpoint for allLogs, invoking it only for the logs being truncated.

Reviewers: Ismael Juma <ismael@juma.me.uk>, Jun Rao <junrao@gmail.com>

5 weeks agoMINOR: Update zstd, easymock, powermock, zkclient and build plugins (#5846)
Ismael Juma [Sat, 10 Nov 2018 21:58:18 +0000 (13:58 -0800)] 
MINOR: Update zstd, easymock, powermock, zkclient and build plugins (#5846)

EasyMock 4.0.x includes a change that relies on the caller for inferring
the return type of mock creator methods. Updated a number of Scala
tests for compilation and execution to succeed.

The versions of EasyMock and PowerMock in this PR include full support
for Java 11.

Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>

5 weeks agoKAFKA-7564: Expose single task details in Trogdor (#5852)
Stanislav Kozlovski [Fri, 9 Nov 2018 18:31:04 +0000 (18:31 +0000)] 
KAFKA-7564: Expose single task details in Trogdor (#5852)

This commit adds a new "/coordinator/tasks/{taskId}" endpoint which fetches details for a single task.

5 weeks agoKAFKA-7412: clarify the doc for producer callback (#5798)
huxi [Fri, 9 Nov 2018 00:58:14 +0000 (08:58 +0800)] 
KAFKA-7412: clarify the doc for producer callback (#5798)

The metadata in the callback is not null with non-null exception.

Reviewers: Jun Rao <junrao@gmail.com>

5 weeks agoKAFKA-7165: Retry the BrokerInfo registration into ZooKeeper (#5575)
Jonathan Santilli [Thu, 8 Nov 2018 21:28:37 +0000 (21:28 +0000)] 
KAFKA-7165: Retry the BrokerInfo registration into ZooKeeper (#5575)

* Add logic to retry the BrokerInfo registration into ZooKeeper

In case the ZooKeeper session has been regenerated and the broker
tries to register the BrokerInfo into Zookeeper, this code deletes
the current BrokerInfo from Zookeeper and creates it again, just if
the znode ephemeral owner belongs to the Broker which tries to register
himself again into ZooKeeper

* Add test to validate the BrokerInfo re-registration into ZooKeeper

5 weeks agoKAFKA-7604; Fix flaky unit test `testRebalanceAfterTopicUnavailableWithPatternSubscri...
Jason Gustafson [Thu, 8 Nov 2018 13:37:05 +0000 (05:37 -0800)] 
KAFKA-7604; Fix flaky unit test `testRebalanceAfterTopicUnavailableWithPatternSubscribe` (#5889)

The problem is the concurrent metadata updates in the foreground and in the heartbeat thread. Changed the code to use ConsumerNetworkClient.poll, which enforces mutual exclusion when accessing the underlying client.

5 weeks agoKAFKA-7431: Clean up connect unit tests
Srinivas Reddy [Wed, 7 Nov 2018 16:23:19 +0000 (08:23 -0800)] 
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

5 weeks agoKAFKA-7560; PushHttpMetricsReporter should not convert metric value to double
Dong Lin [Wed, 7 Nov 2018 16:04:29 +0000 (08:04 -0800)] 
KAFKA-7560; PushHttpMetricsReporter should not convert metric value to double

Currently PushHttpMetricsReporter will convert value from KafkaMetric.metricValue() to double. This will not work for non-numerical metrics such as version in AppInfoParser whose value can be string. This has caused issue for PushHttpMetricsReporter which in turn caused system test kafkatest.tests.client.quota_test.QuotaTest.test_quota to fail.

Since we allow metric value to be object, PushHttpMetricsReporter should also read metric value as object and pass it to the http server.

Author: Dong Lin <lindong28@gmail.com>

Reviewers: Manikumar Reddy O <manikumar.reddy@gmail.com>, Ewen Cheslack-Postava <ewen@confluent.io>

Closes #5886 from lindong28/KAFKA-7560