Re: flink sql timed-window join throw "mismatched type" AssertionError on rowtime column

2018-03-07 Thread Yan Zhou [FDS Science]
Hi Xingcan, Thanks for your help. Attached is a sample code that can reproduce the problem. When I was writing the sample code, if I remove the `distinct` keyword in select clause, the AssertionError doesn't occur. String sql1 = "select distinct id, eventTs, count(*) over (partition by id

Re: Dynamic CEP https://issues.apache.org/jira/browse/FLINK-7129?subTaskView=all

2018-03-07 Thread Fabian Hueske
We hope to pick up FLIP-20 after Flink 1.5.0 has been released. 2018-03-07 22:05 GMT-08:00 Shailesh Jain : > In addition to making the addition of patterns dynamic, any updates on > FLIP 20 ? > https://cwiki.apache.org/confluence/display/FLINK/FLIP- >

Re: Dynamic CEP https://issues.apache.org/jira/browse/FLINK-7129?subTaskView=all

2018-03-07 Thread Shailesh Jain
In addition to making the addition of patterns dynamic, any updates on FLIP 20 ? https://cwiki.apache.org/confluence/display/FLINK/FLIP-20%3A+Integration+of+SQL+and+CEP On Thu, Mar 8, 2018 at 12:23 AM, Vishal Santoshi wrote: > I see

[ANNOUNCE] Apache Flink 1.4.2 released

2018-03-07 Thread Tzu-Li (Gordon) Tai
The Apache Flink community is very happy to announce the release of Apache Flink 1.4.2, which is the second bugfix release for the Apache Flink 1.4 series.  Apache Flink® is an open-source stream processing framework for distributed, high-performing, always-available, and accurate data

Re: flink sql timed-window join throw "mismatched type" AssertionError on rowtime column

2018-03-07 Thread xccui-foxmail
Hi Yan, I’d like to look into this. Can you share more about your queries and the full stack trace? Thank, Xingcan > On 8 Mar 2018, at 11:28 AM, Yan Zhou [FDS Science] > wrote: > > Hi experts, > I am using flink table api to join two tables,

flink sql timed-window join throw "mismatched type" AssertionError on rowtime column

2018-03-07 Thread Yan Zhou [FDS Science]
Hi experts, I am using flink table api to join two tables, which are datastream underneath. However, I got an assertion error of "java.lang.AssertionError: mismatched type $1 TIMESTAMP(3)" on rowtime column. Below is more details: There in only one kafka data source, which is then converted

Job is be cancelled, but the stdout log still prints

2018-03-07 Thread sundy
Hi: I faced a problem, the taskmanagers in 3 nodes are still running, I make sure that all job are cancelled, but I could see that stdout logs are still printing all the way. The job's parallelism is 6. I wrote a scheduled pool like this static {

Re: Failing to recover once checkpoint fails

2018-03-07 Thread Stephan Ewen
The assumption in your previous mail is correct. Just to double check: - The initially affected version you were running was 1.3.2, correct? The issue should be fixed in all active branches (1.4, 1.5, 1.6) and additional in 1.3. Currently released versions with this fix: 1.4.0, 1.4.1 1.5.0

Re: Dynamic CEP https://issues.apache.org/jira/browse/FLINK-7129?subTaskView=all

2018-03-07 Thread Vishal Santoshi
I see https://github.com/dawidwys/flink/tree/cep-dynamic-nfa is almost there. On Wed, Mar 7, 2018 at 1:34 PM, Vishal Santoshi wrote: > What is the state of this ticket ? Is CEP invested in supporting dynamic > patterns that could potentially be where patterns can be

Dynamic CEP https://issues.apache.org/jira/browse/FLINK-7129?subTaskView=all

2018-03-07 Thread Vishal Santoshi
What is the state of this ticket ? Is CEP invested in supporting dynamic patterns that could potentially be where patterns can be added/disabled through a control stream ?

Re: bin/start-cluster.sh won't start jobmanager on master machine

2018-03-07 Thread Yesheng Ma
Oh, I have figured out the problem, which has something to do with my ~/.profile, i cannot remember when i added one line in the ~/.profile, which sources my .zshrc, leading to the login shell always goes to zsh. On Wed, Mar 7, 2018 at 2:13 AM, Yesheng Ma wrote: > Related

Flink UI not responding on Yarn + Flink

2018-03-07 Thread samar kumar
Hi Users, I am currently trying to run flink with yarn. I have written a program which will launch flink jobs on yarn . It works perfectly fine with local yarn cluster . When I run it on a remote yarn cluster it does not work as expected. As I am reading and writing data from kafka , I do not see

Re: Flink Kafka reads too many bytes .... Very rarely

2018-03-07 Thread Philip Doctor
Hi Stephan, Sorry for the slow response. I added some logging inside of my DeserializationSchema’s `deserialize(byte[] message)` method. I see the extra bytes appearing in that method. If there’s another place I should add logging, please let me know and I’m happy to do so. Additionally (and

Re: Table Api and CSV builder

2018-03-07 Thread Stefan Richter
Hi, I think you just need to specify a custom watermark extractor that constructs the watermark from the 3 fields, as described here: https://ci.apache.org/projects/flink/flink-docs-release-1.4/dev/event_timestamp_extractors.html

Re: Simple CEP pattern

2018-03-07 Thread Kostas Kloudas
Hi, You can adjust it to see if your pattern works. I thought that this is what you want to do. If you want to run it on a cluster or run a full job, then the best way is actually do that. Write the job using the quickstart, launch a cluster locally using the start-cluster.sh script, submit

RE: Simple CEP pattern

2018-03-07 Thread Esa Heikkinen
Hi Yes I have access to the flink source code, but could you explain little bit more what to do with it in this case ? Best, Esa From: Kostas Kloudas [mailto:k.klou...@data-artisans.com] Sent: Wednesday, March 7, 2018 3:51 PM To: Esa Heikkinen Cc:

Re: CEP issue

2018-03-07 Thread Vishal Santoshi
Will do. On Wed, Mar 7, 2018 at 9:33 AM, Kostas Kloudas wrote: > Why not opening a JIRA and working on adding some debug > statements that you consider useful? > > This could help the next user that faces the same issues ;) > > Kostas > > On Mar 7, 2018, at 3:29 PM,

Re: CEP issue

2018-03-07 Thread Kostas Kloudas
Why not opening a JIRA and working on adding some debug statements that you consider useful? This could help the next user that faces the same issues ;) Kostas > On Mar 7, 2018, at 3:29 PM, Vishal Santoshi wrote: > > Aah, yes we never had a sink state so never came

Re: CEP issue

2018-03-07 Thread Vishal Santoshi
Aah, yes we never had a sink state so never came across a case where it was ever exercised. When the range expires, it is a prune rather than a stop state ( we were expecting it to be a stop state ) which is some what misleading if we hold stop state to " that invalidates a partial match "

Re: CEP issue

2018-03-07 Thread Kostas Kloudas
Hi Vishal, A stopState is a state that invalidates a partial match, e.g. a.NotFollowedBy(b).followedBy(c). If you have an “a” and then you see a “b” then you invalidate the pattern. A finalState is the one where a match has been found. Kostas > On Mar 7, 2018, at 3:20 PM, Vishal Santoshi

Re: CEP issue

2018-03-07 Thread Vishal Santoshi
Absolutely. For one a simple m out of n true conditions where n is defined by range is a little under optimized as in just using time(m) will not short circuit the partial patterns till the time range is achieved even if there is no way m true conditions can be achieved ( we already have had n-m

Re: Simple CEP pattern

2018-03-07 Thread Kostas Kloudas
Hi Esa, You can always test your pattern in isolation. For an example on how to do that, if you have access to the flink source code, you can check the UntilConditionITCase or any other ITCase in the same package. It can also be found here:

Re: CEP issue

2018-03-07 Thread Kostas Kloudas
Hi Vishal, Thanks a lot for sharing your experience and the potential caveats to consider when specifying your pattern. I agree that there is room for improvement when it comes to the state checkpointed in Flink. We already have some ideas but still, as you also said, the bulk of the space

RE: Simple CEP pattern

2018-03-07 Thread Esa Heikkinen
Hi I have tried this CEP example of Data-artisans and it works. It is also only fully working example I have found, but it little bit too complex for my purpose. I have also tried to read the documentation, but I have not found about simple "Hello World" type example about Flink's CEP. It is

Re: Does Flink support stream-stream outer joins in the latest version?

2018-03-07 Thread Hequn Cheng
Hi kant, You are right. Batch joins require the inputs are bounded. To join two unbounded streams without window, all data will be stored in join's states, so the late right row will join the previous left row when it is input. As for state retention time, if the input tables of join are both

Re: CEP issue

2018-03-07 Thread Vishal Santoshi
Hello all, There were recent changes to the flink master that I pulled in and that *seems* to have solved our issue. Few points * CEP is heavy as the NFA transition matrix as state which can be possibly n^2 ( worst case ) can easily blow up space requirements. The after match skip

Re: Does Flink support stream-stream outer joins in the latest version?

2018-03-07 Thread Xingcan Cui
Hi Kant, the non windowed stream-stream join is not equivalent to the full-history join, though they get the same SQL form. The retention times for records must be set to leverage the storage consumption and completeness of the results. Best, Xingcan > On 7 Mar 2018, at 8:02 PM, kant kodali

RE: CsvTableSource Types.TIMESTAMP

2018-03-07 Thread Esa Heikkinen
Hi It works now. It was because of the missing “import”. Thank you. Best, Esa From: Hequn Cheng [mailto:chenghe...@gmail.com] Sent: Wednesday, March 7, 2018 3:00 PM To: Esa Heikkinen Cc: Timo Walther ; user@flink.apache.org Subject: Re:

Re: CsvTableSource Types.TIMESTAMP

2018-03-07 Thread Hequn Cheng
Hi Esa, Have you ever imported org.apache.flink.table.api.scala._ ? There are some examples here[1]. Best, Hequn [1] https://github.com/apache/flink/blob/master/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/runtime/stream/table/TableSinkITCase.scala On Tue, Mar 6, 2018 at

Re: Simple CEP pattern

2018-03-07 Thread Kostas Kloudas
Hi Esa, You could try the examples either from the documentation or from the training. http://training.data-artisans.com/exercises/CEP.html Kostas > On Mar 7, 2018, at 11:32 AM, Esa Heikkinen > wrote: > >

Re: Does Flink support stream-stream outer joins in the latest version?

2018-03-07 Thread kant kodali
Hi Cheng, The docs here states full outer joins are only available for batch (I am not sure if I am reading that correctly). I am trying to understand how two unbounded streams can be joined like a batch? If we

Re: Does Flink support stream-stream outer joins in the latest version?

2018-03-07 Thread Hequn Cheng
Hi kant, It seems that you mean the Time-windowed Join. The Time-windowed Joins are supported now. You can check more details with the docs given by Xingcan. As for the non-window join, it is used to join two unbounded stream and the semantic is very like batch join. Time-windowed Join: >

Re: Does Flink support stream-stream outer joins in the latest version?

2018-03-07 Thread kant kodali
Hi! Thanks for all this. and yes I was indeed talking about SQL/Table API so I will keep track of these tickets! BTW, What is non-windowed Join? I thought stream-stream-joins by default is a stateful operation so it has to be within some time window right? Also does the output of stream-stream

Simple CEP pattern

2018-03-07 Thread Esa Heikkinen
What would be the simplest working CEP (Scala) pattern ? I want to test if my CEP application works at all. Best, Esa

Re: Rest APIs

2018-03-07 Thread Chesnay Schepler
||The |jobmanager.web.upload.dir| option ||only affects where jars submitted through the WebUI/REST API are stored. As for uploading jars directly through the REST API, this may be useful to you.

Re: Flink is looking for Kafka topic "n/a"

2018-03-07 Thread Mu Kong
Hi Gordon, Thanks for your response. I think I've misspoken about the failure after "n/a" exception. The behavior after this exception would be: switched from RUNNING to CANCELING switched from CANCELING to CANCELED Try to restart or fail the job "X" () if no longer