Re: KIP-800: Add reason to LeaveGroupRequest

2021-11-11 Thread Luke Chen
Hi David,
Thanks for the KIP.
It makes sense to me.

Some comments:
1. Should we bump the `LeaveGroupRequest` protocol version to 5?
2. the description in the new field: "about": "The reason" -> "about": "The
reason why the member left the group"
3. For the `removeMembersFromConsumerGroup` case, do you think we can add a
new "reason" field in `RemoveMembersFromConsumerGroupOptions`, and default
to "the consumer was removed by an admin." as you proposed? I think when
getting the reason "removed by an admin" is not quite helpful for
troubleshooting, right? So, with the new field, it allows users to be able
to customize the reason if necessary.  for troubleshooting, What do you
think?

Thank you.
Luke

On Fri, Nov 12, 2021 at 12:32 AM David Jacot 
wrote:

> Hi folks,
>
> I'd like to discuss this very small KIP which proposes to add a reason
> field
> to the LeaveGroupRequest in order to let the broker know why a member
> left the group. This would be really handy for administrators.
>
> KIP: https://cwiki.apache.org/confluence/x/eYyqCw
>
> Cheers,
> David
>


[DISCUSS] KIP-792: Add "generation" field into consumer protocol

2021-11-11 Thread Luke Chen
Hi all,

I'd like to start the discussion for KIP-792: Add "generation" field into
consumer protocol.

The goal of this KIP is to allow assignor/consumer coordinator/group
coordinator to have a way to identify the out-of-date members/assignments.

Detailed description can be found here:
https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=191336614

Any feedback is welcome.

Thank you.
Luke


Re: [DISCUSS] KIP-796: Interactive Query v2

2021-11-11 Thread John Roesler
Thanks for taking a look, Sophie!

Ah, that was a revision error. I had initially been planning
an Optional> with Optional.empty() meaning to
fetch all partitions, but then decided it was needlessly
complex and changed it to the current proposal with two
methods:

boolean isAllPartitions();
Set getPartitions(); (which would throw an
exception if it's an "all partitions" request).

I've corrected the javadoc and also documented the
exception.

Thanks!
-John

On Thu, 2021-11-11 at 15:03 -0800, Sophie Blee-Goldman
wrote:
> Thanks John, I've been looking forward to this for a while now. It
> was pretty horrifying to learn
> how present-day IQ works  (or rather, doesn't work) with custom state
> stores :/
> 
> One minor cosmetic point, In the InteractiveQueryRequest class, the #
> getPartitions
> method has a return type of Set, but the javadocs refer to Optional.
> Not
> sure which is intended for this API, but if is supposed to be the return
> type, do you perhaps
> mean for it to be  Optional.ofEmpty() and Optional.of(non-empty set)
> rather than Optional.of(empty set) and Optional.of(non-empty set) ?
> 
> On Thu, Nov 11, 2021 at 12:03 PM John Roesler  wrote:
> 
> > Hello again, all,
> > 
> > Just bumping this discussion on a new, more flexible
> > Interactive Query API in Kafka Streams.
> > 
> > If there are no concerns, I'll go ahead and call a vote on
> > Monday.
> > 
> > Thanks!
> > -John
> > 
> > On Tue, 2021-11-09 at 17:37 -0600, John Roesler wrote:
> > > Hello all,
> > > 
> > > I'd like to start the discussion for KIP-796, which proposes
> > > a revamp of the Interactive Query APIs in Kafka Streams.
> > > 
> > > The proposal is here:
> > > https://cwiki.apache.org/confluence/x/34xnCw
> > > 
> > > I look forward to your feedback!
> > > 
> > > Thank you,
> > > -John
> > > 
> > 
> > 
> > 




Re: [DISCUSS] KIP-796: Interactive Query v2

2021-11-11 Thread Sophie Blee-Goldman
Thanks John, I've been looking forward to this for a while now. It
was pretty horrifying to learn
how present-day IQ works  (or rather, doesn't work) with custom state
stores :/

One minor cosmetic point, In the InteractiveQueryRequest class, the #
getPartitions
method has a return type of Set, but the javadocs refer to Optional.
Not
sure which is intended for this API, but if is supposed to be the return
type, do you perhaps
mean for it to be  Optional.ofEmpty() and Optional.of(non-empty set)
rather than Optional.of(empty set) and Optional.of(non-empty set) ?

On Thu, Nov 11, 2021 at 12:03 PM John Roesler  wrote:

> Hello again, all,
>
> Just bumping this discussion on a new, more flexible
> Interactive Query API in Kafka Streams.
>
> If there are no concerns, I'll go ahead and call a vote on
> Monday.
>
> Thanks!
> -John
>
> On Tue, 2021-11-09 at 17:37 -0600, John Roesler wrote:
> > Hello all,
> >
> > I'd like to start the discussion for KIP-796, which proposes
> > a revamp of the Interactive Query APIs in Kafka Streams.
> >
> > The proposal is here:
> > https://cwiki.apache.org/confluence/x/34xnCw
> >
> > I look forward to your feedback!
> >
> > Thank you,
> > -John
> >
>
>
>


Re: [DISCUSS] KIP-795: Add public APIs for AbstractCoordinator

2021-11-11 Thread Hector Geraldino (BLOOMBERG/ 919 3RD A)
Hi Tom,

Thanks for taking time reviewing the KIP. 

I think it's reasonable to ask if Kafka's Group Coordination protocol should be 
used for use cases other than the distributed event log. This was actually 
briefly addressed by Gwen Shapira during her presentation at the strangeloop 
conference in '18 (a link to the video is included in the KIP), in which she 
explain in greater details the protocol internals.

We should also keep in mind that this protocol is already being used for other 
use cases outside of core Kafka: Confluent Schema Registry uses it to determine 
leadership between members of a cluster, Kafka Connect uses it for task 
assignments, same with Kafka Stream for partition and task distributions, and 
so on. So having a public, stable API not just for new use cases (like ours) 
but existing ones is IMHO a good thing to have. I'll amend the KIP and add a 
bit more details to the motivation and alternatives sections, so the usefulness 
of this KIP is better understood.

Now, for the first point of your technical observations (regarding 
protocolTypes()), I don't think it matters in this context, as the protocol 
name and subtype are only relevant in the context of a consumer group and group 
rebalance. It really doesn't matter if two different libraries decide to name 
their protocols the same.

For item #2, I was under the impression that, because these classes all 
implement the org.apache.kafka.common.protocol.[Message, ApiMessage] interface, 
they are implicitly part of the Kafka protocol and the top-level API. Isn't 
that really the case?

And finally, for #3, the goal I had in mind when creating this KPI was a small 
one: to provide an interface that users can rely on when extending the 
AbstactCoordinator. So my thought was that, while the AbstractCoordinator 
itself uses some internal APIs (like ConsumerNetworkClient, ConsumerMetadata 
and so on) those can remain internal. But it probably makes sense to at least 
explore the possibility of moving the whole AbstractCoordinator class to be 
part of the public API. I'll do that exercise, see what it entails, and update 
the KIP with my findings.


Thanks again!
Hector


From: dev@kafka.apache.org At: 11/10/21 06:43:59 UTC-5:00To:  Hector Geraldino 
(BLOOMBERG/ 919 3RD A ) ,  dev@kafka.apache.org
Subject: Re: [DISCUSS] KIP-795: Add public APIs for AbstractCoordinator

Hi Hector,

Thanks for the KIP.

At a high level, I think the question to be answered by the community is
"Should Kafka really be providing this kind of cluster management API?".
While Kafka clients need this to provide their functionality it's a
different thing to expose that as a public API of the project, which is
otherwise about providing a distributed event log/data streaming
platform/. Having a public
API brings a significant commitment for API compatibility, which could
impair the ability of the project to change the API in order to make
improvements to the Kafka clients. The current AbstractCoordinator not
being a supported API means we don't currently have to reason about
compatibility here. So I think it would help the motivation section of the
KIP to describe in a bit more detail the use case(s) you have for
implementing your own coordinators. For example, are these applications
using Kafka otherwise, or just to leverage this API? And what alternatives
to implementing your own coordinators did you consider, and why did you
reject them?

From a technical point of view, there are a number of issues I think would
need addressing in order to do something like this:

1. There probably ought to be a way to ensure that protocolTypes() don't
collide, or at least reduce the chances of a collision. While probably
unlikely in practice the consequences of different protocols having the
same name could be pretty confusing to debug.
2. JoinGroupRequestData and JoinGroupResponseData are not public classes
(none of the *RequestData or *ResponseData classes are, intentionally), so
there would have to be an abstraction for them.
3. It's all well and good having an interface that anyone can implement,
but there is no supported Kafka API which takes an instance as a parameter
(i.e. where do you plug your implementation in without having to use a
bunch of other non-public Kafka APIs?) I assume in your current usage
you're having to make use of other non-supported client APIs to make use of
your coordinator. The KIP isn't really a complete piece of work without a
way to use a custom implementation, in my opinion. It would be confusing if
it looked like we were encouraging people to use those other non-supported
APIs by making this coordinator public.

Kind regards,

Tom


On Mon, Nov 8, 2021 at 2:01 PM Hector Geraldino (BLOOMBERG/ 919 3RD A) <
hgerald...@bloomberg.net> wrote:

> Hi Kafka devs,
>
> I would like to start the discussion of  KIP-795: Add public APIs for
> AbstractCoordinator
>
>
> 
https://cwiki.apache.org/confluence/display/KAFKA/KIP-795%3A+Add+public+APIs+for
+Abstra

Re: [DISCUSS] KIP-779: Allow Source Tasks to Handle Producer Exceptions

2021-11-11 Thread Chris Egerton
Hi Knowles,

I think this looks good for the most part but I'd still like to see an
explicit mention in the KIP (and proposed doc/Javadoc changes) that states
that, with exactly-once support enabled, producer exceptions that result
from failures related to exactly-once support (including but not limited to
ProducerFencedExcecption instances (
https://kafka.apache.org/30/javadoc/org/apache/kafka/common/errors/ProducerFencedException.html))
will not be skipped even with "errors.tolerance" set to "all", and will
instead unconditionally cause the task to fail. Your proposal that
"WorkerSourceTask could check the configuration before handing off the
records and exception to this function" seems great as long as we update
"handing off the records and exceptions to this function" to the
newly-proposed behavior of "logging the exception and continuing to poll
the task for data".

I'm also a little bit wary of updating the existing "errors.tolerance"
configuration to have new behavior that users can't opt out of without also
opting out of the current behavior they get with "errors.tolerance" set to
"all", but I think I've found a decent argument in favor of it. One thought
that came to mind is whether this use case was originally considered when
KIP-298 was being discussed. However, it appears that KAFKA-8586 (
https://issues.apache.org/jira/browse/KAFKA-8586), the fix for which caused
tasks to fail on non-retriable, asynchronous producer exceptions instead of
logging them and continuing, was discovered over a full year after the
changes for KIP-298 (https://github.com/apache/kafka/pull/5065) were
merged. I suspect that the current proposal aligns nicely with the original
design intent of KIP-298, and that if KAFKA-8586 were discovered before or
during discussion for KIP-298, non-retriable, asynchronous producer
exceptions would have been included in its scope. With that in mind,
although it may cause issues for some niche use cases, I think that this is
a valid change and would be worth the tradeoff of potentially complicating
life for a small number of users. I'd be interested in Arjun's thoughts on
this though (as he designed and implemented KIP-298), and if this analysis
is agreeable, we may want to document that information in the KIP as well
to strengthen our case for not introducing a new configuration property and
instead making this behavior tied to the existing "errors.tolerance"
property with no opt-out besides using a new value for that property.

My last thought is that, although it may be outside the scope of this KIP,
I believe your original proposal of giving tasks a hook to handle
downstream exceptions is actually quite valid. The DLQ feature for sink
connectors is an extremely valuable one as it prevents data loss when
"errors.tolerance" is set to "all" by allowing users to reprocess
problematic records at a later date without stopping the flow of data in
their connector entirely. As others have noted, it's difficult if not
outright impossible to provide a Kafka DLQ topic for source connectors with
the same guarantees, and so allowing source connectors the option of
storing problematic records back in the system that they came from seems
like a reasonable alternative. I think we're probably past the point of
making that happen in this KIP, but I don't believe the changes you've
proposed make that any harder in the future than it is now (which is
great!), and I wanted to voice my general support for a mechanism like this
in case you or someone following along think it'd be worth it to pursue at
a later date.

Thanks for your KIP and thanks for your patience with the process!

Cheers,

Chris

On Fri, Nov 5, 2021 at 8:26 AM Knowles Atchison Jr 
wrote:

> Good morning,
>
> If there is no additional feedback, I am going to call a vote for this KIP
> on Monday.
>
> Knowles
>
> On Tue, Nov 2, 2021 at 10:00 AM Knowles Atchison Jr  >
> wrote:
>
> > Third time's the charm.
> >
> > I've added a getter for the RetryWithToleranceOperator to get the
> > ToleranceType. I've updated WorkerSourceTask to check this setting to see
> > if it is ToleranceType.ALL.
> >
> > Setting "errors.tolerance" to "all" solves both problems:
> >
> > 1. Use an existing configuration
> > 2. Moves the configuration back to the connector/task level instead of at
> > the connect worker level.
> >
> > I've updated the KIP and PR.
> >
> > Additional thoughts and feedback are welcome.
> >
> > Knowles
> >
> > On Mon, Nov 1, 2021 at 2:00 AM Arjun Satish 
> > wrote:
> >
> >> Looks really nice. Thanks for the changes. Couple of suggestions:
> >>
> >> 1. Can we reuse any of the existing configs, instead of introducing a
> new
> >> one? I’m wondering if the error.tolerance configuration’s scope can be
> >> increased to include produce errors as well. That’ll help us keep number
> >> of
> >> configs in check. Effectively, if error.tolerance is set to all, then
> the
> >> behavior would be like how you describe the worker would ignore producer
> 

Re: [DISCUSS] KIP-796: Interactive Query v2

2021-11-11 Thread John Roesler
Hello again, all,

Just bumping this discussion on a new, more flexible
Interactive Query API in Kafka Streams.

If there are no concerns, I'll go ahead and call a vote on
Monday.

Thanks!
-John

On Tue, 2021-11-09 at 17:37 -0600, John Roesler wrote:
> Hello all,
> 
> I'd like to start the discussion for KIP-796, which proposes
> a revamp of the Interactive Query APIs in Kafka Streams.
> 
> The proposal is here:
> https://cwiki.apache.org/confluence/x/34xnCw
> 
> I look forward to your feedback!
> 
> Thank you,
> -John
> 




Re: [VOTE] KIP-779: Allow Source Tasks to Handle Producer Exceptions

2021-11-11 Thread John Roesler
Thanks, Knowles!

I'm +1 (binding)

-John

On Wed, 2021-11-10 at 12:42 -0500, Christopher Shannon
wrote:
> +1 (non-binding). This looks good to me and will be useful as a way to
> handle producer errors.
> 
> On Mon, Nov 8, 2021 at 8:55 AM Knowles Atchison Jr 
> wrote:
> 
> > Good morning,
> > 
> > I'd like to start a vote for KIP-779: Allow Source Tasks to Handle Producer
> > Exceptions:
> > 
> > 
> > https://cwiki.apache.org/confluence/display/KAFKA/KIP-779%3A+Allow+Source+Tasks+to+Handle+Producer+Exceptions
> > 
> > The purpose of this KIP is to allow Source Tasks the option to "ignore"
> > kafka producer exceptions. After a few iterations, this is now part of the
> > "errors.tolerance" configuration and provides a null RecordMetadata to
> > commitRecord() in lieu of a new SourceTask interface method or worker
> > configuration item.
> > 
> > PR is here:
> > 
> > https://github.com/apache/kafka/pull/11382
> > 
> > Any comments and feedback are welcome.
> > 
> > Knowles
> > 




Re: [DISCUSS] KIP-799 Align behaviour for producer callbacks with documented behaviour

2021-11-11 Thread John Roesler
Thanks for the reply, Séamus,

Ah, I missed that the actual value of the placeholder is
that otherwise, you wouldn't know the topic/partition of the
error.

I guess, on balance, it doesn't feel like this situation
really justifies moving to a new callback interface (to pass
back the topic/partition separately from the other
metadata), even though that might have been nicer in a
vacuum. So, if you think it's nice to have those metadata,
then I think the fact that it's been the de-facto behavior
for many callback situations for a while now, and the fact
that it's the established pattern of the interceptor
indicates that it's probably fine to do as you propose and
just standardize on the placeholder in error cases.

Thanks!
-John

On Thu, 2021-11-11 at 17:08 +, Séamus Ó Ceanainn wrote:
> Hey John,
> 
> > did you consider just going back to the original behavior?
> 
> I hadn't considered going back to the exact original behaviour as I think
> there's a valid point made in discussions around KAFKA-7412 (I
> forget whether in a JIRA or PR comment) that returning the topic partition
> when available can be useful for users. Something I did consider is to
> include the topic partition separately to the metadata value when
> exceptions occur so that metadata could still be null in those cases while
> still having topic partition data available.
> 
> My opinion is that this other behaviour would be nicer (where returned
> metadata is null but topic partition information is still available),
> however it would not be consistent with the implementation of
> ProducerInterceptor.onAcknowledgement method. I would tend to favour
> consistency in this case (as both methods are handled very similarly in
> code), and I don't think there's a strong argument to make a breaking
> change to ProducerInterceptor when there is nothing currently broken in
> that implementation (like there currently is with Callback).
> 
> Of course if the general consensus is that consistency between the
> behaviour of the two methods (ProducerInterceptor.onAcknowledgement and
> Callback.onCompletion) does not matter, or that a change in the behaviour
> of ProducerInterceptor.onAcknowledgement should also be included in the
> scope of this KIP, I'm open to updating the KIP to reflect that.
> 
> > Although it’s technically an implementation detail (which doesn’t need to
> be in a KIP), I like the fact that you’re planning to refactor the code to
> enforce consistent handling of the callbacks.
> 
> I wasn't entirely sure how to deal with changes to the interfaces within
> the 'clients.producer.internals' package, so I thought it was best to err
> on the side of including too much in the KIP.  I'll remove the unnecessary
> detail to ensure the discussion doesn't get derailed, for anyone interested
> in implementation details there is a draft PR linked in the KIP with that
> refactoring done, so any discussion on that topic can take place in Github
> / JIRA.
> 
> Regards,
> Séamus.
> 
> On Thu, 11 Nov 2021 at 14:33, John Roesler  wrote:
> 
> > Thanks for the KIP, Séamus!
> > 
> > I agree that the current situation you’re describing doesn’t seem ideal,
> > and it’s probably worth a slight behavior change to fix it.
> > 
> > It’s too bad that we introduced that placeholder record, since it seems
> > less error prone for users if we have the invariant that exactly one
> > argument is non-null. I’m concerned (as reported in KAFKA-7412) that people
> > could mistake the placeholder for a successful ack. Since we’re considering
> > some breakage to fix this inconsistency, did you consider just going back
> > to the original behavior?
> > 
> > Although it’s technically an implementation detail (which doesn’t need to
> > be in a KIP), I like the fact that you’re planning to refactor the code to
> > enforce consistent handling of the callbacks.
> > 
> > Thanks,
> > John
> > 
> > On Thu, Nov 11, 2021, at 07:25, Séamus Ó Ceanainn wrote:
> > > Hi,
> > > 
> > > As outlined in KIP-799: Align behaviour for producer callbacks with
> > > documented behaviour
> > > <
> > https://cwiki.apache.org/confluence/display/KAFKA/KIP-799%3A+Align+behaviour+for+producer+callbacks+with+documented+behaviour
> > > ,
> > > there is an inconsistency between the documented behaviour and
> > > implementation of producer callbacks for the Kafka client. The KIP
> > > proposes
> > > breaking changes to the implementation of the Kafka producer client to
> > > align the implementation with the documented behaviour, and includes a
> > > link
> > > to a PR containing a tested implementation of the changes being
> > > recommended.
> > > 
> > > There is a need to take action here as a breaking change was previously
> > > introduced accidentally, and the documentation was later updated to try
> > to
> > > reflect those breaking changes. I believe the main discussion here is
> > > around the most appropriate behaviour for callbacks, which will inform
> > > whether the implementation, documentati

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

2021-11-11 Thread Apache Jenkins Server
See 


Changes:


--
[...truncated 498519 lines...]
[2021-11-11T17:46:19.929Z] > Task :raft:testClasses UP-TO-DATE
[2021-11-11T17:46:19.929Z] > Task :connect:json:testJar
[2021-11-11T17:46:19.929Z] > Task :connect:json:testSrcJar
[2021-11-11T17:46:19.929Z] > Task :metadata:compileTestJava UP-TO-DATE
[2021-11-11T17:46:19.929Z] > Task :metadata:testClasses UP-TO-DATE
[2021-11-11T17:46:19.929Z] > Task 
:clients:generateMetadataFileForMavenJavaPublication
[2021-11-11T17:46:19.929Z] > Task 
:clients:generatePomFileForMavenJavaPublication
[2021-11-11T17:46:19.929Z] > Task :core:compileScala UP-TO-DATE
[2021-11-11T17:46:19.929Z] > Task :core:classes UP-TO-DATE
[2021-11-11T17:46:19.929Z] > Task :core:compileTestJava NO-SOURCE
[2021-11-11T17:46:20.876Z] 
[2021-11-11T17:46:20.876Z] > Task :streams:processMessages
[2021-11-11T17:46:20.876Z] Execution optimizations have been disabled for task 
':streams:processMessages' to ensure correctness due to the following reasons:
[2021-11-11T17:46:20.876Z]   - Gradle detected a problem with the following 
location: 
'/home/jenkins/workspace/Kafka_kafka_trunk/streams/src/generated/java/org/apache/kafka/streams/internals/generated'.
 Reason: Task ':streams:srcJar' uses this output of task 
':streams:processMessages' without declaring an explicit or implicit 
dependency. This can lead to incorrect results being produced, depending on 
what order the tasks are executed. Please refer to 
https://docs.gradle.org/7.2/userguide/validation_problems.html#implicit_dependency
 for more details about this problem.
[2021-11-11T17:46:20.876Z] MessageGenerator: processed 1 Kafka message JSON 
files(s).
[2021-11-11T17:46:20.876Z] 
[2021-11-11T17:46:20.876Z] > Task :streams:compileJava UP-TO-DATE
[2021-11-11T17:46:20.876Z] > Task :streams:classes UP-TO-DATE
[2021-11-11T17:46:20.876Z] > Task :streams:copyDependantLibs UP-TO-DATE
[2021-11-11T17:46:20.876Z] > Task :streams:test-utils:compileJava UP-TO-DATE
[2021-11-11T17:46:20.876Z] > Task :streams:jar UP-TO-DATE
[2021-11-11T17:46:20.876Z] > Task :core:compileTestScala UP-TO-DATE
[2021-11-11T17:46:20.876Z] > Task :core:testClasses UP-TO-DATE
[2021-11-11T17:46:20.876Z] > Task 
:streams:generateMetadataFileForMavenJavaPublication
[2021-11-11T17:46:23.529Z] > Task :connect:api:javadoc
[2021-11-11T17:46:23.529Z] > Task :connect:api:copyDependantLibs UP-TO-DATE
[2021-11-11T17:46:23.529Z] > Task :connect:api:jar UP-TO-DATE
[2021-11-11T17:46:23.529Z] > Task 
:connect:api:generateMetadataFileForMavenJavaPublication
[2021-11-11T17:46:23.529Z] > Task :connect:json:copyDependantLibs UP-TO-DATE
[2021-11-11T17:46:23.529Z] > Task :connect:json:jar UP-TO-DATE
[2021-11-11T17:46:23.529Z] > Task 
:connect:json:generateMetadataFileForMavenJavaPublication
[2021-11-11T17:46:23.529Z] > Task :connect:api:javadocJar
[2021-11-11T17:46:23.529Z] > Task 
:connect:json:publishMavenJavaPublicationToMavenLocal
[2021-11-11T17:46:23.529Z] > Task :connect:json:publishToMavenLocal
[2021-11-11T17:46:23.529Z] > Task :connect:api:compileTestJava UP-TO-DATE
[2021-11-11T17:46:23.529Z] > Task :connect:api:testClasses UP-TO-DATE
[2021-11-11T17:46:24.474Z] > Task :connect:api:testJar
[2021-11-11T17:46:24.474Z] > Task :connect:api:testSrcJar
[2021-11-11T17:46:24.474Z] > Task 
:connect:api:publishMavenJavaPublicationToMavenLocal
[2021-11-11T17:46:24.474Z] > Task :connect:api:publishToMavenLocal
[2021-11-11T17:46:26.248Z] > Task :streams:javadoc
[2021-11-11T17:46:27.194Z] > Task :streams:javadocJar
[2021-11-11T17:46:27.194Z] > Task :streams:compileTestJava UP-TO-DATE
[2021-11-11T17:46:27.194Z] > Task :streams:testClasses UP-TO-DATE
[2021-11-11T17:46:27.194Z] > Task :streams:testJar
[2021-11-11T17:46:27.195Z] > Task :streams:testSrcJar
[2021-11-11T17:46:27.195Z] > Task 
:streams:publishMavenJavaPublicationToMavenLocal
[2021-11-11T17:46:27.195Z] > Task :streams:publishToMavenLocal
[2021-11-11T17:46:28.141Z] > Task :clients:javadoc
[2021-11-11T17:46:29.088Z] > Task :clients:javadocJar
[2021-11-11T17:46:30.034Z] 
[2021-11-11T17:46:30.034Z] > Task :clients:srcJar
[2021-11-11T17:46:30.034Z] Execution optimizations have been disabled for task 
':clients:srcJar' to ensure correctness due to the following reasons:
[2021-11-11T17:46:30.034Z]   - Gradle detected a problem with the following 
location: 
'/home/jenkins/workspace/Kafka_kafka_trunk/clients/src/generated/java'. Reason: 
Task ':clients:srcJar' uses this output of task ':clients:processMessages' 
without declaring an explicit or implicit dependency. This can lead to 
incorrect results being produced, depending on what order the tasks are 
executed. Please refer to 
https://docs.gradle.org/7.2/userguide/validation_problems.html#implicit_dependency
 for more details about this problem.
[2021-11-11T17:46:30.982Z] 
[2021-11-11T17:46:30.982Z] > Task :clients:testJar
[2021-11-11T17:46:30.982Z] > Task :clients:testSrcJar
[2021-11-1

Build failed in Jenkins: Kafka » Kafka Branch Builder » 3.1 #7

2021-11-11 Thread Apache Jenkins Server
See 


Changes:


--
[...truncated 497765 lines...]
[2021-11-11T17:37:51.416Z] 
[2021-11-11T17:37:51.416Z] SaslOAuthBearerSslEndToEndAuthorizationTest > 
testNoConsumeWithoutDescribeAclViaAssign() STARTED
[2021-11-11T17:37:51.579Z] 
[2021-11-11T17:37:51.579Z] MetricsTest > testWindowsStyleTagNames(String) > 
kafka.metrics.MetricsTest.testWindowsStyleTagNames(String)[1] PASSED
[2021-11-11T17:37:51.579Z] 
[2021-11-11T17:37:51.579Z] MetricsTest > testWindowsStyleTagNames(String) > 
kafka.metrics.MetricsTest.testWindowsStyleTagNames(String)[2] STARTED
[2021-11-11T17:37:53.326Z] 
[2021-11-11T17:37:53.326Z] MetricsTest > testWindowsStyleTagNames(String) > 
kafka.metrics.MetricsTest.testWindowsStyleTagNames(String)[2] PASSED
[2021-11-11T17:37:53.326Z] 
[2021-11-11T17:37:53.326Z] MetricsTest > testBrokerStateMetric(String) > 
kafka.metrics.MetricsTest.testBrokerStateMetric(String)[1] STARTED
[2021-11-11T17:37:55.947Z] 
[2021-11-11T17:37:55.947Z] MetricsTest > testBrokerStateMetric(String) > 
kafka.metrics.MetricsTest.testBrokerStateMetric(String)[1] PASSED
[2021-11-11T17:37:55.947Z] 
[2021-11-11T17:37:55.947Z] MetricsTest > testBrokerStateMetric(String) > 
kafka.metrics.MetricsTest.testBrokerStateMetric(String)[2] STARTED
[2021-11-11T17:37:58.584Z] 
[2021-11-11T17:37:58.584Z] MetricsTest > testBrokerStateMetric(String) > 
kafka.metrics.MetricsTest.testBrokerStateMetric(String)[2] PASSED
[2021-11-11T17:37:58.584Z] 
[2021-11-11T17:37:58.584Z] MetricsTest > testZkControllerMetrics(String) > 
kafka.metrics.MetricsTest.testZkControllerMetrics(String)[1] STARTED
[2021-11-11T17:38:01.203Z] 
[2021-11-11T17:38:01.203Z] MetricsTest > testZkControllerMetrics(String) > 
kafka.metrics.MetricsTest.testZkControllerMetrics(String)[1] PASSED
[2021-11-11T17:38:01.203Z] 
[2021-11-11T17:38:01.204Z] MetricsTest > 
testBrokerTopicMetricsBytesInOut(String) > 
kafka.metrics.MetricsTest.testBrokerTopicMetricsBytesInOut(String)[1] STARTED
[2021-11-11T17:38:01.492Z] 
[2021-11-11T17:38:01.492Z] SaslOAuthBearerSslEndToEndAuthorizationTest > 
testNoConsumeWithoutDescribeAclViaAssign() PASSED
[2021-11-11T17:38:01.492Z] 
[2021-11-11T17:38:01.492Z] SaslOAuthBearerSslEndToEndAuthorizationTest > 
testNoGroupAcl() STARTED
[2021-11-11T17:38:05.949Z] 
[2021-11-11T17:38:05.949Z] MetricsTest > 
testBrokerTopicMetricsBytesInOut(String) > 
kafka.metrics.MetricsTest.testBrokerTopicMetricsBytesInOut(String)[1] PASSED
[2021-11-11T17:38:05.949Z] 
[2021-11-11T17:38:05.949Z] MetricsTest > 
testBrokerTopicMetricsBytesInOut(String) > 
kafka.metrics.MetricsTest.testBrokerTopicMetricsBytesInOut(String)[2] STARTED
[2021-11-11T17:38:08.400Z] 
[2021-11-11T17:38:08.401Z] SaslOAuthBearerSslEndToEndAuthorizationTest > 
testNoGroupAcl() PASSED
[2021-11-11T17:38:08.401Z] 
[2021-11-11T17:38:08.401Z] SaslOAuthBearerSslEndToEndAuthorizationTest > 
testNoProduceWithDescribeAcl() STARTED
[2021-11-11T17:38:09.015Z] 
[2021-11-11T17:38:09.015Z] MetricsTest > 
testBrokerTopicMetricsBytesInOut(String) > 
kafka.metrics.MetricsTest.testBrokerTopicMetricsBytesInOut(String)[2] PASSED
[2021-11-11T17:38:09.015Z] 
[2021-11-11T17:38:09.015Z] MetricsTest > testJMXFilter(String) > 
kafka.metrics.MetricsTest.testJMXFilter(String)[1] STARTED
[2021-11-11T17:38:10.763Z] 
[2021-11-11T17:38:10.763Z] MetricsTest > testJMXFilter(String) > 
kafka.metrics.MetricsTest.testJMXFilter(String)[1] PASSED
[2021-11-11T17:38:10.763Z] 
[2021-11-11T17:38:10.763Z] MetricsTest > testJMXFilter(String) > 
kafka.metrics.MetricsTest.testJMXFilter(String)[2] STARTED
[2021-11-11T17:38:13.383Z] 
[2021-11-11T17:38:13.383Z] MetricsTest > testJMXFilter(String) > 
kafka.metrics.MetricsTest.testJMXFilter(String)[2] PASSED
[2021-11-11T17:38:15.706Z] 
[2021-11-11T17:38:15.706Z] SaslOAuthBearerSslEndToEndAuthorizationTest > 
testNoProduceWithDescribeAcl() PASSED
[2021-11-11T17:38:15.706Z] 
[2021-11-11T17:38:15.706Z] SaslOAuthBearerSslEndToEndAuthorizationTest > 
testNoDescribeProduceOrConsumeWithoutTopicDescribeAcl() STARTED
[2021-11-11T17:38:16.337Z] 
[2021-11-11T17:38:16.337Z] Deprecated Gradle features were used in this build, 
making it incompatible with Gradle 8.0.
[2021-11-11T17:38:16.337Z] 
[2021-11-11T17:38:16.337Z] You can use '--warning-mode all' to show the 
individual deprecation warnings and determine if they come from your own 
scripts or plugins.
[2021-11-11T17:38:16.337Z] 
[2021-11-11T17:38:16.337Z] See 
https://docs.gradle.org/7.2/userguide/command_line_interface.html#sec:command_line_warnings
[2021-11-11T17:38:16.337Z] 
[2021-11-11T17:38:16.337Z] BUILD SUCCESSFUL in 1h 47m 32s
[2021-11-11T17:38:16.337Z] 202 actionable tasks: 109 executed, 93 up-to-date
[2021-11-11T17:38:16.337Z] 
[2021-11-11T17:38:16.337Z] See the profiling report at: 
file:///home/jenkins/jenkins-agent/workspace/Kafka_kafka_3.1/build/reports/profile/profile-2021-11-11-15-50-51.html
[2021-11-11T17:38:16.337Z] A fine-grained perfor

Re: [DISCUSS] KIP-799 Align behaviour for producer callbacks with documented behaviour

2021-11-11 Thread Séamus Ó Ceanainn
Hey John,

> did you consider just going back to the original behavior?

I hadn't considered going back to the exact original behaviour as I think
there's a valid point made in discussions around KAFKA-7412 (I
forget whether in a JIRA or PR comment) that returning the topic partition
when available can be useful for users. Something I did consider is to
include the topic partition separately to the metadata value when
exceptions occur so that metadata could still be null in those cases while
still having topic partition data available.

My opinion is that this other behaviour would be nicer (where returned
metadata is null but topic partition information is still available),
however it would not be consistent with the implementation of
ProducerInterceptor.onAcknowledgement method. I would tend to favour
consistency in this case (as both methods are handled very similarly in
code), and I don't think there's a strong argument to make a breaking
change to ProducerInterceptor when there is nothing currently broken in
that implementation (like there currently is with Callback).

Of course if the general consensus is that consistency between the
behaviour of the two methods (ProducerInterceptor.onAcknowledgement and
Callback.onCompletion) does not matter, or that a change in the behaviour
of ProducerInterceptor.onAcknowledgement should also be included in the
scope of this KIP, I'm open to updating the KIP to reflect that.

> Although it’s technically an implementation detail (which doesn’t need to
be in a KIP), I like the fact that you’re planning to refactor the code to
enforce consistent handling of the callbacks.

I wasn't entirely sure how to deal with changes to the interfaces within
the 'clients.producer.internals' package, so I thought it was best to err
on the side of including too much in the KIP.  I'll remove the unnecessary
detail to ensure the discussion doesn't get derailed, for anyone interested
in implementation details there is a draft PR linked in the KIP with that
refactoring done, so any discussion on that topic can take place in Github
/ JIRA.

Regards,
Séamus.

On Thu, 11 Nov 2021 at 14:33, John Roesler  wrote:

> Thanks for the KIP, Séamus!
>
> I agree that the current situation you’re describing doesn’t seem ideal,
> and it’s probably worth a slight behavior change to fix it.
>
> It’s too bad that we introduced that placeholder record, since it seems
> less error prone for users if we have the invariant that exactly one
> argument is non-null. I’m concerned (as reported in KAFKA-7412) that people
> could mistake the placeholder for a successful ack. Since we’re considering
> some breakage to fix this inconsistency, did you consider just going back
> to the original behavior?
>
> Although it’s technically an implementation detail (which doesn’t need to
> be in a KIP), I like the fact that you’re planning to refactor the code to
> enforce consistent handling of the callbacks.
>
> Thanks,
> John
>
> On Thu, Nov 11, 2021, at 07:25, Séamus Ó Ceanainn wrote:
> > Hi,
> >
> > As outlined in KIP-799: Align behaviour for producer callbacks with
> > documented behaviour
> > <
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-799%3A+Align+behaviour+for+producer+callbacks+with+documented+behaviour
> >,
> > there is an inconsistency between the documented behaviour and
> > implementation of producer callbacks for the Kafka client. The KIP
> > proposes
> > breaking changes to the implementation of the Kafka producer client to
> > align the implementation with the documented behaviour, and includes a
> > link
> > to a PR containing a tested implementation of the changes being
> > recommended.
> >
> > There is a need to take action here as a breaking change was previously
> > introduced accidentally, and the documentation was later updated to try
> to
> > reflect those breaking changes. I believe the main discussion here is
> > around the most appropriate behaviour for callbacks, which will inform
> > whether the implementation, documentation or a combination of both should
> > be updated. Right now, the documented behaviour aligns closely with the
> > implementation of ProducerInterceptors, and as a result I would favor
> > keeping the documented behaviour and updating the implementation to
> match,
> > as that would provide greater consistency between Callbacks and
> > Interceptors implementations for producers.
> >
> > Regards,
> > Séamus.
>


KIP-800: Add reason to LeaveGroupRequest

2021-11-11 Thread David Jacot
Hi folks,

I'd like to discuss this very small KIP which proposes to add a reason field
to the LeaveGroupRequest in order to let the broker know why a member
left the group. This would be really handy for administrators.

KIP: https://cwiki.apache.org/confluence/x/eYyqCw

Cheers,
David


[jira] [Created] (KAFKA-13451) Add reason to LeaveGroupRequest

2021-11-11 Thread David Jacot (Jira)
David Jacot created KAFKA-13451:
---

 Summary: Add reason to LeaveGroupRequest
 Key: KAFKA-13451
 URL: https://issues.apache.org/jira/browse/KAFKA-13451
 Project: Kafka
  Issue Type: Improvement
Reporter: David Jacot
Assignee: David Jacot


At the moment, when a member leaves a group, the java clients logs the reason. 
However, we have not information on the broker side. We should include this 
into the LeaveGroup request (as a tagged field).



--
This message was sent by Atlassian Jira
(v8.20.1#820001)


Re: [DISCUSS] KIP-799 Align behaviour for producer callbacks with documented behaviour

2021-11-11 Thread John Roesler
Thanks for the KIP, Séamus!

I agree that the current situation you’re describing doesn’t seem ideal, and 
it’s probably worth a slight behavior change to fix it.

It’s too bad that we introduced that placeholder record, since it seems less 
error prone for users if we have the invariant that exactly one argument is 
non-null. I’m concerned (as reported in KAFKA-7412) that people could mistake 
the placeholder for a successful ack. Since we’re considering some breakage to 
fix this inconsistency, did you consider just going back to the original 
behavior?

Although it’s technically an implementation detail (which doesn’t need to be in 
a KIP), I like the fact that you’re planning to refactor the code to enforce 
consistent handling of the callbacks.

Thanks,
John

On Thu, Nov 11, 2021, at 07:25, Séamus Ó Ceanainn wrote:
> Hi,
>
> As outlined in KIP-799: Align behaviour for producer callbacks with
> documented behaviour
> ,
> there is an inconsistency between the documented behaviour and
> implementation of producer callbacks for the Kafka client. The KIP 
> proposes
> breaking changes to the implementation of the Kafka producer client to
> align the implementation with the documented behaviour, and includes a 
> link
> to a PR containing a tested implementation of the changes being 
> recommended.
>
> There is a need to take action here as a breaking change was previously
> introduced accidentally, and the documentation was later updated to try to
> reflect those breaking changes. I believe the main discussion here is
> around the most appropriate behaviour for callbacks, which will inform
> whether the implementation, documentation or a combination of both should
> be updated. Right now, the documented behaviour aligns closely with the
> implementation of ProducerInterceptors, and as a result I would favor
> keeping the documented behaviour and updating the implementation to match,
> as that would provide greater consistency between Callbacks and
> Interceptors implementations for producers.
>
> Regards,
> Séamus.


[DISCUSS] KIP-799 Align behaviour for producer callbacks with documented behaviour

2021-11-11 Thread Séamus Ó Ceanainn
Hi,

As outlined in KIP-799: Align behaviour for producer callbacks with
documented behaviour
,
there is an inconsistency between the documented behaviour and
implementation of producer callbacks for the Kafka client. The KIP proposes
breaking changes to the implementation of the Kafka producer client to
align the implementation with the documented behaviour, and includes a link
to a PR containing a tested implementation of the changes being recommended.

There is a need to take action here as a breaking change was previously
introduced accidentally, and the documentation was later updated to try to
reflect those breaking changes. I believe the main discussion here is
around the most appropriate behaviour for callbacks, which will inform
whether the implementation, documentation or a combination of both should
be updated. Right now, the documented behaviour aligns closely with the
implementation of ProducerInterceptors, and as a result I would favor
keeping the documented behaviour and updating the implementation to match,
as that would provide greater consistency between Callbacks and
Interceptors implementations for producers.

Regards,
Séamus.


[jira] [Created] (KAFKA-13450) Kafka broker to return alternating sets of listeners to clients for multipathing

2021-11-11 Thread Sudarshan Pathalam (Jira)
Sudarshan Pathalam created KAFKA-13450:
--

 Summary: Kafka broker to return alternating sets of listeners to 
clients for multipathing
 Key: KAFKA-13450
 URL: https://issues.apache.org/jira/browse/KAFKA-13450
 Project: Kafka
  Issue Type: Improvement
  Components: network
Reporter: Sudarshan Pathalam


It is not uncommon to deploy Kafka for high throughput use cases. In this 
context operators could have multiple network paths for client to broker 
traffic.

The use of multiple network paths is not only to cover for failure conditions 
but also to balance load between the available paths. Common link speeds 
(currently) in data centers/cloud are 10Gbps/40Gbps and when there is a need to 
scale past the link speed multiple paths (multipathing) to the target (Kafka) 
is desired. The option of bundling multiple links in to a single channel is not 
commonly offered as a user facing feature in the Cloud.

However when using DNS to load balance between the available network paths, TTL 
caching creates traffic skew between the network paths causing either 
deterioration or failure. 

It would be ideal for the broker to return alternating sets of listener names 
to the metadata requests made by the client for even distribution of traffic 
between the available network paths. KIP 302 provides the ability for the 
clients to attempt to use all IP address for a name but does not cover 
situations when there is a need for multiple names each pointing to a separate 
network path. 



--
This message was sent by Atlassian Jira
(v8.20.1#820001)


Re: [VOTE] KIP-791: Add Record Metadata to State Store Context

2021-11-11 Thread Patrick Stuedi
Thanks everyone for voting.

Voting passed after 3 days with the following +1s:

- Luke Chen
- Vasiliki (Vicky) Papavasileiou
- Guozhang Wang (binding)
- John Roesler (binding)
- Bruno Cadonna (binding)

I'll update the KIP status accordingly.

Best,
  Patrick

On Wed, Nov 10, 2021 at 5:03 PM Bruno Cadonna  wrote:

> Patrick,
>
> Thank you for the KIP!
>
> +1 (binding)
>
> Best,
> Bruno
>
> On 09.11.21 16:58, John Roesler wrote:
> > +1 (binding) from me.
> >
> > Thanks, Patrick!
> >
> > On Mon, 2021-11-08 at 14:08 -0800, Guozhang Wang wrote:
> >> +1, thanks Patrick!
> >>
> >>
> >> Guozhang
> >>
> >> On Mon, Nov 8, 2021 at 5:44 AM Vasiliki Papavasileiou
> >>  wrote:
> >>
> >>> Hi Patrick,
> >>>
> >>> Having the recordMetadata available in the state stores is fundamental
> for
> >>> the consistency work and the proposed approach is reasonable.
> >>>
> >>> +1 (non-binding)
> >>>
> >>> Thank you,
> >>> Vicky
> >>>
> >>> On Mon, Nov 8, 2021 at 10:00 AM Luke Chen  wrote:
> >>>
>  Hi Patrick,
>  Thanks for the KIP.
>  Adding RecordMetadata into StateStoreContext for offset updating makes
>  sense to me.
> 
>  +1 (non-binding)
> 
>  Thank you.
>  Luke
> 
> 
>  On Mon, Nov 8, 2021 at 5:18 PM Patrick Stuedi
> >>>  >
>  wrote:
> 
> > Hi all,
> >
> > Thanks for the feedback on KIP-791, I have updated the KIP and would
> >>> like
> > to start the voting.
> >
> > The KIP can be found here:
> > https://cwiki.apache.org/confluence/x/I5BnCw
> >
> > Please vote in this thread.
> >
> > Thanks!
> > -Patrick
> >
> 
> >>>
> >>
> >>
> >
> >
>


[jira] [Created] (KAFKA-13449) Comment optimization for parameter log.cleaner.delete.retention.ms

2021-11-11 Thread RivenSun (Jira)
RivenSun created KAFKA-13449:


 Summary: Comment optimization for parameter 
log.cleaner.delete.retention.ms 
 Key: KAFKA-13449
 URL: https://issues.apache.org/jira/browse/KAFKA-13449
 Project: Kafka
  Issue Type: Improvement
  Components: config
Affects Versions: 3.0.0
Reporter: RivenSun


You can view the comment of this parameter from Kafka's official website.

https://kafka.apache.org/documentation/#brokerconfigs_log.cleaner.delete.retention.ms
{code:java}
log.cleaner.delete.retention.ms
How long are delete records retained? {code}
I think it should be consistent with the comment of topic level parameter 
*delete.retention.ms* .

https://kafka.apache.org/documentation/#topicconfigs_delete.retention.ms



--
This message was sent by Atlassian Jira
(v8.20.1#820001)