Re: Flink 1.2 Proper Packaging of flink-table with SBT

2017-03-07 Thread Justin Yan
Of course, 15 minutes after I give up and decide to email the mailing list, I figure it out - my flink App was using the CollectionsEnvironment and not the proper RemoteEnvironment. It is still the case, however, that the `flink-table` JAR built by the standard commands doesn't include the

Flink 1.2 Proper Packaging of flink-table with SBT

2017-03-07 Thread Justin Yan
Hello! We are attempting to use the Flink Table API, but are running into a few issues. We initially started with our dependencies looking something like this: libraryDependencies ++= Seq( "org.apache.flink" %% "flink-scala" % "1.2.0" % "provided", "org.apache.flink" %% "flink-clients" %

Re: Issues with Event Time and Kafka

2017-03-07 Thread Dawid Wysakowicz
Hi Ethan, I believe then it is because the Watermark and Timestamps in your implementation are uncorrelated. What Watermark really is a marker that says there will be no elements with timestamp smaller than the value of this watermark. For more info on the concept see [1]

flink/cancel & shutdown hooks

2017-03-07 Thread Dominik Safaric
Hi all, I would appreciate for any help or advice in regard to default Java runtime shutdown hooks and canceling Flink jobs. Namely part of my Flink application I am using a Kafka interceptor class that defines a shutdown hook thread. When stopping the Flink streaming job on my local machine

Re: Issues with Event Time and Kafka

2017-03-07 Thread ext.eformichella
Hi Dawid, I'm working with Max on the project Our code for the TimestampAndWatermarkAssigner is: ``` class TimestampAndWatermarkAssigner(val maxLateness: Long) extends AssignerWithPeriodicWatermarks[Row] { override def extractTimestamp(element: Row, previousElementTimestamp: Long): Long = {

Re: How to use 'dynamic' state

2017-03-07 Thread Steve Jerman
Thanks for your reply. It makes things much clearer for me. I think you are right - Side Inputs are probably the right way long term (I looked at the Team definition), but I think I can construct something in the mean time. Steve On Mar 7, 2017, at 6:11 AM, Aljoscha Krettek

Re: Issues with Event Time and Kafka

2017-03-07 Thread Dawid Wysakowicz
Hi Max, How do you assign timestamps to your events (in event-time case)? Could you post whole code for your TimestampAndWatermarkAssigner? Regards, Dawid 2017-03-07 20:59 GMT+01:00 ext.mwalker : > Hi Stephan, > > The right number of events seem to leave the source

Re: Issues with Event Time and Kafka

2017-03-07 Thread ext.mwalker
Hi Stephan, The right number of events seem to leave the source and enter the windows, but it shows that 0 exit the windows. Also I have tried 30 minutes and not setting the watermark interval, I am not sure what I am supposed to put there the docs seem vague about that. Best, Max On Tue, Mar

Re: Data stream to write to multiple rds instances

2017-03-07 Thread Till Rohrmann
Hi Sathi, if you read data from Kinesis than Flink can offer you exactly once processing guarantees. However, what you see written out to your database depends a little bit on the implementation of your custom sink. If you have synchronous JDBC client which does not lose data and you fail your

Re: AWS exception serialization problem

2017-03-07 Thread Shannon Carey
> is there some shading logic involved in the dependencies, concerning the AWS > libraries? Not that I am aware of. The AWS code is included in the job's fat jar as-is.

Re: Issues with Event Time and Kafka

2017-03-07 Thread Stephan Ewen
Hi! At a first glance, your code looks correct to assign the Watermarks. What is your watermark interval in the config? Can you check with the Flink metrics (if you are using 1.2) to see how many rows leave the source, how many enter/leave the window operators, etc? That should help figuring

Appropriate State to use to buffer events in ProcessFunction

2017-03-07 Thread Yassine MARZOUGUI
Hi all, I want to label events in a stream based on a condition on some future events. For example my stream contains events of type A and B and and I would like to assign a label 1 to an event E of type A if an event of type B happens within a duration x of E. I am using event time and my events

Re: Serialization performance

2017-03-07 Thread Stephan Ewen
I'll try and add more details in a bit. If you have some suggestions on how to make the serialization stack more extensible, please let us know! Some hooks exist, like TypeInfoFactories:

Re: Flink checkpointing gets stuck

2017-03-07 Thread Stephan Ewen
Great to hear it! What do you think about adding a section to the Flink docs about deployment on Azure (there is already AWS and GCE, so Azure would make the cloud-trinity complete) that explains how to set this up and avoid such pitfalls.

Re: AWS exception serialization problem

2017-03-07 Thread Stephan Ewen
@Shannon @Gordon - is there some shading logic involved in the dependencies, concerning the AWS libraries? On Tue, Mar 7, 2017 at 5:50 PM, Tzu-Li (Gordon) Tai wrote: > Hi, > > I just had a quick look on this, but the Kafka fetcher thread’s context > classloader doesn’t

Re: Integrate Flink with S3 on EMR cluster

2017-03-07 Thread Shannon Carey
Generally, using S3 filesystem in EMR with Flink has worked pretty well for me in Flink < 1.2 (unless you run out of connections in your HTTP pool). When you say, "using Hadoop File System class", what do you mean? In my experience, it's sufficient to just use the "s3://" filesystem protocol

Re: FlinkKafkaConsumer010 - creating a data stream of type DataStream<ConsumerRecord<K,V>>

2017-03-07 Thread Dominik Safaric
Hi Gordon, Thanks for the advice. Following it I’ve implemented the Keyed(De)SerializationSchema and am able to further emit the metadata to downstream operators. Regards, Dominik > On 7 Mar 2017, at 07:08, Tzu-Li (Gordon) Tai wrote: > > Hi Dominik, > > I would

Re: AWS exception serialization problem

2017-03-07 Thread Tzu-Li (Gordon) Tai
Hi, I just had a quick look on this, but the Kafka fetcher thread’s context classloader doesn’t seem to be the issue (at least for 1.1.4). In Flink 1.1.4, a separate thread from the task thread is created to run the fetcher, but since the task thread sets the user code classloader as its

Re: Any good ideas for online/offline detection of devices that send events?

2017-03-07 Thread Bruno Aranda
Hi Gordon, Many thanks for your helpful ideas. We tried yesterday the CEP approach, but could not figure it out. The ProcessFunction one looks more promising, and we are investigating it, though we are fighting with some issues related to the event time, where we cannot see so far the timer

Re: Integrate Flink with S3 on EMR cluster

2017-03-07 Thread vinay patil
Hi Guys, Has anyone got this error before ? If yes, have you found any other solution apart from copying the jar files to flink lib folder Regards, Vinay Patil On Mon, Mar 6, 2017 at 8:21 PM, vinay patil [via Apache Flink User Mailing List archive.]

Re: AWS exception serialization problem

2017-03-07 Thread Stephan Ewen
Ah, I see... The issue is that the Kafka fetcher thread apparently do not have the user-code class loader set as the context class loader. Kryo relies on that for class resolution. What Flink version are you on? I think that actual processing and forwarding does not happen in the Kafka Fetchers