Re: [VOTE] KIP-815: Replace KafkaConsumer with AdminClient in GetOffsetShell

2022-02-01 Thread David Jacot
Hey,

Thanks for updating the KIP. I think that there are a few more configs
which could
be used. e.g. all the network related configs - they are in both
consumer and admin
configurations as well. Is `session.timeout.ms` relevant in our
context? It does not
seem to be used when querying offsets.

Regarding the usage of `request.timeout.ms` in the KafkaConsumer, it would be
great if we could be clearer in the KIP. When I read "Will use
default.api.timeout.ms
instead of request.timeout.ms , This is a small bug and will be fixed
in a separate PR",
it is not clear what will be fixed where. We could say that the KafkaConsumer is
inconsistent in its usage of the timeouts so the AdminClient will
behave slightly
differently. However, as it seems to be a bug, we will fix the Consumer and we
can add a link to the Jira.

It is important to get this section as clear as possible because this
is where questions
will be.

Cheers,
David

On Wed, Feb 2, 2022 at 7:23 AM deng ziming  wrote:
>
> Hey David,
> I rechecked the ConsumerConfig and split the Compatibility section into 2 
> sub-sections
> regarding AdminClientConfig and ConsumerConfig, I also find a small bug that 
> we use
>  different timeout in `KafkaConsumer.beginningOffsets` and 
> `KafkaConsumer.endOffsets`,
> I will fix this.
>
> Please help to review the KIP and the bug, thank you.
>
> Best,
> Ziming Deng
>
>
> > On Feb 1, 2022, at 6:31 PM, David Jacot  wrote:
> >
> > Thanks for the updated KIP.
> >
> > Regarding the compatibility section, I think that it would be
> > great if we could really stress that the configurations that
> > could reasonably be used to configure the tool are actually
> > all supported by the admin client. Regarding the retry mechanism,
> > the consumer will retry until `default.api.timeout.ms` is reached
> > and it seems that the admin client does the same by default. Do
> > you confirm this?
> >
> > Best,
> > David
> >
> > On Mon, Jan 31, 2022 at 11:12 AM David Jacot  wrote:
> >>
> >> Hey,
> >>
> >> Thanks for the KIP. I have a few comments:
> >>
> >> 1. I think that it would be better to name the KIP: "GetOffsetShell
> >> should support max-timestamp"
> >> or something like that as this is the initial intent of the change.
> >>
> >> 2. There is a typo: `OffsetSpce` -> `OffsetSpec`.
> >>
> >> 3. It would be great if we could further expand the compatibility
> >> section. It seems that the number
> >> of consumer configurations which could reasonably be used by
> >> `GetOffsetShell` is quite small (timeout,
> >> retries, etc.) and it seems that most of them (if not all) are
> >> supported by the admin client as well. I wonder
> >> if we could be explicit here and argue that the transition won't be
> >> noticed. I might be speculating here.
> >>
> >> 4. For completeness, I think that we should mention extending the
> >> consumer to support max-timestamp
> >> as well in the rejected alternatives. That would be another way to
> >> address the issue. However, I agree
> >> with you that using the admin client is better in the admin tools.
> >>
> >> Best,
> >> David
> >>
> >> On Sun, Jan 30, 2022 at 2:09 PM deng ziming  
> >> wrote:
> >>>
> >>> Sorry all, I mean KIP-851 not KIP-734.
> >>> In KIP-734 we add a new OffsetSpec to AdminClient, in this KIP I just 
> >>> extend this OffsetSpec to GetOffsetShell.
> >>>
>  On Jan 30, 2022, at 6:29 PM, deng ziming  
>  wrote:
> 
>  Hey all, I'm starting the voting on KIP-734.
> 
>  This supports a new OffsetSpec in GetOffsetShell so that we can easily 
>  determine the offset and timestamp of the message with the largest 
>  timestamp on a partition. This seems a simple change but replaced 
>  KafkaConsumer with AdminClient in GetOffsetShell.
> 
>  Thanks,
>  Ziming Deng
> >>>
>


[jira] [Created] (KAFKA-13637) User default.api.timeout.ms config as default timeout for KafkaConsumer.endOffsets

2022-02-01 Thread dengziming (Jira)
dengziming created KAFKA-13637:
--

 Summary: User default.api.timeout.ms config as default timeout for 
KafkaConsumer.endOffsets
 Key: KAFKA-13637
 URL: https://issues.apache.org/jira/browse/KAFKA-13637
 Project: Kafka
  Issue Type: Improvement
Reporter: dengziming
Assignee: dengziming


In KafkaConsumer, we use `request.timeout.ms` in `endOffsets` and 
`default.api.timeout.ms` when in `beginningOffsets`, we should use 
`default.api.timeout.ms` for both.



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


Re: [VOTE] KIP-815: Replace KafkaConsumer with AdminClient in GetOffsetShell

2022-02-01 Thread deng ziming
Hey David,
I rechecked the ConsumerConfig and split the Compatibility section into 2 
sub-sections
regarding AdminClientConfig and ConsumerConfig, I also find a small bug that we 
use
 different timeout in `KafkaConsumer.beginningOffsets` and 
`KafkaConsumer.endOffsets`,
I will fix this.

Please help to review the KIP and the bug, thank you.

Best,
Ziming Deng


> On Feb 1, 2022, at 6:31 PM, David Jacot  wrote:
> 
> Thanks for the updated KIP.
> 
> Regarding the compatibility section, I think that it would be
> great if we could really stress that the configurations that
> could reasonably be used to configure the tool are actually
> all supported by the admin client. Regarding the retry mechanism,
> the consumer will retry until `default.api.timeout.ms` is reached
> and it seems that the admin client does the same by default. Do
> you confirm this?
> 
> Best,
> David
> 
> On Mon, Jan 31, 2022 at 11:12 AM David Jacot  wrote:
>> 
>> Hey,
>> 
>> Thanks for the KIP. I have a few comments:
>> 
>> 1. I think that it would be better to name the KIP: "GetOffsetShell
>> should support max-timestamp"
>> or something like that as this is the initial intent of the change.
>> 
>> 2. There is a typo: `OffsetSpce` -> `OffsetSpec`.
>> 
>> 3. It would be great if we could further expand the compatibility
>> section. It seems that the number
>> of consumer configurations which could reasonably be used by
>> `GetOffsetShell` is quite small (timeout,
>> retries, etc.) and it seems that most of them (if not all) are
>> supported by the admin client as well. I wonder
>> if we could be explicit here and argue that the transition won't be
>> noticed. I might be speculating here.
>> 
>> 4. For completeness, I think that we should mention extending the
>> consumer to support max-timestamp
>> as well in the rejected alternatives. That would be another way to
>> address the issue. However, I agree
>> with you that using the admin client is better in the admin tools.
>> 
>> Best,
>> David
>> 
>> On Sun, Jan 30, 2022 at 2:09 PM deng ziming  wrote:
>>> 
>>> Sorry all, I mean KIP-851 not KIP-734.
>>> In KIP-734 we add a new OffsetSpec to AdminClient, in this KIP I just 
>>> extend this OffsetSpec to GetOffsetShell.
>>> 
 On Jan 30, 2022, at 6:29 PM, deng ziming  wrote:
 
 Hey all, I'm starting the voting on KIP-734.
 
 This supports a new OffsetSpec in GetOffsetShell so that we can easily 
 determine the offset and timestamp of the message with the largest 
 timestamp on a partition. This seems a simple change but replaced 
 KafkaConsumer with AdminClient in GetOffsetShell.
 
 Thanks,
 Ziming Deng
>>> 



Jenkins build is still unstable: Kafka » Kafka Branch Builder » trunk #652

2022-02-01 Thread Apache Jenkins Server
See 




[jira] [Reopened] (KAFKA-13152) Replace "buffered.records.per.partition" with "input.buffer.max.bytes"

2022-02-01 Thread Matthias J. Sax (Jira)


 [ 
https://issues.apache.org/jira/browse/KAFKA-13152?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Matthias J. Sax reopened KAFKA-13152:
-

The PR broke backward compatibility. Needed to revert the commit. Sorry about 
that.

Lets do a new PR based on the old commit that fixes the issue. Thx.

> Replace "buffered.records.per.partition" with "input.buffer.max.bytes" 
> ---
>
> Key: KAFKA-13152
> URL: https://issues.apache.org/jira/browse/KAFKA-13152
> Project: Kafka
>  Issue Type: Improvement
>  Components: streams
>Reporter: Guozhang Wang
>Assignee: Sagar Rao
>Priority: Major
>  Labels: needs-kip
> Fix For: 3.2.0
>
>
> The current config "buffered.records.per.partition" controls how many records 
> in maximum to bookkeep, and hence it is exceed we would pause fetching from 
> this partition. However this config has two issues:
> * It's a per-partition config, so the total memory consumed is dependent on 
> the dynamic number of partitions assigned.
> * Record size could vary from case to case.
> And hence it's hard to bound the memory usage for this buffering. We should 
> consider deprecating that config with a global, e.g. "input.buffer.max.bytes" 
> which controls how much bytes in total is allowed to be buffered. This is 
> doable since we buffer the raw records in .



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


Jenkins build is still unstable: Kafka » Kafka Branch Builder » trunk #651

2022-02-01 Thread Apache Jenkins Server
See 




Jenkins build is still unstable: Kafka » Kafka Branch Builder » trunk #650

2022-02-01 Thread Apache Jenkins Server
See 




[jira] [Created] (KAFKA-13636) Committed offsets could be deleted during a rebalance if a group did not commit for a while

2022-02-01 Thread Damien Gasparina (Jira)
Damien Gasparina created KAFKA-13636:


 Summary: Committed offsets could be deleted during a rebalance if 
a group did not commit for a while
 Key: KAFKA-13636
 URL: https://issues.apache.org/jira/browse/KAFKA-13636
 Project: Kafka
  Issue Type: Bug
  Components: core, offset manager
Affects Versions: 3.0.0, 2.8.1, 2.7.2, 2.6.2, 2.5.1, 2.4.0
Reporter: Damien Gasparina


The group coordinator might delete invalid offsets during a group rebalance. 
During a rebalance, the coordinator is relying on the last commit timestamp 
({_}offsetAndMetadata.commitTimestamp{_}) instead of the last state 
modification {_}timestampt (currentStateTimestamp{_}) to detect expired offsets.

 

This is relatively easy to reproduce by playing with 
group.initial.rebalance.delay.ms, offset.retention.minutes and 
offset.check.retention.interval, I uploaded an example on: 
[https://github.com/Dabz/kafka-example/tree/master/docker/offsets-retention] .

This script does:
 * Start a broker with: offset.retention.minute=2, 
o[ffset.check.retention.interval.ms=|http://offset.check.retention.interval.ms/]1000,
  group.initial.rebalance.delay=2
 * Produced 10 messages
 * Create a consumer group to consume 10 messages, and disable auto.commit to 
only commit a few times
 * Wait 3 minutes, then the Consumer get a {{kill -9}}
 * Restart the consumer after a few seconds
 * The consumer restart from {{auto.offset.reset}} , the offset got removed

 

The cause is due to the GroupMetadata.scala:
 * When the group get emptied, the {{subscribedTopics}} is set to {{Set.empty}} 
([https://github.com/apache/kafka/blob/trunk/core/src/main/scala/kafka/coordinator/group/GroupMetadata.scala#L520-L521])
 * When the new member joins, we add the new member right away in the group ; 
BUT the {{subscribedTopics}} is only updated once the migration is over (in the 
initNewGeneration) (which could take a while due to the 
{{{}group.initial.rebalance.delay{}}})
 * When the log cleaner got executed,  {{subscribedTopics.isDefined}} returns 
true as {{Set.empty != None}} (the underlying condition)
 * Thus we enter 
[https://github.com/apache/kafka/blob/trunk/core/src/main/scala/kafka/coordinator/group/GroupMetadata.scala#L782-L785]
 with an empty {{subscribedTopics}} list and we are relying on the 
{{commitTimestamp}} regardless of the {{currentStateTimestamp}}

 

This seem to be a regression generated by KIP-496 
https://cwiki.apache.org/confluence/display/KAFKA/KIP-496%3A+Administrative+API+to+delete+consumer+offsets#KIP496:AdministrativeAPItodeleteconsumeroffsets-ProposedChanges



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


[jira] [Resolved] (KAFKA-13527) Add top-level error code field to DescribeLogDirsResponse

2022-02-01 Thread Mickael Maison (Jira)


 [ 
https://issues.apache.org/jira/browse/KAFKA-13527?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Mickael Maison resolved KAFKA-13527.

Fix Version/s: 3.2.0
   Resolution: Fixed

> Add top-level error code field to DescribeLogDirsResponse
> -
>
> Key: KAFKA-13527
> URL: https://issues.apache.org/jira/browse/KAFKA-13527
> Project: Kafka
>  Issue Type: Bug
>Reporter: Mickael Maison
>Assignee: Mickael Maison
>Priority: Major
> Fix For: 3.2.0
>
>
> Ticket for KIP-784: 
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-784%3A+Add+top-level+error+code+field+to+DescribeLogDirsResponse



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


[jira] [Resolved] (KAFKA-13595) Allow producing records with null values in Kafka Console Producer

2022-02-01 Thread Mickael Maison (Jira)


 [ 
https://issues.apache.org/jira/browse/KAFKA-13595?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Mickael Maison resolved KAFKA-13595.

Fix Version/s: 3.2.0
   Resolution: Fixed

> Allow producing records with null values in Kafka Console Producer
> --
>
> Key: KAFKA-13595
> URL: https://issues.apache.org/jira/browse/KAFKA-13595
> Project: Kafka
>  Issue Type: Improvement
>Reporter: Mickael Maison
>Assignee: Mickael Maison
>Priority: Major
> Fix For: 3.2.0
>
>
> KIP-810: 
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-810%3A+Allow+producing+records+with+null+values+in+Kafka+Console+Producer



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


Re: [VOTE] KIP-814: Static membership protocol should let the leader skip assignment

2022-02-01 Thread Ryan Leslie (BLOOMBERG/ 919 3RD A)
Thanks, David.

+1 (non-binding)

From: show...@gmail.com At: 01/31/22 22:13:57 UTC-5:00To:  dev@kafka.apache.org
Subject: Re: [VOTE] KIP-814: Static membership protocol should let the leader 
skip assignment

Hi David,

Thanks for the KIP.

I'm +1(non-binding)

Thanks.
Luke

Jason Gustafson  於 2022年2月1日 週二 上午7:11 
寫道:

> +1 Thanks!
>
> On Mon, Jan 31, 2022 at 12:17 AM David Jacot  wrote:
>
> > Hi all,
> >
> > I'd like to start a vote about KIP-814: Static membership protocol
> > should let the leader
> > skip assignment.
> >
> > The KIP is here: https://cwiki.apache.org/confluence/x/C5-kCw.
> >
> > Best,
> > David
> >
>




Re: [VOTE] KIP-801 new authorizer for kip-500 kraft mode

2022-02-01 Thread Jason Gustafson
+1 Thanks!

On Mon, Jan 31, 2022 at 6:20 PM Colin McCabe  wrote:

> Hi all,
>
> It looks like people using gmail are seeing the previous vote thread as
> merged with the discuss thread, so let me create a new thread in order to
> avoid confusion. Usually using a very different thread title works well
> enough to avoid the merging.
>
> Original vote thread:
> https://lists.apache.org/thread/jwjhpdll4jp3y6lo9kox3p5thwo8qpk3
>
> best,
> Colin
>


Re: Understanding the SASL/PLAIN mechanism

2022-02-01 Thread Jeremy Whitlock
Hello all,
Anyone involved in Kafka's SASL support got time to help with this?
Thankfully KIP-255 (
https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=75968876)
helped shed light on things but from the SASL/OAUTHBEARER perspective
instead of SASL/PLAIN, but if anyone has time to help me better understand
what it would take to implement a custom SASL mechanism for Kafka, I'd
greatly appreciate it.

Take care,

Jeremy


On Fri, Dec 17, 2021 at 10:36 AM Jeremy Whitlock 
wrote:

> Hello Kafka Dev,
> I realize that this question might be more SASL than Kafka related,
> but after endless Googling and code browsing, I'm not understanding a few
> things.  I've looked at all of the code for SASL/PLAIN and SASL/OAUTHBEARER
> but when attempting to implement my own custom SASL mechanism, there are
> gaps in my understanding and I'm really trying to make sure I
> understand things before just copying/pasting/refactoring and hoping for
> the best.
>
> Does someone have a little time to explain the execution path for
> SASL/PLAIN so that I can eventually implement my own custom mechanism?
> Here are a few questions I had after spending a good bit of time trying to
> figure this out on my own:
>
> 1. What runs where?  (Where is the LoginModule run, where are the
> callbacks ran, how are SaslClient/SaslServer used, ...)
>
> 2. A follow-up to #1 is that the SASL/PLAIN implementation doesn't seem to
> have a custom SaslClient implementation but does have a custom SaslServer
> implementation.  Why isn't a SaslClient required for SASL/PLAIN?
>
> 3. Are callbacks required for anything more than pluggability?  I ask
> because for PlainLoginModule, JAAS states that the LoginModule should
> perform authentication in login() but PlainLoginModule doesn't do anything
> of the sort, just adding details to the Subject.  SaslChannelBuilder wires
> up a PlainServerCallbackHandler to do the real work but if pluggability
> isn't required, couldn't login() do it?
>
> I think that's it for now.  Ultimately, I want to create my own SASL
> mechanism that works in Kafka to do external authentication using more than
> just username and password.
>
> Take care,
>
> Jeremy
>


Jenkins build is still unstable: Kafka » Kafka Branch Builder » trunk #649

2022-02-01 Thread Apache Jenkins Server
See 




[jira] [Created] (KAFKA-13635) Make Consumer Group Protocol resilient to disk issues with __consumer_offsets

2022-02-01 Thread Akhilesh Dubey (Jira)
Akhilesh Dubey created KAFKA-13635:
--

 Summary: Make Consumer Group Protocol resilient to disk issues 
with __consumer_offsets 
 Key: KAFKA-13635
 URL: https://issues.apache.org/jira/browse/KAFKA-13635
 Project: Kafka
  Issue Type: Improvement
Reporter: Akhilesh Dubey


While working with 6.1.1, we experienced offset reset on some consumer groups 
after a disk full issue (the actual underlying issue was an uncontrolled kafka 
and a machine shutdown).

When the machine and kafka brokers were restarted, consumer applications 
received a {{Found no committed offset for partition }} which triggered 
offset reset which in our case was set to earliest - {{{}Resetting offset for 
partition {}}}.

On further investigation, we noticed that {{GroupMetadataManager}} silently 
handled an offset load issue. 
ERROR [GroupMetadataManager brokerId=1] Error loading offsets from 
__consumer_offsets-33 (kafka.coordinator.group.GroupMetadataManager)
org.apache.kafka.common.errors.CorruptRecordException: Record size 0 is less 
than the minimum record overhead (14)
There's nothing wrong here as the uncontrolled shutdown and possibly pagecache 
issues could have led to disk flush issues and GroupCoordinator cannot do much 
if the offsets themselves are missing.

I would like to request a feature to stop progress/retry if 
{{__consumer_offsets}} partition fails to load.



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


Jenkins build is still unstable: Kafka » Kafka Branch Builder » trunk #648

2022-02-01 Thread Apache Jenkins Server
See 




Re: [VOTE] KIP-815: Replace KafkaConsumer with AdminClient in GetOffsetShell

2022-02-01 Thread David Jacot
Thanks for the updated KIP.

Regarding the compatibility section, I think that it would be
great if we could really stress that the configurations that
could reasonably be used to configure the tool are actually
all supported by the admin client. Regarding the retry mechanism,
the consumer will retry until `default.api.timeout.ms` is reached
and it seems that the admin client does the same by default. Do
you confirm this?

Best,
David

On Mon, Jan 31, 2022 at 11:12 AM David Jacot  wrote:
>
> Hey,
>
> Thanks for the KIP. I have a few comments:
>
> 1. I think that it would be better to name the KIP: "GetOffsetShell
> should support max-timestamp"
> or something like that as this is the initial intent of the change.
>
> 2. There is a typo: `OffsetSpce` -> `OffsetSpec`.
>
> 3. It would be great if we could further expand the compatibility
> section. It seems that the number
> of consumer configurations which could reasonably be used by
> `GetOffsetShell` is quite small (timeout,
> retries, etc.) and it seems that most of them (if not all) are
> supported by the admin client as well. I wonder
> if we could be explicit here and argue that the transition won't be
> noticed. I might be speculating here.
>
> 4. For completeness, I think that we should mention extending the
> consumer to support max-timestamp
> as well in the rejected alternatives. That would be another way to
> address the issue. However, I agree
> with you that using the admin client is better in the admin tools.
>
> Best,
> David
>
> On Sun, Jan 30, 2022 at 2:09 PM deng ziming  wrote:
> >
> > Sorry all, I mean KIP-851 not KIP-734.
> > In KIP-734 we add a new OffsetSpec to AdminClient, in this KIP I just 
> > extend this OffsetSpec to GetOffsetShell.
> >
> > > On Jan 30, 2022, at 6:29 PM, deng ziming  wrote:
> > >
> > >  Hey all, I'm starting the voting on KIP-734.
> > >
> > > This supports a new OffsetSpec in GetOffsetShell so that we can easily 
> > > determine the offset and timestamp of the message with the largest 
> > > timestamp on a partition. This seems a simple change but replaced 
> > > KafkaConsumer with AdminClient in GetOffsetShell.
> > >
> > > Thanks,
> > > Ziming Deng
> >


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

2022-02-01 Thread David Jacot
Hi Hector,

Thanks for the KIP. I finally had a bit of time to read it.

I understand that a few services have been leveraging Kafka's Group Membership
Protocol to do leader election and/or service discovery. However, I am
not entirely
convinced that Kafka should be used in that way because specialized systems
already exist to solve those use cases. That's just my personal opinion.

Assuming that we would want to support such cases in the future, I think that
AbstractCoordinator is not the right abstraction for this.
AbstractCoordinator is more
and internal implementation detail and not really something meant to become a
public API. I think that it would be much better to come up with a new
set of APIs
for these use cases instead of extracting an interface out of
AbstractCoordinator.

Best,
David

On Thu, Nov 11, 2021 at 11:15 PM Hector Geraldino (BLOOMBERG/ 919 3RD
A)  wrote:
>
> 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, in

Re: [VOTE] KIP-591: Add Kafka Streams config to set default state store

2022-02-01 Thread Luke Chen
Hi Matthias,

Good point! I didn't notice that. Yes, I agree it's not good to let store
depends on kstream package. I'll remove the stores API change later. (after
Chinese new year :) )

Thanks for your good suggestion!

Luke


Matthias J. Sax  於 2022年2月1日 週二 下午3:10 寫道:

> Thanks for updating the KIP.
>
> It's much clearer now. I think the improvement to `Materialized` is a
> good one.
>
> However, I still have doubts about `Stores`. While the API change itself
> seems ok (even if I don't see a large benefit), it adds a dependency
> from the DSL classes (ie, package `kstream`) to the state store package
> `state`. I think it would be better to avoid "cyclic" package
> dependencies. While it does not matter atm, as it's all a single module,
> we usually try to avoid "reverse" dependencies: neither the `processor`
> nor `state` packages should use anything from the `kstream` package IMHO.
>
>
> -Matthias
>
>
>
> On 1/30/22 12:00 AM, Luke Chen wrote:
> > Hi John and Guozhang,
> >
> > Thanks for your comments.
> >
> > And @John, yes, you are right.
> > The goal of the improved Materialized API is to provide a way to use
> > im-memory store without providing the name.
> > So, about this comment:
> >
> >> On the other hand, I don't see how the latter of these is
> > more compelling than the former:
> > .count(Materialized.as(Stores.inMemoryKeyValueStore("count-
> > store")));
> > .count(Materialized.as(Stores.keyValueStoreSupplier(StoreTyp
> > e.IN_MEMORY, "count-store")));
> >
> > I think I didn't make it clear here.
> > The improved Materialized API is like this
> >
> .count(Materialized.as(Materialized.withStoreType(StoreImplType.IN_MEMORY));
> > // without name provided.
> >
> > I've updated the KIP to make it clear.
> >
> > Therefore, I'll keep the Materialize/Stores change and complete the PR.
> >
> > Thanks for your comments again!
> >
> > Luke
> >
> >
> >
> > On Sat, Jan 29, 2022 at 7:46 AM John Roesler 
> wrote:
> >
> >> Hi Luke,
> >>
> >> Thanks for the KIP!
> >>
> >> I'm +1 (binding) on your KIP.
> >>
> >> Regarding this last question about chaning Materialized
> >> and/or Stores, I think it might actually be best to drop
> >> that part of the proposal.
> >>
> >> The primary benefit of your proposal is in the cases when
> >> the user doesn't want to specify the store type at all and
> >> just, as a blanket, use in-memory stores across the whole
> >> topology instead of rocksDB ones. For that, we have the
> >> config you proposed.
> >>
> >> As I read it, the Materialized part of the proposal was
> >> secondary; to allow users to override the default storage
> >> engine on a per-operation basis without having to bother
> >> about providing a full-fledged store supplier. In other
> >> words, today, if you want an in-memory store on a grouped
> >> stream, you have to do:
> >>
> >> .count(Materialized.as(Stores.inMemoryKeyValueStore("count-
> >> store")));
> >>
> >> What if you didn't care about the name but wanted it to be
> >> in memory? Well, you're out of luck.
> >>
> >> Therefore, I think there's significant value in modifying
> >> the DSL to allow users to orthogonally specify the storage
> >> engine and the name of the store, as in your KIP as written.
> >>
> >> On the other hand, I don't see how the latter of these is
> >> more compelling than the former:
> >> .count(Materialized.as(Stores.inMemoryKeyValueStore("count-
> >> store")));
> >> .count(Materialized.as(Stores.keyValueStoreSupplier(StoreTyp
> >> e.IN_MEMORY, "count-store")));
> >>
> >>
> >> Regardless, I don't want to let perfect be the enemy of
> >> good. Like I said, I think that the key benefit you're
> >> really going for is the config, so maybe you want to just
> >> drop the Materialize/Stores aspect and simplify the
> >> proposal. Or if you want to keep the latter, I'm fine with
> >> whatever approach you feel is best (which is why I still
> >> voted).
> >>
> >> This feels like the kind of thing that won't really be
> >> crystal clear until the PR is under review (and I'd
> >> encourage you and the reviewer to pay particular attention
> >> to how the new APIs actually look when used in the tests).
> >>
> >> Thanks again! People have been asking for this for a long
> >> time.
> >> -John
> >>
> >>
> >> On Fri, 2022-01-28 at 13:46 -0800, Guozhang Wang wrote:
> >>> Hi Luke,
> >>>
> >>> I'm in favor of using the newly proposed `#sessionStore(StoreType..)`
> and
> >>> deprecating the existing `#persistenSessionStore` etc.
> >>>
> >>> Thanks,
> >>> Guozhang
> >>>
> >>> On Tue, Jan 25, 2022 at 12:17 AM Luke Chen  wrote:
> >>>
>  Thanks Matthias!
> 
>  I agree we could deprecate the existing ones, and add the one with
>  storeType parameter.
> 
>  That is:
>  @deprecated
>  Stores#persistentSessionStore(...)
>  @deprecated
>  Stores#inMemorySessionStore(...)
>  @new added with an additional storeType parameter (IN_MEMORY or
> >> ROCKS_DB)
>  Stores#sessionStoreSupplier(StoreType storeType, ...)

[VOTE] KIP-815: Support max-timestamp in GetOffsetShell

2022-02-01 Thread deng ziming
Thank you David,
I retitled this KIP to be more accurate and supplemented the Compatibility and 
Rejected Alternatives sections, please help to review this again.

Best,
Ziming Deng

> On Jan 31, 2022, at 6:12 PM, David Jacot  wrote:
> 
> Hey,
> 
> Thanks for the KIP. I have a few comments:
> 
> 1. I think that it would be better to name the KIP: "GetOffsetShell
> should support max-timestamp"
> or something like that as this is the initial intent of the change.
> 
> 2. There is a typo: `OffsetSpce` -> `OffsetSpec`.
> 
> 3. It would be great if we could further expand the compatibility
> section. It seems that the number
> of consumer configurations which could reasonably be used by
> `GetOffsetShell` is quite small (timeout,
> retries, etc.) and it seems that most of them (if not all) are
> supported by the admin client as well. I wonder
> if we could be explicit here and argue that the transition won't be
> noticed. I might be speculating here.
> 
> 4. For completeness, I think that we should mention extending the
> consumer to support max-timestamp
> as well in the rejected alternatives. That would be another way to
> address the issue. However, I agree
> with you that using the admin client is better in the admin tools.
> 
> Best,
> David
> 
> On Sun, Jan 30, 2022 at 2:09 PM deng ziming  wrote:
>> 
>> Sorry all, I mean KIP-851 not KIP-734.
>> In KIP-734 we add a new OffsetSpec to AdminClient, in this KIP I just extend 
>> this OffsetSpec to GetOffsetShell.
>> 
>>> On Jan 30, 2022, at 6:29 PM, deng ziming  wrote:
>>> 
>>> Hey all, I'm starting the voting on KIP-734.
>>> 
>>> This supports a new OffsetSpec in GetOffsetShell so that we can easily 
>>> determine the offset and timestamp of the message with the largest 
>>> timestamp on a partition. This seems a simple change but replaced 
>>> KafkaConsumer with AdminClient in GetOffsetShell.
>>> 
>>> Thanks,
>>> Ziming Deng
>>