Delete save point when using incremental checkpoint

2017-10-11 Thread Kien Truong
Hi, When using increment checkpoint mode, can I delete the save point that the job recovered from after sometime ? Or do I have to keep that checkpoint forever because it's a part of the snapshot chain ? Best regards, Kien

Re: RichMapFunction parameters in the Streaming API

2017-10-11 Thread Chesnay Schepler
The Configuration parameter in open() is a relic of the previous java API where operators were instantiated generically. Nowadays, this is no longer the case as they are serialized instead, which simplifies the passing of parameters as you can simply store them in a field of your UDF. The con

Re: Delete save point when using incremental checkpoint

2017-10-11 Thread Chesnay Schepler
Hi, There is an important distinction between checkpoints (triggered by Flink, may be incremental) and savepoints (manually triggered, always self-contained). Your question is unfortunately mixing both terms, please expand which you're referring to. Regards, Chesnay On 11.10.2017 10:31, K

Re: Delete save point when using incremental checkpoint

2017-10-11 Thread Stephan Ewen
Kien, I think what you are asking is: Do incremental checkpoints refer to chunks of state in a previous savepoint. Meaning that if the savepoint is deleted, will the checkpoint be missing some chunks. The answer is: No, savepoints are always full snapshots and incremental checkpoints do not refer

Re: RichMapFunction parameters in the Streaming API

2017-10-11 Thread Aljoscha Krettek
I think we should remove that part from the best-practices documentation. I'll quickly open a PR. > On 11. Oct 2017, at 10:46, Chesnay Schepler wrote: > > The Configuration parameter in open() is a relic of the previous java API > where operators were instantiated generically. > > Nowadays, t

Re: Windows getting created only on first execution

2017-10-11 Thread Aljoscha Krettek
Hi, I think the problem is that your Kafka consumer has the same group-id across those two runs. This means that it will pick up the last "read position" of the previous run, and thus not read anything. If you change the group-id for the second run you should be able to read your data again. B

Re: Subscribe

2017-10-11 Thread Aljoscha Krettek
Hi Stephen, You would have to send a mail to "user-subscr...@flink.apache.org " Best, Aljoscha > On 10. Oct 2017, at 20:08, Stephen Jiang wrote: > >

Re: Manual checkpoint

2017-10-11 Thread Aljoscha Krettek
Hi, Triggering a savepoint is currently not possible from within a job. The job would somehow have to emit a message that an outside system would pick up and then trigger a savepoint. Best, Aljoscha > On 10. Oct 2017, at 17:25, nragon wrote: > > Can I trigger a checkpoint based on a specific

Re: Delete save point when using incremental checkpoint

2017-10-11 Thread Kien Truong
Thanks Stephan. That's exactly what I was asking about. Best regards, Kien On Oct 11, 2017, 16:59, at 16:59, Stephan Ewen wrote: >Kien, > >I think what you are asking is: Do incremental checkpoints refer to >chunks >of state in a previous savepoint. Meaning that if the savepoint is >deleted, >wi

Flink 1.3.2 Netty Exception

2017-10-11 Thread Flavio Pompermaier
Hi to all, we wrote a small JUnit test to reproduce a memory issue we have in a Flink job (that seems related to Netty) . At some point, usually around the 28th loop, the job fails with the following exception (actually we never faced that in production but maybe is related to the memory issue some

Re: Windows getting created only on first execution

2017-10-11 Thread Rahul Raj
Changing the group id didn't work for me, instead using setStartfromEarliest() on kafka consumer worked for me. But it created one confusion, that is in case of failure if I start from a particular checkpoint or savepoint will the application start reading the message from a particular offset where

Re: Manual checkpoint

2017-10-11 Thread Rahul Raj
You can use FLink's REST API to get the job id of running jobs and then cancel it via CLI commands. You can enclose both things i.e. getting job ids and CLI command in a single script. Rahul Raj On 11 October 2017 at 15:47, Aljoscha Krettek wrote: > Hi, > > Triggering a savepoint is currently n

Re: Windows getting created only on first execution

2017-10-11 Thread Aljoscha Krettek
Hi, When you are restoring from a savepoint (or checkpoint) the offsets in Kafka are complete ignored. Flink is checkpointing the offset at the time the checkpoint/savepoint is taken and that will be used as the read offset when restoring. Best, Aljoscha > On 11. Oct 2017, at 12:58, Rahul Raj

Re: Windowing isn't applied per key

2017-10-11 Thread mclendenin
Hi Tony, In the documentation on keyed windows vs non-keyed it says that it will split the stream into parallel keyed streams with windows being executed in parallel across the keys. I would think that this would mean that each key has it's own window managed independently. https://ci.apache.org/

Decouple Kafka partitions and Flink parallelism for ordered streams

2017-10-11 Thread Sanne de Roever
Hi, Currently we need 75 Kafka partitions per topic and a parallelism of 75 to meet required performance, increasing the partitions and parallelism gives diminished returns Currently the performance is approx. 1500 msg/s per core, having one pipeline (source, map, sink) deployed as one instance p

Re: Flink 1.3.2 Netty Exception

2017-10-11 Thread Chesnay Schepler
I can confirm that the issue is reproducible with the given test, from the command-line and IDE. While cutting down the test case, by replacing the outputformat with a DiscardingOutputFormat and the JDBCInputFormat with a simple collection, i stumbled onto a new Exception after ~200 iterations

Re: How flink monitor source stream task(Time Trigger) is running?

2017-10-11 Thread Aljoscha Krettek
I think this might not actually be resolved. What YunFan was referring to in the initial mail is the Thread factory that is used for the processing-time service: https://github.com/apache/flink/blob/5af463a9c0ff62603bc342a78dfd5483d834e8a7/flink-streaming-java/src/main/java/org/apache/flink/stre

Re: javax.net.ssl.SSLPeerUnverifiedException: peer not authenticated for S3 access

2017-10-11 Thread Patrick Lucas
I thought it might be a CA certificates issue, but it looks like openjdk:8-jre-alpine includes the proper certificates. You could just this just to make sure: exec into the container and run curl -v https://s3.amazonaws.com. You may have to run apk add --no-cache curl first. Apart from that, a se

Re: Finding things not seen in the last window

2017-10-11 Thread Aljoscha Krettek
Hi Ron, I think your colleague might be able to do that using a ProcessFunction with MapState and timers. The MapState is used to determine if a record is new. Timers would be used to schedule emission and also to schedule cleanup of of entries from the MapState. For doing cleanup, the entries

Re: Windowing isn't applied per key

2017-10-11 Thread Tony Wei
Hi Marcus, Yes, each key would has it's own window managed, so the aggregation on window is sum of the value by each key, not sum of all element. You can imagine that each key has its own sliding window assignor that decides each element in each keyed stream belong to which windows, but all keyed

Re: Decouple Kafka partitions and Flink parallelism for ordered streams

2017-10-11 Thread Chesnay Schepler
It is correct that keyBy and partition operations will distribute messages over the network as they distribute the data across all subtasks. For this use-case we only want to consider subtasks that are subsequent to our operator, like a local keyBy. I don't think there is an obvious way to impl

Re: Question about checkpointing with stateful operators and state recovery

2017-10-11 Thread Aljoscha Krettek
Hi Frederico, I'll try and give some answers: 1. Generally speaking, no. If you use keyed state, for example via RuntimeContext you don't need to implement CheckpointedFunction. 2. You don't have to set setCommitOffsetsOnCheckpoints(true), this only affects how offsets are committed to Kafka i

Re: Flink 1.3.2 Netty Exception

2017-10-11 Thread Ufuk Celebi
@Chesnay: Recycling of network resources happens after the tasks go into state FINISHED. Since we are submitting new jobs in a local loop here it can easily happen that the new job is submitted before enough buffers are available again. At least, previously that was the case. I'm CC'ing Nico who r

Re: Flink Savepoint Config parameter

2017-10-11 Thread Aljoscha Krettek
Hi, I'm assuming you're running via YARN? I think you need to add the -yid parameter to your incantation. Best, Aljoscha > On 28. Sep 2017, at 10:54, ant burton wrote: > > Hey, > > When running in EMR and taking a savepoint with > > flink cancel -s SAVEPOINT_DIR JOB_ID > > results in

Re: Using latency markers

2017-10-11 Thread Aljoscha Krettek
This is the Jira issue: https://issues.apache.org/jira/browse/FLINK-7608 > On 27. Sep 2017, at 12:22, Martin Eden wrote: > > Any follow-up on this? Jira? PR? > > On Wed, Sep 13, 2017 at 11:30 AM, Tzu-Li (Gordon) Tai > wrote: > Hi Aitozi, > > Yes, I think we haven’t

Re: got Warn message - "the expected leader session ID did not equal the received leader session ID " when using LocalFlinkMiniCluster to interpret scala code

2017-10-11 Thread Aljoscha Krettek
Hi, I think you can make it start the Web Frontend via conf.setBoolean(ConfigConstants.LOCAL_START_WEBSERVER, true); In the future, this will become moot, though, when the JobManager has a proper REST API that is always there. Best, Aljoscha > On 27. Sep 2017, at 11:40, XiangWei Huang wrote

Re: Decouple Kafka partitions and Flink parallelism for ordered streams

2017-10-11 Thread Chesnay Schepler
I couldn't find a proper solution for this. The easiest solution might be to use the Async I/O , and do the validation with an ExecutionService or similar in the map function. I've CC'd aljoscha, may

Write each group to its own file

2017-10-11 Thread rlazoti
Hi, Is there a way to write each group to its own file using the Dataset api (Batch)? For example, lets use the following class: And the following Dataset: So in this example my output should be these 3 files: - cpu.csv i7, cpu R5, cpu - gpu.csv gtx1080, gpu vega64, gpu - ssd.csv evo250gb

Write each group to its own file

2017-10-11 Thread rlazoti
Hi, Is there a way to write each group to its own file using the Dataset api (Batch)? For example, lets use the following class: case class Product(name: String, category: String) And the following Dataset: val products = env.fromElements(Product("i7", "cpu"), Product("R5", "cpu"), Product("gt

Re: NoResourceAvailable exception

2017-10-11 Thread Aljoscha Krettek
Btw, what load are you putting on the cluster, i.e. what is your computation? If you don't have load, the cluster and job just keep on running, right? Best, Aljoscha > On 19. Sep 2017, at 12:00, AndreaKinn wrote: > > Thank you, unfortunately it had no effects. > > As I add more load on the co

R/W traffic estimation between Flink and Zookeeper

2017-10-11 Thread Hao Sun
Hi Is there a way to estimate read/write traffic between flink and zk? I am looking for something like 1000 reads/sec or 1000 writes/sec. And the size of the message. Thanks

Re: DataStream joining without window

2017-10-11 Thread Yan Zhou [FDS Science] ­
Thank you for the reply. It's very helpful. Best Yan On Tue, Oct 10, 2017 at 7:57 AM, Aljoscha Krettek wrote: > Hi, > > Yes, using a TwoInputStreamOperator (or even better, a CoProcessFunction, > because TwoInputStreamOperator is a low-level interface that might change > in the future) is the r

Re: NoResourceAvailable exception

2017-10-11 Thread AndreaKinn
the program is composed by: 6 Kafka /source/ connector with custom timestamp and watermark /extractor/ and /map/ function each. then I use 6 instance of an external library called flink-htm (quite heavy) moreover I have 6 /process/ method and 2 /union/ method to merge result streams. Finally I hav

Implement bunch of transformations applied to same source stream in Apache Flink in parallel and combine result

2017-10-11 Thread Andrey Salnikov
Hi! Could you please help me - I'm trying to use Apache Flink for machine learning tasks with external ensemble/tree libs like XGBoost, so my workflow will be like this: - receive single stream of data which atomic event looks like a simple vector event=(X1, X2, X3...Xn) and it can be imagi

Writing to an HDFS file from a Flink stream job

2017-10-11 Thread Isuru Suriarachchi
Hi all, I'm just trying to use an HDFS file as the sink for my flink stream job. I use the following line to do so. stream.writeAsText("hdfs://hadoop-master:9000/user/isuru/foo"); I have not set "fs.hdfs.hadoopconf" in my flink configuration as it should work with the full hdfs file name accord

Re: RichMapFunction parameters in the Streaming API

2017-10-11 Thread Colin Williams
Thanks for the detailed explanation regarding the reasoning behind not using opens' configuration parameters! On Wed, Oct 11, 2017 at 1:46 AM, Chesnay Schepler wrote: > The Configuration parameter in open() is a relic of the previous java API > where operators were instantiated generically. > >

Writing an Integration test for flink-metrics

2017-10-11 Thread Colin Williams
I have a RichMapFunction and I'd like to ensure Meter fields are properly incremented. I've been trying to think of the best way to do this. Currently I think that I'd need to either implement my own reporter (or use JMX) and write to a socket, create a listener and wait for the reporter to send th