Re: [Discuss] FLIP-13 Side Outputs in Flink

2016-11-02 Thread Chen Qin
Hi Fabian Thanks for your feedback. sorry for late reply. Some of comments inline. Will update FLIP-13 wiki reflect your comments. - Will multiple side outputs of the same type be supported? > It wasn't implemented in prototype. But should be easy to support, we have unique id in stream record

Re: [DISCUSS] FLIP-2 Extending Window Function Metadata

2016-11-02 Thread Manu Zhang
No problem, Aljoscha. I'll follow up with my use cases. Thanks, Manu On Wed, Nov 2, 2016 at 9:15 PM Aljoscha Krettek wrote: > Hi Manu, > it's great that you want to work on this but another contributor has also > started looking into this and has some code already. It's unfortunate that > there

[jira] [Created] (FLINK-5002) Lack of synchronization in LocalBufferPool#getNumberOfUsedBuffers

2016-11-02 Thread Ted Yu (JIRA)
Ted Yu created FLINK-5002: - Summary: Lack of synchronization in LocalBufferPool#getNumberOfUsedBuffers Key: FLINK-5002 URL: https://issues.apache.org/jira/browse/FLINK-5002 Project: Flink Issue Type

[jira] [Created] (FLINK-5001) Ensure that the Kafka 0.9+ connector is compatible with kafka-consumer-groups.sh

2016-11-02 Thread Robert Metzger (JIRA)
Robert Metzger created FLINK-5001: - Summary: Ensure that the Kafka 0.9+ connector is compatible with kafka-consumer-groups.sh Key: FLINK-5001 URL: https://issues.apache.org/jira/browse/FLINK-5001 Proj

[jira] [Created] (FLINK-5000) Rename Methods in ManagedInitializationContext

2016-11-02 Thread Aljoscha Krettek (JIRA)
Aljoscha Krettek created FLINK-5000: --- Summary: Rename Methods in ManagedInitializationContext Key: FLINK-5000 URL: https://issues.apache.org/jira/browse/FLINK-5000 Project: Flink Issue Type

Re: [DISCUSS] Releasing Flink 1.1.4

2016-11-02 Thread Till Rohrmann
It might make sense to backport - [FLINK-4944] Replace Akka's death watch with own heartbeat on the TM side: https://github.com/apache/flink/pull/2742 as well. This will allow us to activate the quarantine monitoring per default in 1.1.4 without risking to kill all TMs in case of a JM failure. C

[jira] [Created] (FLINK-4999) Improve accuracy of SingleInputGate#getNumberOfQueuedBuffers

2016-11-02 Thread Ted Yu (JIRA)
Ted Yu created FLINK-4999: - Summary: Improve accuracy of SingleInputGate#getNumberOfQueuedBuffers Key: FLINK-4999 URL: https://issues.apache.org/jira/browse/FLINK-4999 Project: Flink Issue Type: Bug

[jira] [Created] (FLINK-4998) ResourceManager fails when num task slots > Yarn vcores

2016-11-02 Thread Maximilian Michels (JIRA)
Maximilian Michels created FLINK-4998: - Summary: ResourceManager fails when num task slots > Yarn vcores Key: FLINK-4998 URL: https://issues.apache.org/jira/browse/FLINK-4998 Project: Flink

[jira] [Created] (FLINK-4996) Make CrossHint @Public

2016-11-02 Thread Greg Hogan (JIRA)
Greg Hogan created FLINK-4996: - Summary: Make CrossHint @Public Key: FLINK-4996 URL: https://issues.apache.org/jira/browse/FLINK-4996 Project: Flink Issue Type: Improvement Components:

[jira] [Created] (FLINK-4997) Extending Window Function Metadata

2016-11-02 Thread Ventura Del Monte (JIRA)
Ventura Del Monte created FLINK-4997: Summary: Extending Window Function Metadata Key: FLINK-4997 URL: https://issues.apache.org/jira/browse/FLINK-4997 Project: Flink Issue Type: New Feat

[jira] [Created] (FLINK-4995) YarnFlinkResourceManagerTest JobManager Lost Leadership test failed

2016-11-02 Thread Ufuk Celebi (JIRA)
Ufuk Celebi created FLINK-4995: -- Summary: YarnFlinkResourceManagerTest JobManager Lost Leadership test failed Key: FLINK-4995 URL: https://issues.apache.org/jira/browse/FLINK-4995 Project: Flink

[FLINK-4492] Cleanup files from canceled snapshots

2016-11-02 Thread Nikolay Vasilishin
Hi, dev. I've been working on FLINK-4492 issue, but couldn't reproduce the bug. As I mentioned in comment, FsCheckpointStreamFactory

[jira] [Created] (FLINK-4994) Don't Clear Trigger State and Merging Window Set When Purging

2016-11-02 Thread Aljoscha Krettek (JIRA)
Aljoscha Krettek created FLINK-4994: --- Summary: Don't Clear Trigger State and Merging Window Set When Purging Key: FLINK-4994 URL: https://issues.apache.org/jira/browse/FLINK-4994 Project: Flink

Re: [DISCUSS] FLIP-2 Extending Window Function Metadata

2016-11-02 Thread Aljoscha Krettek
Hi Manu, it's great that you want to work on this but another contributor has also started looking into this and has some code already. It's unfortunate that there is no Jira issue yet for this but I think he'll open one this week. Cheers, Aljoscha On Wed, 2 Nov 2016 at 13:00 Manu Zhang wrote:

Re: [FLINK-3848] Add ProjectableTableSource

2016-11-02 Thread Fabian Hueske
Hi Anton, a regular TableSource does not accept a predicate and return the whole table. A ProjectableTableSource is able to evaluate a predicate while scanning. TableSources that evaluate predicates while (or rather before) scanning can significantly reduce IO compared to a full scan. Sources tha

Re: Assign a unique id to each line of a dataset

2016-11-02 Thread Fabian Hueske
Hi Thomas, have a look at DataSetUtils [1]. Best, Fabian [1] https://github.com/apache/flink/blob/master/flink-java/src/main/java/org/apache/flink/api/java/utils/DataSetUtils.java 2016-11-02 13:17 GMT+01:00 Thomas FOURNIER : > Hello, > > Is it possible with the current Flink-API to give a uniq

[FLINK-3848] Add ProjectableTableSource

2016-11-02 Thread Anton Solovev
Hi folks, Can you explain me what ProjectableTableSource is ? What is it being invented for? Best, Anton

Assign a unique id to each line of a dataset

2016-11-02 Thread Thomas FOURNIER
Hello, Is it possible with the current Flink-API to give a unique id to each line of a dataset ? More precisely, I've globally sorted my Dataset with partitionByRange and I'd like to perform a kind of "zipWithIndex" operation, so that I can retrieve a Map (such as a collectAsMap with Spark). Tha

Re: [DISCUSS] FLIP-2 Extending Window Function Metadata

2016-11-02 Thread Manu Zhang
Hi Aljoscha, Have you started working on ProcessWindowFunction ? If not, may I take this task ? Thanks, Manu On Wed, Nov 2, 2016 at 5:16 PM Aljoscha Krettek wrote: > I think we reached consensus here so I would like to mark this FLIP as > accepted. We will now process with implementing the fi

Re: [DISCUSS] Releasing Flink 1.1.4

2016-11-02 Thread Ufuk Celebi
As a quick update: the "pending review" issues have all been resolved. The open issues are still open: - FLINK-4904: Add a limit for how much data may be spilled in checkpoint alignments => fix pending - FLINK-4910: Introduce safety net for closing file system streams Any updates here? – Ufuk

[jira] [Created] (FLINK-4993) Don't Allow Trigger.onMerge() to return TriggerResult

2016-11-02 Thread Aljoscha Krettek (JIRA)
Aljoscha Krettek created FLINK-4993: --- Summary: Don't Allow Trigger.onMerge() to return TriggerResult Key: FLINK-4993 URL: https://issues.apache.org/jira/browse/FLINK-4993 Project: Flink Iss

Re: [DISCUSS] FLIP-2 Extending Window Function Metadata

2016-11-02 Thread Aljoscha Krettek
I think we reached consensus here so I would like to mark this FLIP as accepted. We will now process with implementing the first step, i.e. adding the new ProcessWindowFunction. On Mon, 1 Aug 2016 at 18:08 Aljoscha Krettek wrote: > Alright, that seems reasonable. I updated the doc to add the Col

Re: [DISCUSSION] Timely function interface and timer params

2016-11-02 Thread Gyula Fóra
Hi, I opened this: https://issues.apache.org/jira/browse/FLINK-4992 With this users can implement practically anything depending how they use the parameter so changing the state access logic is not necessary. Cheers, Gyula Aljoscha Krettek ezt írta (időpont: 2016. nov. 1., K, 16:39): > Hi, > y

[jira] [Created] (FLINK-4992) Expose String parameter for timers in Timely functions and TimerService

2016-11-02 Thread Gyula Fora (JIRA)
Gyula Fora created FLINK-4992: - Summary: Expose String parameter for timers in Timely functions and TimerService Key: FLINK-4992 URL: https://issues.apache.org/jira/browse/FLINK-4992 Project: Flink

Re: [DISCUSS] FLIP-11: Table API Stream Aggregations

2016-11-02 Thread Fabian Hueske
Thanks everybody for the input. I updated the FLIP-11 document [1] and did the following changes: - made over() mandatory also for single row windows - removed allowedLateness() from windows which would mean to add some kind of retraction output mode. This is out-of-scope for FLIP-11 and is addre

[jira] [Created] (FLINK-4991) TestTask hangs in testWatchDogInterruptsTask

2016-11-02 Thread Ufuk Celebi (JIRA)
Ufuk Celebi created FLINK-4991: -- Summary: TestTask hangs in testWatchDogInterruptsTask Key: FLINK-4991 URL: https://issues.apache.org/jira/browse/FLINK-4991 Project: Flink Issue Type: Bug