Re: Kafka Per-Partition Watermarks

2018-10-04 Thread Taher Koitawala
Hey Andrew, We face the same problem in our application where some of the kafka partitions are empty. In this case what we do is use the rebalance() method on the source streams. Ex: DataStream srcStream= Env.addSource(new FlinkKafkaConsumer09<>(topic,Ser,props)); SrcStream.rebalance();

Re: Streaming to Parquet Files in HDFS

2018-10-04 Thread Averell
Hi Fabian, Kostas, >From the description of this ticket https://issues.apache.org/jira/browse/FLINK-9753, I understand that now my output parquet file with StreamingFileSink will span multiple checkpoints. However, when I tried (as in the here below code snippet) I still see that one "part-X-X"

Re: Kafka Per-Partition Watermarks

2018-10-04 Thread Elias Levy
Does your job perform a keyBy or broadcast that would result in data from different partitions being distributed among tasks? If so, then that would be the cause. On Thu, Oct 4, 2018 at 12:58 PM Andrew Kowpak wrote: > Hi all, > > I apologize if this has been discussed to death in the past,

Kafka Per-Partition Watermarks

2018-10-04 Thread Andrew Kowpak
Hi all, I apologize if this has been discussed to death in the past, but, I'm finding myself very confused, and google is not proving helpful. Based on the documentation, I understand that if there are idle partitions in a kafka stream, watermarks will not advance for the entire application. I

Unable to start session cluster using Docker

2018-10-04 Thread Vinay Patil
Hi, I have used the docker-compose file for creating the cluster as shown in the documentation. The web ui is started successfully, however, the task managers are unable to join. Job Manager container logs: 018-10-04 18:13:13,907 INFO org.apache.flink.runtime.dispatcher.DispatcherRestEndpoint

Re: HDFS HA issue in Flink 1.4 caused by failing to use fs.hdfs.hadoopconf in Flink conf

2018-10-04 Thread Yan Yan
Hi Aljoscha, Thanks for looking into this! Yes, we toggled Flink 1.4 back to Flink 1.3.2 and it works. So seems truly a feature disparity between 1.3.2 and 1.4. Best, Yan On Thu, Oct 4, 2018 at 6:36 AM Aljoscha Krettek wrote: > Another thing: when you retry this again with Flink 1.3.2 it

[deserialization schema] skip data, that couldn't be properly deserialized

2018-10-04 Thread Rinat
Hi mates, in accordance with the contract of org.apache.flink.formats.avro.DeserializationSchema, it should return null value, when content couldn’t be deserialized. But in most cases (for example org.apache.flink.formats.avro.AvroDeserializationSchema) method fails if data is corrupted.

Re: Rowtime for Table from DataStream without explixit fieldnames

2018-10-04 Thread Johannes Schulte
Thanks for the answer Dawid and the helper function, Timo it's not too bad for my use case (small number of fields), I just wanted to make sure I am not missing something here. Cheers, Johannes On Thu, Oct 4, 2018 at 5:07 PM Timo Walther wrote: > Hi Johannes, > > this is not supported so

Re: Data loss when restoring from savepoint

2018-10-04 Thread Stefan Richter
Hi, > Am 04.10.2018 um 16:08 schrieb Juho Autio : > > > you could take a look at Bravo [1] to query your savepoints and to check if > > the state in the savepoint complete w.r.t your expectations > > Thanks. I'm not 100% if this is the case, but to me it seemed like the missed > ids were

[ANNOUNCE] Weekly community update #40

2018-10-04 Thread Till Rohrmann
Dear community, this is the weekly community update thread #40. Please post any news and updates you want to share with the community to this thread. # Discussing feature freeze for Flink 1.7 The community is currently discussing the feature freeze for Flink 1.7 [1]. The 22nd of October is

Identifying missing events in keyed streams

2018-10-04 Thread Averell
Hi everyone, I have a keyed stream which is expecting events every fixed interval (let's say 1 minute). I want to raise alarms for any key which has received no events in n-periods. What should be the cheapest way (in term of performance ) to do this? I thought of some solutions, but don't know

Re: Data loss when restoring from savepoint

2018-10-04 Thread Juho Autio
> you could take a look at Bravo [1] to query your savepoints and to check if the state in the savepoint complete w.r.t your expectations Thanks. I'm not 100% if this is the case, but to me it seemed like the missed ids were being logged by the reducer soon after the job had started (after

Re: Rowtime for Table from DataStream without explixit fieldnames

2018-10-04 Thread Dawid Wysakowicz
Hi Johannes, I am afraid that this is currently not possible and indeed you have to pass all fields again, but Timo cced might want to correct me if I am wrong. Best, Dawid On 04/10/18 15:08, Johannes Schulte wrote: > Hi, > > when converting a DataStream (with Watermarks) to a table like >

Re: Using FlinkKinesisConsumer through a proxy

2018-10-04 Thread Tzu-Li (Gordon) Tai
Hi, Since Flink 1.5, you should be able to set all available configurations on the ClientConfiguration through the consumer Properties (see FLINK-9188 [1]). The way to do that would be to prefix the configuration you want to set with "aws.clientconfig" and add that to the properties, as such:

Re: HDFS HA issue in Flink 1.4 caused by failing to use fs.hdfs.hadoopconf in Flink conf

2018-10-04 Thread Aljoscha Krettek
Another thing: when you retry this again with Flink 1.3.2 it works? I'm trying to rule out another problem in the setup. > On 4. Oct 2018, at 15:17, Aljoscha Krettek wrote: > > Hi Yan, > > This seems to be a bug in the FileSystems and how they're initialized. I'm > looking into this myself

Rowtime for Table from DataStream without explixit fieldnames

2018-10-04 Thread Johannes Schulte
Hi, when converting a DataStream (with Watermarks) to a table like described here https://ci.apache.org/projects/flink/flink-docs-release-1.6/dev/table/streaming.html#event-time I wonder on how to use the rowtime in a following window operation _without_ explicitly specifying all field names

Re: Data loss when restoring from savepoint

2018-10-04 Thread Stefan Richter
Hi, you could take a look at Bravo [1] to query your savepoints and to check if the state in the savepoint complete w.r.t your expectations. I somewhat doubt that there is a general problem with the state/savepoints because many users are successfully running it on a large state and I am not

Re: Avro serialization problem after updating to flink 1.6.0

2018-10-04 Thread Aljoscha Krettek
Hi, can you check whether AlertEvent actually has a field called "SCHEMA$"? You can do that via javap path/to/AlertEvent.class Best, Aljoscha > On 27. Sep 2018, at 10:03, Mark Harris wrote: > > Hi, > > I recently tried to update a flink job from 1.3.2 to 1.6.0. It deploys > successfully

Re: Data loss when restoring from savepoint

2018-10-04 Thread Juho Autio
Thanks for the suggestions! > In general, it would be tremendously helpful to have a minimal working example which allows to reproduce the problem. Definitely. The problem with reproducing has been that this only seems to happen in the bigger production data volumes. That's why I'm hoping to

Re: Flink support for multiple data centers

2018-10-04 Thread Andrey Zagrebin
Hi Olga, At the moment Flink does not have any embedded support for multi region deployment or failover. You can try to automate it, for example, using savepoints [1] and backing them up in multiple data centres. This way you could restore failed job in other regions from the latest savepoint.

Re: hadoopInputFormat and elasticsearch

2018-10-04 Thread Andrey Zagrebin
Hi, At the moment if the processing of any data input split fails, Flink will restart the batch job completely from scratch. There is an ongoing effort to improve fine-grained recovery in FLINK-4256. Best, Andrey > On 2 Oct 2018, at 13:52, aviad wrote: > > Hi, > > I want to write batch job

Re: Data loss when restoring from savepoint

2018-10-04 Thread Till Rohrmann
Hi Juho, another idea to further narrow down the problem could be to simplify the job to not use a reduce window but simply a time window which outputs the window events. Then counting the input and output events should allow you to verify the results. If you are not seeing missing events, then

Re: Difference between BucketingSink and StreamingFileSink

2018-10-04 Thread Aljoscha Krettek
No worries! :-) it's nice that you also posted the solution > On 4. Oct 2018, at 13:23, Averell wrote: > > Hi, > > Sorry for wasting your time. I found the solution for that question > regarding event-time: a class that extends BucketAssigner would do the > needful: > > class

Re: Data loss when restoring from savepoint

2018-10-04 Thread Andrey Zagrebin
Hi Juho, can you try to reduce the job to minimal reproducible example and share the job and input? For example: - some simple records as input, e.g. tuples of primitive types saved as cvs - minimal deduplication job which processes them and misses records - check if it happens for shorter

Re: BucketingSink to S3: Missing class com/amazonaws/AmazonClientException

2018-10-04 Thread Aljoscha Krettek
Hi, they are actually using different interfaces and dependencies. Checkpointing uses Flink FileSystem and the shaded Hadoop Filesystem is a special implementation of this based on the Hadoop S3 FileSystem that has all dependencies bundled in. The BucketingSink uses HDFS/Hadoop FileSystem,

Re: Using FlinkKinesisConsumer through a proxy

2018-10-04 Thread Aljoscha Krettek
Hi, I'm looping in Gordon and Thomas, they might have some idea about how to resolve this. Best, Aljoscha > On 3. Oct 2018, at 17:29, Vijay Balakrishnan wrote: > > I have been trying with all variations to no avail of java > -Dhttp.nonProxyHosts=.. -Dhttps.proxyHost=http://...

Re: Memory Allocate/Deallocate related Thread Deadlock encountered when running a large job > 10k tasks

2018-10-04 Thread Aljoscha Krettek
Hi, this looks like a potential Flink bug. Looping in Nico and Piotr who have looked into that in the past. Could you please comment on that? Best, Aljoscha > On 3. Oct 2018, at 12:12, Narayanaswamy, Krishna > wrote: > > Hi, > > I am trying to run one large single job graph which has >

Re: How can I set the Flink Job Id to enable flink checkpoint

2018-10-04 Thread Aljoscha Krettek
Hi, for starting a job from the state of a previous job you should look into savepoints: https://ci.apache.org/projects/flink/flink-docs-release-1.6/ops/state/savepoints.html Checkpoints are only meant

Re: Difference between BucketingSink and StreamingFileSink

2018-10-04 Thread Averell
Hi, Sorry for wasting your time. I found the solution for that question regarding event-time: a class that extends BucketAssigner would do the needful: class SdcTimeBucketAssigner[T <: MyClass](prefix: String, formatString: String) extends BucketAssigner[T, String]{ @transient

FlinkUserClassLoader in AggregateFunction

2018-10-04 Thread Chirag Dewan
Hi All, Is there any other way to get hold of the FlinkUserClassLoaderother than the RuntimeContext? The problem is, AggregateFunction cant be a RichFunction. I understand that's  because of the state merging issue(from a thread here earlier). Now, I need DynamicClassLoading in

Re: In-Memory Lookup in Flink Operators

2018-10-04 Thread Chirag Dewan
Thanks a lot, David and Fabian. I will give this a try. Cheers,Chirag On Monday, 1 October, 2018, 3:48:42 PM IST, David Anderson wrote: Hi Chirag, The community is also looking at an approach that involves using Bravo[1][2] to bootstrap state by loading the initial version of the state

Re: Data loss when restoring from savepoint

2018-10-04 Thread Juho Autio
Sorry to insist, but we seem to be blocked for any serious usage of state in Flink if we can't rely on it to not miss data in case of restore. Would anyone have suggestions for how to troubleshoot this? So far I have verified with DEBUG logs that our reduce function gets to process also the data

Re: Difference between BucketingSink and StreamingFileSink

2018-10-04 Thread Averell
Hi, https://issues.apache.org/jira/browse/FLINK-9749 <<< as per this ticket, StreamingFileSink is a newer option, which is better than BucketingSink for Parquet. Would love to see some example one using that. Thanks and best regards, Averell -- Sent from:

Difference between BucketingSink and StreamingFileSink

2018-10-04 Thread Averell
Hi everyone, I am trying to persist my stream into parquet files. In the documents, I can see two different file sinks: BucketingSink (Rolling File Sink) and StreamingFileSink. I could not see any information regarding the differences between these two types. Which one should I choose for writing