Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Chris Egerton
Hi Arjun, This is great news. Given that we're already willing to ask developers to catch a method-not-found exception and it sounds like a new interface can be handled in a similar try/catch block in a same place, I like the idea of a new fleshed-out interface instead of a BiConsumer or a BiFunct

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Arjun Satish
ok folks, this is my POC PR: https://github.com/wicknicks/kafka/tree/kip-610-2.4.1. connectors built from this were copied into a fresh installation of Kafka Connect (v2.5, the latest version), and ran. Without proper try-catch, the tasks would fail. But when the appropriate exceptions were handled

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Randall Hauch
Thanks for updating the KIP, Aakash. A few comments on the updated content there: In order to avoid error records being written out of order (for example, > due to retries), the developer can use > `max.in.flight.requests.per.connection=1` in their implementation for > writing error records. > IM

Re: [DISCUSS] KIP-612: Ability to Limit Connection Creation Rate on Brokers

2020-05-16 Thread Anna Povzner
Hi Rajini, Thanks for reviewing the KIP! I agree with your suggestion to make per-IP connection rate quota a dynamic quota for entity name IP. This will allow configuring connection rate for a particular IP as well. I updated the wiki accordingly. Your second concern makes sense -- rejecting the

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Arjun Satish
Yeah I had tried this locally on java 8 and 11, and it had seemed to work. Let me clean up and publish my code in a branch somewhere so we can take a look at it. Thanks, On Sat, May 16, 2020 at 3:39 PM Randall Hauch wrote: > Have you tried this? IIUC the problem is with the new type, and any cl

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Randall Hauch
Have you tried this? IIUC the problem is with the new type, and any class that uses ‘ErrantRecordReporter’ with an import would fail to be loaded by the classloader if the type does not exist (I.e., pre-2.9 Connect runtimes). Catching that ClassNotFoundException and dynamically importing the type i

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Aakash Shah
Hi Arjun, Thanks for this suggestion. I actually like this a lot because a defined interface looks more appealing and is clearer in its intention. Since we are still using NoSuchMethodException to account for backwards compatibility, this works for me. I can't see any drawbacks besides having to c

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Arjun Satish
Thanks Konstantine, happy to write something up in a KIP. But I think it would be redundant if we add this kip. What do you think? Also, Randall, yes that API would work. But, if we expect the developers to catch NoSuchMethodErrors, then should we also go ahead and make a class that would have a r

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Aakash Shah
Hi Randall, Thanks for the suggestion. I've updated the KIP with the agreed upon changes as well as the new suggestions Randall mentioned: https://cwiki.apache.org/confluence/display/KAFKA/KIP-610%3A+Error+Reporting+in+Sink+Connectors Please let me know what you think. Thanks, Aakash On Sat, M

Build failed in Jenkins: kafka-trunk-jdk14 #96

2020-05-16 Thread Apache Jenkins Server
See Changes: [github] KAFKA-9617 Replica Fetcher can mark partition as failed when -- [...truncated 3.09 MB...] org.apache.kafka.streams.TopologyTestDriverTest > shouldTh

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Konstantine Karantasis
Thanks for following up Randall. I agree with your latest suggestion. It was good that we explored several options but accessing the context to obtain the reporter in Kafka Connect versions that support this feature makes the most sense. The burden for connector developers that want to use this re

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Randall Hauch
Thanks again for the active discussion! Regarding the future-vs-callback discussion: I did like where Chris was going with the Callback, but he raises good point that it's unclear what to use for the reporter type, since we'd need three parameters. Introducing a new interface makes it much harder

Re: [DISCUSS] KIP-609: Use Pre-registration and Blocking Calls for Better Transaction Efficiency

2020-05-16 Thread Boyang Chen
Thanks Guozhang for the context. The producer batch is either bounded by the size or the linger time. For the default 10ms linger and 100ms transaction commit time, the producer will be capped by AddPartitionToTxn 10 times in the worst case. I think the improvement here aims for the worst case scen

Build failed in Jenkins: kafka-trunk-jdk8 #4537

2020-05-16 Thread Apache Jenkins Server
See Changes: [github] KAFKA-9617 Replica Fetcher can mark partition as failed when -- [...truncated 3.07 MB...] org.apache.kafka.streams.MockTimeTest > shouldAdvanceTime

[jira] [Resolved] (KAFKA-9955) Exceptions thrown from SinkTask::close shadow other exceptions

2020-05-16 Thread Konstantine Karantasis (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-9955?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Konstantine Karantasis resolved KAFKA-9955. --- Resolution: Fixed > Exceptions thrown from SinkTask::close shadow other excep

Re: [DISCUSS] KIP-317 - Add end-to-end data encryption functionality to Apache Kafka

2020-05-16 Thread Ryanne Dolan
Maybe worth taking a look at TDE in HDFS: https://hadoop.apache.org/docs/current/hadoop-project-dist/hadoop-hdfs/TransparentEncryption.html A complete solution requires several Hadoop services. I suspect that would scare the Kafka community a bit, but maybe it's unreasonable to expect Kafka broke

Re: [VOTE] KIP-572: Improve timeouts and retires in Kafka Streams

2020-05-16 Thread Guozhang Wang
Hello Matthias, Thanks for the updated KIP, overall I'm +1 on this proposal. Some minor comments (I know gmail mixed that again for me so I'm leaving it as a combo for both DISCUSS and VOTE :) 1) There are some inconsistent statements in the proposal regarding what to deprecated: at the beginning

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Konstantine Karantasis
Thanks for the quick response Aakash. With respect to deprecation, this refers to deprecating this method in newer versions of Kafka Connect (and eventually removing it). As a connector developer, if you want your connector to run across a wide spectrum of Connect versions, you'll have to take th

Re: [DISCUSS] KIP-609: Use Pre-registration and Blocking Calls for Better Transaction Efficiency

2020-05-16 Thread Guozhang Wang
Hello Boyang, Thanks for the proposed KIP, overall it makes sense to me. One non-public API related point that I'd like to make though, is that in KafkaProducer.send call we can potentially defer sending AddPartitionsToTxn request until the sender is about to send the first batch -- this is what

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Chris Egerton
Hi Konstantine, I don't believe a callback-based implementation would require additional threads to be spun up. The callback could be invoked by the framework whenever the work for reporting a record is done, on that same thread or, in the case of a DLQ topic reporter, in a callback from the produ

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Aakash Shah
+1 On Sat, May 16, 2020 at 9:55 AM Konstantine Karantasis < konstant...@confluent.io> wrote: > Hi Arjun, > > I think I agree with you that subject is interesting. Yet, I feel it > belongs to a separate future KIP. Reading the proposal in the KIP format > will help, at least myself, to understand

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Aakash Shah
Hi Konstantine, Thanks a lot for your feedback. These are all good points, especially that we already have the threads we need and that we'd rather not spin up additional. It is also true we should consider the level of control we want to provide to the developer rather than overstating the burde

[jira] [Resolved] (KAFKA-9617) Replica Fetcher can mark partition as failed when max.message.bytes is changed

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-9617?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-9617. -- Fix Version/s: 2.6.0 Resolution: Fixed > Replica Fetcher can mark partition as failed wh

Re: [DISCUSS] KIP-613: Add end-to-end latency metrics to Streams

2020-05-16 Thread Guozhang Wang
We do have a big JIRA for making sensor recording to be sufficient. But it is a bit hard to try to reduce the num.system clock calls in general (but admittedly, in nowadays OS they are supposed to be very cheap as well), I've just recently merged a small PR trying to make less SystemTime.currentMil

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Konstantine Karantasis
Hi Arjun, I think I agree with you that subject is interesting. Yet, I feel it belongs to a separate future KIP. Reading the proposal in the KIP format will help, at least myself, to understand it better. Having said that, for the purpose of simplifying error handling for sink tasks, the discussi

Re: [DISCUSS] KIP-613: Add end-to-end latency metrics to Streams

2020-05-16 Thread John Roesler
FWIW, I’d be fine with TRACE, if it makes the processor-level metrics more palatable to everyone. Ultimately, I think this is just kicking the can down the road, and we should really just refactor our sensors/metrics to be lower overhead. But that’s too much work to consider right before the re

Re: [DISCUSS] KIP-610: Error Reporting in Sink Connectors

2020-05-16 Thread Konstantine Karantasis
> I believe it is important to relieve as much of the burden of > implementation as possible from the developer in this case, and thus I > think using a Callback rather than a Future would be easier on the > developer, while adding asynchronous functionality with the ability to > opt-in synchronous

Re: [DISCUSS] KIP-613: Add end-to-end latency metrics to Streams

2020-05-16 Thread Guozhang Wang
Cool, that's good points -- we should not let our current metrics proposal to depend on future roadmaps, since they can still be removed later if ever becoming not valuable after all. On the other hand, what do you think about introducing TRACE level metrics recording and make the the processor-no

[jira] [Resolved] (KAFKA-10001) Store's own restore listener should be triggered in store changelog reader

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-10001?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-10001. --- Fix Version/s: 2.6.0 Resolution: Fixed > Store's own restore listener should be trigg

[jira] [Resolved] (KAFKA-8918) Flaky Test org.apache.kafka.trogdor.coordinator.CoordinatorTest.testTaskCancellation

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-8918?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-8918. -- Resolution: Fixed > Flaky Test > org.apache.kafka.trogdor.coordinator.CoordinatorTest.testTask

[jira] [Resolved] (KAFKA-8197) Flaky Test kafka.server.DynamicBrokerConfigTest > testPasswordConfigEncoderSecretChange

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-8197?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-8197. -- Resolution: Fixed > Flaky Test kafka.server.DynamicBrokerConfigTest > > testPasswordConfigEnco

[jira] [Resolved] (KAFKA-7042) Fall back to the old behavior when the broker is too old to recognize LIST_OFFSET versions

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-7042?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-7042. -- Resolution: Not A Problem > Fall back to the old behavior when the broker is too old to recogni

[jira] [Resolved] (KAFKA-5719) Create a quickstart archetype project for Kafka Streams

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-5719?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-5719. -- Resolution: Fixed > Create a quickstart archetype project for Kafka Streams > -

[jira] [Resolved] (KAFKA-1985) Document on possible error codes for each response type

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-1985?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-1985. -- Resolution: Fixed > Document on possible error codes for each response type > -

[jira] [Resolved] (KAFKA-1617) Move Metadata Cache to TopicManager and handling of Offset Request to LogManager

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-1617?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-1617. -- Resolution: Won't Fix > Move Metadata Cache to TopicManager and handling of Offset Request to

[jira] [Resolved] (KAFKA-3103) Transient Failure in kafka.integration.PlaintextTopicMetadataTest.testIsrAfterBrokerShutDownAndJoinsBack

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-3103?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-3103. -- Resolution: Fixed > Transient Failure in > kafka.integration.PlaintextTopicMetadataTest.testIs

[jira] [Resolved] (KAFKA-3168) Failure in kafka.integration.PrimitiveApiTest.testPipelinedProduceRequests

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-3168?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-3168. -- Resolution: Fixed > Failure in kafka.integration.PrimitiveApiTest.testPipelinedProduceRequests

[jira] [Resolved] (KAFKA-3738) Add load system tests for Streams

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-3738?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-3738. -- Resolution: Fixed We have soak tests and perf benchmarks today. > Add load system tests for St

[jira] [Resolved] (KAFKA-1602) Use single error code for offset commit response

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-1602?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-1602. -- Resolution: Not A Problem > Use single error code for offset commit response >

[jira] [Resolved] (KAFKA-4143) Transient failure in kafka.server.SaslSslReplicaFetchTest.testReplicaFetcherThread

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-4143?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-4143. -- Resolution: Fixed Did not see this recently. > Transient failure in > kafka.server.SaslSslRep

[jira] [Resolved] (KAFKA-3220) Failure in kafka.server.ClientQuotaManagerTest.testQuotaViolation

2020-05-16 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-3220?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-3220. -- Resolution: Fixed Did not see this recently. > Failure in kafka.server.ClientQuotaManagerTest.

Re: [DISCUSS] KIP-317 - Add end-to-end data encryption functionality to Apache Kafka

2020-05-16 Thread Adam Bellemare
Hi Sönke I've been giving it more thought over the last few days, and looking into other systems as well, and I think that I've derailed your proposal a bit with suggesting that at-rest encryption may be sufficient. I believe that many of us are lacking the context of the sorts of discussions you

Jenkins build is back to normal : kafka-2.4-jdk8 #209

2020-05-16 Thread Apache Jenkins Server
See

[jira] [Resolved] (KAFKA-6959) Any impact we foresee if we upgrade Linux version or move to VM instead of physical Linux server

2020-05-16 Thread Alexandre Dupriez (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-6959?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Alexandre Dupriez resolved KAFKA-6959. -- Resolution: Fixed > Any impact we foresee if we upgrade Linux version or move to VM ins

[jira] [Resolved] (KAFKA-8916) Unreliable kafka-reassign-partitions.sh affecting performance

2020-05-16 Thread Alexandre Dupriez (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-8916?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Alexandre Dupriez resolved KAFKA-8916. -- Resolution: Invalid Closing this as there is no bug or development required. Please ki

[jira] [Resolved] (KAFKA-8815) Kafka broker blocked on I/O primitive

2020-05-16 Thread Alexandre Dupriez (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-8815?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Alexandre Dupriez resolved KAFKA-8815. -- Resolution: Not A Problem System failure. Not related to Kafka. > Kafka broker blocked

[jira] [Created] (KAFKA-10009) Add method for getting last record offset in kafka partition

2020-05-16 Thread Yuriy Badalyantc (Jira)
Yuriy Badalyantc created KAFKA-10009: Summary: Add method for getting last record offset in kafka partition Key: KAFKA-10009 URL: https://issues.apache.org/jira/browse/KAFKA-10009 Project: Kafka

Jenkins build is back to normal : kafka-trunk-jdk11 #1469

2020-05-16 Thread Apache Jenkins Server
See

Re: [DISCUSS] KIP-612: Ability to Limit Connection Creation Rate on Brokers

2020-05-16 Thread Alexandre Dupriez
Hi Anna, Thank you for your answers and explanations. A couple of additional comments: 900. KIP-612 does not intend to dedicate a metric to the throttling of incoming connections. I wonder if such a metric would be handy for monitoring and help set-up metric-based alarming if one wishes to captu

Jenkins build is back to normal : kafka-trunk-jdk14 #95

2020-05-16 Thread Apache Jenkins Server
See

Build failed in Jenkins: kafka-2.5-jdk8 #125

2020-05-16 Thread Apache Jenkins Server
See Changes: [rhauch] KAFKA-9537 - Cleanup error messages for abstract transformations (#8090) [konstantine] KAFKA-9955: Prevent SinkTask::close from shadowing other exceptions -

Build failed in Jenkins: kafka-trunk-jdk8 #4536

2020-05-16 Thread Apache Jenkins Server
See Changes: [github] KAFKA-9955: Prevent SinkTask::close from shadowing other exceptions -- [...truncated 6.14 MB...] org.apache.kafka.streams.TopologyTestDriverTest >