[ https://issues.apache.org/jira/browse/KAFKA-7895?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16763268#comment-16763268 ]
John Roesler commented on KAFKA-7895: ------------------------------------- Thanks for the reminder, [~guozhang]. I think that the same mechanism could cause duplicates as well during recovery, since stream time is currently determined by the input partitions, but the recovery is from the changelog topic, so recovered events may all appear to be arriving at the same stream time. If I'm right about this, then the fix I have in mind should address that as well. However, it is troubling that Michael also said that he observed duplicates during steady-state processing, post recovery, with caching disabled. There may be something else going on as well. The system tests already exercise crash and shutdown recovery, so to complete the picture, I'll make sure I add a system test with caching disabled (if there's not already one). > Ktable supress operator emitting more than one record for the same key per > window > --------------------------------------------------------------------------------- > > Key: KAFKA-7895 > URL: https://issues.apache.org/jira/browse/KAFKA-7895 > Project: Kafka > Issue Type: Bug > Components: streams > Affects Versions: 2.1.0, 2.1.1 > Reporter: prasanthi > Assignee: John Roesler > Priority: Major > > Hi, We are using kstreams to get the aggregated counts per vendor(key) within > a specified window. > Here's how we configured the suppress operator to emit one final record per > key/window. > {code:java} > KTable<Windowed<Integer>, Long> windowedCount = groupedStream > .windowedBy(TimeWindows.of(Duration.ofMinutes(1)).grace(ofMillis(5L))) > .count(Materialized.with(Serdes.Integer(),Serdes.Long())) > .suppress(Suppressed.untilWindowCloses(unbounded())); > {code} > But we are getting more than one record for the same key/window as shown > below. > {code:java} > [KTABLE-TOSTREAM-0000000010]: [131@1549067040000/1549067100000], 1039 > [KTABLE-TOSTREAM-0000000010]: [131@1549067040000/1549067100000], 1162 > [KTABLE-TOSTREAM-0000000010]: [9@1549067040000/1549067100000], 6584 > [KTABLE-TOSTREAM-0000000010]: [88@1549067040000/1549067100000], 107 > [KTABLE-TOSTREAM-0000000010]: [108@1549067040000/1549067100000], 315 > [KTABLE-TOSTREAM-0000000010]: [119@1549067040000/1549067100000], 119 > [KTABLE-TOSTREAM-0000000010]: [154@1549067040000/1549067100000], 746 > [KTABLE-TOSTREAM-0000000010]: [154@1549067040000/1549067100000], 809{code} > Could you please take a look? > Thanks -- This message was sent by Atlassian JIRA (v7.6.3#76005)