Flink application has slightly data loss using Processing Time

2021-03-08 Thread Rainie Li
Hello Flink Community, Our flink application in v1.9, the basic logic of this application is consuming one large kafka topic and filter some fields, then produce data to a new kafka topic. After comparing the original kafka topic count with the generated kafka topic based on the same field by usin

Re: Flink application has slightly data loss using Processing Time

2021-03-08 Thread Smile
Hi Rainie, Could you please provide more information about your processing logic? Do you use window operators? If there's no time-based operator in your logic, late arrival data won't be dropped by default and there might be something wrong with your flat map or filter operator. Otherwise, you ca

Re: Flink application has slightly data loss using Processing Time

2021-03-08 Thread Rainie Li
Thanks for the quick response, Smile. I don't use window operators or flatmap. Here is the core logic of my filter, it only iterates on filters list. Will *rebalance() *cause it? Thanks again. Best regards Rainie SingleOutputStreamOperator> matchedRecordsStream = eventStream .rebalanc

Re: How do I call an algorithm written in C++ in Flink?

2021-03-08 Thread Yun Gao
Hi Suxi, Do you mean you want to call the algorithm in C++ ? If so, I think you could do it the same with as you wrap it in SpringBoot project via JNI. I think you do not need to add a new operator, and you could use existing Flink API, and you could load you library in open() and call the algori

Re: java.lang.OutOfMemoryError: GC overhead limit exceeded

2021-03-08 Thread bat man
The Java options should not have the double quotes. That was the issue. I was able to generate the heap dump. based on the dump have made some changes in the code to fix this issue. This worked - env.java.opts: -XX:+HeapDumpOnOutOfMemoryError -XX:HeapDumpPath=/tmp/dump.hprof Thanks. On Mon, 8 M

Re: java options to generate heap dump in EMR not working

2021-03-08 Thread bat man
Issue was with double quotes around the Java options. This worked - env.java.opts: -XX:+HeapDumpOnOutOfMemoryError -XX:HeapDumpPath=/tmp/dump.hprof On Mon, 8 Mar 2021 at 12:02 PM, Yun Gao wrote: > Hi, > > I tried with the standalone session (sorry I do not have a yarn cluster in > hand) and it

Re: Stop vs Cancel with savepoint

2021-03-08 Thread Thomas Eckestad
OK, thank you for validating my thoughts =) I created https://issues.apache.org/jira/browse/FLINK-21666# Thanks, Thomas On 3 Mar 2021, at 22:02, Chesnay Schepler mailto:ches...@apache.org>> wrote: Your understanding of cancel vs stop(-with-savepoint) is correct. I agree that we should update

Re: LocalWatermarkAssigner causes predicate pushdown to be skipped

2021-03-08 Thread Yuval Itzchakov
Thank you Shenkai, That does explain what I'm seeing. Jark / Shenkai - Is there any workaround to get Flink to work with push watermarks and predicate pushdown until this is resolved? On Mon, Mar 8, 2021 at 4:54 AM Shengkai Fang wrote: > Hi, Yuval, Jark, Timo. > > Currently the watermark push d

KafkaSource Problem

2021-03-08 Thread Bobby Richard
I'm receiving the following exception when trying to use a KafkaSource from the new DataSource API. Exception in thread "main" java.lang.NullPointerException at org.apache.flink.connector.kafka.source.reader.deserializer.ValueDeserializerWrapper.getProducedType(ValueDeserializerWrapper.java:79) at

Re: Re: Flink application has slightly data loss using Processing Time

2021-03-08 Thread Yun Gao
Hi Rainie, From the code it seems the current problem does not use the time-related functionality like window/timer? If so, the problem would be indepdent with the time type used. Also, it would not likely due to rebalance() since the network layer has the check of sequence number. If there are

Missing support for `TestStreamEnvironment#executeAsync`

2021-03-08 Thread Bob Tiernay
Hi all, I have been trying to test a Flink 1.11 streaming job using the `DataStreamUtils#collect` utility against a `MiniCluster` based test. However, I noticed an issue when doing so. `TestStreamEnvironment` does not implement `executeAsync`. Thus when `DataStreamUtils#collect` is called, it inv

Re: Gradually increasing checkpoint size

2021-03-08 Thread Yun Gao
Hi Dan, Have you use a too large upperBound or lowerBound? If not, could you also check the watermark strategy ? The interval join operator depends on the event-time timer for cleanup, and the event-time timer would be triggered via watermark. Best, Yun --Original Mail -

Re: Re: Re: Checkpoint Error

2021-03-08 Thread Yun Gao
Hi Navneeth, Is the attached exception the root cause for the checkpoint failure ? Namely is it also reported in job manager log? Also, have you enabled concurrent checkpoint? Best, Yun --Original Mail -- Sender:Navneeth Krishnan Send Date:Mon Mar 8 13:10:4

Re: questions about broadcasts

2021-03-08 Thread Yun Gao
Hi Marco, (a) It is possible for an operator to receive two different kind of broadcasts, DataStream ints = DataStream strs = ... ints.broadcast().connect(strs.broadcast()) ​.process(new CoProcessFunction(){...}); (b) Traditional Flink operator could not accept three different inputs.

Re: New settings are not honored unless checkpoint is cleared.

2021-03-08 Thread Yun Gao
Hi Yordan, What are the settings that are changed during the tests? Best, Yun -- From:Yordan Pavlov Send Time:2021 Mar. 5 (Fri.) 23:36 To:user Subject:New settings are not honored unless checkpoint is cleared. Hello there, I am

failure checkpoint counts

2021-03-08 Thread Abdullah bin Omar
Hi, I faced this exception at the time of checkpoint counts. Could you please inform me what the problem is here? the exception: org.apache.flink.runtime.JobException: Recovery is suppressed by FixedDelayRestartBackoffTimeStrategy(maxNumberRestartAttempts=3, backoffTimeMS=100) at org.apache

Re: Gradually increasing checkpoint size

2021-03-08 Thread Dan Hill
Hi Yun! Thanks for the quick reply. One of the lowerBounds is large but the table being joined with is ~500 rows. I also have my own operator that only outputs the first value. public class OnlyFirstUser extends RichFlatMapFunction { private transient ValueState alreadyOutputted; @O

idleTimeMsPerSecond on Flink 1.9?

2021-03-08 Thread Lakshmi Gururaja Rao
Hi I'm trying to understand the implementation of idleTimeMsPerSecond. Specifically what I'm trying to do is, adapt this metric to be used with Flink 1.9 (for a fork). I tried an approach similar to this PR and measuring the time to request a new

Re: Flink application has slightly data loss using Processing Time

2021-03-08 Thread David Anderson
Rainie, Were there any failures/restarts, or is this discrepancy observed without any disruption to the processing? Regards, David On Mon, Mar 8, 2021 at 10:14 AM Rainie Li wrote: > Thanks for the quick response, Smile. > I don't use window operators or flatmap. > Here is the core logic of my

Re: Flink application has slightly data loss using Processing Time

2021-03-08 Thread Rainie Li
Thanks Yun and David. There were some tasks that got restarted. We configured the restart policy and the job didn't fail. Will task restart cause data loss? Thanks Rainie On Mon, Mar 8, 2021 at 10:42 AM David Anderson wrote: > Rainie, > > Were there any failures/restarts, or is this discrepanc

Re: Running Pyflink job on K8s Flink Cluster Deployment?

2021-03-08 Thread Kevin Lam
Awesome, thanks Shuiqiang! I was able to get an example running by referencing your configs. On Sat, Mar 6, 2021 at 7:12 AM Shuiqiang Chen wrote: > Hi Kevin, > > For your information, bellow is an example for running a PyFlink table API > WordCount job. > > 1. Building a Docker image with Python

Re: Flink application has slightly data loss using Processing Time

2021-03-08 Thread David Anderson
Rainie, A restart after a failure can cause data loss if you aren't using checkpointing, or if you experience a transaction timeout. A manual restart can also lead to data loss, depending on how you manage the offsets, transactions, and other state during the restart. What happened in this case?

Flink Read S3 Intellij IDEA Error

2021-03-08 Thread sri hari kali charan Tummala
> Hi Flink Experts, > I am trying to read an S3 file from my Intellij using Flink I am.comimg > across Aws Auth error can someone help below are all the details. > > I have Aws credentials in homefolder/.aws/credentials > My Intellij Environment Variables:- > ENABLE_BUILT_IN_PLUGINS=flink-s3-fs

Re: Flink application has slightly data loss using Processing Time

2021-03-08 Thread Rainie Li
Thanks for the info, David. The job has checkpointing. I saw some tasks failed due to "org.apache.flink.streaming.connectors.kafka.FlinkKafkaException: Failed to send data to Kafka" Here is stacktrack from JM log: container_e17_1611597945897_8007_01_000240 @ worker-node-host (dataPort=42321). 2021

Re: Stateful functions 2.2 and stop with savepoint

2021-03-08 Thread Meissner, Dylan
Thank you for this information, Piotr. The comment from Igal Shilman in FLINK-18894 issue says, "Obtaining a MAX_PRIO mailbox from StreamTask, solves this issue." I'm unclear what this means -- is this a workaround I can leverage? Dylan From: Piotr Nowojski Sen

Call for Presentations for ApacheCon 2021 now open

2021-03-08 Thread Rich Bowen
[Note: You are receiving this because you are subscribed to a users@ list on one or more Apache Software Foundation projects.] The ApacheCon Planners and the Apache Software Foundation are pleased to announce that ApacheCon@Home will be held online, September 21-23, 2021. Once again, we’ll be

Re: Trigger and completed Checkpointing do not appeared

2021-03-08 Thread Alexey Trenikhun
The picture in first e-mail shows that job was completed in 93ms From: Abdullah bin Omar Sent: Monday, March 8, 2021 3:53 PM To: user@flink.apache.org Subject: Re: Trigger and completed Checkpointing do not appeared Hi, Please read the previous email (and also

Re: LocalWatermarkAssigner causes predicate pushdown to be skipped

2021-03-08 Thread Shengkai Fang
Hi, Yuval. I have opened a ticket about this[1]. But I don't think we have any solution to solve. Do you have time to help us to solve this? I think it will take too much time. [1] https://issues.apache.org/jira/browse/FLINK-21675 Yuval Itzchakov 于2021年3月8日周一 下午9:18写道: > Thank you Shenkai, >

Re: LocalWatermarkAssigner causes predicate pushdown to be skipped

2021-03-08 Thread Shengkai Fang
Sorry for the typo... I mean it will not take too much time. Best, Shengkai Shengkai Fang 于2021年3月9日周二 上午10:25写道: > Hi, Yuval. > > I have opened a ticket about this[1]. But I don't think we have any > solution to solve. > > Do you have time to help us to solve this? I think it will take too mu

Re: Re: How to check checkpointing mode

2021-03-08 Thread Yun Gao
Hi Alexey, Sorry I also do not see problems in the attached code. Could you add a breakpoint at `see.execute(name)` and have a look at the value of see#checkpointCfg#checkpointingMode ? Best, Yun --Original Mail -- Sender:Alexey Trenikhun Send Date:Tue Mar 9 0

Re: [ANNOUNCE] Apache Flink 1.12.2 released

2021-03-08 Thread Zhu Zhu
Thanks Roman and Yuan for being the release managers! Thanks everyone who has made this release possible! Cheers, Zhu Piotr Nowojski 于2021年3月6日周六 上午12:38写道: > Thanks Roman and Yuan for your work and driving the release process :) > > pt., 5 mar 2021 o 15:53 Till Rohrmann napisał(a): > >> Great

Is Ververica Connector Redis open source?

2021-03-08 Thread Yik San Chan
Hi community, I found this package https://mvnrepository.com/artifact/com.alibaba.ververica/ververica-connector-redis/1.11-vvr-2.1.3 in Maven Repository. However, I cannot find it anywhere in GitHub. Does anyone know this is open source or not? Thank you! Best, Yik San Chan

Re: Trigger and completed Checkpointing do not appeared

2021-03-08 Thread Smile
Hi, Could you please change the source to an endless one? For example a Kafka source or a custom source that implements SourceFunction([1])? env.readTextFile() won't wait for all data to be finished, but exit immediately after telling readers what to read. So it may exit before the first checkpoi

Hadoop Integration Link broken in downloads page

2021-03-08 Thread Debraj Manna
Hi It appears the Hadoop Interation link is broken on downloads page. Apache Flink® 1.12.2 is our latest stable release. > If you plan to use Apache Flink together w

Future of QueryableState

2021-03-08 Thread Maciek Próchniak
Hello, We are using QueryableState in some of Nussknacker deployments as a nice addition, allowing end users to peek inside job state for a given key (we mostly use custom operators). Judging by mailing list and feature radar proposition by Stephan: https://github.com/StephanEwen/flink-web

Re: Re: Gradually increasing checkpoint size

2021-03-08 Thread Yun Gao
Hi Dan, Regarding the original checkpoint size problem, could you also have a check which tasks' state are increasing from the checkpoint UI ? For example, the attached operator has a `alreadyOutputed` value state, which seems to keep increasing if there are always new keys ? Best, Yun

Re: [ANNOUNCE] Apache Flink 1.12.2 released

2021-03-08 Thread Leonard Xu
Well done! Thanks to Roman and Yuan, and everyone who contributed to the release. Best, Leonard > 在 2021年3月9日,11:39,Zhu Zhu 写道: > > Thanks Roman and Yuan for being the release managers! Thanks everyone who has > made this release possible! > > Cheers, > Zhu > > Piotr Nowojski mailto:pnowoj.

Re: Re: Gradually increasing checkpoint size

2021-03-08 Thread Dan Hill
Hi Yun! That advice was useful. The state for that operator is very small (31kb). Most of the checkpoint size is in a couple simple DataStream.intervalJoin operators. The time intervals are fairly short. I'm going to try running the code with some small configuration changes. One thing I did no