Re: Flink ML - Vector and DenseVector

2016-01-19 Thread Hilmi Yildirim
We had a discussion about the "LabeledVector" class. But now this is a 
discussion about the Vector and DenseVector :)


Am 18.01.2016 um 22:29 schrieb Till Rohrmann:

Didn't we just had the discussion in another email thread?
On Jan 18, 2016 8:55 PM, "Hilmi Yildirim"  wrote:


Hi,
the Vector and DenseVector implementations of Flink ML only allow Double
values. But there are cases where the values are not Doubles, e.g. in NLP.
Does it make sense to make the implementations generic, i.e. Vector[T] and
DenseVector[T]?

Best Regards,
Hilmi

--
==
Hilmi Yildirim, M.Sc.
Researcher

DFKI GmbH
Intelligente Analytik für Massendaten
DFKI Projektbüro Berlin
Alt-Moabit 91c
D-10559 Berlin
Phone: +49 30 23895 1814

E-Mail: hilmi.yildi...@dfki.de

-
Deutsches Forschungszentrum fuer Kuenstliche Intelligenz GmbH
Firmensitz: Trippstadter Strasse 122, D-67663 Kaiserslautern

Geschaeftsfuehrung:
Prof. Dr. Dr. h.c. mult. Wolfgang Wahlster (Vorsitzender)
Dr. Walter Olthoff

Vorsitzender des Aufsichtsrats:
Prof. Dr. h.c. Hans A. Aukes

Amtsgericht Kaiserslautern, HRB 2313
-





--
==
Hilmi Yildirim, M.Sc.
Researcher

DFKI GmbH
Intelligente Analytik für Massendaten
DFKI Projektbüro Berlin
Alt-Moabit 91c
D-10559 Berlin
Phone: +49 30 23895 1814

E-Mail: hilmi.yildi...@dfki.de

-
Deutsches Forschungszentrum fuer Kuenstliche Intelligenz GmbH
Firmensitz: Trippstadter Strasse 122, D-67663 Kaiserslautern

Geschaeftsfuehrung:
Prof. Dr. Dr. h.c. mult. Wolfgang Wahlster (Vorsitzender)
Dr. Walter Olthoff

Vorsitzender des Aufsichtsrats:
Prof. Dr. h.c. Hans A. Aukes

Amtsgericht Kaiserslautern, HRB 2313
-



Re: [ANNOUNCE] Chengxiang Li added as committer

2016-01-19 Thread Chiwan Park
Congrats! Welcome Chengxiang Li!

> On Jan 19, 2016, at 7:13 PM, Vasiliki Kalavri  
> wrote:
> 
> Congratulations! Welcome Chengxiang Li!
> 
> On 19 January 2016 at 11:02, Fabian Hueske  wrote:
> 
>> Hi everybody,
>> 
>> I'd like to announce that Chengxiang Li accepted the PMC's offer to become
>> a committer of the Apache Flink project.
>> 
>> Please join me in welcoming Chengxiang Li!
>> 
>> Best, Fabian
>> 

Regards,
Chiwan Park



Re: [ANNOUNCE] Chengxiang Li added as committer

2016-01-19 Thread Vasiliki Kalavri
Congratulations! Welcome Chengxiang Li!

On 19 January 2016 at 11:02, Fabian Hueske  wrote:

> Hi everybody,
>
> I'd like to announce that Chengxiang Li accepted the PMC's offer to become
> a committer of the Apache Flink project.
>
> Please join me in welcoming Chengxiang Li!
>
> Best, Fabian
>


What is the topic & offset used for in KeyedDeserializationSchema.deserialize()?

2016-01-19 Thread Tzu-Li (Gordon) Tai
Hi devs,

I need a little help on clarification of what the arguments "topic" and
"offset" is used for in KeyedDeserializationSchema.deserialize(). The main
issue is that I'm currently in progress of implementing Flink Kinesis
Consumer, and Kinesis offsets, unlike Kafka offsets which are incremental
starting from 0, are digits that can only by stored in BigIntegers and
generally doesn't increment by 1 between each data record.

Just need to make sure that I won't be messing things up with these two
values. A point to any part of the codebase where I can understand how Flink
uses "topic" and "offset" in the deserialization schema would be perfect.

Many thanks in advance!

Cheers,
Gordon



--
View this message in context: 
http://apache-flink-mailing-list-archive.1008284.n3.nabble.com/What-is-the-topic-offset-used-for-in-KeyedDeserializationSchema-deserialize-tp9911.html
Sent from the Apache Flink Mailing List archive. mailing list archive at 
Nabble.com.


Re: [ANNOUNCE] Chengxiang Li added as committer

2016-01-19 Thread Maximilian Michels
Pleased to have you with us Chengxiang!

Cheers,
Max

On Tue, Jan 19, 2016 at 11:13 AM, Chiwan Park  wrote:
> Congrats! Welcome Chengxiang Li!
>
>> On Jan 19, 2016, at 7:13 PM, Vasiliki Kalavri  
>> wrote:
>>
>> Congratulations! Welcome Chengxiang Li!
>>
>> On 19 January 2016 at 11:02, Fabian Hueske  wrote:
>>
>>> Hi everybody,
>>>
>>> I'd like to announce that Chengxiang Li accepted the PMC's offer to become
>>> a committer of the Apache Flink project.
>>>
>>> Please join me in welcoming Chengxiang Li!
>>>
>>> Best, Fabian
>>>
>
> Regards,
> Chiwan Park
>


Re: What is the topic & offset used for in KeyedDeserializationSchema.deserialize()?

2016-01-19 Thread Maximilian Michels
Hi Gordon,

You may use "topic" and "offset" for whatever you like. Note that this
is just an interface. If it does not work for your Kinesis adapter,
you may create a new interface. For existing usage of the
KeyedDeserializationSchema, please have a look at the
FlinkKafkaConsumer.

Cheers,
Max

On Tue, Jan 19, 2016 at 11:27 AM, Tzu-Li (Gordon) Tai
 wrote:
> Hi devs,
>
> I need a little help on clarification of what the arguments "topic" and
> "offset" is used for in KeyedDeserializationSchema.deserialize(). The main
> issue is that I'm currently in progress of implementing Flink Kinesis
> Consumer, and Kinesis offsets, unlike Kafka offsets which are incremental
> starting from 0, are digits that can only by stored in BigIntegers and
> generally doesn't increment by 1 between each data record.
>
> Just need to make sure that I won't be messing things up with these two
> values. A point to any part of the codebase where I can understand how Flink
> uses "topic" and "offset" in the deserialization schema would be perfect.
>
> Many thanks in advance!
>
> Cheers,
> Gordon
>
>
>
> --
> View this message in context: 
> http://apache-flink-mailing-list-archive.1008284.n3.nabble.com/What-is-the-topic-offset-used-for-in-KeyedDeserializationSchema-deserialize-tp9911.html
> Sent from the Apache Flink Mailing List archive. mailing list archive at 
> Nabble.com.


Re: What is the topic & offset used for in KeyedDeserializationSchema.deserialize()?

2016-01-19 Thread Tzu-Li (Gordon) Tai
Hi Max,

Thanks for the quick response and clarification :)
I got a bit confused and thought that Flink internals would be accessing
this interface too.

Cheers,
Gordon



--
View this message in context: 
http://apache-flink-mailing-list-archive.1008284.n3.nabble.com/What-is-the-topic-offset-used-for-in-KeyedDeserializationSchema-deserialize-tp9911p9914.html
Sent from the Apache Flink Mailing List archive. mailing list archive at 
Nabble.com.


Questions for a better understanding of the internals of data exchange between tasks

2016-01-19 Thread Camelia Elena Ciolac
Hello,

I list some questions gathered while reading documentation on Flink's internals 
and I am grateful to receive your answers.

1) How is the JobManager involved in the communication between tasks running in 
task slots on TaskManagers?

>From [1] it appears to me that, as part of the control flow for data exchange, 
>every time a task has a a result that is consumable, it notifies the JobManager
This is repeated in [2] "When producing an intermediate result, the producing 
task is responsible to notify the JobManager about available data".
However, the result's size might as well be 1, in the case of data streams, so 
I ask this question to understand better if for each data exchange the 
JobManager is involved.

2) In case my understanding of the aforementioned control flow is correct, then 
from the following options, which is the data outcome that triggers a 
notification of the JobManager: a ResultPartition, a ResultSubpartition or a 
Buffer?

3) Idem, then are the Akka actors involved in this notification of data 
availability for consumption?

4) How is orchestrated this co-existence of receiver-initiated data transfers 
(the consumer requests the data partitions) with the push data transfers [2] 
between 2 tasks ?
>From [3] I retain the paragraph:

"The JobManager also acts as the input split assigner for data sources. It is 
responsible for distributing the work across all TaskManager such that data 
locality is preserved where possible. In order to dynamically balance the load, 
the Tasks request a new input split after they have finished processing the old 
one. This request is realized by sending a RequestNextInputSplit to the 
JobManager. The JobManager responds with a NextInputSplit message. If there are 
no more input splits, then the input split contained in the message is null.

The Tasks are deployed lazily to the TaskManagers. This means that tasks which 
consume data are only deployed after one of its producers has finished 
producing some data. Once the producer has done so, it sends a 
ScheduleOrUpdateConsumers message to the JobManager. This messages says that 
the consumer can now read the newly produced data. If the consuming task is not 
yet running, it will be deployed to a TaskManager."

5) Can you please summarize how this data exchange occurs in Flink, based on an 
example?

Thank you!

Best regards,
Camelia


[1] 
https://cwiki.apache.org/confluence/display/FLINK/Data+exchange+between+tasks
[2] 
https://cwiki.apache.org/confluence/display/FLINK/Network+Stack%2C+Plugable+Data+Exchange
[3] https://cwiki.apache.org/confluence/display/FLINK/Akka+and+Actors



Re: What is the topic & offset used for in KeyedDeserializationSchema.deserialize()?

2016-01-19 Thread Robert Metzger
Hi Gordon,

thank you for starting the discussion. I think in fact the
KeyedDeserializationSchema is located in the wrong package. Its methods are
very Kafka specific, maybe I should move them there.

How would the deserializationSchema for Kinesis look like? Does the Kinesis
API return byte[] ? Is there any other information which is useful for
users?


On Tue, Jan 19, 2016 at 11:49 AM, Tzu-Li (Gordon) Tai 
wrote:

> Hi Max,
>
> Thanks for the quick response and clarification :)
> I got a bit confused and thought that Flink internals would be accessing
> this interface too.
>
> Cheers,
> Gordon
>
>
>
> --
> View this message in context:
> http://apache-flink-mailing-list-archive.1008284.n3.nabble.com/What-is-the-topic-offset-used-for-in-KeyedDeserializationSchema-deserialize-tp9911p9914.html
> Sent from the Apache Flink Mailing List archive. mailing list archive at
> Nabble.com.
>


Re: What is the topic & offset used for in KeyedDeserializationSchema.deserialize()?

2016-01-19 Thread Tzu-Li (Gordon) Tai
Hi Robert,

+1 for a change to where the KeyedDeserializationSchema is located. I was
just starting to wonder how I should name the Kinesis's
deserializationSchema if I were to create another one in the same package.

For Kinesis, the API returns String for key, byte[] for value, String for
streamName (similar to Kafka topic), and a String for the offset. So I would
definitely need to create a new deserializationSchema for this.

For https://issues.apache.org/jira/browse/FLINK-3229, I'll create the
required new interfaces in Kinesis connector specific packages. I'd be happy
to help with relocating the current KeyedDeserializationSchema related
interfaces and classes to Kafka specific package as a seperate issue, if you
want to =)

Cheers,
Gordon



--
View this message in context: 
http://apache-flink-mailing-list-archive.1008284.n3.nabble.com/What-is-the-topic-offset-used-for-in-KeyedDeserializationSchema-deserialize-tp9911p9917.html
Sent from the Apache Flink Mailing List archive. mailing list archive at 
Nabble.com.


Re: What is the topic & offset used for in KeyedDeserializationSchema.deserialize()?

2016-01-19 Thread Robert Metzger
I'll relocate the KeyedDeserializationSchema as part of the Kafka 0.9.0.0
support (its a pending pull request I'll merge soon)

On Tue, Jan 19, 2016 at 12:20 PM, Tzu-Li (Gordon) Tai 
wrote:

> Hi Robert,
>
> +1 for a change to where the KeyedDeserializationSchema is located. I was
> just starting to wonder how I should name the Kinesis's
> deserializationSchema if I were to create another one in the same package.
>
> For Kinesis, the API returns String for key, byte[] for value, String for
> streamName (similar to Kafka topic), and a String for the offset. So I
> would
> definitely need to create a new deserializationSchema for this.
>
> For https://issues.apache.org/jira/browse/FLINK-3229, I'll create the
> required new interfaces in Kinesis connector specific packages. I'd be
> happy
> to help with relocating the current KeyedDeserializationSchema related
> interfaces and classes to Kafka specific package as a seperate issue, if
> you
> want to =)
>
> Cheers,
> Gordon
>
>
>
> --
> View this message in context:
> http://apache-flink-mailing-list-archive.1008284.n3.nabble.com/What-is-the-topic-offset-used-for-in-KeyedDeserializationSchema-deserialize-tp9911p9917.html
> Sent from the Apache Flink Mailing List archive. mailing list archive at
> Nabble.com.
>


Re: What is the topic & offset used for in KeyedDeserializationSchema.deserialize()?

2016-01-19 Thread Stephan Ewen
While the de-serializations schema is not used by the Flink internals, I
think the initial idea  was to use it across different sources/sinks (like
Kafka, Socket, RabbitMQ, ...)

Does it make sense to have a KafkaDeSerializationSchema, and then wrap the
common serialization schemata?

On Tue, Jan 19, 2016 at 12:25 PM, Robert Metzger 
wrote:

> I'll relocate the KeyedDeserializationSchema as part of the Kafka 0.9.0.0
> support (its a pending pull request I'll merge soon)
>
> On Tue, Jan 19, 2016 at 12:20 PM, Tzu-Li (Gordon) Tai 
> wrote:
>
> > Hi Robert,
> >
> > +1 for a change to where the KeyedDeserializationSchema is located. I was
> > just starting to wonder how I should name the Kinesis's
> > deserializationSchema if I were to create another one in the same
> package.
> >
> > For Kinesis, the API returns String for key, byte[] for value, String for
> > streamName (similar to Kafka topic), and a String for the offset. So I
> > would
> > definitely need to create a new deserializationSchema for this.
> >
> > For https://issues.apache.org/jira/browse/FLINK-3229, I'll create the
> > required new interfaces in Kinesis connector specific packages. I'd be
> > happy
> > to help with relocating the current KeyedDeserializationSchema related
> > interfaces and classes to Kafka specific package as a seperate issue, if
> > you
> > want to =)
> >
> > Cheers,
> > Gordon
> >
> >
> >
> > --
> > View this message in context:
> >
> http://apache-flink-mailing-list-archive.1008284.n3.nabble.com/What-is-the-topic-offset-used-for-in-KeyedDeserializationSchema-deserialize-tp9911p9917.html
> > Sent from the Apache Flink Mailing List archive. mailing list archive at
> > Nabble.com.
> >
>


[jira] [Created] (FLINK-3258) Merge AbstractInvokable's registerInputOutput and invoke

2016-01-19 Thread Ufuk Celebi (JIRA)
Ufuk Celebi created FLINK-3258:
--

 Summary: Merge AbstractInvokable's registerInputOutput and invoke
 Key: FLINK-3258
 URL: https://issues.apache.org/jira/browse/FLINK-3258
 Project: Flink
  Issue Type: Improvement
Reporter: Ufuk Celebi
Assignee: Ufuk Celebi
 Fix For: 1.0.0


The separation between {{registerInputOutput}} and {{invoke}} in 
{{AbstractInvokable}} is an artifact of previous Flink versions. The 
{{registerInputOutput}} method used to trigger the creation of input and output 
gates. This has become obsolete as the task itself takes care of this now and 
{{registerInputOutput}} only requests the input/output gates.

The contract of having a single {{invoke}} method, which cleans up after itself 
is easier to implement than applying it to two methods.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


Re: [ANNOUNCE] Chengxiang Li added as committer

2016-01-19 Thread Stephan Ewen
Good to have you on board!

On Tue, Jan 19, 2016 at 11:29 AM, Maximilian Michels  wrote:

> Pleased to have you with us Chengxiang!
>
> Cheers,
> Max
>
> On Tue, Jan 19, 2016 at 11:13 AM, Chiwan Park 
> wrote:
> > Congrats! Welcome Chengxiang Li!
> >
> >> On Jan 19, 2016, at 7:13 PM, Vasiliki Kalavri <
> vasilikikala...@gmail.com> wrote:
> >>
> >> Congratulations! Welcome Chengxiang Li!
> >>
> >> On 19 January 2016 at 11:02, Fabian Hueske  wrote:
> >>
> >>> Hi everybody,
> >>>
> >>> I'd like to announce that Chengxiang Li accepted the PMC's offer to
> become
> >>> a committer of the Apache Flink project.
> >>>
> >>> Please join me in welcoming Chengxiang Li!
> >>>
> >>> Best, Fabian
> >>>
> >
> > Regards,
> > Chiwan Park
> >
>


[jira] [Created] (FLINK-3259) Redirect programming guides to new layout

2016-01-19 Thread Ufuk Celebi (JIRA)
Ufuk Celebi created FLINK-3259:
--

 Summary: Redirect programming guides to new layout
 Key: FLINK-3259
 URL: https://issues.apache.org/jira/browse/FLINK-3259
 Project: Flink
  Issue Type: Improvement
  Components: Documentation
Reporter: Ufuk Celebi
Assignee: Ufuk Celebi
 Fix For: 1.0.0


The recent changes to the docs have broken old links. At least for the main 
streaming and batch programming guides we should add a redirect to the new urls.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


Re: [ANNOUNCE] Chengxiang Li added as committer

2016-01-19 Thread Kostas Tzoumas
Welcome Chengxiang!!

On Tue, Jan 19, 2016 at 12:31 PM, Stephan Ewen  wrote:

> Good to have you on board!
>
> On Tue, Jan 19, 2016 at 11:29 AM, Maximilian Michels 
> wrote:
>
> > Pleased to have you with us Chengxiang!
> >
> > Cheers,
> > Max
> >
> > On Tue, Jan 19, 2016 at 11:13 AM, Chiwan Park 
> > wrote:
> > > Congrats! Welcome Chengxiang Li!
> > >
> > >> On Jan 19, 2016, at 7:13 PM, Vasiliki Kalavri <
> > vasilikikala...@gmail.com> wrote:
> > >>
> > >> Congratulations! Welcome Chengxiang Li!
> > >>
> > >> On 19 January 2016 at 11:02, Fabian Hueske  wrote:
> > >>
> > >>> Hi everybody,
> > >>>
> > >>> I'd like to announce that Chengxiang Li accepted the PMC's offer to
> > become
> > >>> a committer of the Apache Flink project.
> > >>>
> > >>> Please join me in welcoming Chengxiang Li!
> > >>>
> > >>> Best, Fabian
> > >>>
> > >
> > > Regards,
> > > Chiwan Park
> > >
> >
>


Re: [ANNOUNCE] Chengxiang Li added as committer

2016-01-19 Thread Matthias J. Sax
Congrats and welcome Chengxiang!! :)

On 01/19/2016 12:56 PM, Kostas Tzoumas wrote:
> Welcome Chengxiang!!
> 
> On Tue, Jan 19, 2016 at 12:31 PM, Stephan Ewen  wrote:
> 
>> Good to have you on board!
>>
>> On Tue, Jan 19, 2016 at 11:29 AM, Maximilian Michels 
>> wrote:
>>
>>> Pleased to have you with us Chengxiang!
>>>
>>> Cheers,
>>> Max
>>>
>>> On Tue, Jan 19, 2016 at 11:13 AM, Chiwan Park 
>>> wrote:
 Congrats! Welcome Chengxiang Li!

> On Jan 19, 2016, at 7:13 PM, Vasiliki Kalavri <
>>> vasilikikala...@gmail.com> wrote:
>
> Congratulations! Welcome Chengxiang Li!
>
> On 19 January 2016 at 11:02, Fabian Hueske  wrote:
>
>> Hi everybody,
>>
>> I'd like to announce that Chengxiang Li accepted the PMC's offer to
>>> become
>> a committer of the Apache Flink project.
>>
>> Please join me in welcoming Chengxiang Li!
>>
>> Best, Fabian
>>

 Regards,
 Chiwan Park

>>>
>>
> 



signature.asc
Description: OpenPGP digital signature


[jira] [Created] (FLINK-3263) Log task statistics on TaskManager

2016-01-19 Thread Greg Hogan (JIRA)
Greg Hogan created FLINK-3263:
-

 Summary: Log task statistics on TaskManager
 Key: FLINK-3263
 URL: https://issues.apache.org/jira/browse/FLINK-3263
 Project: Flink
  Issue Type: Improvement
  Components: TaskManager
Affects Versions: 1.0.0
Reporter: Greg Hogan
Assignee: Greg Hogan
Priority: Minor


Similar to how memory statistics can be written to the TaskMangers' log files 
by configuring {{taskmanager.debug.memory.startLogThread}} and 
{{taskmanager.debug.memory.logIntervalMs}}, it would be useful to have 
statistics written for each task within a job.

One use case is to reconstruct progress to analyze why TaskManagers take 
different amounts of time to process the same quantity of data.

I envision this being the same statistics which are displayed on the web 
frontend.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


Re: [DISCUSS] Git force pushing and deletion of branchs

2016-01-19 Thread Ufuk Celebi
Thank you :D

On Tue, Jan 19, 2016 at 6:41 PM, Till Rohrmann  wrote:

> Thanks Max :-)
>
> On Tue, Jan 19, 2016 at 6:05 PM, Fabian Hueske  wrote:
>
> > Thanks Max!
> >
> > 2016-01-19 18:04 GMT+01:00 Maximilian Michels :
> >
> > > I've filed an issue at infra to protect the master:
> > > https://issues.apache.org/jira/browse/INFRA-11088
> > >
> > > On Fri, Jan 15, 2016 at 3:40 PM, Maximilian Michels 
> > > wrote:
> > > > +1 for a protected master.
> > > > +1 for creating release tags under rel/.
> > > >
> > > > On Thu, Jan 14, 2016 at 10:07 AM, Chiwan Park  >
> > > wrote:
> > > >> +1 for protecting all branches including master.
> > > >>
> > > >>> On Jan 14, 2016, at 1:20 AM, Aljoscha Krettek  >
> > > wrote:
> > > >>>
> > > >>> +1 on protecting the master
> > >  On 13 Jan 2016, at 14:46, Márton Balassi <
> balassi.mar...@gmail.com>
> > > wrote:
> > > 
> > >  +1
> > > 
> > >  On Wed, Jan 13, 2016 at 12:37 PM, Matthias J. Sax <
> mj...@apache.org
> > >
> > > wrote:
> > > 
> > > > +1
> > > >
> > > > On 01/13/2016 11:51 AM, Fabian Hueske wrote:
> > > >> @Stephan: You mean all tags should be protected, not only those
> > > under
> > > > rel?
> > > >>
> > > >> 2016-01-13 11:43 GMT+01:00 Till Rohrmann  >:
> > > >>
> > > >>> +1 for protecting the master branch.
> > > >>>
> > > >>> On Wed, Jan 13, 2016 at 11:42 AM, Li, Chengxiang <
> > > > chengxiang...@intel.com>
> > > >>> wrote:
> > > >>>
> > >  +1 on the original style.
> > >  Master branch disable force pushing in case of misusing and
> > > feature
> > > >>> branch
> > >  enable force pushing for flexible developing.
> > > 
> > >  -Original Message-
> > >  From: Gyula Fóra [mailto:gyf...@apache.org]
> > >  Sent: Wednesday, January 13, 2016 6:36 PM
> > >  To: dev@flink.apache.org
> > >  Subject: Re: [DISCUSS] Git force pushing and deletion of
> branchs
> > > 
> > >  +1 for protecting the master branch.
> > > 
> > >  I also don't see any reason why anyone should force push there
> > > 
> > >  Gyula
> > > 
> > >  Fabian Hueske  ezt írta (időpont: 2016.
> jan.
> > > 13.,
> > > >>> Sze,
> > >  11:07):
> > > 
> > > > Hi everybody,
> > > >
> > > > Lately, ASF Infra has changed the write permissions of all
> Git
> > > > repositories twice.
> > > >
> > > > Originally, it was not possible to force into the master
> > branch.
> > > > A few weeks ago, infra disabled also force pushing into other
> > > > branches.
> > > >
> > > > Now, this has changed again after the issue was discussed
> with
> > > the ASF
> > > > board.
> > > > The current situation is the following:
> > > > - force pushing is allowed on all branched, including master
> > > > - branches and tags can be deleted (not sure if this applies
> as
> > > well
> > > > for the master branch)
> > > > - "the 'protected' portions of git to primarily focus on
> > > refs/tags/rel
> > > > - thus any tags under rel, will have their entire commit
> > > history."
> > > >
> > > > I am not 100% sure which exact parts of the repository are
> > > protected
> > > > now as I am not very much into the details of Git.
> > > > However, I believe we need to create new tags under rel for
> our
> > > > previous releases to protect them.
> > > >
> > > > In addition, I would like to propose to ask Infra to add
> > > protection
> > > > for the master branch. I can only recall very few situations
> > > where
> > > > changes had to be reverted. I am much more in favor of a
> > > reverting
> > > > commit now and then compared to a branch that can be
> > arbitrarily
> > > >>> changed.
> > > >
> > > > What do you think about this?
> > > >
> > > > Best, Fabian
> > > >
> > > >>
> > > >> Regards,
> > > >> Chiwan Park
> > > >>
> > > >>
> > >
> >
>


Re: Questions for a better understanding of the internals of data exchange between tasks

2016-01-19 Thread Ufuk Celebi
On Tue, Jan 19, 2016 at 11:37 AM, Camelia Elena Ciolac 
wrote:

> Hello,
>
> I list some questions gathered while reading documentation on Flink's
> internals and I am grateful to receive your answers.
>
> 1) How is the JobManager involved in the communication between tasks
> running in task slots on TaskManagers?
>
> From [1] it appears to me that, as part of the control flow for data
> exchange, every time a task has a a result that is consumable, it notifies
> the JobManager
> This is repeated in [2] "When producing an intermediate result, the
> producing task is responsible to notify the JobManager about available
> data".
> However, the result's size might as well be 1, in the case of data
> streams, so I ask this question to understand better if for each data
> exchange the JobManager is involved.
>

The JobManager is involved when the result is consumable. Depending on the
result characteristic this is either the case when the first record/buffer
has been produced (pipelined results) or when all records/buffers have been
produced (blocking results). The notification can either kick of the
deployment of the consuming tasks or update them with more specific
information about the location of the to be consumed results. The
notification does not happen for each buffer that is transferred. The data
exchange happens from task manager to task manager via a different TCP
connection.


> 2) In case my understanding of the aforementioned control flow is correct,
> then from the following options, which is the data outcome that triggers a
> notification of the JobManager: a ResultPartition, a ResultSubpartition or
> a Buffer?
>

The ResultPartition having at least one (pipelined) or all (blocking)
buffers produced. The sub partitions contain all data for a parallel sub
task consumer (for example reducer sub task 1 out of a total of 4 reducer
sub tasks).

3) Idem, then are the Akka actors involved in this notification of data
> availability for consumption?
>

Yes, all distributed coordinations happens via Akka. The actual data
exchange happens via a custom TCP stack with Netty. As an example: the Akka
messages contain information about which result to request from which task
manager and the request and data transfer happens via TCP.


> 4) How is orchestrated this co-existence of receiver-initiated data
> transfers (the consumer requests the data partitions) with the push data
> transfers [2] between 2 tasks ?
>

The JobManager acts as the coordinator of the system and holds all required
information in the ExecutionGraph data structure. The execution graph is an
asynchronous state machine used for scheduling and tracking the progress of
deployed tasks. During deployment, the tasks either know where to request
data from or they get updates during runtime if the result location is not
known at deployment time.


> From [3] I retain the paragraph:
>
> "The JobManager also acts as the input split assigner for data sources. It
> is responsible for distributing the work across all TaskManager such that
> data locality is preserved where possible. In order to dynamically balance
> the load, the Tasks request a new input split after they have finished
> processing the old one. This request is realized by sending a
> RequestNextInputSplit to the JobManager. The JobManager responds with a
> NextInputSplit message. If there are no more input splits, then the input
> split contained in the message is null.
>
> The Tasks are deployed lazily to the TaskManagers. This means that tasks
> which consume data are only deployed after one of its producers has
> finished producing some data. Once the producer has done so, it sends a
> ScheduleOrUpdateConsumers message to the JobManager. This messages says
> that the consumer can now read the newly produced data. If the consuming
> task is not yet running, it will be deployed to a TaskManager."
>
> 5) Can you please summarize how this data exchange occurs in Flink, based
> on an example?
>

WordCount with parallelism 2:

[ Source -> Map 0 ] |-> ResultPartition 0 with 2 sub partitions <--+--- [
Reduce 0 -> Sink 0 ]
[ Source -> Map 1 ] |-> ResultPartition 1 with 2 sub partitions <--+--- [
Reduce 1 -> Sink 1 ]

Assume that Result 0 is pipelined. At runtime, result 0 is made up from of
two ResultPartitions (for each parallel Source->Map pipeline) with two sub
partitions (for each consuming Reduce->Sink pipeline). The JobManager
schedules the source tasks (which are chained with the mappers). As soon as
the Source->Map pipeline produces the first data, the job manager will
receive an Akka message to deploy the consuming tasks (it will receive
multiple of these, of which one will be the first). Because Result 0 is
consumed in an all-to-all fashion, this triggers the deployment of both
reducers. At this point the JobManager tries to schedule the reducers,
which then request their respective sub partitions from both
ResultPartition 0 and 1, e.g. [Reduce 0->Sink 0] 

Fwd: integration with a scheduler

2016-01-19 Thread Serkan Taş
> 
> Hi,
> 
> I am planning to integrate flink with our job scheduler product to execute
> jobs - especially bathc like - on flink which may be the part of some
> other DAG style job chain.
> 
> I need some control ablities like start, stop, suspend, get status...
> 
> Where shold i go through ?
> 
> -- 
> Serkan Tas
> Likya Bilgi Teknolojileri
> ve Iletişim Hiz. Ltd.
> www.likyateknoloji.com
> Tel : 0 216 471 81 55
> Gsm : 0 542 242 00 92
> Faks:  0 216 661 14 92
> 
> --
> Bu elektronik posta ve onunla iletilen bütün dosyalar gizlidir. Sadece
> yukarıda isimleri belirtilen kişiler arasında özel haberleşme amacını
> taşımaktadır. Size yanlışlıkla ulaşmışsa bu elektonik postanın içeriğini
> açıklamanız, kopyalamanız, yönlendirmeniz ve kullanmanız kesinlikle
> yasaktır. Lütfen mesajı geri gönderiniz ve sisteminizden siliniz. Likya
> Bilgi Teknolojileri ve İletişim Hiz. Ltd. Şti. bu mesajın içeriği ile
> ilgili olarak hiç bir hukuksal sorumluluğu kabul etmez.
> 
> This electonic mail and any files transmitted with it are intended for the
> private use of  the persons named above. If you received this message in
> error, forwarding, copying or use of any of the information is strictly
> prohibited. Please immediately notify the sender and delete it from your
> system. Likya Bilgi Teknolojileri ve İletişim Hiz. Ltd. Şti. does not
> accept legal responsibility for the contents of this message.
> --
> 
> 
> Bu e-postayı yazdırmadan önce, çevreye olan sorumluluğunuzu tekrar düşünün.
> Please consider your environmental responsibility before printing this
> e-mail.
> 
> 
> 



Serkan Taş
Mobil : +90 532 250 07 71
Likya Bilgi Teknolojileri
ve İletişim Hiz. Ltd. Şti.
www.likyateknoloji.com 
 
--
Bu elektronik posta ve onunla iletilen bütün dosyalar gizlidir. Sadece yukarıda 
isimleri belirtilen kişiler arasında özel haberleşme amacını taşımaktadır. Size 
yanlışlıkla ulaşmışsa bu elektonik postanın içeriğini açıklamanız, 
kopyalamanız, yönlendirmeniz ve kullanmanız kesinlikle yasaktır. Lütfen mesajı 
geri gönderiniz ve sisteminizden siliniz. Likya Bilgi Teknolojileri ve İletişim 
Hiz. Ltd. Şti. bu mesajın içeriği ile ilgili olarak hiç bir hukuksal 
sorumluluğu kabul etmez.
 
This electronic mail and any files transmitted with it are intended for the 
private use of  the persons named above. If you received this message in error, 
forwarding, copying or use of any of the information is strictly prohibited. 
Please immediately notify the sender and delete it from your system. Likya 
Bilgi Teknolojileri ve İletişim Hiz. Ltd. Şti. does not accept legal 
responsibility for the contents of this message.
--








P
Bu e-postayı yazdırmadan önce, çevreye olan sorumluluğunuzu tekrar düşünün.
Please consider your environmental responsibility before printing this e-mail.
 



Re: Weird test-source issue

2016-01-19 Thread Stephan Ewen
Yeah, we saw this as well this morning, in a job that triggers checkpoints
super fast (50msecs).

I think we have a good fix figured out, let's solve this for 1.0...

On Tue, Jan 19, 2016 at 3:25 PM, Gyula Fóra  wrote:

> I just got back to this issue. The problem wasn't with the locking but that
> the StreamTask wasn't in running state before the first checkpoint trigger
> message.
> I actually just saw your JIRA as well, funny... :)
>
> Regards,
> Gyula
>
> Stephan Ewen  ezt írta (időpont: 2016. jan. 8., P,
> 15:36):
>
> > Hmm, strange issue indeed.
> >
> > So, checkpoints are definitely triggered (log message by coordinator to
> > trigger checkpoint) but are not completing?
> > Can you check which is the first checkpoint to complete? Is it Checkpoint
> > 1, or a later one (indicating that checkpoint 1 was somehow subsumed).
> >
> > Can you check in the stacktrace on which lock the checkpoint runables are
> > waiting, and who is holding that lock?
> >
> > Two thoughts:
> >
> > 1) What I mistakenly did once in one of my tests is to have the sleep()
> in
> > a downstream task. That would simply prevent the fast generated data
> > elements (and the inline checkpoint barriers) from passing though and
> > completing the checkpoint.
> >
> > 2) Is this another issue with the non-fair lock? Does the checkpoint
> > runnable simply not get the lock before the checkpoint. Not sure why it
> > would suddenly work after the failure. We could try and swap the lock
> > Object by a "ReentrantLock(true)" and see what would happen.
> >
> >
> > Stephan
> >
> >
> > On Fri, Jan 8, 2016 at 11:49 AM, Gyula Fóra  wrote:
> >
> > > Hey,
> > >
> > > I have encountered a weird issue in a checkpointing test I am trying to
> > > write. The logic is the same as with the previous checkpointing tests,
> > > there is a OnceFailingReducer.
> > >
> > > My problem is that before the reducer fails, my job cannot take any
> > > snapshots. The Runnables executing the checkpointing logic in the
> sources
> > > keep waiting on some lock.
> > >
> > > After the failure and the restart, everything is fine and the
> > checkpointing
> > > can succeed properly.
> > >
> > > Also if I remove the failure from the reducer, the job doesnt take any
> > > snapshots (waiting on lock) and the job will finish.
> > >
> > > Here is the code:
> > >
> > >
> >
> https://github.com/gyfora/flink/blob/d1f12c2474413c9af357b6da33f1fac30549fbc3/flink-contrib/flink-streaming-contrib/src/test/java/org/apache/flink/contrib/streaming/state/TfileStateCheckpointingTest.java#L83
> > >
> > > I assume there is no problem with the source as the Thread.sleep(..) is
> > > outside of the synchronized block. (and as I said after the failure it
> > > works fine).
> > >
> > > Any ideas?
> > >
> > > Thanks,
> > > Gyula
> > >
> >
>


Re: Weird test-source issue

2016-01-19 Thread Stephan Ewen
It is nice to see that we converge on the issues we find.

Means that this is getting pretty stable :-)

On Tue, Jan 19, 2016 at 8:17 PM, Stephan Ewen  wrote:

> Yeah, we saw this as well this morning, in a job that triggers checkpoints
> super fast (50msecs).
>
> I think we have a good fix figured out, let's solve this for 1.0...
>
> On Tue, Jan 19, 2016 at 3:25 PM, Gyula Fóra  wrote:
>
>> I just got back to this issue. The problem wasn't with the locking but
>> that
>> the StreamTask wasn't in running state before the first checkpoint trigger
>> message.
>> I actually just saw your JIRA as well, funny... :)
>>
>> Regards,
>> Gyula
>>
>> Stephan Ewen  ezt írta (időpont: 2016. jan. 8., P,
>> 15:36):
>>
>> > Hmm, strange issue indeed.
>> >
>> > So, checkpoints are definitely triggered (log message by coordinator to
>> > trigger checkpoint) but are not completing?
>> > Can you check which is the first checkpoint to complete? Is it
>> Checkpoint
>> > 1, or a later one (indicating that checkpoint 1 was somehow subsumed).
>> >
>> > Can you check in the stacktrace on which lock the checkpoint runables
>> are
>> > waiting, and who is holding that lock?
>> >
>> > Two thoughts:
>> >
>> > 1) What I mistakenly did once in one of my tests is to have the sleep()
>> in
>> > a downstream task. That would simply prevent the fast generated data
>> > elements (and the inline checkpoint barriers) from passing though and
>> > completing the checkpoint.
>> >
>> > 2) Is this another issue with the non-fair lock? Does the checkpoint
>> > runnable simply not get the lock before the checkpoint. Not sure why it
>> > would suddenly work after the failure. We could try and swap the lock
>> > Object by a "ReentrantLock(true)" and see what would happen.
>> >
>> >
>> > Stephan
>> >
>> >
>> > On Fri, Jan 8, 2016 at 11:49 AM, Gyula Fóra  wrote:
>> >
>> > > Hey,
>> > >
>> > > I have encountered a weird issue in a checkpointing test I am trying
>> to
>> > > write. The logic is the same as with the previous checkpointing tests,
>> > > there is a OnceFailingReducer.
>> > >
>> > > My problem is that before the reducer fails, my job cannot take any
>> > > snapshots. The Runnables executing the checkpointing logic in the
>> sources
>> > > keep waiting on some lock.
>> > >
>> > > After the failure and the restart, everything is fine and the
>> > checkpointing
>> > > can succeed properly.
>> > >
>> > > Also if I remove the failure from the reducer, the job doesnt take any
>> > > snapshots (waiting on lock) and the job will finish.
>> > >
>> > > Here is the code:
>> > >
>> > >
>> >
>> https://github.com/gyfora/flink/blob/d1f12c2474413c9af357b6da33f1fac30549fbc3/flink-contrib/flink-streaming-contrib/src/test/java/org/apache/flink/contrib/streaming/state/TfileStateCheckpointingTest.java#L83
>> > >
>> > > I assume there is no problem with the source as the Thread.sleep(..)
>> is
>> > > outside of the synchronized block. (and as I said after the failure it
>> > > works fine).
>> > >
>> > > Any ideas?
>> > >
>> > > Thanks,
>> > > Gyula
>> > >
>> >
>>
>
>


RE: Questions for a better understanding of the internals of data exchange between tasks

2016-01-19 Thread Camelia Elena Ciolac
Dear Ufuk,

Thank you very much for this detailed explanation, it helped me understand.
So, many many thanks!

Camelia

 

From: Ufuk Celebi [u...@apache.org]
Sent: Tuesday, January 19, 2016 7:16 PM
To: dev@flink.apache.org
Subject: Re: Questions for a better understanding of the internals of data 
exchange between tasks

On Tue, Jan 19, 2016 at 11:37 AM, Camelia Elena Ciolac 
wrote:

> Hello,
>
> I list some questions gathered while reading documentation on Flink's
> internals and I am grateful to receive your answers.
>
> 1) How is the JobManager involved in the communication between tasks
> running in task slots on TaskManagers?
>
> From [1] it appears to me that, as part of the control flow for data
> exchange, every time a task has a a result that is consumable, it notifies
> the JobManager
> This is repeated in [2] "When producing an intermediate result, the
> producing task is responsible to notify the JobManager about available
> data".
> However, the result's size might as well be 1, in the case of data
> streams, so I ask this question to understand better if for each data
> exchange the JobManager is involved.
>

The JobManager is involved when the result is consumable. Depending on the
result characteristic this is either the case when the first record/buffer
has been produced (pipelined results) or when all records/buffers have been
produced (blocking results). The notification can either kick of the
deployment of the consuming tasks or update them with more specific
information about the location of the to be consumed results. The
notification does not happen for each buffer that is transferred. The data
exchange happens from task manager to task manager via a different TCP
connection.


> 2) In case my understanding of the aforementioned control flow is correct,
> then from the following options, which is the data outcome that triggers a
> notification of the JobManager: a ResultPartition, a ResultSubpartition or
> a Buffer?
>

The ResultPartition having at least one (pipelined) or all (blocking)
buffers produced. The sub partitions contain all data for a parallel sub
task consumer (for example reducer sub task 1 out of a total of 4 reducer
sub tasks).

3) Idem, then are the Akka actors involved in this notification of data
> availability for consumption?
>

Yes, all distributed coordinations happens via Akka. The actual data
exchange happens via a custom TCP stack with Netty. As an example: the Akka
messages contain information about which result to request from which task
manager and the request and data transfer happens via TCP.


> 4) How is orchestrated this co-existence of receiver-initiated data
> transfers (the consumer requests the data partitions) with the push data
> transfers [2] between 2 tasks ?
>

The JobManager acts as the coordinator of the system and holds all required
information in the ExecutionGraph data structure. The execution graph is an
asynchronous state machine used for scheduling and tracking the progress of
deployed tasks. During deployment, the tasks either know where to request
data from or they get updates during runtime if the result location is not
known at deployment time.


> From [3] I retain the paragraph:
>
> "The JobManager also acts as the input split assigner for data sources. It
> is responsible for distributing the work across all TaskManager such that
> data locality is preserved where possible. In order to dynamically balance
> the load, the Tasks request a new input split after they have finished
> processing the old one. This request is realized by sending a
> RequestNextInputSplit to the JobManager. The JobManager responds with a
> NextInputSplit message. If there are no more input splits, then the input
> split contained in the message is null.
>
> The Tasks are deployed lazily to the TaskManagers. This means that tasks
> which consume data are only deployed after one of its producers has
> finished producing some data. Once the producer has done so, it sends a
> ScheduleOrUpdateConsumers message to the JobManager. This messages says
> that the consumer can now read the newly produced data. If the consuming
> task is not yet running, it will be deployed to a TaskManager."
>
> 5) Can you please summarize how this data exchange occurs in Flink, based
> on an example?
>

WordCount with parallelism 2:

[ Source -> Map 0 ] |-> ResultPartition 0 with 2 sub partitions <--+--- [
Reduce 0 -> Sink 0 ]
[ Source -> Map 1 ] |-> ResultPartition 1 with 2 sub partitions <--+--- [
Reduce 1 -> Sink 1 ]

Assume that Result 0 is pipelined. At runtime, result 0 is made up from of
two ResultPartitions (for each parallel Source->Map pipeline) with two sub
partitions (for each consuming Reduce->Sink pipeline). The JobManager
schedules the source tasks (which are chained with the mappers). As soon as
the Source->Map pipeline produces the first data, the job manager will
receive an Akka message to deploy 

Re: What is the topic & offset used for in KeyedDeserializationSchema.deserialize()?

2016-01-19 Thread Tzu-Li (Gordon) Tai
Hi Stephan,

A comment on this. For KeyedDeserializationSchema, I don't think it is
necessary.
As previously explained, the interfaces for the KeyedDeserializationSchema
of Kafka / Kinesis can be quite different, and may also be specific for
future external systems that we might implement connectors to. Wrapper
classes for a common KeyedDeserializationSchema doesn't seem to make sense,
since in the end we will still need to expose system-specific interfaces for
the user.

It may be reasonable to keep the most simple DeSerializationSchema
interfaces and wrappers in flink-streaming-java. By a simple
KeyDeserializationSchema, I mean deserialize() methods that only take key as
byte[] and message as byte[]. If new connectors happen to require more
specific interfaces, then they create them in their own module
(flink-connector-*).

Cheers,
Gordon



--
View this message in context: 
http://apache-flink-mailing-list-archive.1008284.n3.nabble.com/What-is-the-topic-offset-used-for-in-KeyedDeserializationSchema-deserialize-tp9911p9944.html
Sent from the Apache Flink Mailing List archive. mailing list archive at 
Nabble.com.


[jira] [Created] (FLINK-3260) ExecutionGraph gets stuck in state FAILING

2016-01-19 Thread Stephan Ewen (JIRA)
Stephan Ewen created FLINK-3260:
---

 Summary: ExecutionGraph gets stuck in state FAILING
 Key: FLINK-3260
 URL: https://issues.apache.org/jira/browse/FLINK-3260
 Project: Flink
  Issue Type: Bug
  Components: JobManager
Affects Versions: 0.10.1
Reporter: Stephan Ewen
Priority: Blocker
 Fix For: 1.0.0


It is a bit of a rare case, but the following can currently happen:

  1. Jobs runs for a while, some tasks are already finished.
  2. Job fails, goes to state failing and restarting. Non-finished tasks fail 
or are canceled.
  3. For the finished tasks, ask-futures from certain messages (for example for 
releasing intermediate result partitions) can fail (timeout) and cause the 
execution to go from FINISHED to FAILED
  4. This triggers the execution graph to go to FAILING without ever going 
further into RESTARTING again
  5. The job is stuck

It initially looks like this is mainly an issue for batch jobs (jobs where 
tasks do finish, rather than run infinitely).

The log that shows how this manifests:
{code}

17:19:19,782 INFO  akka.event.slf4j.Slf4jLogger 
 - Slf4jLogger started
17:19:19,844 INFO  Remoting 
 - Starting remoting
17:19:20,065 INFO  Remoting 
 - Remoting started; listening on addresses :[akka.tcp://flink@127.0.0.1:56722]
17:19:20,090 INFO  org.apache.flink.runtime.blob.BlobServer 
 - Created BLOB server storage directory 
/tmp/blobStore-6766f51a-1c51-4a03-acfb-08c2c29c11f0
17:19:20,096 INFO  org.apache.flink.runtime.blob.BlobServer 
 - Started BLOB server at 0.0.0.0:43327 - max concurrent requests: 50 - max 
backlog: 1000
17:19:20,113 INFO  org.apache.flink.runtime.jobmanager.MemoryArchivist  
 - Started memory archivist akka://flink/user/archive
17:19:20,115 INFO  org.apache.flink.runtime.checkpoint.SavepointStoreFactory
 - No savepoint state backend configured. Using job manager savepoint state 
backend.
17:19:20,118 INFO  org.apache.flink.runtime.jobmanager.JobManager   
 - Starting JobManager at akka.tcp://flink@127.0.0.1:56722/user/jobmanager.
17:19:20,123 INFO  org.apache.flink.runtime.jobmanager.JobManager   
 - JobManager akka.tcp://flink@127.0.0.1:56722/user/jobmanager was granted 
leadership with leader session ID None.
17:19:25,605 INFO  org.apache.flink.runtime.instance.InstanceManager
 - Registered TaskManager at testing-worker-linux-docker-e6d6931f-3200-linux-4 
(akka.tcp://flink@172.17.0.253:43702/user/taskmanager) as 
f213232054587f296a12140d56f63ed1. Current number of registered hosts is 1. 
Current number of alive task slots is 2.
17:19:26,758 INFO  org.apache.flink.runtime.instance.InstanceManager
 - Registered TaskManager at testing-worker-linux-docker-e6d6931f-3200-linux-4 
(akka.tcp://flink@172.17.0.253:43956/user/taskmanager) as 
f9e78baa14fb38c69517fb1bcf4f419c. Current number of registered hosts is 2. 
Current number of alive task slots is 4.
17:19:27,064 INFO  org.apache.flink.api.java.ExecutionEnvironment   
 - The job has 0 registered types and 0 default Kryo serializers
17:19:27,071 INFO  org.apache.flink.client.program.Client   
 - Starting client actor system
17:19:27,072 INFO  org.apache.flink.runtime.client.JobClient
 - Starting JobClient actor system
17:19:27,110 INFO  akka.event.slf4j.Slf4jLogger 
 - Slf4jLogger started
17:19:27,121 INFO  Remoting 
 - Starting remoting
17:19:27,143 INFO  org.apache.flink.runtime.client.JobClient
 - Started JobClient actor system at 127.0.0.1:51198
17:19:27,145 INFO  Remoting 
 - Remoting started; listening on addresses :[akka.tcp://flink@127.0.0.1:51198]
17:19:27,325 INFO  org.apache.flink.runtime.client.JobClientActor   
 - Disconnect from JobManager null.
17:19:27,362 INFO  org.apache.flink.runtime.client.JobClientActor   
 - Received job Flink Java Job at Mon Jan 18 17:19:27 UTC 2016 
(fa05fd25993a8742da09cc5023c1e38d).
17:19:27,362 INFO  org.apache.flink.runtime.client.JobClientActor   
 - Could not submit job Flink Java Job at Mon Jan 18 17:19:27 UTC 2016 
(fa05fd25993a8742da09cc5023c1e38d), because there is no connection to a 
JobManager.
17:19:27,379 INFO  org.apache.flink.runtime.client.JobClientActor   
 - Connect to JobManager 
Actor[akka.tcp://flink@127.0.0.1:56722/user/jobmanager#-1489998809].
17:19:27,379 INFO  org.apache.flink.runtime.client.JobClientActor   
 - Connected to new JobManager 

Re: Weird test-source issue

2016-01-19 Thread Gyula Fóra
I just got back to this issue. The problem wasn't with the locking but that
the StreamTask wasn't in running state before the first checkpoint trigger
message.
I actually just saw your JIRA as well, funny... :)

Regards,
Gyula

Stephan Ewen  ezt írta (időpont: 2016. jan. 8., P, 15:36):

> Hmm, strange issue indeed.
>
> So, checkpoints are definitely triggered (log message by coordinator to
> trigger checkpoint) but are not completing?
> Can you check which is the first checkpoint to complete? Is it Checkpoint
> 1, or a later one (indicating that checkpoint 1 was somehow subsumed).
>
> Can you check in the stacktrace on which lock the checkpoint runables are
> waiting, and who is holding that lock?
>
> Two thoughts:
>
> 1) What I mistakenly did once in one of my tests is to have the sleep() in
> a downstream task. That would simply prevent the fast generated data
> elements (and the inline checkpoint barriers) from passing though and
> completing the checkpoint.
>
> 2) Is this another issue with the non-fair lock? Does the checkpoint
> runnable simply not get the lock before the checkpoint. Not sure why it
> would suddenly work after the failure. We could try and swap the lock
> Object by a "ReentrantLock(true)" and see what would happen.
>
>
> Stephan
>
>
> On Fri, Jan 8, 2016 at 11:49 AM, Gyula Fóra  wrote:
>
> > Hey,
> >
> > I have encountered a weird issue in a checkpointing test I am trying to
> > write. The logic is the same as with the previous checkpointing tests,
> > there is a OnceFailingReducer.
> >
> > My problem is that before the reducer fails, my job cannot take any
> > snapshots. The Runnables executing the checkpointing logic in the sources
> > keep waiting on some lock.
> >
> > After the failure and the restart, everything is fine and the
> checkpointing
> > can succeed properly.
> >
> > Also if I remove the failure from the reducer, the job doesnt take any
> > snapshots (waiting on lock) and the job will finish.
> >
> > Here is the code:
> >
> >
> https://github.com/gyfora/flink/blob/d1f12c2474413c9af357b6da33f1fac30549fbc3/flink-contrib/flink-streaming-contrib/src/test/java/org/apache/flink/contrib/streaming/state/TfileStateCheckpointingTest.java#L83
> >
> > I assume there is no problem with the source as the Thread.sleep(..) is
> > outside of the synchronized block. (and as I said after the failure it
> > works fine).
> >
> > Any ideas?
> >
> > Thanks,
> > Gyula
> >
>


Re: [ANNOUNCE] Chengxiang Li added as committer

2016-01-19 Thread Paris Carbone
Congrats Chengxiang! Really pleased to have you on board

> On 19 Jan 2016, at 13:16, Matthias J. Sax  wrote:
> 
> Congrats and welcome Chengxiang!! :)
> 
> On 01/19/2016 12:56 PM, Kostas Tzoumas wrote:
>> Welcome Chengxiang!!
>> 
>> On Tue, Jan 19, 2016 at 12:31 PM, Stephan Ewen  wrote:
>> 
>>> Good to have you on board!
>>> 
>>> On Tue, Jan 19, 2016 at 11:29 AM, Maximilian Michels 
>>> wrote:
>>> 
 Pleased to have you with us Chengxiang!
 
 Cheers,
 Max
 
 On Tue, Jan 19, 2016 at 11:13 AM, Chiwan Park 
 wrote:
> Congrats! Welcome Chengxiang Li!
> 
>> On Jan 19, 2016, at 7:13 PM, Vasiliki Kalavri <
 vasilikikala...@gmail.com> wrote:
>> 
>> Congratulations! Welcome Chengxiang Li!
>> 
>> On 19 January 2016 at 11:02, Fabian Hueske  wrote:
>> 
>>> Hi everybody,
>>> 
>>> I'd like to announce that Chengxiang Li accepted the PMC's offer to
 become
>>> a committer of the Apache Flink project.
>>> 
>>> Please join me in welcoming Chengxiang Li!
>>> 
>>> Best, Fabian
>>> 
> 
> Regards,
> Chiwan Park
> 
 
>>> 
>> 
> 



[jira] [Created] (FLINK-3261) Tasks should eagerly report back when they cannot start a checkpoint

2016-01-19 Thread Stephan Ewen (JIRA)
Stephan Ewen created FLINK-3261:
---

 Summary: Tasks should eagerly report back when they cannot start a 
checkpoint
 Key: FLINK-3261
 URL: https://issues.apache.org/jira/browse/FLINK-3261
 Project: Flink
  Issue Type: Bug
  Components: Distributed Runtime
Affects Versions: 0.10.1
Reporter: Stephan Ewen
Priority: Blocker
 Fix For: 1.0.0


With very fast checkpoint intervals (few 100 msecs), it can happen that a Task 
is not ready to start a checkpoint by the time it gets the first checkpoint 
trigger message.

If some other tasks are ready already and commence a checkpoint, the stream 
alignment will make the non-participating task wait until the checkpoint 
expires (default: 10 minutes).

A simple way to fix this is that tasks report back when they could not start a 
checkpoint. The checkpoint coordinator can then abort that checkpoint and 
unblock the streams by starting new checkpoint (where all tasks will 
participate).

An optimization would be to send a special "abort checkpoints barrier" that 
tells the barrier buffers for stream alignment to unblock a checkpoint.




--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


[jira] [Created] (FLINK-3262) Remove fuzzy versioning from Bower dependencies

2016-01-19 Thread Greg Hogan (JIRA)
Greg Hogan created FLINK-3262:
-

 Summary: Remove fuzzy versioning from Bower dependencies
 Key: FLINK-3262
 URL: https://issues.apache.org/jira/browse/FLINK-3262
 Project: Flink
  Issue Type: Improvement
  Components: Webfrontend
Affects Versions: 1.00
Reporter: Greg Hogan
Assignee: Greg Hogan
Priority: Trivial


{{bower.json}} is currently defined with fuzzy versions, i.e. {{"bootstrap": 
"~3.3.5"}}, which silently pull in patch updates. When a user compiles the web 
frontend the new versions are creating changes in the compiled Javascript and 
CSS.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


RE: [ANNOUNCE] Chengxiang Li added as committer

2016-01-19 Thread Li, Chengxiang
Thanks everyone, it's always great to collaborate with you guys, look forward 
to contribute more on Flink.

Thanks
Chengxiang 

-Original Message-
From: Paris Carbone [mailto:par...@kth.se] 
Sent: Tuesday, January 19, 2016 9:24 PM
To: dev@flink.apache.org
Subject: Re: [ANNOUNCE] Chengxiang Li added as committer

Congrats Chengxiang! Really pleased to have you on board

> On 19 Jan 2016, at 13:16, Matthias J. Sax  wrote:
> 
> Congrats and welcome Chengxiang!! :)
> 
> On 01/19/2016 12:56 PM, Kostas Tzoumas wrote:
>> Welcome Chengxiang!!
>> 
>> On Tue, Jan 19, 2016 at 12:31 PM, Stephan Ewen  wrote:
>> 
>>> Good to have you on board!
>>> 
>>> On Tue, Jan 19, 2016 at 11:29 AM, Maximilian Michels 
>>> wrote:
>>> 
 Pleased to have you with us Chengxiang!
 
 Cheers,
 Max
 
 On Tue, Jan 19, 2016 at 11:13 AM, Chiwan Park 
 wrote:
> Congrats! Welcome Chengxiang Li!
> 
>> On Jan 19, 2016, at 7:13 PM, Vasiliki Kalavri <
 vasilikikala...@gmail.com> wrote:
>> 
>> Congratulations! Welcome Chengxiang Li!
>> 
>> On 19 January 2016 at 11:02, Fabian Hueske  wrote:
>> 
>>> Hi everybody,
>>> 
>>> I'd like to announce that Chengxiang Li accepted the PMC's offer to
 become
>>> a committer of the Apache Flink project.
>>> 
>>> Please join me in welcoming Chengxiang Li!
>>> 
>>> Best, Fabian
>>> 
> 
> Regards,
> Chiwan Park
> 
 
>>> 
>> 
> 



Re: Emitting to non-declared output stream

2016-01-19 Thread Matthias J. Sax
Please ignore. Wrong list. Sorry!

On 01/19/2016 03:25 PM, Matthias J. Sax wrote:
> Hi,
> 
> currently, I am using Storm 0.9.3. For first tests on a new topology, I
> use LocalCluster. It happened to me, that I emitted tuples to an output
> stream, that I did never declare (and thus not connect to). For this, I
> would expect an error message in the log. However, I don't get anything
> which makes debugging very hard.
> 
> What do you think about it? Should I open a JIRA for it?
> 
> For real cluster deployment, I think the overhead of checking the output
> stream ID is too large and one can easily see the problem in the UI --
> the non-declared output streams that gets tuples show up there. However,
> for LocalCluster, there is not UI and an error log message would be nice.
> 
> 
> -Matthias
> 



signature.asc
Description: OpenPGP digital signature


Re: [ANNOUNCE] Chengxiang Li added as committer

2016-01-19 Thread Kostas Tzoumas
Usually the first thing is to add yourself to the "team" page :-)

http://flink.apache.org/community.html#people

On Tue, Jan 19, 2016 at 4:29 PM, Li, Chengxiang 
wrote:

> Thanks everyone, it's always great to collaborate with you guys, look
> forward to contribute more on Flink.
>
> Thanks
> Chengxiang
>
> -Original Message-
> From: Paris Carbone [mailto:par...@kth.se]
> Sent: Tuesday, January 19, 2016 9:24 PM
> To: dev@flink.apache.org
> Subject: Re: [ANNOUNCE] Chengxiang Li added as committer
>
> Congrats Chengxiang! Really pleased to have you on board
>
> > On 19 Jan 2016, at 13:16, Matthias J. Sax  wrote:
> >
> > Congrats and welcome Chengxiang!! :)
> >
> > On 01/19/2016 12:56 PM, Kostas Tzoumas wrote:
> >> Welcome Chengxiang!!
> >>
> >> On Tue, Jan 19, 2016 at 12:31 PM, Stephan Ewen 
> wrote:
> >>
> >>> Good to have you on board!
> >>>
> >>> On Tue, Jan 19, 2016 at 11:29 AM, Maximilian Michels 
> >>> wrote:
> >>>
>  Pleased to have you with us Chengxiang!
> 
>  Cheers,
>  Max
> 
>  On Tue, Jan 19, 2016 at 11:13 AM, Chiwan Park 
>  wrote:
> > Congrats! Welcome Chengxiang Li!
> >
> >> On Jan 19, 2016, at 7:13 PM, Vasiliki Kalavri <
>  vasilikikala...@gmail.com> wrote:
> >>
> >> Congratulations! Welcome Chengxiang Li!
> >>
> >> On 19 January 2016 at 11:02, Fabian Hueske 
> wrote:
> >>
> >>> Hi everybody,
> >>>
> >>> I'd like to announce that Chengxiang Li accepted the PMC's offer to
>  become
> >>> a committer of the Apache Flink project.
> >>>
> >>> Please join me in welcoming Chengxiang Li!
> >>>
> >>> Best, Fabian
> >>>
> >
> > Regards,
> > Chiwan Park
> >
> 
> >>>
> >>
> >
>
>


Re: [DISCUSS] Git force pushing and deletion of branchs

2016-01-19 Thread Maximilian Michels
I've filed an issue at infra to protect the master:
https://issues.apache.org/jira/browse/INFRA-11088

On Fri, Jan 15, 2016 at 3:40 PM, Maximilian Michels  wrote:
> +1 for a protected master.
> +1 for creating release tags under rel/.
>
> On Thu, Jan 14, 2016 at 10:07 AM, Chiwan Park  wrote:
>> +1 for protecting all branches including master.
>>
>>> On Jan 14, 2016, at 1:20 AM, Aljoscha Krettek  wrote:
>>>
>>> +1 on protecting the master
 On 13 Jan 2016, at 14:46, Márton Balassi  wrote:

 +1

 On Wed, Jan 13, 2016 at 12:37 PM, Matthias J. Sax  wrote:

> +1
>
> On 01/13/2016 11:51 AM, Fabian Hueske wrote:
>> @Stephan: You mean all tags should be protected, not only those under
> rel?
>>
>> 2016-01-13 11:43 GMT+01:00 Till Rohrmann :
>>
>>> +1 for protecting the master branch.
>>>
>>> On Wed, Jan 13, 2016 at 11:42 AM, Li, Chengxiang <
> chengxiang...@intel.com>
>>> wrote:
>>>
 +1 on the original style.
 Master branch disable force pushing in case of misusing and feature
>>> branch
 enable force pushing for flexible developing.

 -Original Message-
 From: Gyula Fóra [mailto:gyf...@apache.org]
 Sent: Wednesday, January 13, 2016 6:36 PM
 To: dev@flink.apache.org
 Subject: Re: [DISCUSS] Git force pushing and deletion of branchs

 +1 for protecting the master branch.

 I also don't see any reason why anyone should force push there

 Gyula

 Fabian Hueske  ezt írta (időpont: 2016. jan. 13.,
>>> Sze,
 11:07):

> Hi everybody,
>
> Lately, ASF Infra has changed the write permissions of all Git
> repositories twice.
>
> Originally, it was not possible to force into the master branch.
> A few weeks ago, infra disabled also force pushing into other
> branches.
>
> Now, this has changed again after the issue was discussed with the ASF
> board.
> The current situation is the following:
> - force pushing is allowed on all branched, including master
> - branches and tags can be deleted (not sure if this applies as well
> for the master branch)
> - "the 'protected' portions of git to primarily focus on refs/tags/rel
> - thus any tags under rel, will have their entire commit history."
>
> I am not 100% sure which exact parts of the repository are protected
> now as I am not very much into the details of Git.
> However, I believe we need to create new tags under rel for our
> previous releases to protect them.
>
> In addition, I would like to propose to ask Infra to add protection
> for the master branch. I can only recall very few situations where
> changes had to be reverted. I am much more in favor of a reverting
> commit now and then compared to a branch that can be arbitrarily
>>> changed.
>
> What do you think about this?
>
> Best, Fabian
>
>>
>> Regards,
>> Chiwan Park
>>
>>


Re: [DISCUSS] Git force pushing and deletion of branchs

2016-01-19 Thread Fabian Hueske
Thanks Max!

2016-01-19 18:04 GMT+01:00 Maximilian Michels :

> I've filed an issue at infra to protect the master:
> https://issues.apache.org/jira/browse/INFRA-11088
>
> On Fri, Jan 15, 2016 at 3:40 PM, Maximilian Michels 
> wrote:
> > +1 for a protected master.
> > +1 for creating release tags under rel/.
> >
> > On Thu, Jan 14, 2016 at 10:07 AM, Chiwan Park 
> wrote:
> >> +1 for protecting all branches including master.
> >>
> >>> On Jan 14, 2016, at 1:20 AM, Aljoscha Krettek 
> wrote:
> >>>
> >>> +1 on protecting the master
>  On 13 Jan 2016, at 14:46, Márton Balassi 
> wrote:
> 
>  +1
> 
>  On Wed, Jan 13, 2016 at 12:37 PM, Matthias J. Sax 
> wrote:
> 
> > +1
> >
> > On 01/13/2016 11:51 AM, Fabian Hueske wrote:
> >> @Stephan: You mean all tags should be protected, not only those
> under
> > rel?
> >>
> >> 2016-01-13 11:43 GMT+01:00 Till Rohrmann :
> >>
> >>> +1 for protecting the master branch.
> >>>
> >>> On Wed, Jan 13, 2016 at 11:42 AM, Li, Chengxiang <
> > chengxiang...@intel.com>
> >>> wrote:
> >>>
>  +1 on the original style.
>  Master branch disable force pushing in case of misusing and
> feature
> >>> branch
>  enable force pushing for flexible developing.
> 
>  -Original Message-
>  From: Gyula Fóra [mailto:gyf...@apache.org]
>  Sent: Wednesday, January 13, 2016 6:36 PM
>  To: dev@flink.apache.org
>  Subject: Re: [DISCUSS] Git force pushing and deletion of branchs
> 
>  +1 for protecting the master branch.
> 
>  I also don't see any reason why anyone should force push there
> 
>  Gyula
> 
>  Fabian Hueske  ezt írta (időpont: 2016. jan.
> 13.,
> >>> Sze,
>  11:07):
> 
> > Hi everybody,
> >
> > Lately, ASF Infra has changed the write permissions of all Git
> > repositories twice.
> >
> > Originally, it was not possible to force into the master branch.
> > A few weeks ago, infra disabled also force pushing into other
> > branches.
> >
> > Now, this has changed again after the issue was discussed with
> the ASF
> > board.
> > The current situation is the following:
> > - force pushing is allowed on all branched, including master
> > - branches and tags can be deleted (not sure if this applies as
> well
> > for the master branch)
> > - "the 'protected' portions of git to primarily focus on
> refs/tags/rel
> > - thus any tags under rel, will have their entire commit
> history."
> >
> > I am not 100% sure which exact parts of the repository are
> protected
> > now as I am not very much into the details of Git.
> > However, I believe we need to create new tags under rel for our
> > previous releases to protect them.
> >
> > In addition, I would like to propose to ask Infra to add
> protection
> > for the master branch. I can only recall very few situations
> where
> > changes had to be reverted. I am much more in favor of a
> reverting
> > commit now and then compared to a branch that can be arbitrarily
> >>> changed.
> >
> > What do you think about this?
> >
> > Best, Fabian
> >
> >>
> >> Regards,
> >> Chiwan Park
> >>
> >>
>


Re: [DISCUSS] Git force pushing and deletion of branchs

2016-01-19 Thread Till Rohrmann
Thanks Max :-)

On Tue, Jan 19, 2016 at 6:05 PM, Fabian Hueske  wrote:

> Thanks Max!
>
> 2016-01-19 18:04 GMT+01:00 Maximilian Michels :
>
> > I've filed an issue at infra to protect the master:
> > https://issues.apache.org/jira/browse/INFRA-11088
> >
> > On Fri, Jan 15, 2016 at 3:40 PM, Maximilian Michels 
> > wrote:
> > > +1 for a protected master.
> > > +1 for creating release tags under rel/.
> > >
> > > On Thu, Jan 14, 2016 at 10:07 AM, Chiwan Park 
> > wrote:
> > >> +1 for protecting all branches including master.
> > >>
> > >>> On Jan 14, 2016, at 1:20 AM, Aljoscha Krettek 
> > wrote:
> > >>>
> > >>> +1 on protecting the master
> >  On 13 Jan 2016, at 14:46, Márton Balassi 
> > wrote:
> > 
> >  +1
> > 
> >  On Wed, Jan 13, 2016 at 12:37 PM, Matthias J. Sax  >
> > wrote:
> > 
> > > +1
> > >
> > > On 01/13/2016 11:51 AM, Fabian Hueske wrote:
> > >> @Stephan: You mean all tags should be protected, not only those
> > under
> > > rel?
> > >>
> > >> 2016-01-13 11:43 GMT+01:00 Till Rohrmann :
> > >>
> > >>> +1 for protecting the master branch.
> > >>>
> > >>> On Wed, Jan 13, 2016 at 11:42 AM, Li, Chengxiang <
> > > chengxiang...@intel.com>
> > >>> wrote:
> > >>>
> >  +1 on the original style.
> >  Master branch disable force pushing in case of misusing and
> > feature
> > >>> branch
> >  enable force pushing for flexible developing.
> > 
> >  -Original Message-
> >  From: Gyula Fóra [mailto:gyf...@apache.org]
> >  Sent: Wednesday, January 13, 2016 6:36 PM
> >  To: dev@flink.apache.org
> >  Subject: Re: [DISCUSS] Git force pushing and deletion of branchs
> > 
> >  +1 for protecting the master branch.
> > 
> >  I also don't see any reason why anyone should force push there
> > 
> >  Gyula
> > 
> >  Fabian Hueske  ezt írta (időpont: 2016. jan.
> > 13.,
> > >>> Sze,
> >  11:07):
> > 
> > > Hi everybody,
> > >
> > > Lately, ASF Infra has changed the write permissions of all Git
> > > repositories twice.
> > >
> > > Originally, it was not possible to force into the master
> branch.
> > > A few weeks ago, infra disabled also force pushing into other
> > > branches.
> > >
> > > Now, this has changed again after the issue was discussed with
> > the ASF
> > > board.
> > > The current situation is the following:
> > > - force pushing is allowed on all branched, including master
> > > - branches and tags can be deleted (not sure if this applies as
> > well
> > > for the master branch)
> > > - "the 'protected' portions of git to primarily focus on
> > refs/tags/rel
> > > - thus any tags under rel, will have their entire commit
> > history."
> > >
> > > I am not 100% sure which exact parts of the repository are
> > protected
> > > now as I am not very much into the details of Git.
> > > However, I believe we need to create new tags under rel for our
> > > previous releases to protect them.
> > >
> > > In addition, I would like to propose to ask Infra to add
> > protection
> > > for the master branch. I can only recall very few situations
> > where
> > > changes had to be reverted. I am much more in favor of a
> > reverting
> > > commit now and then compared to a branch that can be
> arbitrarily
> > >>> changed.
> > >
> > > What do you think about this?
> > >
> > > Best, Fabian
> > >
> > >>
> > >> Regards,
> > >> Chiwan Park
> > >>
> > >>
> >
>