Commit Graph

5051 Commits

Author SHA1 Message Date
Ismael Juma b751321486 MINOR: Remove MaxPermSize from gradle.properties
No longer needed since we dropped support for Java 7.

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

Reviewers: Dong Lin <lindong28@gmail.com>

Closes #5083 from ijuma/remove-max-perm-size
2018-05-27 10:04:58 -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
Chia-Ping Tsai 8d1e96181d MINOR: Replace unused variables by underscore (#5003)
And remove one unused expression.

Reviewers: Ismael Juma <ismael@juma.me.uk>
2018-05-26 00:54:41 -07:00
Ron Dagostino 8c5d7e0408 KAFKA-6562: OAuth Authentication via SASL/OAUTHBEARER (KIP-255) (#4994)
This KIP adds the following functionality related to SASL/OAUTHBEARER:

1) Allow clients (both brokers when SASL/OAUTHBEARER is the inter-broker protocol as well as non-broker clients) to flexibly retrieve an access token from an OAuth 2 authorization server based on the declaration of a custom login CallbackHandler implementation and have that access token transparently and automatically transmitted to a broker for authentication.

2) Allow brokers to flexibly validate provided access tokens when a client establishes a connection based on the declaration of a custom SASL Server CallbackHandler implementation.

3) Provide implementations of the above retrieval and validation features based on an unsecured JSON Web Token that function out-of-the-box with minimal configuration required (i.e. implementations of the two types of callback handlers mentioned above will be used by default with no need to explicitly declare them).

4) Allow clients (both brokers when SASL/OAUTHBEARER is the inter-broker protocol as well as non-broker clients) to transparently retrieve a new access token in the background before the existing access token expires in case the client has to open new connections.
2018-05-26 08:18:41 +01:00
Manikumar Reddy O d45d7ec781 KAFKA-2951; Add a test to verify produce, consume with ACLs for topic/group wildcard resources (#5054) 2018-05-25 23:49:31 -07:00
Jason Gustafson 7f19df29ac MINOR: AdminClient should respect retry backoff
AdminClient should backoff when retrying a Call. Fixed and added a unit test

Author: Jason Gustafson <jason@confluent.io>

Reviewers: Dong Lin <lindong28@gmail.com>

Closes #5077 from hachikuji/admin-client-retry-backoff
2018-05-25 17:55:21 -07:00
maytals 39fe105dfd Minor: Fixed ConsumerOffset#path (#5060)
consumer offset path in zookeeper should be /consumers/${group}/offsets/${topic}/${partition} instead of /consumers/${group}/offset/${topic}/${partition}. Added `s` to the word `offset`.

Reviewers: Ismael Juma <ismael@juma.me.uk>, Manikumar Reddy O <manikumar.reddy@gmail.com>, Jun Rao <junrao@gmail.com>
2018-05-25 17:10:59 -07:00
yaphet ab3ff7101c KAFKA-6930: Convert byte array to string in KafkaZkClient debug log (#5061)
Reviewers: Ismael Juma <ismael@juma.me.uk>
2018-05-25 16:26:44 -07:00
Jun Rao 345db59650
KAFKA-6937: In-sync replica delayed during fetch if replica throttle is exceeded (#5074)
Reviewers: Ismael Juma <ismael@juma.me.uk>, Dong Lin <lindong28@gmail.com>, Ben Stopford <benstopford@gmail.com>
2018-05-25 16:19:12 -07:00
Radai Rosenblatt c9ec292135 Improve kafka client sensor registration performance by lazily calculating JMX attributes
When any metric (e.g. per-partition metric) is created or deleted,
registerMBean() is called which in turn calls getMBeanInfo().getClassName().
However, KafkaMbean.getMBeanInfo() instantiates an array of all sensors even
though we only need the class name. This costs a lot of CPU to register
sensors when consumer with large partition assignment starts. For example, it
takes 5 minutes to start a consumer with 35k partitions. This patch reduces the
consumer startup time seconds.

Author: radai-rosenblatt <radai.rosenblatt@gmail.com>

Reviewers: Satish Duggana <satish.duggana@gmail.com>, Dong Lin <lindong28@gmail.com>

Closes #5011 from radai-rosenblatt/fun-with-jmx
2018-05-25 15:39:05 -07:00
Ismael Juma 70f0d0bd3f
MINOR: Use reflection for signal handler and do not enable it for IBM JDK (#5047)
The Signal classes are not available in the compile classpath
if --release is used so we use reflection as a workaround.
As part of that moved the code to Java and added a simple
unit test.

Also disabled the signal handler if the IBM JDK is being used
due to KAFKA-6918.

Manually tested shutdown via ctrl+c and verified that
the message is printed.
2018-05-25 07:00:56 -07:00
Sasaki Toru 440445e7c5 KAFKA-2061; Offer a --version flag to print the kafka version [KIP-278] (#639)
Reviewers: Andy Lindeman, Jeremy Donahue, Jason Gustafson <jason@confluent.io>
2018-05-25 00:47:54 -07:00
Ismael Juma 7132a85fc3 KAFKA-6921; Remove old Scala producer and related code
* Removed Scala producers, request classes, kafka.tools.ProducerPerformance, encoders,
tests.
* Updated ConsoleProducer to remove Scala producer support (removed `BaseProducer`
and several options that are not used by the Java producer).
* Updated a few Scala consumer tests to use the new producer (including a minor
refactor of `produceMessages` methods in `TestUtils`).
* Updated `ClientUtils.fetchTopicMetadata` to use `SimpleConsumer` instead of
`SyncProducer`.
* Removed `TestKafkaAppender` as it looks useless and it defined an `Encoder`.
* Minor import clean-ups

No new tests added since behaviour should remain the same after these changes.

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

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

Closes #5045 from ijuma/kafka-6921-remove-old-producer
2018-05-24 17:32:49 -07:00
Rajini Sivaram ff9f928c16 KAFKA-6911; Fix dynamic keystore/truststore update check (#5029)
Fix the check, add unit test to verify the change, update `DynamicBrokerReconfigurationTest` to avoid dynamic keystore update in tests which are not expected to update keystores.
2018-05-24 16:24:37 -07:00
Jason Gustafson 0f86e68840 MINOR: Remove dependence on __consumer_offsets in AdminClient listConsumerGroups
Avoid dependence on the internal __consumer_offsets topic to handle `listConsumerGroups()` since it unnecessarily requires users to have Describe access on an internal topic. Instead we query each broker independently. For most clusters, this amounts to the same thing since the default number of partitions for __consumer_offsets is 50. This also provides better encapsulation since it avoids exposing the use of __consumer_offsets, which gives us more flexibility in the future.

Author: Jason Gustafson <jason@confluent.io>

Reviewers: Dong Lin <lindong28@gmail.com>

Closes #5007 from hachikuji/remove-admin-use-of-offsets-topic
2018-05-23 12:13:04 -07:00
Jason Gustafson e8847205f9
MINOR: Fix transiently failing consumer group admin integration test (#5067)
Since the producer is using retries=0, we need to await topic creation before sending any records.

Reviewers: Ismael Juma <ismael@juma.me.uk>
2018-05-22 16:24:54 -07:00
Jorge Quilcate Otoya 133108cdac KAFKA-6850: Add Record Header support to Kafka Streams Processor API (KIP-244) (#4955)
Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <guozhang@confluent.io>
2018-05-22 15:44:37 -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
Jason Gustafson 5be47a2f26
MINOR: AdminClient consumer group domain objects should have public constructors (#5063)
These constructors should be public to allow users to write test cases using them. We follow a similar pattern for the other domain objects that we expose in `AdminClient` (e.g. `TopicDescription`).

Reviewers: Ismael Juma <ismael@juma.me.uk>
2018-05-22 11:23:47 -07:00
Jason Gustafson c1b30a12b1 MINOR: AdminClient metadata manager should reset state on failure
If the internal metadata request fails, we must reset the state inside `AdminClientMetadataManager` or we will be stuck indefinitely in the `UPDATE_PENDING` state and have no way to fetch new metadata.

Author: Jason Gustafson <jason@confluent.io>

Reviewers: Dong Lin <lindong28@gmail.com>

Closes #5057 from hachikuji/fix-admin-client-metadata-update-failure
2018-05-22 10:12:18 -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
Filipe Agapito 6281fbcb6a KAFKA-6474: Rewrite tests to use new public TopologyTestDriver [part 3] (#5052)
* KAFKA-6474: Rewrite tests to use new public TopologyTestDriver [part 3]

* Refactor:
  - KStreamWindowReduceTest
  - KTableMapKeysTest
  - SessionWindowedKStreamImplTest
  - TimeWindowedKStreamImplTest

* Remove unnecessary @SuppressWarnings(unchecked)

Reviewers: Bill Bejeck <bill@confluent.io>, Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2018-05-22 08:48:54 -07:00
Ismael Juma 193c779682
MINOR: Remove unnecessary conditional in KafkaAdminClient to fix checkstyle (#5058) 2018-05-22 00:15:40 -07:00
Ismael Juma e70a191d30
KAFKA-4423: Drop support for Java 7 (KIP-118) and update deps (#5046)
* Set --source, --target and --release to 1.8.
* Build Scala 2.12 by default.
* Remove some conditionals in the build file now that Java 8
is the minimum version.
* Bump the version of Jetty, Jersey and Checkstyle (the newer
versions require Java 8).
* Fixed issues uncovered by the new version if Checkstyle.
* A couple of minor updates to handle an incompatible source
change in the new version of Jetty.
* Add dependency to jersey-hk2 to fix failing tests caused
by Jersey upgrade.
* Update release script to use Java 8 and to take into account
that Scala 2.12 is now built by default.
* While we're at it, bump the version of Gradle, Gradle plugins,
ScalaLogging, JMH and apache directory api.
* Minor documentation updates including the readme and upgrade
notes. A number of Streams Java 7 examples can be removed
subsequently.
2018-05-21 23:17:42 -07:00
Jagadesh Adireddi 95b46a12e5 KAFKA-6685: Added Exception to distinguish message Key from Value during deserializing.
https://issues.apache.org/jira/browse/KAFKA-6685

Added Exception message in `WorkerSinkTask.convertMessages` to distinguish message Key from Value during deserialization to Kafka connect format.

*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: Jagadesh Adireddi <adireddijagadesh@gmail.com>

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

Closes #4765 from jadireddi/KAFKA-6685---log-message-should-distinguish-key-from-value
2018-05-21 21:10:17 -07:00
Guozhang Wang 70a506b983
MINOR: Ignore test_broker_type_bounce_at_start system test (#5055)
test_broker_type_bounce_at_start tries to validate that when the controller is down, the streams client will always fail trying to create the topic; with the current behavior of admin client it is actually not always true: the actual behavior depends on the admin client internals as well as when the controller becomes unavailable during the leader assign partitions phase. I'd suggest at least ignore this test for now until the admin client has more stable (personally I'd even suggest removing this test as its coverage benefits is smaller than its introduced issues to me).

Also adding a few more log4j entries as a result of investigating this issue.

Reviewers: Matthias J. Sax <matthias@confluent.io>
2018-05-21 17:28:40 -07:00
Guozhang Wang cbce95d9a5
MINOR: Reduce required occurrance from 100 to 10 (#5048)
Due to #4644 the consumer connector logs will be much more clean with fewer "broker may not be available" entries. We need to reduce the required frequency from 100 to a smaller number.

I've thought about reducing to just 1, but it may still be transient (i.e. even if broker is starting up you may see a few entries) so I reduced it to 10.

Reviewers: Bill Bejeck <bill@confluent.io>, Matthias J. Sax <matthias@confluent.io>
2018-05-21 11:50:19 -07:00
Colin Patrick McCabe 16ad358d64 KAFKA-6868; Fix buffer underflow and expose group state in the consumer groups API (#4980)
* The consumer groups API should expose group state and coordinator information.  This information is needed by administrative tools and scripts that access consume groups.

* The partition assignment will be empty when the group is rebalancing. Fix an issue where the adminclient attempted to deserialize this empty buffer.

* Remove nulls from the API and make all collections immutable.

* DescribeConsumerGroupsResult#all should return a result as expected, rather than Void

* Fix exception text for GroupIdNotFoundException, GroupNotEmptyException. It was being filled in as "The group id The group id does not exist was not found" and similar.

Reviewers: Attila Sasvari <asasvari@apache.org>, Andras Beni <andrasbeni@cloudera.com>, Dong Lin <lindong28@gmail.com>, Jason Gustafson <jason@confluent.io>
2018-05-21 08:37:35 -07:00
Joan Goyeau 96cda0e07a MINOR: Fix type inference on joins and aggregates (#5019)
The type inference doesn't currently work for the join functions in Scala as it doesn't know yet the types of the given KStream[K, V] or KTable[K, V].

The fix here is to curry the joiner function. I personally prefer this notation but this also means it differs more from the Java API.
I believe the diff with the Java API is worth in this case as it's not only solving the type inference but also fits better the Scala way of coding (ex: fold).

Moreover any Scala dev will bug and spend little time on these functions trying to understand why the type inference is not working and then get frustrated to be obliged to be explicit here where it's not harmful to be inferred.

Reviewers: Debasish Ghosh <dghosh@acm.org>, Guozhang Wang <guozhang@confluent.io>, Ismael Juma <ismael@juma.me.uk>
2018-05-20 16:25:16 -07:00
Guozhang Wang 9752ccad55
KAFKA-6729: Follow up; disable logging for source KTable. (#5038)
Reviewers: Matthias J. Sax <matthias@confluent.io>
2018-05-20 10:24:07 -07:00
David Glasser f65f3a878f KAFKA-6905: Document that Transformers may be re-used by Streams (#5026)
This is a follow-up to #5022 which added documentation to the Processor
interface. This commit adds similar documentation to Transformer and
ValueTransformer.

Also, s/processor/transformer/ in the close() docs.

Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2018-05-19 18:44:20 -07:00
Andy Coates 4e1c8ffd0d KAFKA-6849: add transformValues methods to KTable. (#4959)
See the KIP: https://cwiki.apache.org/confluence/display/KAFKA/KIP-292%3A+Add+transformValues%28%29+method+to+KTable

This PR adds the transformValues method to the KTable interface. The semantics of the call are the same as the methods of the same name on the KStream interface.

Fixes KAFKA-6849

Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2018-05-18 16:06:50 -07:00
Rajini Sivaram c53e274d31 KAFKA-6917; Process txn completion asynchronously to avoid deadlock (#5036)
Reviewers: Guozhang Wang <wangguoz@gmail.com>, Jason Gustafson <jason@confluent.io>
2018-05-18 13:30:12 -07:00
Guozhang Wang 05ea580091
MINOR: Remove unused class (#5037)
Reviewers: Bill Bejeck <bill@confluent.io>, Matthias J. Sax <matthias@confluent.io>
2018-05-18 10:58:10 -07:00
Robert Yokota ee8abb2f70 KAFKA-6566: Improve Connect Resource Cleanup
This is a change to improve resource cleanup for sink tasks and source tasks.  Now `Task.stop()` is called from both `WorkerSinkTask.close()` and `WorkerSourceTask.close()`.

It is called from `WorkerXXXTask.close()` since this method is called in the `finally` block of `WorkerTask.run()`, and Connect developers use `stop()` to clean up resources.

Author: Robert Yokota <rayokota@gmail.com>

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

Closes #5020 from rayokota/K6566-improve-connect-resource-cleanup
2018-05-18 10:39:34 -07:00
John Roesler 58a910f0a7 KAFKA-5697: revert wakeup-based impl (#5035)
The wakeup-based strategy caused more problems than it
solved, so we'll instead focus on KIP-266.

Revert commit 2d8049b.

Keep the metrics addition and the new test util.

Also keep the tests for shutdown, although they must be ignored until
poll(Duration) is done in the scope of KIP-266.

Reviewers: Guozhang Wang <wangguoz@gmail.com>
2018-05-18 08:05:09 -07:00
Guozhang Wang ba237c5d21 HOTFIX: use ConsumedInternal in StreamsBuilder 2018-05-17 17:20:12 -07:00
Guozhang Wang 6b8e79b137
HOTFIX: move Conusmed to o.a.k.streams.kstream (#5033)
Reviewers: Bill Bejeck <bill@confluent.io>, Matthias J. Sax <matthias@confluent.io>
2018-05-17 11:56:07 -07:00
Guozhang Wang 1a324d784c
KAFKA-6729: Reuse source topics for source KTable's materialized store's changelog (#5017)
1. In InternalTopologyBuilder#topicGroups, which is used in StreamsPartitionAssignor, look for book-kept storeToChangelogTopic map before creating a new internal changelog topics. In this way if the source KTable is created, its source topic stored in storeToChangelogTopic will be used.

2. Added unit test (confirmed that without 1) it will fail).

3. MINOR: removed TODOs that are related to removed KStreamBuilder.

4. MINOR: removed TODOs in StreamsBuilderTest util functions and replaced with TopologyWrapper.

5. MINOR: removed StreamsBuilderTest#testFrom as it is already covered by TopologyTest#shouldNotAllowToAddSourcesWithSameName, plus it requires KStreamImpl.SOURCE_NAME which should be a package private field of the KStreamImpl.

Reviewers: John Roesler <john@confluent.io>, Bill Bejeck <bill@confluent.io>, Matthias
 J. Sax <matthias@confluent.io>
2018-05-17 11:28:45 -07:00
Joan Goyeau ac9de822b2 MINOR: Use Set instead of List for multiple topics (#5024)
Debasish Ghosh <dghosh@acm.org>, Guozhang Wang <guozhang@confluent.io>
2018-05-17 08:44:50 -07:00
Matthias J. Sax 0b3712d8a5
MINOR: add missing parameter `processing.guaratees` to Streams docs (#5023)
Reviewers: Guozhang Wang <guozhang@confluent.io>, Bill Bejeck <bill@confluent.io>
2018-05-16 13:58:25 -07:00
Guozhang Wang d4204e8b14
MINOR: fix broken links in streams doc (#5025)
Reviewers: Guozhang Wang <wangguoz@gmail.com>
2018-05-16 13:23:20 -07:00
David Glasser e9154b7960 KAFKA-6905: Document that Processors may be re-used by Streams (#5022)
Reviewers: Guozhang Wang <wangguoz@gmail.com>
2018-05-16 13:10:21 -07:00
Guozhang Wang c9161afda9
MINOR: doc change for deprecate removal (#5006)
Reviewers: John Roesler <john@confluent.io>, Bill Bejeck <bill@confluent.io>, Matthias J. Sax <matthias@confluent.io>
2018-05-15 20:26:19 -07:00
Boyang Chen 1e207b2ef8 KAFKA-6896: Add producer metrics exporting in KafkaStreams (#4998)
We would like to also export the producer metrics from StreamThread just like consumer metrics, so that we could gain more visibility of stream application. The approach is to pass in the threadProducer into the StreamThread so that we could export its metrics in dynamic.

Note that this is a pure internal change that doesn't require a KIP, and in the future we also want to export admin client metrics. A followup KIP for admin client will be created once this is merged.

Reviewers: Guozhang Wang <wangguoz@gmail.com>
2018-05-15 14:29:07 -07:00
Dong Lin 0bb48a1669 KAFKA-3473; More Controller Health Metrics (KIP-237)
This patch adds a few metrics that are useful for monitoring controller health. See KIP-237 for more detail.

Author: Dong Lin <lindong28@gmail.com>

Reviewers: Jun Rao <junrao@gmail.com>

Closes #4392 from lindong28/KAFKA-3473
2018-05-14 23:04:56 -07:00
Matthias J. Sax 9947cd40c6
MINOR: Ensure sensor names are unique in Kafka Streams (#5009)
Reviewer: Guozhang Wang <guozhang@confluent.io>
2018-05-11 17:19:11 -07:00
Matthias J. Sax adeced2997 HOTFIX: RegexSourceIntegrationTest needs to cleanup shared output topic (#5008)
Reviewers: Guozhang Wang <wangguoz@gmail.com>
2018-05-11 13:53:27 -07:00
Guozhang Wang caca1fdc90
KAFKA-6813: Remove deprecated APIs in KIP-182, Part III (#4991)
1. Remove TopologyBuilder, TopologyBuilderException, KStreamBuilder,

2. Completed the leftover work of https://issues.apache.org/jira/browse/KAFKA-5660, when we remove TopologyBuilderException.

3. Added MockStoreBuilder to replace MockStateStoreSupplier, remove all XXStoreSupplier except StateStoreSupplier as it is still referenced in the logical streams graph.

4. Minor: rename KStreamsFineGrainedAutoResetIntegrationTest.java to FineGrainedAutoResetIntegrationTest.java.

Reviewers: Matthias J. Sax <matthias@confluent.io>
2018-05-11 11:38:02 -07:00
Joel Hamill c14b0ad9ee MINOR - Fix typo in Streams Dev Guide (#4972)
Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <guozhang@confluent.io>
2018-05-11 10:27:07 -07:00