Commit Graph

5997 Commits

Author SHA1 Message Date
John Roesler db98c6f8cd KAFKA-5998: fix checkpointableOffsets handling (#7030)
fix checkpoint file warning by filtering checkpointable offsets per task
clean up state manager hierarchy to prevent similar bugs

Reviewers: Bruno Cadonna <bruno@confluent.io>, Bill Bejeck <bbejeck@gmail.com>
2019-07-12 09:58:42 -04:00
Nacho Muñoz Gómez d956cc72dd 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:24:08 -07:00
cadonna 9618c5bfa4 MINOR: Fix version in environment setup (#7004)
Reviewers: Guozhang Wang <wangguoz@gmail.com>, John Roesler <john@confluent.io>, Boyang Chen <boyang@confluent.io>, Bill Bejeck <bbejeck@gmail.com>
2019-06-26 14:26:55 -04:00
Manikumar Reddy 72c74e06bb MINOR: Add missing Alter Operation to Topic supported operations list in AclCommand
- Update the AclCommandTest

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

Reviewers: Jun Rao <junrao@gmail.com>

Closes #6263 from omkreddy/aclcommand
2019-06-23 00:39:40 +05:30
Dhruvil Shah a297722cb1 KAFKA-8570; Grow buffer to hold down converted records if it was insufficiently sized (#6974)
When the log contains out of order message formats (for example v2 message followed by v1 message) and consists of compressed batches typically greater than 1kB in size, it is possible for down-conversion to fail. With compressed batches, we estimate the size of down-converted batches using:

```
    private static int estimateCompressedSizeInBytes(int size, CompressionType compressionType) {
        return compressionType == CompressionType.NONE ? size : Math.min(Math.max(size / 2, 1024), 1 << 16);
    }
```

This almost always underestimates size of down-converted records if the batch is between 1kB-64kB in size. In general, this means we may under estimate the total size required for compressed batches.

Because of an implicit assumption in the code that messages with a lower message format appear before any with a higher message format, we do not grow the buffer we copy the down converted records into when we see a message <= the target message format. This assumption becomes incorrect when the log contains out of order message formats, for example because of leaders flapping while upgrading the message format.

Reviewers: Jason Gustafson <jason@confluent.io>
2019-06-21 09:45:05 -07:00
Mickael Maison 73ab486fa0 KAFKA-8564; Fix NPE on deleted partition dir when no segments remain (#6968)
Kafka should not NPE while loading a deleted partition dir with no log segments. This patch ensures that there will always be at least one segment after initialization.

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

Reviewers: Ismael Juma <ismael@juma.me.uk>, Jason Gustafson <jason@confluent.io>
2019-06-19 12:45:44 -07:00
wenhoujx 7d75a2085e KAFKA-8488: Reduce logging-related string allocation in FetchSessionHandler
Reviewers: Colin P. McCabe <cmccabe@apache.org>, Ismael Juma <ismael@juma.me.uk>
(cherry picked from commit 35814298e1)
(cherry picked from commit 5daf9f1811)
2019-06-14 11:23:14 -07:00
Stanislav Kozlovski d797428cd0 MINOR: Fix merge conflict in version.py (#6931)
Fix a regression introduced in commit c450bfc291

Reviewers: Ismael Juma <ismael@confluent.io>, Matthias J. Sax <matthias@confluent.io>
2019-06-14 08:39:34 -07:00
Colin P. Mccabe bda2a02ce7 KAFKA-4893; Fix deletion and moving of topics with long names
Author: Colin P. Mccabe <cmccabe@confluent.io>

Reviewers: Gwen Shapira, David Arthur, James Cheng, Vahid Hashemian

Closes #6869 from cmccabe/KAFKA-4893

(cherry picked from commit e6563aab72)
Signed-off-by: Gwen Shapira <cshapi@gmail.com>
2019-06-11 10:33:21 -07:00
Bill Bejeck b0dc6ed6fb MINOR: Increase timeouts to 30 seconds (#6852)
The ResetIntegrationTest has experienced several failures and it seems the current timeout of 10 seconds may not be enough time

Reviewers: Matthias J. Sax <mjsax@apache.org>, Boyang Chen <boyang@confluent.io>
2019-06-10 20:53:26 -04:00
Victoria Bialas b1f189b2e3 KAFKA-7315 DOCS update TOC internal links serdes all versions (#6875)
Reviewers: Joel Hamill <joel@confluent.io>, Jim Galasyn <jim.galasyn@confluent.io>, Matthias J. Sax <matthias@confluent.io>
2019-06-10 10:10:20 -07:00
Jason Gustafson 91d229df60 MINOR: Fix race condition on shutdown of verifiable producer
We've seen `ReplicaVerificationToolTest.test_replica_lags` fail occasionally due to errors such as the following:
```
RemoteCommandError: ubuntuworker7: Command 'kill -15 2896' returned non-zero exit status 1. Remote error message: bash: line 0: kill: (2896) - No such process
```
The problem seems to be a shutdown race condition when using `max_messages` with the producer. The process may already be gone which will cause the signal to fail.

Author: Jason Gustafson <jason@confluent.io>

Reviewers: Gwen Shapira

Closes #6906 from hachikuji/fix-failing-replicat-verification-test
2019-06-07 17:06:11 -07:00
Jason Gustafson be1bfadde7 MINOR: Lower producer throughput in flaky upgrade system test
We see the upgrade test failing from time to time. I looked into it and found that the root cause is basically that the test throughput can be too high for the 0.9 producer to make progress. Eventually it reaches a point where it has a huge backlog of timed out requests in the accumulator which all have to be expired. We see a long run of messages like this in the output:

```
{"exception":"class org.apache.kafka.common.errors.TimeoutException","time_ms":1559907386132,"name":"producer_send_error","topic":"test_topic","message":"Batch Expired","class":"class org.apache.kafka.tools.VerifiableProducer","value":"335160","key":null}
{"exception":"class org.apache.kafka.common.errors.TimeoutException","time_ms":1559907386132,"name":"producer_send_error","topic":"test_topic","message":"Batch Expired","class":"class org.apache.kafka.tools.VerifiableProducer","value":"335163","key":null}
{"exception":"class org.apache.kafka.common.errors.TimeoutException","time_ms":1559907386133,"name":"producer_send_error","topic":"test_topic","message":"Batch Expired","class":"class org.apache.kafka.tools.VerifiableProducer","value":"335166","key":null}
{"exception":"class org.apache.kafka.common.errors.TimeoutException","time_ms":1559907386133,"name":"producer_send_error","topic":"test_topic","message":"Batch Expired","class":"class org.apache.kafka.tools.VerifiableProducer","value":"335169","key":null}
```
This can continue for a long time (I have observed up to 1 min) and prevents the producer from successfully writing any new data. While it is busy expiring the batches, no data is getting delivered to the consumer, which causes it to eventually raise a timeout.
```
kafka.consumer.ConsumerTimeoutException
at kafka.consumer.NewShinyConsumer.receive(BaseConsumer.scala:50)
at kafka.tools.ConsoleConsumer$.process(ConsoleConsumer.scala:109)
at kafka.tools.ConsoleConsumer$.run(ConsoleConsumer.scala:69)
at kafka.tools.ConsoleConsumer$.main(ConsoleConsumer.scala:47)
at kafka.tools.ConsoleConsumer.main(ConsoleConsumer.scala)
```
The fix here is to reduce the throughput, which seems reasonable since the purpose of the test is to verify the upgrade, which does not demand heavy load. Note that I investigated several failing instances of this test going back to 1.0 and saw a similar pattern, so there does not appear to be a regression.

Author: Jason Gustafson <jason@confluent.io>

Reviewers: Gwen Shapira

Closes #6907 from hachikuji/lower-throughput-for-upgrade-test
2019-06-07 17:06:03 -07:00
Lucas Bradstreet ed47e09967 KAFKA-8499: ensure java is in PATH for ducker system tests (#6898)
Reviewers: Colin P. McCabe <cmccabe@apache.org>
2019-06-07 14:24:41 -07:00
A. Sophie Blee-Goldman 0db64f6828 HOTFIX: Close unused ColumnFamilyHandle (#6893)
In RocksDBTimestampedStore#openRocksDB we try to open a db with two column families. If this succeeds but the first column family is empty (db.newIterator.seekToFirst.isValid() == false) we never actually close its ColumnFamilyHandle

Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2019-06-06 11:29:39 -07:00
Hai-Dang Dam 8ea05ee01b 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>

# Conflicts:
#	connect/runtime/src/main/java/org/apache/kafka/connect/runtime/rest/resources/ConnectorsResource.java
#	connect/runtime/src/test/java/org/apache/kafka/connect/runtime/rest/resources/ConnectorsResourceTest.java
2019-06-03 21:34:09 -05:00
Vahid Hashemian 79a42ae793 Update versions to 2.2.2-SNAPSHOT 2019-06-01 22:55:42 -07:00
Vahid Hashemian ed80742bed Merge tag '2.2.1-rc1' into 2.2
2.2.1-rc1
2019-06-01 22:50:50 -07:00
Matthias J. Sax c450bfc291 KAFKA-8155: Add 2.1.1 release to system tests (#6596)
Reviewers: Bill Bejeck <bill@confluent.io>, John Roesler <john@confluent.io>, Guozhang Wang <guozhang@confluent.io>
2019-05-30 12:58:32 -07:00
Alex Diachenko 46f59e7331 KAFKA-8418: Wait until REST resources are loaded when starting a Connect Worker. (#6840)
Author: Alex Diachenko <sansanichfb@gmail.com>
Reviewers: Arjun Satish <arjun@confluent.io>, Konstantine Karantasis <konstantine@confluent.io>, Randall Hauch <rhauch@gmail.com>
2019-05-30 14:03:40 -05:00
Lifei Chen ed659fe73d KAFKA-8187: Add wait time for other thread in the same jvm to free the locks (#6818)
Fix KAFKA-8187: State store record loss across multiple reassignments when using standby tasks.
Do not let the thread to transit to RUNNING until all tasks (including standby tasks) are ready.

Reviewers: Guozhang Wang <wangguoz@gmail.com>,  Bill Bejeck <bbejeck@gmail.com>
2019-05-30 11:51:52 -04:00
Alex Diachenko 0779740f9b MINOR: Fix red herring when ConnectDistributedTest.test_bounce fails. (#6838)
Author: Alex Diachenko <sansanichfb@gmail.com>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-05-29 17:35:24 -05:00
Jason Gustafson 2ae66eba6a KAFKA-8351; Cleaner should handle transactions spanning multiple segments (#6722)
When cleaning transactional data, we need to keep track of which transactions still have data associated with them so that we do not remove the markers. We had logic to do this, but the state was not being carried over when beginning cleaning for a new set of segments. This could cause the cleaner to incorrectly believe a transaction marker was no longer needed. The fix here carries the transactional state between groups of segments to be cleaned.

Reviewers: Dhruvil Shah <dhruvil@confluent.io>, Viktor Somogyi <viktorsomogyi@gmail.com>, Jun Rao <junrao@gmail.com>
2019-05-24 23:18:40 -07:00
sdreynolds 39a088e376 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:56:42 -07:00
Matthias J. Sax 9d559da39e MINOR: fix Streams version-probing system test (#6763)
A version probing rebalance, triggers a second rebalance. If the second rebalance happens quickly, we see the log about successful group assignment twice.

Reviewers: Bill Bejeck <bill@confluent.io>, John Roesler <john@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2019-05-20 14:17:05 -07:00
Bill Bejeck 2039e834a2 KAFKA-8290: Close producer for zombie task (#6636)
When we close a task and EOS is enabled we should always close the producer regardless if the task is in a zombie state (the broker fenced the producer) or not.

I've added tests that fail without this change.

Reviewers: Matthias J. Sax <mjsax@apache.org>, Jason Gustafson <jason@confluent.io>
2019-05-20 09:12:17 -04:00
Jason Gustafson 7b8db0d790 MINOR: Increase security test timeouts for transient failures (#6760)
Reviewers: Ismael Juma <ismael@juma.me.uk>
2019-05-18 16:53:15 -07:00
A. Sophie Blee-Goldman 722e854bfe KAFKA-8347: Choose next record to process by timestamp (#6719)
When choosing the next record to process, we should look at the head record's timestamp of each partition and choose the lowest rather than choosing the lowest of the partition's streamtime.

This change effectively makes RecordQueue return the timestamp of the head record rather than its streamtime. Streamtime is removed (replaced) from RecordQueue as it was only being tracked in order to choose the next partition to poll from.

Reviewers: Matthias J. Sax <mjsax@apache.org>,  Bill Bejeck <bbejeck@gmail.com>
2019-05-17 17:19:43 -04:00
Konstantine Karantasis f8fd17d256 MINOR: Enable console logs in Connect tests (#6745)
Author: Konstantine Karantasis <konstantine@confluent.io>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-05-15 22:07:03 -05:00
Magesh Nandakumar 04b516b984 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:21:42 -05:00
A. Sophie Blee-Goldman bb3c61b355 Moved initialization of bloom filter (#6734)
Bug-fix that moves the initialization of the Bloom Filter Rocks object out of the @before test setup method. Don't use KIP-453 so we can apply to previous versions

Reviewers: Bill Bejeck <bbejeck@gmail.com>
2019-05-14 22:55:45 -04:00
Chris Egerton 9d22438e5f KAFKA-8363: Fix parsing bug for config providers (#6726)
Author: Chris Egerton <cegerton@oberlin.edu>
Reviewers: Robert Yokota <rayokota@gmail.com>, Randall Hauch <rhauch@gmail.com>
2019-05-14 14:33:47 -05:00
Vahid Hashemian 55783d3133 Bump version to 2.2.1 2019-05-13 09:10:09 -07:00
Jason Gustafson 0bf7fa85de KAFKA-8335; Clean empty batches when sequence numbers are reused (#6715)
The log cleaner attempts to preserve the last entry for each producerId in order to ensure that sequence/epoch state is not lost. The current validation checks only the last sequence number for each producerId in order to decide whether a batch should be retained. There are two problems with this:

1. Sequence numbers are not unique alone. It is the tuple of sequence number and epoch which is uniquely defined.
2. The group coordinator always writes batches beginning with sequence number 0, which means there could be many batches which have the same sequence number.

The complete fix for the second issue would probably add proper sequence number bookkeeping in the coordinator. For now, we have left the coordinator implementation unchanged and changed the cleaner logic to use the last offset written by a producer instead of the last sequence number. 

Reviewers: Guozhang Wang <wangguoz@gmail.com>
2019-05-13 08:49:15 -07:00
Jason Gustafson 220af14ac3
MINOR: Use request version to ensure metadata reflects subscription changes (#6718)
This is a backport to 2.2 of a fix which was merged in KAFKA-7831. When there is a subscription change with a metadata request in flight, we may incorrectly interpret the response as applying to the changed subscription. The fix here is to track a separate request version in `Metadata` so that we do not confuse the topic set that was requested.

Reviewers: Rajini Sivaram <rajinisivaram@googlemail.com>
2019-05-13 08:36:21 -07:00
Arabelle Hou 5a4d608491 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:08:24 -07:00
Matthias J. Sax 4ed5efe7ba MINOR: Align KTableAgg and KTableReduce (#6712)
Reviewers: John Roesler <john@confluent.io>, Bill Bejeck <bill@confluent.io>, Jeff Kim <kimkb2011@gmail.com>, Guozhang Wang <guozhang@confluent.io>
2019-05-11 11:56:24 +02:00
Magesh Nandakumar b5b2c5af2f KAFKA-8352 : Fix Connect System test failure 404 Not Found (#6713)
Corrects the system tests to check for either a 404 or a 409 error and sleeping until the Connect REST API becomes available. This corrects a previous change to how REST extensions are initialized (#6651), which added the ability of Connect throwing a 404 if the resources are not yet started. The integration tests were already looking for 409.

Author: Magesh Nandakumar <magesh.n.kumar@gmail.com>
Reviewer: Randall Hauch <rhauch@gmail.com>
2019-05-10 18:21:07 -05:00
Lifei Chen c9ebcd6652 KAFKA-8348: Fix KafkaStreams JavaDocs (#6707) 2019-05-10 11:56:02 +02:00
Chris Egerton c50573904e 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:36:02 -05:00
Manikumar Reddy f8447a831e KAFKA-6789; Handle retriable group errors in AdminClient API (#5578)
This patch adds support to retry all group operations after COORDINATOR_LOAD_IN_PROGRESS and COORDINATOR_NOT_AVAILABLE in AdminClient group operations. Previously we only had logic to retry after FindCoordinator failures.

Reviewers: Yishun Guan <gyishun@gmail.com>, Viktor Somogyi <viktorsomogyi@gmail.com>, Jason Gustafson <jason@confluent.io>
2019-05-09 15:48:53 -07:00
Matthias J. Sax 17708df8c6
KAFKA-8240: Fix NPE in Source.equals() (#6685)
- backport of PR #6589 to 2.2 branch

Reviewers: Bruno Cadonna <bruno@confluent.io>, Bill Bejeck <bill@confluent.io>
2019-05-09 15:48:11 +02:00
Chris Egerton a569a81a6c 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:29:47 -05:00
Dhruvil Shah f000dab544 KAFKA-8306; Initialize log end offset accurately when start offset is non-zero (#6652)
This patch ensures that the log end offset of each partition is initialized consistently with the checkpointed log start offset.

Reviewers: Ismael Juma <ismael@juma.me.uk>, Jason Gustafson <jason@confluent.io>
2019-05-04 14:09:52 -07:00
A. Sophie Blee-Goldman b334786b66 Should close filter in RocksDBStoreTest as well (#6676)
Forgot to also close the filter in RocksDBStoreTest in time.

Reviewers: Bill Bejeck <bbejeck@gmail.com>
2019-05-04 10:56:47 -04:00
A. Sophie Blee-Goldman 40ee41518d KAFKA-8323: Close RocksDBStore's BloomFilter (#6672)
Any RocksJava object that inherits from org.rocksdb.AbstractNativeReference must be closed explicitly in order to free up the memory of the backing C++ object. The BloomFilter extends RocksObject (which implements AbstractNativeReference) and should be also be closed in RocksDBStore#close to avoid leaking memory.

Reviewers: Bill Bejeck <bbejeck@gmail.com>
2019-05-03 19:52:48 -04:00
John Roesler bfad7f99f7 KAFKA-8289: Fix Session Expiration and Suppression (#6654) (#6670)
Fix two problems in Streams:
* Session windows expired prematurely (off-by-one error), since the window end is inclusive, unlike other windows
* Suppress duration for sessions incorrectly waited only the grace period, but session windows aren't closed until gracePeriod + sessionGap

cherry-pick of 6654 into trunk

Reviewers: Bill Bejeck <bill@confluent.io>
2019-05-03 16:23:59 -04:00
Jason Gustafson 8d6e2a2aac KAFKA-7601; Clear leader epoch cache on downgraded format in append (#6568)
During a partial message format upgrade, it is possible for the message format to flap between new and old versions. If we detect that data appended to the log is on an old format, we can clear the leader epoch cache so that we revert to truncation by high watermark. Once the upgrade completes and all replicas are on the same format, we will append to the epoch cache as usual. Note this is related to KAFKA-7897, which handles message format downgrades through configuration. 

Reviewers: Jun Rao <junrao@gmail.com>
2019-05-02 17:46:43 -07:00
Jason Gustafson 98e440c5b5 KAFKA-8248; Ensure time updated before sending transactional request (#6613)
This patch fixes a bug in the sending of transactional requests. We need to call `KafkaClient.send` with an updated current time. Failing to do so can result in an `IllegalStateExcepton` which leaves the producer effectively dead since the in-flight correlation id has been set, but no request has been sent. To avoid the same problem in the future, we update the in flight correlationId only after sending the request.

Reviewers: Matthias J. Sax <matthias@confluent.io>, Apurva Mehta <apurva@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2019-05-02 13:59:29 -07:00
Zhanxiang (Patrick) Huang 2cb1efd77f 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:46:40 -07:00