Re: [DISCUSS] KIP-622 Add currentSystemTimeMs and currentStreamTimeMs to ProcessorContext

2020-07-04 Thread Richard Yu
Hi all, This reminds me of a previous issue I think that we were discussing. @John Roesler I think you should remember this one. A while back, we were talking about having suppress operator emit records by wall-clock time instead of stream time. If we are adding this, wouldn't that make it more

[jira] [Created] (KAFKA-9808) Refactor State Store Hierarchy

2020-04-02 Thread Richard Yu (Jira)
Richard Yu created KAFKA-9808: - Summary: Refactor State Store Hierarchy Key: KAFKA-9808 URL: https://issues.apache.org/jira/browse/KAFKA-9808 Project: Kafka Issue Type: Improvement

[jira] [Created] (KAFKA-9733) Consider addition to Kafka's replication model

2020-03-18 Thread Richard Yu (Jira)
Richard Yu created KAFKA-9733: - Summary: Consider addition to Kafka's replication model Key: KAFKA-9733 URL: https://issues.apache.org/jira/browse/KAFKA-9733 Project: Kafka Issue Type

Re: [VOTE] KIP-557: Add emit on change support for Kafka Streams

2020-03-07 Thread Richard Yu
ard for this case, because it's computed as maximum over > all input record for the window). > > > (3) The discussion about stream time is very interesting. I agree that > it's an orthogonal concern to this KIP. > > > > - -Matthias > > > On 3/6/20 1

Re: [VOTE] KIP-557: Add emit on change support for Kafka Streams

2020-03-06 Thread Richard Yu
Hi all, I have decided to pass this KIP with 2 binding votes and 3 non-binding votes (including mine). I will update KIP status shortly after this. Best, Richard On Thu, Mar 5, 2020 at 3:45 PM Richard Yu wrote: > Hi all, > > Just polling for some last changes on the name. > I thin

Re: [VOTE] KIP-557: Add emit on change support for Kafka Streams

2020-03-05 Thread Richard Yu
quot;skipped-idempotent-updates" to be consistent with the > > "dropped-records". > > > > Best, > > Bruno > > > > On Tue, Mar 3, 2020 at 11:57 PM Richard Yu > > wrote: > > > > > > Hi all, > > > > > > Thanks fo

Re: [VOTE] KIP-557: Add emit on change support for Kafka Streams

2020-03-03 Thread Richard Yu
ating all types of > > > > scenarios that can cause dropped records to the same metrics: > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-557%3A+Add+emit+on+change+support+for+Kafka+Streams > > > > > > > > late-records-drop: I

Re: [VOTE] KIP-557: Add emit on change support for Kafka Streams

2020-03-02 Thread Richard Yu
nks for the KIP! > > > > I'm +1 (binding) > > > > -john > > > > On Thu, Feb 27, 2020, at 14:40, Richard Yu wrote: > > > Hi all, > > > > > > I am proposing a new optimization to Kafka Streams which

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-28 Thread Richard Yu
Hi all, Just some updates. Below is the vote thread: https://sematext.com/opensee/m/Kafka/uyzND1h1NPW1tLVQR?subj=+VOTE+KIP+557+Add+emit+on+change+support+for+Kafka+Streams It would be great if we can include this change to Kafka. :) Cheers, Richard On Thu, Feb 27, 2020 at 6:45 PM Richard Yu

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-27 Thread Richard Yu
7;t result in any computations. > > But practically speaking, I'm really not confident that there's anyone > really using the > untilTimeLimit suppression, and even if they are, if they would really see > consecutive > idempotent updates for long enough to really have

[VOTE] KIP-557: Add emit on change support for Kafka Streams

2020-02-27 Thread Richard Yu
Hi all, I am proposing a new optimization to Kafka Streams which would greatly reduce the number of idempotent updates (or no-ops) in the Kafka Streams DAG. A number of users have been interested in this feature, so it would be nice to pass this one in. For information, the KIP is described below

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-27 Thread Richard Yu
Hi all, I might've made a minor mistake. The processor node level is level 3, not level 1. I will correct the KIP accordingly. After looking over things, I decided to start the voting thread this afternoon. Cheers, Richard On Thu, Feb 27, 2020 at 12:29 PM Richard Yu wrote: > Hi B

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-27 Thread Richard Yu
, Bruno Cadonna wrote: > > > Hi Richard, > > > > > > 1. Could you change "idempotent update operations will only be dropped > > > from KTables, not from other classes." -> idempotent update operations > > > will only be dropped from material

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-25 Thread Richard Yu
rialize both the "old" > and > "new" result. > > It's certainly a good observation, but I think we can just make a note of > it > in "rejected alternatives" for now, and plan to refine it later, if it does > pose a big performance problem. > &

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-22 Thread Richard Yu
the underlying values of the result might be the same. (therefore equals() might return true) Do you think this would be plausible? Cheers, Richard On Fri, Feb 21, 2020 at 2:37 PM Richard Yu wrote: > Hello, > > Just to make some updates. I changed the name of the metric so that it was

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-21 Thread Richard Yu
the number of records dropped. I hope that this is more on point. Best, Richard On Fri, Feb 21, 2020 at 2:20 PM Richard Yu wrote: > Hi all, > > Thanks for the clarification. I was just confused a little on what was > going on. > > So I guess then that for the actual pr

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-21 Thread Richard Yu
> do you mean? I have the feeling that we agreed to get rid of > > > > idempotent updates if the aggregate is updated with the same key, > > > > value, AND timestamp. I am also fine if we do not include this into > > > > this KIP (remember: baby steps). > > > &

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-18 Thread Richard Yu
re demand, we could > > still add an option to send `oldValues` instead of materialization (this > > would at least save the store overhead). As we consider the KIP an > > optimization, a "config" seems to make sense. > > > > > > -Matthias > > > &g

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-17 Thread Richard Yu
is going to struggle with high-volume updates, and it would be > nice > for them to know that this feature is saving them X-thousand updates per > second, > etc. > > What does everyone think about this? Note, as I read it, what I've said > above is > already ref

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-11 Thread Richard Yu
Hi all, Bumping this. If you feel that this KIP is not too urgent. Then let me know. :) Cheers, Richard On Thu, Feb 6, 2020 at 4:55 PM Richard Yu wrote: > Hi all, > > I've had just a few thoughts regarding the forwarding of change>. As Matthias already mentioned, there a

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-06 Thread Richard Yu
:30 -0600, John Roesler wrote: [EXTERNAL > > > >> EMAIL] Attention: This email was sent from outside TiVo. DO NOT > > > >> CLICK any links or attachments unless you expected them. > > > >> > > > >> > > > >>

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-02 Thread Richard Yu
. In the discussion about materializing results of stateless > operations, could you please add that those stateless operations are > on KTables? IMO adding this information makes the KIP easier > approachable by people that are not that familiar with the matter. > Best, > Bruno >

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-02-01 Thread Richard Yu
of bullet points? I.e., please drop glue phrases like > "there has > >>> been some discussion" or "possibly we could do X". For the final > version of the > >>> KIP, it should just say, "Streams will do X, Streams will do Y". Feel > free

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-01-27 Thread Richard Yu
, Jan 25, 2020 at 11:12 AM Richard Yu wrote: > Hi all, > > Thanks for all the discussion! > > @John and @Bruno I will survey other possible systems and see what I can > do. > Just a question, by systems, I suppose you would mean the pros and cons of > different repo

Re: [KAFKA-557] Add emit on change support for Kafka Streams

2020-01-25 Thread Richard Yu
, it seems like we should > propose > > >>> to add an opt-out config. Can you update the KIP to mention the exact > > >>> config key and value(s) you'd propose? > > >>> > > >>> Just to move the discussion forward, maybe something lik

Re: [VOTE] KIP-373: Allow users to create delegation tokens for other users

2020-01-16 Thread Richard Yu
Hi Jun, Can the SSL username really include the comma? >From what I could tell, when I searched it up, I couldn't find anything that indicated comma can be a delimiter. A related doc below: https://knowledge.digicert.com/solution/SO12401.html Cheers, Richard On Thu, Jan 16, 2020 at 1:37 PM J

[KAFKA-557] Add emit on change support for Kafka Streams

2020-01-10 Thread Richard Yu
Hi everybody! I'd like to propose a change that we probably should've added for a long time now. The key benefit of this KIP would be reduced traffic in Kafka Streams since a lot of no-op results would no longer be sent downstream. Here is the KIP for reference. https://cwiki.apache.org/confluen

[jira] [Resolved] (KAFKA-9285) Implement failed message topic to account for processing lag during failure

2019-12-07 Thread Richard Yu (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-9285?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Richard Yu resolved KAFKA-9285. --- Resolution: Fixed Already resolved by Kafka Connect. > Implement failed message topic to acco

[jira] [Created] (KAFKA-9285) Implement failed message topic to account for processing lag during failure

2019-12-06 Thread Richard Yu (Jira)
Richard Yu created KAFKA-9285: - Summary: Implement failed message topic to account for processing lag during failure Key: KAFKA-9285 URL: https://issues.apache.org/jira/browse/KAFKA-9285 Project: Kafka

[DISCUSS] KIP-540: Implement per key stream time tracking

2019-11-10 Thread Richard Yu
Hi all, After some thought, I thought that we could move forward with this KIP. The low traffic suppression buffer issue is not necessarily a blocker issue, since there are ways to mitigate the problem when implementing this KIP. Below is the KIP link: https://cwiki.apache.org/confluence/display/

Re: [DISCUSS] KIP-539: Implement mechanism to flush out records in low volume suppression buffers

2019-10-22 Thread Richard Yu
're willing to > get events flushed out of the suppression buffer after some wall clock > time passes, rather than solely based on stream time. > > WDYT? > Thanks, > -John > > On Sun, Oct 20, 2019 at 11:50 AM Richard Yu > wrote: > > > > Hi Bill, John

Re: [DISCUSS] KIP-539: Implement mechanism to flush out records in low volume suppression buffers

2019-10-20 Thread Richard Yu
2019 at 7:16 PM Richard Yu wrote: > Hi Bill, > > Thanks for the input! > TBH, I am think that suppression buffers are not used *in response *to > low traffic conditions. > Rather, we are trying to fix the situation when low traffic conditions > occur in a suppression buffe

Re: [DISCUSS] KIP-539: Implement mechanism to flush out records in low volume suppression buffers

2019-10-18 Thread Richard Yu
hanks, > Bill > > On Thu, Oct 17, 2019 at 7:42 PM Richard Yu > wrote: > > > Hi all, > > > > I wish to discuss this KIP which would help us in resolving some issues > we > > have with suppression buffers. > > Below is the link: > > > > >

Re: [VOTE] KIP-534: Retain tombstones for approximately delete.retention.ms milliseconds

2019-10-18 Thread Richard Yu
Hi all, Seeing that we got all out votes needed with 3 binding votes and 0 nonbinding. I consider this KIP passed. Cheers, Richard On Fri, Oct 18, 2019 at 9:17 AM Guozhang Wang wrote: > Thanks Richard, I'm +1 on the KIP > > On Thu, Oct 17, 2019 at 3:51 PM Richard Yu >

[DISCUSS] KIP-539: Implement mechanism to flush out records in low volume suppression buffers

2019-10-17 Thread Richard Yu
Hi all, I wish to discuss this KIP which would help us in resolving some issues we have with suppression buffers. Below is the link: https://cwiki.apache.org/confluence/display/KAFKA/KIP-539%3A+Implement+mechanism+to+flush+out+records+in+low+volume+suppression+buffers @John Roesler if you have t

Re: [VOTE] KIP-534: Retain tombstones for approximately delete.retention.ms milliseconds

2019-10-17 Thread Richard Yu
ore bytes to encode. > With a > > record batch of 512 in practice, and suppose after compaction each record > > would take 2 more byte for encoding deltas, that would be 1K more per > > batch. Usually it would not be too big of an issue with reasonable sized > >

Re: [VOTE] KIP-534: Retain tombstones for approximately delete.retention.ms milliseconds

2019-10-16 Thread Richard Yu
delta, that would > take more bytes to encode. > > Guozhang > > On Wed, Oct 16, 2019 at 6:48 PM Jason Gustafson > wrote: > > > +1. Thanks Richard. > > > > On Wed, Oct 16, 2019 at 10:04 AM Richard Yu > > wrote: > > > > > Hi all, >

Re: [VOTE] KIP-534: Retain tombstones for approximately delete.retention.ms milliseconds

2019-10-16 Thread Richard Yu
Hi all, Want to try to get this KIP wrapped up. So it would be great if we can get some votes. Cheers, Richard On Tue, Oct 15, 2019 at 12:58 PM Jun Rao wrote: > Hi, Richard, > > Thanks for the updated KIP. +1 from me. > > Jun > > On Tue, Oct 15, 2019 at 12:46 PM Richard

Re: [VOTE] KIP-534: Retain tombstones for approximately delete.retention.ms milliseconds

2019-10-15 Thread Richard Yu
;t cover the transaction > markers. For proposal 2, one reason is that the interval record header > could be exposed to the clients. > > > Jun > > > On Mon, Oct 14, 2019 at 4:42 PM Richard Yu > wrote: > > > Hi all, > > > > The discussion for KIP-534 seem

[VOTE] KIP-534: Retain tombstones for approximately delete.retention.ms milliseconds

2019-10-14 Thread Richard Yu
Hi all, The discussion for KIP-534 seems to have concluded. So I wish to vote this in so that we can get it done. Its a small bug fix. :) Below is the KIP link: https://cwiki.apache.org/confluence/display/KAFKA/KIP-534%3A+Retain+tombstones+for+approximately+delete.retention.ms+milliseconds Cheer

Re: [DISCUSS] KIP-515: Reorganize checkpoint system in log cleaner to be per partition

2019-10-12 Thread Richard Yu
the new message format. Perhaps we can just document this > limitation. > > Hi, Richard, > > Could you update the KIP with Jason's approach? Also, it seems that KIP-515 > is already taken by another KIP. Could you use a new KIP number for this? > > Thanks, > > Ju

Re: [DISCUSS] KIP-515: Reorganize checkpoint system in log cleaner to be per partition

2019-09-27 Thread Richard Yu
current time + log.cleaner.delete.retention.ms. > > What do you think? > > -Jason > > > > On Thu, Sep 19, 2019 at 3:21 PM Richard Yu > wrote: > > > Hi Jason, > > > > That hadn't occurred to me. > > > > I think I missed your comment in

Re: [DISCUSS] KIP-515: Reorganize checkpoint system in log cleaner to be per partition

2019-09-19 Thread Richard Yu
ee that earlier transactions > will be eligible for deletion before later ones. It all depends on the keys > written in the transaction. I don't see an obvious way to solve this > problem without some record-level bookkeeping, but I might be missing > something. > > T

Re: [DISCUSS] KIP-515: Reorganize checkpoint system in log cleaner to be per partition

2019-09-09 Thread Richard Yu
set. We could use this timestamp to determine whether the > tombstone should be removed in subsequent rounds of cleaning. This way, we > can still keep the current per disk checkpoint file, which is more > efficient. Personally, I think this approach may be better. Could you > document th

[DISCUSS] KIP-515: Reorganize checkpoint system in log cleaner to be per partition

2019-09-01 Thread Richard Yu
Hi all, A KIP has been written that wishes to upgrade the checkpoint file system in log cleaner. If anybody wishes to comment, feel free to do so. :) https://cwiki.apache.org/confluence/display/KAFKA/KIP-515%3A+Reorganize+checkpoint+file+system+in+log+cleaner+to+be+per+partition Above is the link

Re: KIP-457: Add DISCONNECTED state to Kafka Streams

2019-06-27 Thread Richard Yu
nces as to which approach we should take? It would greatly help in the implementation of the issue. Cheers,Richard On Thursday, June 13, 2019, 4:55:29 PM GMT+8, Richard Yu wrote: Hi Guozhang, Thanks for the input! Then I guess from the approach you have listed above, no API changes will be

Re: KIP-457: Add DISCONNECTED state to Kafka Streams

2019-06-13 Thread Richard Yu
ignment indeed, which is not the same as b), but I feel consolidating these to cases with a single metric seem also fine. Guozhang On Wed, Apr 17, 2019 at 2:30 PM Richard Yu wrote: > Alright, so I made a few changes to the KIP. > I realized that there might be an easier way to give the user

[jira] [Created] (KAFKA-8516) Consider allowing all replicas to have read/write permissions

2019-06-09 Thread Richard Yu (JIRA)
Richard Yu created KAFKA-8516: - Summary: Consider allowing all replicas to have read/write permissions Key: KAFKA-8516 URL: https://issues.apache.org/jira/browse/KAFKA-8516 Project: Kafka Issue

Re: [DISCUSS] KIP-472: Add header to RecordContext

2019-05-31 Thread Richard Yu
lication but would be considered "internal" similar to transaction > markers. However, changing the message format is a mayor change and > hence, I am not sure if it worth doing at all atm. > > > -Matthias > > > > On 5/20/19 7:20 PM, Richard Yu wrote: > >

[jira] [Created] (KAFKA-8438) Add API to allow user to define end behavior of consumer failure

2019-05-27 Thread Richard Yu (JIRA)
Richard Yu created KAFKA-8438: - Summary: Add API to allow user to define end behavior of consumer failure Key: KAFKA-8438 URL: https://issues.apache.org/jira/browse/KAFKA-8438 Project: Kafka

[jira] [Resolved] (KAFKA-8434) Make global stream time consistent over all stream tasks

2019-05-26 Thread Richard Yu (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-8434?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Richard Yu resolved KAFKA-8434. --- Resolution: Fixed > Make global stream time consistent over all stream ta

[jira] [Created] (KAFKA-8434) Make global stream time consistent over all stream tasks

2019-05-26 Thread Richard Yu (JIRA)
Richard Yu created KAFKA-8434: - Summary: Make global stream time consistent over all stream tasks Key: KAFKA-8434 URL: https://issues.apache.org/jira/browse/KAFKA-8434 Project: Kafka Issue Type

[jira] [Created] (KAFKA-8431) Add a onTimeoutExpired callback to Kafka Consumer

2019-05-26 Thread Richard Yu (JIRA)
Richard Yu created KAFKA-8431: - Summary: Add a onTimeoutExpired callback to Kafka Consumer Key: KAFKA-8431 URL: https://issues.apache.org/jira/browse/KAFKA-8431 Project: Kafka Issue Type

[DISCUSS] KIP-472: Add header to RecordContext

2019-05-20 Thread Richard Yu
: https://cwiki.apache.org/confluence/display/KAFKA/KIP-472%3A+%5BSTREAMS%5D+Add+partition+time+field+to+RecordContext Cheers, Richard Yu

[jira] [Created] (KAFKA-8388) Add methods to query for entries in KTable using timestamp

2019-05-17 Thread Richard Yu (JIRA)
Richard Yu created KAFKA-8388: - Summary: Add methods to query for entries in KTable using timestamp Key: KAFKA-8388 URL: https://issues.apache.org/jira/browse/KAFKA-8388 Project: Kafka Issue

Re: KIP-457: Add DISCONNECTED state to Kafka Streams

2019-04-28 Thread Richard Yu
Alright, I made some changes. Matthias, if you had time, it would be good if you made another pass. This should be close to completion. Cheers, Richard On Sat, Apr 27, 2019 at 3:46 PM Richard Yu wrote: > Hi Matthias, > > Sure, I could do the DISCONNECTED state. > > > On Sat,

Re: KIP-457: Add DISCONNECTED state to Kafka Streams

2019-04-27 Thread Richard Yu
better user experience. > > Your current proposal does not add a new state, even if it mentions this > in the beginning. Compare: > > https://github.com/apache/kafka/blob/trunk/streams/src/main/java/org/apache/kafka/streams/processor/internals/StreamThread.java#L74-L153 > > > -Matthia

[DISCUSS] KIP-463: Auto-configure serdes passed alongside TopologyBuilder

2019-04-25 Thread Richard Yu
the pros and cons of each approach. https://cwiki.apache.org/confluence/display/KAFKA/KIP-463%3A+Auto-configure+non-default+Serdes+passed+alongside+the+TopologyBuilder Hope this helps, Richard Yu

Re: KIP-457: Add DISCONNECTED state to Kafka Streams

2019-04-23 Thread Richard Yu
Oh, so if possible. I thought it would be good if we could finish this KIP up. Matthias, or Michael, if you have any further comments, please let me know. :) Otherwise, I might restart the voting thread in a few days. Cheers, Richard On Wed, Apr 17, 2019 at 2:30 PM Richard Yu wrote: > Alri

Re: KIP-457: Add DISCONNECTED state to Kafka Streams

2019-04-17 Thread Richard Yu
approach that is now outlined in the KIP. Instead, we could just add a method which I think achieves the same effect. If any of you thinks there is wrong with this approach, please let me know. :) Cheers, Richard On Wed, Apr 17, 2019 at 11:49 AM Richard Yu wrote: > I just realized someth

Re: KIP-457: Add DISCONNECTED state to Kafka Streams

2019-04-17 Thread Richard Yu
t mean we would also be dealing with consumer API changes as well? I don't think consumer has any methods which would give us the state of a connection either. - Richard On Wed, Apr 17, 2019 at 8:43 AM Richard Yu wrote: > Hi Micheal, > > Yeah, those are some points I should'

Re: [VOTE] KIP-457: Add DISCONNECTED status to Kafka Streams

2019-04-17 Thread Richard Yu
Sorry everybody, if you don't mind holding off voting for a second. Something came up, take a look at the discussion thread. - Richard On Wed, Apr 17, 2019 at 8:46 AM Richard Yu wrote: > Hi all, > > I would like to propose a minor change to the current KafkaStreams#st

[VOTE] KIP-457: Add DISCONNECTED status to Kafka Streams

2019-04-17 Thread Richard Yu
Hi all, I would like to propose a minor change to the current KafkaStreams#state() method. Considering the small size of this proposal, I thought it would be good if we could pass it quickly. (It does not have large scale ramifications) Here is the KIP link: https://cwiki.apache.org/confluence/di

Re: KIP-457: Add DISCONNECTED state to Kafka Streams

2019-04-17 Thread Richard Yu
a KIP vote to pass if these basic questions aren't > properly sorted out in the KIP. > > Best, > Michael > > > > On Wed, Apr 17, 2019 at 3:35 AM Richard Yu > wrote: > > > Hi all, > > > > Considering that this is a simple KIP, I would pr

Re: KIP-457: Add DISCONNECTED state to Kafka Streams

2019-04-16 Thread Richard Yu
Hi all, Considering that this is a simple KIP, I would probably start the voting tomorrow. I think it would be good if we could get this in fast. On Tue, Apr 16, 2019 at 3:31 PM Richard Yu wrote: > Oh, I probably misunderstood the difference between DISCONNECTED and DEAD. > I will upda

Re: KIP-457: Add DISCONNECTED state to Kafka Streams

2019-04-16 Thread Richard Yu
nnect to the brokers. It seems reasonable to > add a DISCONNECT for this case though. > > > > -Matthias > > > > On 4/16/19 9:30 AM, Richard Yu wrote: > > Hi all, > > > > I like to propose a small KIP on adding a new state to > KafkaStreams#state(). > > I

KIP-457: Add DISCONNECTED state to Kafka Streams

2019-04-16 Thread Richard Yu
Hi all, I like to propose a small KIP on adding a new state to KafkaStreams#state(). It is very simple, so this should pass relatively quickly! Here is the discussion link: https://cwiki.apache.org/confluence/display/KAFKA/KIP-457%3A+Add+DISCONNECTED+status+to+Kafka+Streams Cheers, Richard

[jira] [Created] (KAFKA-8020) Consider making ThreadCache a time-aware LRU Cache

2019-02-28 Thread Richard Yu (JIRA)
Richard Yu created KAFKA-8020: - Summary: Consider making ThreadCache a time-aware LRU Cache Key: KAFKA-8020 URL: https://issues.apache.org/jira/browse/KAFKA-8020 Project: Kafka Issue Type

[jira] [Created] (KAFKA-7850) Remove deprecated KStreamTestDriver

2019-01-21 Thread Richard Yu (JIRA)
Richard Yu created KAFKA-7850: - Summary: Remove deprecated KStreamTestDriver Key: KAFKA-7850 URL: https://issues.apache.org/jira/browse/KAFKA-7850 Project: Kafka Issue Type: Improvement

Re: [DISCUSS] KIP-408: Add Asynchronous Processing to Kafka Streams

2019-01-04 Thread Richard Yu
Hi all, Just want to hear some opinions on this KIP from the PMCs. It would be nice if we got input from them. Don't want to drag this KIP for too long! :) Hope we get some input :) Thanks, Richard On Thu, Jan 3, 2019 at 8:26 PM Richard Yu wrote: > Hi Boyang, > > Inter

Re: [DISCUSS] KIP-408: Add Asynchronous Processing to Kafka Streams

2019-01-03 Thread Richard Yu
could skip records and leave certain records remain on the > queue for late processing. This should be something similar to KIP-408 > which also shares some motivations for us to invest. > > Boyang > > ____ > From: Richard Yu > Sent: Frida

Re: [DISCUSS] KIP-408: Add Asynchronous Processing to Kafka Streams

2019-01-03 Thread Richard Yu
Hi all, Just bumping this KIP. Would be great if we got some discussion. On Sun, Dec 30, 2018 at 5:13 PM Richard Yu wrote: > Hi all, > > I made some recent changes to the KIP. It should be more relevant with the > issue now (involves Processor API in detail). > It would be gre

Re: [DISCUSS] KIP-262 Metadata should include the number of state stores for task

2019-01-03 Thread Richard Yu
> >>> Hi Richard, > >>> > >>> with KIP-268 in place (should be accepted soon) the upgrade path is > >>> covered. Thus, you can update your KIP accordingly, referring to > KIP-268. > >>> > >>> Can you also update your KIP similar

Re: [DISCUSS] KIP-408: Add Asynchronous Processing to Kafka Streams

2018-12-30 Thread Richard Yu
Hi all, I made some recent changes to the KIP. It should be more relevant with the issue now (involves Processor API in detail). It would be great if you could comment. Thanks, Richard On Wed, Dec 26, 2018 at 10:01 PM Richard Yu wrote: > Hi all, > > Just changing the title o

Re: KIP-408: Add Asynchronous Processing to Kafka Streams

2018-12-26 Thread Richard Yu
Hi all, just saying. We are migrating to a different discussion thread. (Forgot that the discussion thread's name was incorrect.) Sorry for the confusion. On Mon, Dec 24, 2018 at 7:57 PM Richard Yu wrote: > Sorry, just making a correction. > > Even if we are processing records ou

Re: [DISCUSS] KIP-408: Add Asynchronous Processing to Kafka Streams

2018-12-26 Thread Richard Yu
Hi all, Just changing the title of the KIP. Discovered it wasn't right. Thats about it. :) On Mon, Dec 24, 2018 at 7:57 PM Richard Yu wrote: > Sorry, just making a correction. > > Even if we are processing records out of order, we will still have to > checkpoint offset ranges

Re: KIP-408: Add Asynchronous Processing to Kafka Streams

2018-12-24 Thread Richard Yu
guarantee. Although when implementing this change, there might be some kinks that we have not thought about which could throw a monkey wrench into the works. But definitely worth trying out, Richard On Mon, Dec 24, 2018 at 6:51 PM Richard Yu wrote: > Hi Boyang, > > I could see where you

Re: KIP-408: Add Asynchronous Processing to Kafka Streams

2018-12-24 Thread Richard Yu
e a good idea to break > consumer ordering guarantee by default. > > Best, > Boyang > > > From: Richard Yu > Sent: Saturday, December 22, 2018 9:08 AM > To: dev@kafka.apache.org > Subject: Re: KIP-408: Add Asynchronous Processing to Kafka Streams > > Hi Boyang, >

Re: KIP-408: Add Asynchronous Processing to Kafka Streams

2018-12-21 Thread Richard Yu
antee we are gonna > provide with this new API, or there is no ordering guarantee at all? Could > we discuss any potential issues if consumer needs to process out-of-order > messages? > > Best, > Boyang > > From: Richard Yu > Sent:

KIP-408: Add Asynchronous Processing to Kafka Streams

2018-12-21 Thread Richard Yu
this problem. Thanks, Richard Yu

Re: KIP-406: GlobalStreamThread should honor custom reset policy

2018-12-17 Thread Richard Yu
Hi Matthias, It would be great if we got your input on this. On Sun, Dec 16, 2018 at 3:06 PM Richard Yu wrote: > Hi everybody, > > There is a new KIP regarding the resilience of GlobalStreamThread which > could be seen below: > > https://cwiki.apache.org/confluence/displa

KIP-406: GlobalStreamThread should honor custom reset policy

2018-12-16 Thread Richard Yu
great if you could pitch in! Thanks, Richard Yu

Re: [DISCUSS] KIP-333 Consider a faster form of rebalancing

2018-07-17 Thread Richard Yu
Hi Becket, I made some changes and clarified the motivation for this KIP. :)It should be easier to understand now since I included a diagram. Thanks,Richard Yu On Tuesday, July 17, 2018, 4:38:11 PM GMT+8, Richard Yu wrote: Hi Becket, Thanks for reviewing this KIP. :) I probably did

Re: [DISCUSS] KIP-333 Consider a faster form of rebalancing

2018-07-17 Thread Richard Yu
your questions answered. I will update the KIP soon, so please stay tuned.  Thanks,Richard Yu On Tuesday, July 17, 2018, 2:14:07 PM GMT+8, Becket Qin wrote: Hi Richard, Thanks for the KIP. I am a little confused on what is proposed. The KIP suggests that after recovery from a

Re: [DISCUSS] KIP-335 Consider configurations for Kafka Streams

2018-07-08 Thread Richard Yu
Hi Matthias, It would be nice to get your opinions on this. On Monday, July 9, 2018, 12:17:33 PM GMT+8, Richard Yu wrote: Hi all, Eversince KIP-266 was concluded, there has been a pressing need to migrate Kafka Streams as well. For the link, please click here: https

[DISCUSS] KIP-335 Consider configurations for Kafka Streams

2018-07-08 Thread Richard Yu
Hi all, Eversince KIP-266 was concluded, there has been a pressing need to migrate Kafka Streams as well. For the link, please click here: https://cwiki.apache.org/confluence/display/KAFKA/KIP-335%3A+Consider+configurations+for+KafkaStreams Thanks, Richard Yu

Re: [VOTE] KIP-331 Add default implementation to close() and configure() for Serializer, Deserializer and Serde

2018-07-05 Thread Richard Yu
Nice KIP! +1 (non-binding) -Richard On Friday, July 6, 2018, 9:10:43 AM GMT+8, Matthias J. Sax wrote: Thanks for the KIP! +1 (binding) -Matthias On 7/5/18 7:45 AM, Chia-Ping Tsai wrote: > hi all, > > I would like to start voting on "KIP-331 Add default implementation to > close()

[DISCUSS] KIP-333 Add faster mode of rebalancing.

2018-07-05 Thread Richard Yu
... | | | Thanks,Richard Yu

[DISCUSS] KIP-333 Consider a faster form of rebalancing

2018-07-05 Thread Richard Yu
Hi all, I would like to discuss KIP-333 (which proposes a faster mode of rebalancing). Here is the link for the KIP: https://cwiki.apache.org/confluence/display/KAFKA/KIP-333%3A+Add+faster+mode+of+rebalancing Thanks, Richard Yu

[DISCUSS] KIP-333 Consider adding faster form of rebalancing

2018-07-05 Thread Richard Yu
... | | | Thanks,Richard Yu

[jira] [Created] (KAFKA-7132) Consider adding multithreaded form of recovery

2018-07-04 Thread Richard Yu (JIRA)
Richard Yu created KAFKA-7132: - Summary: Consider adding multithreaded form of recovery Key: KAFKA-7132 URL: https://issues.apache.org/jira/browse/KAFKA-7132 Project: Kafka Issue Type

[jira] [Created] (KAFKA-7127) Add asynchronous support for methods in KafkaConsumer

2018-07-02 Thread Richard Yu (JIRA)
Richard Yu created KAFKA-7127: - Summary: Add asynchronous support for methods in KafkaConsumer Key: KAFKA-7127 URL: https://issues.apache.org/jira/browse/KAFKA-7127 Project: Kafka Issue Type

[jira] [Created] (KAFKA-7118) Currently, it was discovered that KafkaConsumer's close() method might not be multi-thread safe when multiple cores are calling the same consumer.

2018-06-28 Thread Richard Yu (JIRA)
Richard Yu created KAFKA-7118: - Summary: Currently, it was discovered that KafkaConsumer's close() method might not be multi-thread safe when multiple cores are calling the same consumer. Key: KAFKA-7118 URL:

Re: [VOTE] KIP-266: Add TimeoutException for KafkaConsumer#position

2018-05-13 Thread Richard Yu
gt;>> Thanks for the KIP, +1 (binding). > > >>>>> > > >>>>> One small correction: the KIP mentions that close() will be > > >> deprecated, > > >>>> but > > >>>>> we do not want to do this because it is needed

Re: [VOTE] KIP-266: Add TimeoutException for KafkaConsumer#position

2018-05-04 Thread Richard Yu
Hi all, I would like to bump this thread since discussion in the KIP appears to be reaching its conclusion. On Thu, Mar 15, 2018 at 3:30 PM, Richard Yu wrote: > Hi all, > > Since there does not seem to be too much discussion in KIP-266, I will be > starting a voting thread. > H

[VOTE] KIP-266: Add TimeoutException to KafkaConsumer#position()

2018-05-04 Thread Richard Yu
Hi all, It appears that discussion is coming to a close for KIP-266. I would like to start a voting thread for this KIP. Here is the link for reference. https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=75974886 Thanks, Richard

Re: [DISCUSSION] KIP-266: Add TimeoutException to KafkaConsumer#position()

2018-05-02 Thread Richard Yu
> > > > > > Thanks for the tip, Ted! > > > > > > > > > > On Thu, Apr 19, 2018 at 12:12 PM, Ted Yu > > wrote: > > > > > > > > > >> John: > > > > >> In case you want to pursue async poll, it seems

Re: [DISCUSSION] KIP-266: Add TimeoutException to KafkaConsumer#position()

2018-04-17 Thread Richard Yu
you'll have a clean slate for the rest of the work. > > On Tue, Apr 17, 2018 at 3:39 PM, Richard Yu > wrote: > > > Hi John, > > > > I think that you could finish your PR that corresponds with KIP-288 and > > merge it. I can finish my side of the work after

Re: [DISCUSSION] KIP-266: Add TimeoutException to KafkaConsumer#position()

2018-04-17 Thread Richard Yu
reading this discussion gave me a new idea for > > providing a non-breaking update path... What if we introduce a new > variant > > 'poll(long timeout, TimeUnit unit)' that displays the new, desired > > behavior, and just leave the old method alone? > > > >

Re: [DISCUSS] KIP-288: Consumer poll timeout change and new waitForAssignment method

2018-04-17 Thread Richard Yu
Hi John, bq. #1 (wait for metadata) is infinite. Some of what you stated in this KIP has already been previously discussed in a older KIP. (KIP-266) Just for your reference. Thanks, Richard On Tue, Apr 17, 2018 at 11:08 AM, John Roesler wrote: > Hello all, > > I am proposing KIP-288 to improv

  1   2   >