Hi, Jan.

Could you tell us which Flink version you use? As far as I know, the kafka
sql connector has implemented `SupportWatermarkPushDown` in Flink-1.12. The
`SupportWatermarkPushDown` pushes the watermark generator into the source
and emits the minimum watermark among all the partitions. For more details,
you can refer to the doc for more details[1].

If you use the version before FLINK-1.12,  I think the best approach to
solve this problem is to increase source tasks.

Best,
Shengkai

[1]
https://ci.apache.org/projects/flink/flink-docs-master/docs/connectors/table/kafka/#source-per-partition-watermarks

Jan Oelschlegel <oelschle...@integration-factory.de> 于2021年2月25日周四 下午4:24写道:

> Hi Benchao,
>
>
>
> i’m observing this behaviour only for the SQL API. With the Datastream API
> i can take more or less source-tasks then kafka partition count. And
> FLIP-27 seems to belong to the Datastream API.
>
>
>
> The problem is only on the SQL site.
>
>
>
>
>
> Best,
>
> Jan
>
>
>
> *Von:* Benchao Li <libenc...@apache.org>
> *Gesendet:* Donnerstag, 25. Februar 2021 00:04
> *An:* Jan Oelschlegel <oelschle...@integration-factory.de>
> *Cc:* Arvid Heise <ar...@apache.org>; user <user@flink.apache.org>; Timo
> Walther <twal...@apache.org>
> *Betreff:* Re: Kafka SQL Connector: dropping events if more partitions
> then source tasks
>
>
>
> Hi Jan,
>
>
>
> What you are observing is correct for the current implementation.
>
>
>
> Current watermark generation is based on subtask instead of partition.
> Hence if there are
>
> more than on partition in the same subtask, it's very easy to see more
> data dropped.
>
>
>
> AFAIK, FLIP-27 could solve this problem, however the Kafka Connector has
> not been
>
> migrated to FLIP-27 for now.
>
>
>
>
>
> Jan Oelschlegel <oelschle...@integration-factory.de> 于2021年2月24日周三 下午10:07
> 写道:
>
> Hi Arvid,
>
>
>
> thanks for bringing back this topic.
>
>
>
> Yes, I’m running on historic data, but as you mentioned that should not be
> the problem, even there is a event-time skew between partitions.
>
>
>
> But maybe this issue with the missing watermark pushdown per partition  is
> the important fact:
>
>
>
> https://issues.apache.org/jira/browse/FLINK-20041
>
>
>
>
>
> Best,
>
> Jan
>
>
>
> *Von:* Arvid Heise <ar...@apache.org>
> *Gesendet:* Mittwoch, 24. Februar 2021 14:10
> *An:* Jan Oelschlegel <oelschle...@integration-factory.de>
> *Cc:* user <user@flink.apache.org>; Timo Walther <twal...@apache.org>
> *Betreff:* Re: Kafka SQL Connector: dropping events if more partitions
> then source tasks
>
>
>
> Hi Jan,
>
>
>
> Are you running on historic data? Then your partitions might drift apart
> quickly.
>
>
>
> However, I still suspect that this is a bug (Watermark should only be from
> the slowest partition). I'm pulling in Timo who should know more.
>
>
>
>
>
>
>
> On Fri, Feb 19, 2021 at 10:50 AM Jan Oelschlegel <
> oelschle...@integration-factory.de> wrote:
>
> If i increase the watermark, the dropped events getting lower. But why is
> the DataStream API Job still running with 12 hours watermark delay?
>
> By the way, I’m using Flink 1.11. It would be nice if someone could give
> me some advice.
>
>
>
> Best,
>
> Jan
>
>
>
> *Von:* Jan Oelschlegel <oelschle...@integration-factory.de>
> *Gesendet:* Donnerstag, 18. Februar 2021 09:51
> *An:* Jan Oelschlegel <oelschle...@integration-factory.de>; user <
> user@flink.apache.org>
> *Betreff:* AW: Kafka SQL Connector: dropping events if more partitions
> then source tasks
>
>
>
> By  using the DataStream API with the same business logic I’m getting no
> dropped events.
>
>
>
> *Von:* Jan Oelschlegel <oelschle...@integration-factory.de>
> *Gesendet:* Mittwoch, 17. Februar 2021 19:18
> *An:* user <user@flink.apache.org>
> *Betreff:* Kafka SQL Connector: dropping events if more partitions then
> source tasks
>
>
>
> Hi,
>
>
>
> i have a question regarding FlinkSQL connector for Kafka. I have 3 Kafka
> partitions and 1 Kafka SQL source connector (Parallelism 1). The data
> within the Kafka parttitons are sorted based on a event-time field, which
> is also my event-time in Flink. My Watermark is generated with a delay of
> 12 hours
>
>
>
> WATERMARK FOR eventtime as eventtime - INTERVAL '12' HOUR
>
>
>
>
>
> But the problem is that I see dropping events due arriving late in
> Prometheus.  But with parallelism of 3  there are no drops.
>
>
>
> Do I always have to have as much source-tasks as I have Kafka partitions?
>
>
>
>
>
>
>
> Best,
>
> Jan
>
> HINWEIS: Dies ist eine vertrauliche Nachricht und nur für den Adressaten
> bestimmt. Es ist nicht erlaubt, diese Nachricht zu kopieren oder Dritten
> zugänglich zu machen. Sollten Sie diese Nachricht irrtümlich erhalten
> haben, bitte ich um Ihre Mitteilung per E-Mail oder unter der oben
> angegebenen Telefonnummer.
>
> HINWEIS: Dies ist eine vertrauliche Nachricht und nur für den Adressaten
> bestimmt. Es ist nicht erlaubt, diese Nachricht zu kopieren oder Dritten
> zugänglich zu machen. Sollten Sie diese Nachricht irrtümlich erhalten
> haben, bitte ich um Ihre Mitteilung per E-Mail oder unter der oben
> angegebenen Telefonnummer.
>
> HINWEIS: Dies ist eine vertrauliche Nachricht und nur für den Adressaten
> bestimmt. Es ist nicht erlaubt, diese Nachricht zu kopieren oder Dritten
> zugänglich zu machen. Sollten Sie diese Nachricht irrtümlich erhalten
> haben, bitte ich um Ihre Mitteilung per E-Mail oder unter der oben
> angegebenen Telefonnummer.
>
>
>
>
> --
>
>
> Best,
> Benchao Li
> HINWEIS: Dies ist eine vertrauliche Nachricht und nur für den Adressaten
> bestimmt. Es ist nicht erlaubt, diese Nachricht zu kopieren oder Dritten
> zugänglich zu machen. Sollten Sie diese Nachricht irrtümlich erhalten
> haben, bitte ich um Ihre Mitteilung per E-Mail oder unter der oben
> angegebenen Telefonnummer.
>

Reply via email to