Re: [VOTE] KIP-618: Exactly-once support for source connectors

2021-06-03 Thread Ryanne Dolan
+1 (non-binding)

On Thu, Jun 3, 2021, 10:23 AM Chris Egerton 
wrote:

> Hi all,
>
> I'd like to call for a vote on KIP-618, which adds support for exactly-once
> delivery guarantees for source connectors in the Kafka Connect framework.
>
> I suspect there might be a little more discussion to be had but with the
> KIP freeze deadline approaching, I wanted to give anyone following along
> the chance to cast a +1 as soon as they feel that we've gotten to a
> reasonable state.
>
> The KIP:
>
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-618%3A+Exactly-Once+Support+for+Source+Connectors
>
> The discussion thread:
>
> https://mail-archives.apache.org/mod_mbox/kafka-dev/202005.mbox/%3CCAMdOrUX-VK5OSB3OOdJNXW_YKEJH9FjQZ4eyzr2GXUhSeDnF3Q%40mail.gmail.com%3E
>
> Cheers,
>
> Chris
>


Re: [DISCUSS] KIP-655: Windowed "Distinct" Operation for KStream

2021-06-03 Thread Matthias J. Sax
Just skimmed over the thread -- first of all, I am glad that we could
merge KIP-418 and ship it :)

About the re-partitioning concerns, there are already two tickets for it:

 - https://issues.apache.org/jira/browse/KAFKA-4835
 - https://issues.apache.org/jira/browse/KAFKA-10844

Thus, it seems best to exclude this topic from this KIP, and do a
separate KIP for it (if necessary, we can "pause" this KIP until the
repartition KIP is done). It's a long standing "issue" and we should
resolve it in a general way I guess.

(Did not yet ready all responses in detail yet, so keeping this comment
short.)


-Matthias

On 6/2/21 6:35 AM, John Roesler wrote:
> Thanks, Ivan!
> 
> That sounds like a great plan to me. Two smaller KIPs are easier to agree on 
> than one big one. 
> 
> I agree hopping and sliding windows will actually have a duplicating effect. 
> We can avoid adding distinct() to the sliding window interface, but hopping 
> windows are just a different parameterization of epoch-aligned windows. It 
> seems we can’t do much about that except document the issue.
> 
> Thanks,
> John
> 
> On Wed, May 26, 2021, at 10:14, Ivan Ponomarev wrote:
>> Hi John!
>>
>> I think that your proposal is just fantastic, it simplifies things a lot!
>>
>> I also felt uncomfortable due to the fact that the proposed `distinct()` 
>> is not somewhere near `count()` and `reduce(..)`. But 
>> `selectKey(..).groupByKey().windowedBy(..).distinct()` didn't look like 
>> a correct option for  me because of the issue with the unneeded 
>> repartitioning.
>>
>> The bold idea that we can just CANCEL the repartitioning didn't came to 
>> my mind.
>>
>> What seemed to me a single problem is in fact two unrelated problems: 
>> `distinct` operation and cancelling the unneeded repartitioning.
>>
>>  > what if we introduce a parameter to `selectKey()` that specifies that 
>> the caller asserts that the new key does _not_ change the data partitioning?
>>
>> I think a more elegant solution would be not to add a new parameter to 
>> `selectKey` and all the other key-changing operations (`map`, 
>> `transform`, `flatMap`, ...), but add a new operator 
>> `KStream#cancelRepartitioning()` that resets `keyChangingOperation` flag 
>> for the upstream node. Of course, "use it only if you know what you're 
>> doing" warning is to be added. Well, it's a topic for a separate KIP!
>>
>> Concerning `distinct()`. If we use `XXXWindowedKStream` facilities, then 
>> changes to the API are minimally invasive: we're just adding 
>> `distinct()` to TimeWindowedKStream and SessionWindowedKStream, and 
>> that's all.
>>
>> We can now define `distinct` as an operation that returns only a first 
>> record that falls into a new window, and filters out all the other 
>> records that fall into an already existing window. BTW, we can mock the 
>> behaviour of such an operation with `TopologyTestDriver` using 
>> `reduce((l, r) -> STOP)`.filterNot((k, v)->STOP.equals(v)).  ;-)
>>
>> Consider the following example (record times are in seconds):
>>
>> //three bursts of variously ordered records
>> 4, 5, 6
>> 23, 22, 24
>> 34, 33, 32
>> //'late arrivals'
>> 7, 22, 35
>>
>>
>> 1. 'Epoch-aligned deduplication' using tumbling windows:
>>
>> .groupByKey().windowedBy(TimeWindows.of(Duration.ofSeconds(10))).distinct()
>>
>> produces
>>
>> (key@[0/1], 4)
>> (key@[2/3], 23)
>> (key@[3/4], 34)
>>
>> -- that is, one record per epoch-aligned window.
>>
>> 2. Hopping and sliding windows do not make much sense here, because they 
>> produce multiple intersected windows, so that one record can be 
>> multiplied, but we want deduplication.
>>
>> 3. SessionWindows work for 'data-aligned deduplication'.
>>
>> .groupByKey().windowedBy(SessionWindows.with(Duration.ofSeconds(10))).distinct()
>>  
>>
>>
>> produces only
>>
>> ([key@4000/4000], 4)
>> ([key@23000/23000], 23)
>>
>> because all the records bigger than 7 are stuck together in one session. 
>> Setting inactivity gap to 9 seconds will return three records:
>>
>> ([key@4000/4000], 4)
>> ([key@23000/23000], 23)
>> ([key@34000/34000], 34)
>>
>> WDYT? If you like this variant, I will re-write KIP-655 and propose a 
>> separate KIP for `cancelRepartitioning` (or whatever name we will choose 
>> for it).
>>
>> Regards,
>>
>> Ivan
>>
>>
>> 24.05.2021 22:32, John Roesler пишет:
>>> Hey there, Ivan!
>>>
>>> In typical fashion, I'm going to make a somewhat outlandish
>>> proposal. I'm hoping that we can side-step some of the
>>> complications that have arisen. Please bear with me.
>>>
>>> It seems like `distinct()` is not fundamentally unlike other windowed
>>> "aggregation" operations. Your concern about unnecessary
>>> repartitioning seems to apply just as well to `count()` as to `distinct()`.
>>> This has come up before, but I don't remember when: what if we
>>> introduce a parameter to `selectKey()` that specifies that the caller
>>> asserts that the new key does _not_ change the data partitioning?
>>> The docs on 

Build failed in Jenkins: Kafka » Kafka Branch Builder » trunk #195

2021-06-03 Thread Apache Jenkins Server
See 


Changes:


--
[...truncated 473306 lines...]
[2021-06-04T03:45:18.877Z] 
[2021-06-04T03:45:18.877Z] TopicCommandIntegrationTest > 
testDescribeUnderMinIsrPartitionsMixed() STARTED
[2021-06-04T03:45:21.977Z] 
[2021-06-04T03:45:21.977Z] SaslSslAdminIntegrationTest > 
testAttemptToCreateInvalidAcls() PASSED
[2021-06-04T03:45:21.977Z] 
[2021-06-04T03:45:21.977Z] SaslSslAdminIntegrationTest > 
testAclAuthorizationDenied() STARTED
[2021-06-04T03:45:27.661Z] 
[2021-06-04T03:45:27.661Z] TopicCommandIntegrationTest > 
testDescribeUnderMinIsrPartitionsMixed() PASSED
[2021-06-04T03:45:27.661Z] 
[2021-06-04T03:45:27.661Z] TopicCommandIntegrationTest > 
testCreateWhenAlreadyExists() STARTED
[2021-06-04T03:45:31.337Z] 
[2021-06-04T03:45:31.337Z] TopicCommandIntegrationTest > 
testCreateWhenAlreadyExists() PASSED
[2021-06-04T03:45:31.337Z] 
[2021-06-04T03:45:31.337Z] TopicCommandIntegrationTest > testDescribe() STARTED
[2021-06-04T03:45:36.070Z] 
[2021-06-04T03:45:36.070Z] TopicCommandIntegrationTest > testDescribe() PASSED
[2021-06-04T03:45:36.070Z] 
[2021-06-04T03:45:36.070Z] TopicCommandIntegrationTest > testAlterAssignment() 
STARTED
[2021-06-04T03:45:54.931Z] 
[2021-06-04T03:45:54.931Z] SaslSslAdminIntegrationTest > 
testAclAuthorizationDenied() PASSED
[2021-06-04T03:45:54.931Z] 
[2021-06-04T03:45:54.931Z] SaslSslAdminIntegrationTest > testAclOperations() 
STARTED
[2021-06-04T03:46:12.332Z] 
[2021-06-04T03:46:12.332Z] TopicCommandIntegrationTest > testAlterAssignment() 
PASSED
[2021-06-04T03:46:12.332Z] 
[2021-06-04T03:46:12.332Z] DynamicConnectionQuotaTest > 
testDynamicConnectionQuota() STARTED
[2021-06-04T03:46:14.136Z] 
[2021-06-04T03:46:14.136Z] DynamicConnectionQuotaTest > 
testDynamicConnectionQuota() PASSED
[2021-06-04T03:46:14.136Z] 
[2021-06-04T03:46:14.136Z] DynamicConnectionQuotaTest > 
testDynamicListenerConnectionQuota() STARTED
[2021-06-04T03:46:21.311Z] 
[2021-06-04T03:46:21.311Z] DynamicConnectionQuotaTest > 
testDynamicListenerConnectionQuota() PASSED
[2021-06-04T03:46:21.311Z] 
[2021-06-04T03:46:21.311Z] DynamicConnectionQuotaTest > 
testDynamicListenerConnectionCreationRateQuota() STARTED
[2021-06-04T03:46:28.605Z] 
[2021-06-04T03:46:28.605Z] SaslSslAdminIntegrationTest > testAclOperations() 
PASSED
[2021-06-04T03:46:28.605Z] 
[2021-06-04T03:46:28.605Z] SaslSslAdminIntegrationTest > testAclOperations2() 
STARTED
[2021-06-04T03:46:56.049Z] 
[2021-06-04T03:46:56.049Z] SaslSslAdminIntegrationTest > testAclOperations2() 
PASSED
[2021-06-04T03:46:56.049Z] 
[2021-06-04T03:46:56.049Z] SaslSslAdminIntegrationTest > testAclDelete() STARTED
[2021-06-04T03:47:27.767Z] 
[2021-06-04T03:47:27.767Z] SaslSslAdminIntegrationTest > testAclDelete() PASSED
[2021-06-04T03:47:27.767Z] 
[2021-06-04T03:47:27.767Z] TransactionsTest > testBumpTransactionalEpoch() 
STARTED
[2021-06-04T03:47:29.377Z] 
[2021-06-04T03:47:29.377Z] DynamicConnectionQuotaTest > 
testDynamicListenerConnectionCreationRateQuota() PASSED
[2021-06-04T03:47:29.377Z] 
[2021-06-04T03:47:29.377Z] DynamicConnectionQuotaTest > 
testDynamicIpConnectionRateQuota() STARTED
[2021-06-04T03:47:44.248Z] 
[2021-06-04T03:47:44.249Z] TransactionsTest > testBumpTransactionalEpoch() 
PASSED
[2021-06-04T03:47:44.249Z] 
[2021-06-04T03:47:44.249Z] TransactionsTest > 
testSendOffsetsWithGroupMetadata() STARTED
[2021-06-04T03:47:59.598Z] 
[2021-06-04T03:47:59.598Z] TransactionsTest > 
testSendOffsetsWithGroupMetadata() PASSED
[2021-06-04T03:47:59.598Z] 
[2021-06-04T03:47:59.598Z] TransactionsTest > testBasicTransactions() STARTED
[2021-06-04T03:48:06.500Z] 
[2021-06-04T03:48:06.500Z] TransactionsTest > testBasicTransactions() PASSED
[2021-06-04T03:48:06.500Z] 
[2021-06-04T03:48:06.500Z] TransactionsTest > testSendOffsetsWithGroupId() 
STARTED
[2021-06-04T03:48:11.347Z] 
[2021-06-04T03:48:11.347Z] DynamicConnectionQuotaTest > 
testDynamicIpConnectionRateQuota() PASSED
[2021-06-04T03:48:11.347Z] 
[2021-06-04T03:48:11.347Z] Deprecated Gradle features were used in this build, 
making it incompatible with Gradle 8.0.
[2021-06-04T03:48:11.347Z] Use '--warning-mode all' to show the individual 
deprecation warnings.
[2021-06-04T03:48:11.347Z] See 
https://docs.gradle.org/7.0.2/userguide/command_line_interface.html#sec:command_line_warnings
[2021-06-04T03:48:11.347Z] 
[2021-06-04T03:48:11.347Z] BUILD SUCCESSFUL in 1h 50m 5s
[2021-06-04T03:48:11.347Z] 196 actionable tasks: 105 executed, 91 up-to-date
[2021-06-04T03:48:11.347Z] 
[2021-06-04T03:48:11.347Z] See the profiling report at: 
file:///home/jenkins/jenkins-agent/workspace/Kafka_kafka_trunk/build/reports/profile/profile-2021-06-04-01-58-09.html
[2021-06-04T03:48:11.347Z] A fine-grained performance profile is available: use 
the --scan option.
[Pipeline] junit
[2021-06-04T03:48:12.241Z] Recording test results
[2021-06-04T03:48:18.211Z] 
[2021-06-04T03:48:18.211Z] TransactionsTest > 

Build failed in Jenkins: Kafka » Kafka Branch Builder » trunk #194

2021-06-03 Thread Apache Jenkins Server
See 


Changes:


--
[...truncated 472601 lines...]
[2021-06-04T01:29:52.582Z] > Task :clients:jar UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :server-common:compileJava UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :storage:api:compileJava UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :connect:api:compileJava UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :connect:api:classes UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :streams:compileJava UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :streams:classes UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :connect:json:compileJava UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :storage:compileJava UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :connect:json:classes UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :raft:compileJava UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :connect:json:javadoc SKIPPED
[2021-06-04T01:29:52.582Z] > Task :connect:json:javadocJar
[2021-06-04T01:29:52.582Z] > Task :clients:compileTestJava UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :clients:testClasses UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :streams:test-utils:compileJava UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :metadata:compileJava UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :core:compileJava NO-SOURCE
[2021-06-04T01:29:52.582Z] > Task :connect:json:compileTestJava UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :connect:json:testClasses UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task :connect:json:testJar
[2021-06-04T01:29:52.582Z] > Task :connect:json:testSrcJar
[2021-06-04T01:29:52.582Z] > Task :streams:copyDependantLibs
[2021-06-04T01:29:52.582Z] > Task 
:clients:generateMetadataFileForMavenJavaPublication
[2021-06-04T01:29:52.582Z] > Task :streams:jar UP-TO-DATE
[2021-06-04T01:29:52.582Z] > Task 
:streams:generateMetadataFileForMavenJavaPublication
[2021-06-04T01:29:55.997Z] > Task :connect:api:javadoc
[2021-06-04T01:29:57.044Z] > Task :connect:api:copyDependantLibs UP-TO-DATE
[2021-06-04T01:29:57.044Z] > Task :connect:api:jar UP-TO-DATE
[2021-06-04T01:29:57.044Z] > Task 
:connect:api:generateMetadataFileForMavenJavaPublication
[2021-06-04T01:29:57.044Z] > Task :connect:json:copyDependantLibs UP-TO-DATE
[2021-06-04T01:29:57.044Z] > Task :connect:json:jar UP-TO-DATE
[2021-06-04T01:29:57.044Z] > Task 
:connect:json:generateMetadataFileForMavenJavaPublication
[2021-06-04T01:29:57.044Z] > Task :connect:api:javadocJar
[2021-06-04T01:29:57.044Z] > Task 
:connect:json:publishMavenJavaPublicationToMavenLocal
[2021-06-04T01:29:57.044Z] > Task :connect:json:publishToMavenLocal
[2021-06-04T01:29:57.044Z] > Task :connect:api:compileTestJava UP-TO-DATE
[2021-06-04T01:29:57.044Z] > Task :connect:api:testClasses UP-TO-DATE
[2021-06-04T01:29:57.044Z] > Task :connect:api:testJar
[2021-06-04T01:29:57.044Z] > Task :connect:api:testSrcJar
[2021-06-04T01:29:57.044Z] > Task 
:connect:api:publishMavenJavaPublicationToMavenLocal
[2021-06-04T01:29:57.044Z] > Task :connect:api:publishToMavenLocal
[2021-06-04T01:29:59.142Z] > Task :streams:javadoc
[2021-06-04T01:29:59.142Z] > Task :streams:javadocJar
[2021-06-04T01:30:01.401Z] > Task :clients:javadoc
[2021-06-04T01:30:01.401Z] > Task :clients:javadocJar
[2021-06-04T01:30:02.350Z] > Task :clients:testJar
[2021-06-04T01:30:02.350Z] > Task :clients:testSrcJar
[2021-06-04T01:30:02.350Z] > Task 
:clients:publishMavenJavaPublicationToMavenLocal
[2021-06-04T01:30:02.350Z] > Task :clients:publishToMavenLocal
[2021-06-04T01:30:22.466Z] > Task :core:compileScala
[2021-06-04T01:31:35.014Z] > Task :core:classes
[2021-06-04T01:31:35.014Z] > Task :core:compileTestJava NO-SOURCE
[2021-06-04T01:31:57.264Z] > Task :core:compileTestScala
[2021-06-04T01:32:42.735Z] > Task :core:testClasses
[2021-06-04T01:33:12.399Z] > Task :streams:compileTestJava
[2021-06-04T01:34:13.792Z] > Task :streams:testClasses
[2021-06-04T01:34:13.792Z] > Task :streams:testJar
[2021-06-04T01:34:13.792Z] > Task :streams:testSrcJar
[2021-06-04T01:34:13.792Z] > Task 
:streams:publishMavenJavaPublicationToMavenLocal
[2021-06-04T01:34:13.792Z] > Task :streams:publishToMavenLocal
[2021-06-04T01:34:13.792Z] 
[2021-06-04T01:34:13.792Z] Deprecated Gradle features were used in this build, 
making it incompatible with Gradle 8.0.
[2021-06-04T01:34:13.792Z] Use '--warning-mode all' to show the individual 
deprecation warnings.
[2021-06-04T01:34:13.792Z] See 
https://docs.gradle.org/7.0.2/userguide/command_line_interface.html#sec:command_line_warnings
[2021-06-04T01:34:13.792Z] 
[2021-06-04T01:34:13.792Z] Execution optimizations have been disabled for 2 
invalid unit(s) of work during this build to ensure correctness.
[2021-06-04T01:34:13.792Z] Please consult deprecation warnings for more details.
[2021-06-04T01:34:13.792Z] 
[2021-06-04T01:34:13.792Z] BUILD SUCCESSFUL in 4m 35s
[2021-06-04T01:34:13.792Z] 71 actionable tasks: 37 executed, 34 up-to-date
[Pipeline] sh
[2021-06-04T01:34:17.286Z] + 

[jira] [Created] (KAFKA-12888) Add transaction tool

2021-06-03 Thread Jason Gustafson (Jira)
Jason Gustafson created KAFKA-12888:
---

 Summary: Add transaction tool
 Key: KAFKA-12888
 URL: https://issues.apache.org/jira/browse/KAFKA-12888
 Project: Kafka
  Issue Type: Sub-task
Reporter: Jason Gustafson
Assignee: Jason Gustafson
 Fix For: 3.0.0


Implement the transaction tool described in KIP-664: 
https://cwiki.apache.org/confluence/display/KAFKA/KIP-664%3A+Provide+tooling+to+detect+and+abort+hanging+transactions.



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


Re: [VOTE] KIP-726: Make the "cooperative-sticky, range" as the default assignor

2021-06-03 Thread Guozhang Wang
Thanks Luke, I'm +1 on this proposal.

Guozhang

On Wed, Jun 2, 2021 at 8:16 PM Luke Chen  wrote:

> Hi all,
> I'd like to call for a vote on KIP-726: Make the "cooperative-sticky,
> range" as the default assignor.
>
> https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=177048248
>
> This KIP is still waiting for the prerequisite stories to get completed,
> but it should be soon. Hopefully this can be put into V3.0 since
> cooperative rebalancing is a major
> improvement to the life of a consumer group (and its operator).
>
> The discussion thread can be found here:
>
> https://lists.apache.org/thread.html/ref63417ea84a58c9068ea025b3ad38ca2cc340f5818ac07cd83452b7%40%3Cdev.kafka.apache.org%3E
>
> Thank you.
> Luke
>


-- 
-- Guozhang


[jira] [Created] (KAFKA-12887) Do not trigger user-customized ExceptionalHandler for RTE

2021-06-03 Thread Guozhang Wang (Jira)
Guozhang Wang created KAFKA-12887:
-

 Summary: Do not trigger user-customized ExceptionalHandler for RTE
 Key: KAFKA-12887
 URL: https://issues.apache.org/jira/browse/KAFKA-12887
 Project: Kafka
  Issue Type: Improvement
  Components: streams
Reporter: Guozhang Wang


Today in StreamThread we have a try-catch block that captures all {{Throwable 
e}} and then triggers {{this.streamsUncaughtExceptionHandler.accept(e)}}. 
However, there are possible RTEs such as IllegalState/IllegalArgument 
exceptions which are usually caused by bugs, etc. In such cases we should not 
let users to decide what to do with these exceptions, but should let Streams 
itself to enforce the decision, e.g. in the IllegalState/IllegalArgument we 
should fail fast to notify the potential error.



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


Re: [DISCUSS] KIP-746: Revise KRaft Metadata Records

2021-06-03 Thread Ismael Juma
Quick question: given that upgrades are not supported, do we actually want
to introduce a new version for the RPCs?

Ismael

On Wed, Jun 2, 2021 at 11:02 AM Colin McCabe  wrote:

> Hi all,
>
> I have posted a KIP about updating the KRaft metadata records for 3.0.
>
> Check it out at : https://cwiki.apache.org/confluence/x/34zOCg
>
> best,
> Colin
>


Re: [DISCUSS] KIP-746: Revise KRaft Metadata Records

2021-06-03 Thread Colin McCabe
On Thu, Jun 3, 2021, at 16:29, Jun Rao wrote:
> Hi, Colin,
> 
> Thanks for the KIP. Just a couple of minor comments.
> 

Hi Jun,

Thanks for taking a look. Sorry I just started the vote thread before I saw 
this! :)

> 1. Fields like RemovingReplicas are added as tagged fields in
> PartitionChangeRecord, but as non-tagged fields in PartitionRecord. Should
> we make them consistent?
> 

I think it makes sense to make them normal fields in PartitionRecord, since 
they will always be present there. In PartitionChangeRecord, these fields will 
only be present if they are changing, so it makes sense to make them tagged 
fields.

> 2. Should we add BrokerRegistrationChangeRecord later when it has more
> fields than what's already covered in FenceBrokerRecord and
> UnfenceBrokerRecord?
> 

Hmm... eventually we want to have the ability to change the broker endpoints 
dynamically (just like we can do in the ZK-enabled broker). That will certainly 
belong in BrokerRegistrationChangeRecord. If I add this to the record, does 
that give enough motivation to add it now rather than later? I like the 
consistency of having a single change record.

best,
Colin


> Jun
> 
> 
> On Wed, Jun 2, 2021 at 11:02 AM Colin McCabe  wrote:
> 
> > Hi all,
> >
> > I have posted a KIP about updating the KRaft metadata records for 3.0.
> >
> > Check it out at : https://cwiki.apache.org/confluence/x/34zOCg
> >
> > best,
> > Colin
> >
> 


[VOTE] KIP-746: Revise KRaft Metadata Records

2021-06-03 Thread Colin McCabe
Hi all,

I'd like to call a vote for KIP-746: Revise KRaft Metadata Records. This is a 
minor KIP which revises the KRaft metadata records slightly for the upcoming 
3.0 release.

The KIP is at: https://cwiki.apache.org/confluence/x/34zOCg

best,
Colin


Re: [DISCUSS] KIP-746: Revise KRaft Metadata Records

2021-06-03 Thread Jun Rao
Hi, Colin,

Thanks for the KIP. Just a couple of minor comments.

1. Fields like RemovingReplicas are added as tagged fields in
PartitionChangeRecord, but as non-tagged fields in PartitionRecord. Should
we make them consistent?

2. Should we add BrokerRegistrationChangeRecord later when it has more
fields than what's already covered in FenceBrokerRecord and
UnfenceBrokerRecord?

Jun


On Wed, Jun 2, 2021 at 11:02 AM Colin McCabe  wrote:

> Hi all,
>
> I have posted a KIP about updating the KRaft metadata records for 3.0.
>
> Check it out at : https://cwiki.apache.org/confluence/x/34zOCg
>
> best,
> Colin
>


Re: [DISCUSS] KIP-748: Add Broker Count Metrics

2021-06-03 Thread Colin McCabe
Hi Ryan,

Thanks for the KIP. I think it would be good to provide the 
RegisteredBrokerCount metric for the ZK controller as well as for the Quorum 
controller. Looks good aside from that!

best,
Colin

On Thu, Jun 3, 2021, at 14:09, Ryan Dielhenn wrote:
> Hey kafka-dev,
> 
> I created KIP-748 as a proposal to add broker count metrics to the 
> Quorum Controller.
> 
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-748%3A+Add+Broker+Count+Metrics#KIP748:AddBrokerCountMetrics
> 
> Best, 
> Ryan Dielhenn
> 


Build failed in Jenkins: Kafka » Kafka Branch Builder » trunk #193

2021-06-03 Thread Apache Jenkins Server
See 


Changes:


--
[...truncated 405148 lines...]
[2021-06-03T22:38:02.534Z] KafkaZkClientTest > testConditionalUpdatePath() 
PASSED
[2021-06-03T22:38:02.534Z] 
[2021-06-03T22:38:02.534Z] KafkaZkClientTest > 
testGetAllTopicsInClusterTriggersWatch() STARTED
[2021-06-03T22:38:03.498Z] 
[2021-06-03T22:38:03.498Z] KafkaZkClientTest > 
testGetAllTopicsInClusterTriggersWatch() PASSED
[2021-06-03T22:38:03.498Z] 
[2021-06-03T22:38:03.498Z] KafkaZkClientTest > testDeleteTopicZNode() STARTED
[2021-06-03T22:38:03.498Z] 
[2021-06-03T22:38:03.498Z] KafkaZkClientTest > testDeleteTopicZNode() PASSED
[2021-06-03T22:38:03.498Z] 
[2021-06-03T22:38:03.498Z] KafkaZkClientTest > testDeletePath() STARTED
[2021-06-03T22:38:03.498Z] 
[2021-06-03T22:38:03.498Z] KafkaZkClientTest > testDeletePath() PASSED
[2021-06-03T22:38:03.498Z] 
[2021-06-03T22:38:03.498Z] KafkaZkClientTest > testGetBrokerMethods() STARTED
[2021-06-03T22:38:04.462Z] 
[2021-06-03T22:38:04.462Z] KafkaZkClientTest > testGetBrokerMethods() PASSED
[2021-06-03T22:38:04.462Z] 
[2021-06-03T22:38:04.462Z] KafkaZkClientTest > 
testCreateTokenChangeNotification() STARTED
[2021-06-03T22:38:04.462Z] 
[2021-06-03T22:38:04.462Z] KafkaZkClientTest > 
testCreateTokenChangeNotification() PASSED
[2021-06-03T22:38:04.462Z] 
[2021-06-03T22:38:04.462Z] KafkaZkClientTest > testGetTopicsAndPartitions() 
STARTED
[2021-06-03T22:38:04.462Z] 
[2021-06-03T22:38:04.462Z] KafkaZkClientTest > testGetTopicsAndPartitions() 
PASSED
[2021-06-03T22:38:05.426Z] 
[2021-06-03T22:38:05.426Z] KafkaZkClientTest > testChroot(boolean) > 
kafka.zk.KafkaZkClientTest.testChroot(boolean)[1] STARTED
[2021-06-03T22:38:05.426Z] 
[2021-06-03T22:38:05.426Z] KafkaZkClientTest > testChroot(boolean) > 
kafka.zk.KafkaZkClientTest.testChroot(boolean)[1] PASSED
[2021-06-03T22:38:05.426Z] 
[2021-06-03T22:38:05.426Z] KafkaZkClientTest > testChroot(boolean) > 
kafka.zk.KafkaZkClientTest.testChroot(boolean)[2] STARTED
[2021-06-03T22:38:06.389Z] 
[2021-06-03T22:38:06.389Z] KafkaZkClientTest > testChroot(boolean) > 
kafka.zk.KafkaZkClientTest.testChroot(boolean)[2] PASSED
[2021-06-03T22:38:06.389Z] 
[2021-06-03T22:38:06.389Z] KafkaZkClientTest > testRegisterBrokerInfo() STARTED
[2021-06-03T22:38:06.389Z] 
[2021-06-03T22:38:06.389Z] KafkaZkClientTest > testRegisterBrokerInfo() PASSED
[2021-06-03T22:38:06.389Z] 
[2021-06-03T22:38:06.389Z] KafkaZkClientTest > testRetryRegisterBrokerInfo() 
STARTED
[2021-06-03T22:38:06.389Z] 
[2021-06-03T22:38:06.389Z] KafkaZkClientTest > testRetryRegisterBrokerInfo() 
PASSED
[2021-06-03T22:38:06.389Z] 
[2021-06-03T22:38:06.389Z] KafkaZkClientTest > testConsumerOffsetPath() STARTED
[2021-06-03T22:38:07.353Z] 
[2021-06-03T22:38:07.353Z] KafkaZkClientTest > testConsumerOffsetPath() PASSED
[2021-06-03T22:38:07.353Z] 
[2021-06-03T22:38:07.353Z] KafkaZkClientTest > 
testDeleteRecursiveWithControllerEpochVersionCheck() STARTED
[2021-06-03T22:38:07.353Z] 
[2021-06-03T22:38:07.353Z] KafkaZkClientTest > 
testDeleteRecursiveWithControllerEpochVersionCheck() PASSED
[2021-06-03T22:38:07.353Z] 
[2021-06-03T22:38:07.353Z] KafkaZkClientTest > testTopicAssignments() STARTED
[2021-06-03T22:38:07.353Z] 
[2021-06-03T22:38:07.353Z] KafkaZkClientTest > testTopicAssignments() PASSED
[2021-06-03T22:38:07.353Z] 
[2021-06-03T22:38:07.353Z] KafkaZkClientTest > 
testControllerManagementMethods() STARTED
[2021-06-03T22:38:08.316Z] 
[2021-06-03T22:38:08.316Z] KafkaZkClientTest > 
testControllerManagementMethods() PASSED
[2021-06-03T22:38:08.316Z] 
[2021-06-03T22:38:08.316Z] KafkaZkClientTest > testTopicAssignmentMethods() 
STARTED
[2021-06-03T22:38:08.316Z] 
[2021-06-03T22:38:08.316Z] KafkaZkClientTest > testTopicAssignmentMethods() 
PASSED
[2021-06-03T22:38:08.316Z] 
[2021-06-03T22:38:08.316Z] KafkaZkClientTest > testConnectionViaNettyClient() 
STARTED
[2021-06-03T22:38:09.280Z] 
[2021-06-03T22:38:09.280Z] KafkaZkClientTest > testConnectionViaNettyClient() 
PASSED
[2021-06-03T22:38:09.280Z] 
[2021-06-03T22:38:09.280Z] KafkaZkClientTest > testPropagateIsrChanges() STARTED
[2021-06-03T22:38:09.280Z] 
[2021-06-03T22:38:09.280Z] KafkaZkClientTest > testPropagateIsrChanges() PASSED
[2021-06-03T22:38:09.280Z] 
[2021-06-03T22:38:09.280Z] KafkaZkClientTest > testControllerEpochMethods() 
STARTED
[2021-06-03T22:38:10.067Z] 
[2021-06-03T22:38:10.067Z] KafkaZkClientTest > testControllerEpochMethods() 
PASSED
[2021-06-03T22:38:10.067Z] 
[2021-06-03T22:38:10.067Z] KafkaZkClientTest > testDeleteRecursive() STARTED
[2021-06-03T22:38:10.067Z] 
[2021-06-03T22:38:10.067Z] KafkaZkClientTest > testDeleteRecursive() PASSED
[2021-06-03T22:38:10.067Z] 
[2021-06-03T22:38:10.067Z] KafkaZkClientTest > testGetTopicPartitionStates() 
STARTED
[2021-06-03T22:38:10.067Z] 
[2021-06-03T22:38:10.067Z] KafkaZkClientTest > testGetTopicPartitionStates() 
PASSED
[2021-06-03T22:38:10.067Z] 
[2021-06-03T22:38:10.067Z] KafkaZkClientTest > 

[jira] [Created] (KAFKA-12886) Enable request forwarding by default in 3.0

2021-06-03 Thread Jason Gustafson (Jira)
Jason Gustafson created KAFKA-12886:
---

 Summary: Enable request forwarding by default in 3.0
 Key: KAFKA-12886
 URL: https://issues.apache.org/jira/browse/KAFKA-12886
 Project: Kafka
  Issue Type: Improvement
Reporter: Jason Gustafson
Assignee: Jason Gustafson


KIP-590 documents that request forwarding will be enabled in 3.0 by default: 
https://cwiki.apache.org/confluence/display/KAFKA/KIP-590%3A+Redirect+Zookeeper+Mutation+Protocols+to+The+Controller.
 This makes it a requirement for users with custom principal implementations to 
provide a `KafkaPrincipalSerde` implementation. We waited until 3.0 because we 
saw this as a compatibility break. 

The KIP documents that use of forwarding will be controlled by the IBP. So once 
the IBP has been configured to 3.0 or above, then the brokers will begin 
forwarding.

(Note that forwarding has always been a requirement for kraft.)





--
This message was sent by Atlassian Jira
(v8.3.4#803005)


[DISCUSS] KIP-748: Add Broker Count Metrics

2021-06-03 Thread Ryan Dielhenn
Hey kafka-dev,

I created KIP-748 as a proposal to add broker count metrics to the Quorum 
Controller.

https://cwiki.apache.org/confluence/display/KAFKA/KIP-748%3A+Add+Broker+Count+Metrics#KIP748:AddBrokerCountMetrics

Best, 
Ryan Dielhenn


Re: [VOTE] KIP-738: Removal of Connect's internal converter properties

2021-06-03 Thread Tom Bentley
+1 (binding).

Thanks Chris!

On Wed, Jun 2, 2021 at 11:00 PM Randall Hauch  wrote:

> Thanks for putting this together, Chris.
>
> Technically, we don't need a new KIP to explicitly remove an API, config,
> etc. that was previously deprecated under an earlier approved KIP. But
> KIP-174 could have been a bit more explicit (note taken for future KIPs)
> that deprecation means "deprecation and future removal", and KIP-738 does
> add a nice migration path for anyone stuck using a non-default internal
> converter.
>
> So I'm +1 (binding) to just wrap this up and remove these configs.
>
> I'd also point out there is a wiki page [1] and dev discussion thread [2]
> highlighting all Connect-related KIPs, including all deprecations. I'll
> update that page to include KIP-736, but it'd be great to have more
> discussion there.
>
> Randall
>
> [1]
> https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=177047362
> [2]
>
> https://lists.apache.org/thread.html/r01e9fa52998a337e435e5d0effca02e74b0552bdec271c1eeca39cd2%40%3Cdev.kafka.apache.org%3E
>
> On Tue, May 18, 2021 at 11:26 AM Ryanne Dolan 
> wrote:
>
> > +1 (non-binding)
> >
> > Thanks!
> >
> > Ryanne
> >
> > On Tue, May 18, 2021, 6:38 AM Chris Egerton  >
> > wrote:
> >
> > > Hi all,
> > >
> > > I'd like to call for a vote on KIP-738:
> > >
> > >
> >
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-738%3A+Removal+of+Connect%27s+internal+converter+properties
> > >
> > > The discussion thread (which was originally titled with "KIP-736") can
> be
> > > found here:
> > >
> > >
> >
> https://mail-archives.apache.org/mod_mbox/kafka-dev/202105.mbox/%3CCAMdOrUV0bqqs-ry7Q6FkNNn21ZhODTrg2d61zE5WZJw1MpQvSQ%40mail.gmail.com%3E
> > >
> > > Cheers,
> > >
> > > Chris
> > >
> >
>


Re: [DISCUSS] KIP-745: Connect API to restart connector and tasks

2021-06-03 Thread Kalpesh Patel
Agreed, this would be a great enhancement.

On Thu, Jun 3, 2021 at 9:26 AM Nikita Kretov  wrote:

> Hello! It's really not intuitive that you need to restart connect and
> tasks separately! I'd like to see this KIP landed in 3.0.0 release!
>
> On 6/2/21 7:40 PM, Randall Hauch wrote:
> > Hello all,
> >
> > Many users struggle with the connector restart REST API only restarting
> the
> > Connector instance rather than everything they associated with a "named"
> > connector. I've created a KIP that attempts to solve this problem via new
> > query parameters on an existing REST API method, though by default it
> > remains backward compatible with older behavior:
> >
> >
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-745%3A+Connect+API+to+restart+connector+and+tasks
> >
> > Please take a look and respond here with questions. I'd love to get this
> > into AK 3.0.0, and the KIP freeze is coming up soon.
> >
> > Best regards,
> >
> > Randall
> >
>


Re: Proposed breaking changes for Connect in AK 3.0.0

2021-06-03 Thread Randall Hauch
I've updated the proposal for 3.0 [1] with the following changes:

   - Remove the `rest.host.name` and `rest.port` worker config properties
   deprecated way back in AK 1.1, which was released over three years ago in
   Mar 2018.
   - Mention KIP-738 for removal of the internal worker configs deprecated
   in AK 2.0, which was released almost three years ago in July 2018.

Respond here if you have any comments on these or any of the other items
mentioned on [1].

Thanks,

Randall

[1]
https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=177047362
[2]
https://cwiki.apache.org/confluence/display/KAFKA/KIP-738%3A+Removal+of+Connect%27s+internal+converter+properties


On Tue, Mar 16, 2021 at 3:52 PM Israel Ekpo  wrote:

> Thanks Randall for kicking this off.
>
> Are there similar pages for the other ecosystem components and APIs that is
> part of the 3.0.0 release?
>
> It will be great to have a central page and then have this page as a subset
> of that collection
>
> On Tue, Mar 16, 2021 at 4:38 PM Randall Hauch  wrote:
>
> > The next release of AK will be 3.0.0. Since this is a major release, we
> > have an opportunity to:
> >
> >- remove previously deprecated worker configuration properties; and
> >- change some of Connect's defaults that were chosen previously to
> >maintain backward compatibility, but for which there are more sensible
> >defaults.
> >
> > I've taken the liberty of creating a wiki page [1] that lists all of the
> > Connect-related KIPs since AK 0.10.0.0 (the release after Connect was
> > introduced), and identifies a small set of changes that are appropriate
> > only for major releases.
> >
> > This page is not a KIP, but will hopefully help us identify any behaviors
> > or APIs that we may wish to change in AK 3.0.0. Note that some changes
> have
> > been already approved, and we need to decide whether to make those
> changes
> > in AK 3.0.0. Other changes will still require a formal KIP with
> discussion
> > and approval.
> >
> > Please use this thread to discuss these or other proposed changes.
> >
> > Thanks,
> >
> > Randall
> >
> > [1]
> >
> https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=177047362
> >
>


[VOTE] KIP-618: Exactly-once support for source connectors

2021-06-03 Thread Chris Egerton
Hi all,

I'd like to call for a vote on KIP-618, which adds support for exactly-once
delivery guarantees for source connectors in the Kafka Connect framework.

I suspect there might be a little more discussion to be had but with the
KIP freeze deadline approaching, I wanted to give anyone following along
the chance to cast a +1 as soon as they feel that we've gotten to a
reasonable state.

The KIP:
https://cwiki.apache.org/confluence/display/KAFKA/KIP-618%3A+Exactly-Once+Support+for+Source+Connectors

The discussion thread:
https://mail-archives.apache.org/mod_mbox/kafka-dev/202005.mbox/%3CCAMdOrUX-VK5OSB3OOdJNXW_YKEJH9FjQZ4eyzr2GXUhSeDnF3Q%40mail.gmail.com%3E

Cheers,

Chris


Kafka EOL policy

2021-06-03 Thread ChienHsing Wu
Hi,

I have a question about the EOL policy in the link at the end. The paragraph 
below from the link says the releases in the past year will be supported. The 3 
releases from last year 2020 are 2.5 in Mar, 2.6 in June and 2.7 in Nov. We 
have in this year 2021 2.8 in April, 3.0 in July and maybe 3.1 later. When we 
have 3.0 or 3.1, do we still support those 3 version in 2020?

The reason I ask for the clarification is the parenthesis in the paragraph 
"(i.e. last 3 releases)" since when we have 3.0 for example, the last 3 
releases are 2.6, 2.7 and 2.8. That would leave 2.5 not supported.

Given 3 releases a year and the fact that no one upgrades three times a year, 
we propose making sure (by testing!) that rolling upgrade can be done from each 
release in the past year (i.e. last 3 releases) to the latest version.

https://cwiki.apache.org/confluence/display/KAFKA/Time+Based+Release+Plan#TimeBasedReleasePlan-WhatIsOurEOLPolicy?

Appreciate your clarification.

Thanks, CH


Re: [DISCUSS] KIP-745: Connect API to restart connector and tasks

2021-06-03 Thread Nikita Kretov
Hello! It's really not intuitive that you need to restart connect and 
tasks separately! I'd like to see this KIP landed in 3.0.0 release!


On 6/2/21 7:40 PM, Randall Hauch wrote:

Hello all,

Many users struggle with the connector restart REST API only restarting the
Connector instance rather than everything they associated with a "named"
connector. I've created a KIP that attempts to solve this problem via new
query parameters on an existing REST API method, though by default it
remains backward compatible with older behavior:

https://cwiki.apache.org/confluence/display/KAFKA/KIP-745%3A+Connect+API+to+restart+connector+and+tasks

Please take a look and respond here with questions. I'd love to get this
into AK 3.0.0, and the KIP freeze is coming up soon.

Best regards,

Randall



[jira] [Created] (KAFKA-12885) Add the --timeout property to kafka-leader-election.sh

2021-06-03 Thread loboxu (Jira)
loboxu created KAFKA-12885:
--

 Summary: Add the --timeout property to kafka-leader-election.sh
 Key: KAFKA-12885
 URL: https://issues.apache.org/jira/browse/KAFKA-12885
 Project: Kafka
  Issue Type: Improvement
  Components: admin
Reporter: loboxu
Assignee: loboxu


https://issues.apache.org/jira/browse/KAFKA-9220 mentions 
kafka-preferred-replica-election.sh script hard-coded timeout problems. I see a 
similar problem with kafka-leader-election.sh.

I would like to add a --timeout parameter to kafka-leader-election.sh to 
control the request timeout. To solve similar problems.



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


[jira] [Created] (KAFKA-12884) Remove "--zookeeper" in system tests

2021-06-03 Thread Luke Chen (Jira)
Luke Chen created KAFKA-12884:
-

 Summary: Remove "--zookeeper" in system tests
 Key: KAFKA-12884
 URL: https://issues.apache.org/jira/browse/KAFKA-12884
 Project: Kafka
  Issue Type: Sub-task
Reporter: Luke Chen
Assignee: Luke Chen






--
This message was sent by Atlassian Jira
(v8.3.4#803005)


Re: Best Tool : Apache kafka Monitoring

2021-06-03 Thread Nikita Kretov
Hello. Prometheus + grafana is a standard stack for metric collection 
and monitoring. Jmx exporter is a good metrics exporter for 
broker\zookeeper. Maybe you can take advantage from some additional 
exporters like linkeding burrow too.


On 6/3/21 10:29 AM, Girish L wrote:

Hello Apache Kafka Experts,
What is the best free tool for Apache kafka monitoring at enterprise level
, with UI for admin team /support team? Could you please suggest something?
Any links/ pointers

Thanks



Best Tool : Apache kafka Monitoring

2021-06-03 Thread Girish L
Hello Apache Kafka Experts,
What is the best free tool for Apache kafka monitoring at enterprise level
, with UI for admin team /support team? Could you please suggest something?
Any links/ pointers

Thanks


Jenkins build is unstable: Kafka » Kafka Branch Builder » trunk #192

2021-06-03 Thread Apache Jenkins Server
See