[ 
https://issues.apache.org/jira/browse/KAFKA-6323?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16298166#comment-16298166
 ] 

Frederic Arno commented on KAFKA-6323:
--------------------------------------

I'm fine with most of that, I only have a doubt about not aligning wall-clock 
punctuations on {{now + N*interval}}, which could effectively make punctuations 
calls drift away. Do you have use cases where spacing punctuations by at least 
interval is critical and requires that behavior?

I've pushed updated code, in which I do not allow punctuation time drift (this 
makes the behavior more coherent between stream-time and wall-clock-time 
punctuation).

By default, the new code aligns punctuations as discussed above.

I've also added an overload, enabling users to choose the first punctuation 
time, the first punctuation time then becomes the reference on which further 
punctuations are aligned.
{code:java}
public Cancellable schedule(final long startTime, final long interval, final 
PunctuationType type, final Punctuator punctuator)
{code}

In my use case, I use wall-clock time punctuation to punctuate every day at 
2am. I would use the new API the following way, allowing me to call 
{{context.schedule()}} once instead of twice currently:
{code:java}
context.schedule(timeUntil2Am, 24 * 60 * 60 * 1000, WALL_CLOCK_TIME, (callTime) 
-> doStuffRightAfter2am(callTime))
{code}


> punctuate with WALL_CLOCK_TIME triggered immediately
> ----------------------------------------------------
>
>                 Key: KAFKA-6323
>                 URL: https://issues.apache.org/jira/browse/KAFKA-6323
>             Project: Kafka
>          Issue Type: Bug
>          Components: streams
>    Affects Versions: 1.0.0
>            Reporter: Frederic Arno
>            Assignee: Frederic Arno
>             Fix For: 1.1.0, 1.0.1
>
>
> When working on a custom Processor from which I am scheduling a punctuation 
> using WALL_CLOCK_TIME. I've noticed that whatever the punctuation interval I 
> set, a call to my Punctuator is always triggered immediately.
> Having a quick look at kafka-streams' code, I could find that all 
> PunctuationSchedule's timestamps are matched against the current time in 
> order to decide whether or not to trigger the punctuator 
> (org.apache.kafka.streams.processor.internals.PunctuationQueue#mayPunctuate). 
> However, I've only seen code that initializes PunctuationSchedule's timestamp 
> to 0, which I guess is what is causing an immediate punctuation.
> At least when using WALL_CLOCK_TIME, shouldn't the PunctuationSchedule's 
> timestamp be initialized to current time + interval?



--
This message was sent by Atlassian JIRA
(v6.4.14#64029)

Reply via email to