Re: Flink CEP Pattern Matching

2016-03-02 Thread Vitor Vieira
Hi Jerry, I'm currently evaluating the CEP library too, probably doing something similar. Something like... comparing the 'offset' of the last event in different time windows, each window, based on the event type, occurring like realtime, with this same day/hour/minute a week ago/15d/1month/etc..

Re: Flink CEP Pattern Matching

2016-03-02 Thread Fabian Hueske
Hi Jerry, I haven't used the CEP features yet, so I cannot comment on your requirements. In case you are looking for the CEP documentation, here it is: --> https://ci.apache.org/projects/flink/flink-docs-master/apis/streaming/libs/cep.html The CEP features will be included in the upcoming 1.0.0

Flink CEP Pattern Matching

2016-03-02 Thread Jerry Lam
Hi Flink users and developers, I'm trying to learn the CEP library. How can I express A-followBy->B-next->C where B is 5 days after A occurs. What I'm trying to get a hold of is the events that matches A when I'm processing B. Is this supported? Best Regards, Jerry

Re: Flink packaging makes life hard for SBT fat jar's

2016-03-02 Thread Till Rohrmann
Hi Shikhar, that is a problem we just found out today. The problem is that the scala.binary.version was not properly replaced in the parent pom so that it resolves to 2.10 [1]. Max already opened a PR to fix this problem. With the next release candidate, this should be fixed. [1] https://issues.a

Re: Flink packaging makes life hard for SBT fat jar's

2016-03-02 Thread shikhar
Hi Till, I just tried creating an assembly with RC4: ``` "org.apache.flink" %% "flink-clients" % flinkVersion % "provided", "org.apache.flink" %% "flink-streaming-scala" % flinkVersion % "provided", "org.apache.flink" %% "flink-connector-kafka-0.8" % flinkVersion, ``` It actually succeeds

Re: Job Manager HA manual setup

2016-03-02 Thread Welly Tambunan
Hi Ufuk, Thanks, it's working fine now with your suggestion. Cheers On Sun, Feb 28, 2016 at 10:10 PM, Ufuk Celebi wrote: > Hey Welly! > > Yes, it is possible to do manually via the jobmanager.sh and > taskmanager.sh script like this: > > jobmanager.sh start cluster $HOST $WEB-UI-PORT > taskman

Re: readTextFile is not working for StreamExecutionEnvironment

2016-03-02 Thread Maximilian Michels
Hi Balaji, You forgot to execute your Flink program using env.execute(); Cheers, Max On Wed, Mar 2, 2016 at 1:36 PM, Balaji Rajagopalan wrote: > def main(args: Array[String]): Unit = { > > > > val env: StreamExecutionEnvironment = > StreamExecutionEnvironment.getExecutionEnvironment(); > e

Re: readTextFile is not working for StreamExecutionEnvironment

2016-03-02 Thread Robert Metzger
Hi, you need to explicitly trigger the execution by calling "env.execute()" On Wed, Mar 2, 2016 at 1:36 PM, Balaji Rajagopalan < balaji.rajagopa...@olacabs.com> wrote: > def main(args: Array[String]): Unit = { > > > > val env: StreamExecutionEnvironment = > StreamExecutionEnvironment.getExecut

readTextFile is not working for StreamExecutionEnvironment

2016-03-02 Thread Balaji Rajagopalan
def main(args: Array[String]): Unit = { val env: StreamExecutionEnvironment = StreamExecutionEnvironment.getExecutionEnvironment(); env.setStreamTimeCharacteristic(TimeCharacteristic.EventTime) try { val data1:DataStream[String] = env.readTextFile(“somefile.txt”); data1.print()

Re: Java Maps and Type Information

2016-03-02 Thread Aljoscha Krettek
I’m afraid so, yes. I also tried out your example and it works if I add a .returns() after the .map() (as you did). Somehow the TypeExtractor seems to be acting up. > On 02 Mar 2016, at 11:12, Simone Robutti wrote: > > >Error:(43, 87) java: no suitable method found for > >returns(java.lang.Cl

Re: Java Maps and Type Information

2016-03-02 Thread Simone Robutti
Ok, I made it work but there's still an issue. I used .returns(java.util.Map.class) after the "map" call and it works with this simple function but it doesn't compile with my CustomMapFunction that extends MapFunction. It gives a compilation error on the .returns() call. This is the case only if t

Re: Windows, watermarks, and late data

2016-03-02 Thread Kostas Kloudas
Hello Mike, The code that Aljiosha mentioned is here: https://github.com/kl0u/flink-examples/blob/master/src/main/java/com/dataartisans/flinksolo/customTriggers/EventTimeTriggerWithEarlyAndLateFiring.java

Re: Windows, watermarks, and late data

2016-03-02 Thread Aljoscha Krettek
Hi, I did some initial work on extending the EventTimeTrigger a bit to allow more complex behavior. Specifically, this allows setting an “allowed lateness” after which elements should no longer lead to windows being emitted. Also, it allows to specify to keep an emitted window in memory and when