[jira] [Created] (FLINK-4022) Partition discovery / regex topic subscription for the Kafka consumer

2016-06-05 Thread Tzu-Li (Gordon) Tai (JIRA)
Tzu-Li (Gordon) Tai created FLINK-4022:
--

 Summary: Partition discovery / regex topic subscription for the 
Kafka consumer
 Key: FLINK-4022
 URL: https://issues.apache.org/jira/browse/FLINK-4022
 Project: Flink
  Issue Type: New Feature
Affects Versions: 1.0.0
Reporter: Tzu-Li (Gordon) Tai
 Fix For: 1.1.0


Allow users to subscribe to "topic-n*", so that the consumer automatically 
reads from "topic-n1", "topic-n2", ... and so on as they are added to Kafka.

I propose to implement this feature by the following description:

Since the overall list of partitions to read will change after job submission, 
the main big change required for this feature will be dynamic partition 
assignment to subtasks while the Kafka consumer is running. This will mainly be 
accomplished using Kafka 0.9.x API 
`KafkaConsumer#subscribe(java.util.regex.Pattern, ConsumerRebalanceListener)`. 
Each KafkaConsumers in each subtask will be added to the same consumer group 
when instantiated, and rely on Kafka to dynamically reassign partitions to them 
whenever a rebalance happens. The registered `ConsumerRebalanceListener` is a 
callback that is called right before and after rebalancing happens. We'll use 
this callback to let each subtask commit its last offsets of partitions its 
currently responsible of to an external store (or Kafka) before a rebalance; 
after rebalance and the substasks gets the new partitions it'll be reading 
from, they'll read from the external store to get the last offsets for their 
new partitions (partitions which don't have offset entries in the store are new 
partitions causing the rebalancing).

The tricky part will be restoring Flink checkpoints when the partition 
assignment is dynamic. Snapshotting will remain the same - subtasks snapshot 
the offsets of partitions they are currently holding. Restoring will be  a bit 
different in that subtasks might not be assigned matching partitions to the 
snapshot the subtask is restored with (since we're letting Kafka dynamically 
assign partitions). There will need to be a coordination process where, if a 
restore state exists, all subtasks first commit the offsets they receive (as a 
result of the restore state) to the external store, and then all subtasks 
attempt to find a last offset for the partitions it is holding.

However, if the globally merged restore state feature mentioned by 
[~StephanEwen] in https://issues.apache.org/jira/browse/FLINK-3231 is 
available, then the restore will be simple again, as each subtask has full 
access to previous global state therefore coordination is not required.

I think changing to dynamic partition assignment is also good in the long run 
for handling topic repartitioning.

Overall,

User-facing API changes:

- New constructor - FlinkKafkaConsumer09(java.util.regex.Pattern, 
DeserializationSchema, Properties)
- New constructor - FlinkKafkaConsumer09(java.util.regex.Pattern,
KeyedDeserializationSchema, Properties)


Implementation changes:

1. Dynamic partition assigning depending on KafkaConsumer#subscribe
- Remove partition list querying from constructor
- Remove static partition assigning to substasks in run()
- Instead of using KafkaConsumer#assign() in fetchers to manually assign static 
partitions, use subscribe() registered with the callback implementation 
explained above.

2. Restoring from checkpointed states
- Snapshotting should remain unchanged
- Restoring requires subtasks to coordinate the restored offsets they hold 
before continuing (unless we are able to have merged restore states).

3. For previous consumer functionality (consume from fixed list of topics), the 
KafkaConsumer#subscribe() has a corresponding overload method for fixed list of 
topics. We can simply decide which subscribe() overload to use depending on 
whether a regex Pattern or list of topics is supplied.



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


[jira] [Commented] (FLINK-4022) Partition discovery / regex topic subscription for the Kafka consumer

2016-06-05 Thread Tzu-Li (Gordon) Tai (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-4022?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15315787#comment-15315787
 ] 

Tzu-Li (Gordon) Tai commented on FLINK-4022:


I'd like to start working on this feature if it's ok with [~rmetzger], after 
confirming that this implementation is acceptable.

It'll introduce more complexity for state restoring for now, but once merged 
restore states are available it'll be simple again.
On the other hand, if we want to stick to defined partition assignments, then 
the complexity will probably be in implementing coordination for discovering 
new partitions / rebalancing between subtasks ourselves.

> Partition discovery / regex topic subscription for the Kafka consumer
> -
>
> Key: FLINK-4022
> URL: https://issues.apache.org/jira/browse/FLINK-4022
> Project: Flink
>  Issue Type: New Feature
>Affects Versions: 1.0.0
>Reporter: Tzu-Li (Gordon) Tai
> Fix For: 1.1.0
>
>
> Allow users to subscribe to "topic-n*", so that the consumer automatically 
> reads from "topic-n1", "topic-n2", ... and so on as they are added to Kafka.
> I propose to implement this feature by the following description:
> Since the overall list of partitions to read will change after job 
> submission, the main big change required for this feature will be dynamic 
> partition assignment to subtasks while the Kafka consumer is running. This 
> will mainly be accomplished using Kafka 0.9.x API 
> `KafkaConsumer#subscribe(java.util.regex.Pattern, 
> ConsumerRebalanceListener)`. Each KafkaConsumers in each subtask will be 
> added to the same consumer group when instantiated, and rely on Kafka to 
> dynamically reassign partitions to them whenever a rebalance happens. The 
> registered `ConsumerRebalanceListener` is a callback that is called right 
> before and after rebalancing happens. We'll use this callback to let each 
> subtask commit its last offsets of partitions its currently responsible of to 
> an external store (or Kafka) before a rebalance; after rebalance and the 
> substasks gets the new partitions it'll be reading from, they'll read from 
> the external store to get the last offsets for their new partitions 
> (partitions which don't have offset entries in the store are new partitions 
> causing the rebalancing).
> The tricky part will be restoring Flink checkpoints when the partition 
> assignment is dynamic. Snapshotting will remain the same - subtasks snapshot 
> the offsets of partitions they are currently holding. Restoring will be  a 
> bit different in that subtasks might not be assigned matching partitions to 
> the snapshot the subtask is restored with (since we're letting Kafka 
> dynamically assign partitions). There will need to be a coordination process 
> where, if a restore state exists, all subtasks first commit the offsets they 
> receive (as a result of the restore state) to the external store, and then 
> all subtasks attempt to find a last offset for the partitions it is holding.
> However, if the globally merged restore state feature mentioned by 
> [~StephanEwen] in https://issues.apache.org/jira/browse/FLINK-3231 is 
> available, then the restore will be simple again, as each subtask has full 
> access to previous global state therefore coordination is not required.
> I think changing to dynamic partition assignment is also good in the long run 
> for handling topic repartitioning.
> Overall,
> User-facing API changes:
> - New constructor - FlinkKafkaConsumer09(java.util.regex.Pattern, 
> DeserializationSchema, Properties)
> - New constructor - FlinkKafkaConsumer09(java.util.regex.Pattern,
> KeyedDeserializationSchema, Properties)
> Implementation changes:
> 1. Dynamic partition assigning depending on KafkaConsumer#subscribe
> - Remove partition list querying from constructor
> - Remove static partition assigning to substasks in run()
> - Instead of using KafkaConsumer#assign() in fetchers to manually assign 
> static partitions, use subscribe() registered with the callback 
> implementation explained above.
> 2. Restoring from checkpointed states
> - Snapshotting should remain unchanged
> - Restoring requires subtasks to coordinate the restored offsets they hold 
> before continuing (unless we are able to have merged restore states).
> 3. For previous consumer functionality (consume from fixed list of topics), 
> the KafkaConsumer#subscribe() has a corresponding overload method for fixed 
> list of topics. We can simply decide which subscribe() overload to use 
> depending on whether a regex Pattern or list of topics is supplied.



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


[jira] [Updated] (FLINK-4022) Partition discovery / regex topic subscription for the Kafka consumer

2016-06-05 Thread Tzu-Li (Gordon) Tai (JIRA)

 [ 
https://issues.apache.org/jira/browse/FLINK-4022?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Tzu-Li (Gordon) Tai updated FLINK-4022:
---
Description: 
Allow users to subscribe to "topic-n*", so that the consumer automatically 
reads from "topic-n1", "topic-n2", ... and so on as they are added to Kafka.

I propose to implement this feature by the following description:

Since the overall list of partitions to read will change after job submission, 
the main big change required for this feature will be dynamic partition 
assignment to subtasks while the Kafka consumer is running. This will mainly be 
accomplished using Kafka 0.9.x API 
`KafkaConsumer#subscribe(java.util.regex.Pattern, ConsumerRebalanceListener)`. 
Each KafkaConsumers in each subtask will be added to the same consumer group 
when instantiated, and rely on Kafka to dynamically reassign partitions to them 
whenever a rebalance happens. The registered `ConsumerRebalanceListener` is a 
callback that is called right before and after rebalancing happens. We'll use 
this callback to let each subtask commit its last offsets of partitions its 
currently responsible of to an external store (or Kafka) before a rebalance; 
after rebalance and the substasks gets the new partitions it'll be reading 
from, they'll read from the external store to get the last offsets for their 
new partitions (partitions which don't have offset entries in the store are new 
partitions causing the rebalancing).

The tricky part will be restoring Flink checkpoints when the partition 
assignment is dynamic. Snapshotting will remain the same - subtasks snapshot 
the offsets of partitions they are currently holding. Restoring will be  a bit 
different in that subtasks might not be assigned matching partitions to the 
snapshot the subtask is restored with (since we're letting Kafka dynamically 
assign partitions). There will need to be a coordination process where, if a 
restore state exists, all subtasks first commit the offsets they receive (as a 
result of the restore state) to the external store, and then all subtasks 
attempt to find a last offset for the partitions it is holding.

However, if the globally merged restore state feature mentioned by 
[~StephanEwen] in https://issues.apache.org/jira/browse/FLINK-3231 is 
available, then the restore will be simple again, as each subtask has full 
access to previous global state therefore coordination is not required.

I think changing to dynamic partition assignment is also good in the long run 
for handling topic repartitioning.

Overall,

User-facing API changes:

- New constructor - FlinkKafkaConsumer09(java.util.regex.Pattern, 
DeserializationSchema, Properties)
- New constructor - FlinkKafkaConsumer09(java.util.regex.Pattern,
KeyedDeserializationSchema, Properties)


Implementation changes:

1. Dynamic partition assigning depending on KafkaConsumer#subscribe
- Remove partition list querying from constructor
- Remove static partition assigning to substasks in run()
- Instead of using KafkaConsumer#assign() in fetchers to manually assign static 
partitions, use subscribe() registered with the callback implementation 
explained above.

2. Restoring from checkpointed states
- Snapshotting should remain unchanged
- Restoring requires subtasks to coordinate the restored offsets they hold 
before continuing (unless we are able to have merged restore states).

3. For previous consumer functionality (consume from fixed list of topics), the 
KafkaConsumer#subscribe() has a corresponding overload method for fixed list of 
topics. We can simply decide which subscribe() overload to use depending on 
whether a regex Pattern or list of topics is supplied.

4. If subtasks don't initially have any assigned partitions, we shouldn't emit 
MAX_VALUE watermark, since it may hold partitions after a rebalance. Instead, 
un-assigned subtasks should be running a fetcher instance too and take part as 
a process pool for the consumer group of the subscribed topics.

  was:
Allow users to subscribe to "topic-n*", so that the consumer automatically 
reads from "topic-n1", "topic-n2", ... and so on as they are added to Kafka.

I propose to implement this feature by the following description:

Since the overall list of partitions to read will change after job submission, 
the main big change required for this feature will be dynamic partition 
assignment to subtasks while the Kafka consumer is running. This will mainly be 
accomplished using Kafka 0.9.x API 
`KafkaConsumer#subscribe(java.util.regex.Pattern, ConsumerRebalanceListener)`. 
Each KafkaConsumers in each subtask will be added to the same consumer group 
when instantiated, and rely on Kafka to dynamically reassign partitions to them 
whenever a rebalance happens. The registered `ConsumerRebalanceListener` is a 
callback that is called right before and after rebalancing happens. We'll use 
this callback to let each subtask 

[GitHub] flink issue #2063: [FLINK-4002] [py] Improve testing infraestructure

2016-06-05 Thread omaralvarez
Github user omaralvarez commented on the issue:

https://github.com/apache/flink/pull/2063
  
I have fixed the last errors in the test functions. But, while trying to 
refactor the utility code, that now is repeated in both test files, I think I 
found another bug.

The thing is that, in order to be able to have another `utils.py` file, we 
need to execute the tests as:

`pyflink2.sh test_main.py utlis.py`

Right now, if HDFS is not used, our case with ` env.execute(local=True)`, 
the packages are not copied to the temp folder along with the script file, and 
the runner fails not being able to locate the module that has been imported. If 
we add this module to the `PYTHONPATH`everything works fine, but I believe this 
should happen. This is probably a matter for another JIRA issue altogether.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (FLINK-4002) [py] Improve testing infraestructure

2016-06-05 Thread ASF GitHub Bot (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-4002?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15315835#comment-15315835
 ] 

ASF GitHub Bot commented on FLINK-4002:
---

Github user omaralvarez commented on the issue:

https://github.com/apache/flink/pull/2063
  
I have fixed the last errors in the test functions. But, while trying to 
refactor the utility code, that now is repeated in both test files, I think I 
found another bug.

The thing is that, in order to be able to have another `utils.py` file, we 
need to execute the tests as:

`pyflink2.sh test_main.py utlis.py`

Right now, if HDFS is not used, our case with ` env.execute(local=True)`, 
the packages are not copied to the temp folder along with the script file, and 
the runner fails not being able to locate the module that has been imported. If 
we add this module to the `PYTHONPATH`everything works fine, but I believe this 
should happen. This is probably a matter for another JIRA issue altogether.


> [py] Improve testing infraestructure
> 
>
> Key: FLINK-4002
> URL: https://issues.apache.org/jira/browse/FLINK-4002
> Project: Flink
>  Issue Type: Bug
>  Components: Python API
>Affects Versions: 1.0.3
>Reporter: Omar Alvarez
>Priority: Minor
>  Labels: Python, Testing
>   Original Estimate: 24h
>  Remaining Estimate: 24h
>
> The Verify() test function errors out when array elements are missing:
> {code}
> env.generate_sequence(1, 5)\
>  .map(Id()).map_partition(Verify([1,2,3,4], "Sequence")).output()
> {code}
> {quote}
> IndexError: list index out of range
> {quote}
> There should also be more documentation in test functions.
> I am already working on a pull request to fix this.



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


[jira] [Updated] (FLINK-4022) Partition discovery / regex topic subscription for the Kafka consumer

2016-06-05 Thread Tzu-Li (Gordon) Tai (JIRA)

 [ 
https://issues.apache.org/jira/browse/FLINK-4022?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Tzu-Li (Gordon) Tai updated FLINK-4022:
---
Description: 
Example: allow users to subscribe to "topic-n*", so that the consumer 
automatically reads from "topic-n1", "topic-n2", ... and so on as they are 
added to Kafka.

I propose to implement this feature by the following description:

Since the overall list of partitions to read will change after job submission, 
the main big change required for this feature will be dynamic partition 
assignment to subtasks while the Kafka consumer is running. This will mainly be 
accomplished using Kafka 0.9.x API 
`KafkaConsumer#subscribe(java.util.regex.Pattern, ConsumerRebalanceListener)`. 
Each KafkaConsumers in each subtask will be added to the same consumer group 
when instantiated, and rely on Kafka to dynamically reassign partitions to them 
whenever a rebalance happens. The registered `ConsumerRebalanceListener` is a 
callback that is called right before and after rebalancing happens. We'll use 
this callback to let each subtask commit its last offsets of partitions its 
currently responsible of to an external store (or Kafka) before a rebalance; 
after rebalance and the substasks gets the new partitions it'll be reading 
from, they'll read from the external store to get the last offsets for their 
new partitions (partitions which don't have offset entries in the store are new 
partitions causing the rebalancing).

The tricky part will be restoring Flink checkpoints when the partition 
assignment is dynamic. Snapshotting will remain the same - subtasks snapshot 
the offsets of partitions they are currently holding. Restoring will be  a bit 
different in that subtasks might not be assigned matching partitions to the 
snapshot the subtask is restored with (since we're letting Kafka dynamically 
assign partitions). There will need to be a coordination process where, if a 
restore state exists, all subtasks first commit the offsets they receive (as a 
result of the restore state) to the external store, and then all subtasks 
attempt to find a last offset for the partitions it is holding.

However, if the globally merged restore state feature mentioned by 
[~StephanEwen] in https://issues.apache.org/jira/browse/FLINK-3231 is 
available, then the restore will be simple again, as each subtask has full 
access to previous global state therefore coordination is not required.

I think changing to dynamic partition assignment is also good in the long run 
for handling topic repartitioning.

Overall,

User-facing API changes:

- New constructor - FlinkKafkaConsumer09(java.util.regex.Pattern, 
DeserializationSchema, Properties)
- New constructor - FlinkKafkaConsumer09(java.util.regex.Pattern,
KeyedDeserializationSchema, Properties)


Implementation changes:

1. Dynamic partition assigning depending on KafkaConsumer#subscribe
- Remove partition list querying from constructor
- Remove static partition assigning to substasks in run()
- Instead of using KafkaConsumer#assign() in fetchers to manually assign static 
partitions, use subscribe() registered with the callback implementation 
explained above.

2. Restoring from checkpointed states
- Snapshotting should remain unchanged
- Restoring requires subtasks to coordinate the restored offsets they hold 
before continuing (unless we are able to have merged restore states).

3. For previous consumer functionality (consume from fixed list of topics), the 
KafkaConsumer#subscribe() has a corresponding overload method for fixed list of 
topics. We can simply decide which subscribe() overload to use depending on 
whether a regex Pattern or list of topics is supplied.

4. If subtasks don't initially have any assigned partitions, we shouldn't emit 
MAX_VALUE watermark, since it may hold partitions after a rebalance. Instead, 
un-assigned subtasks should be running a fetcher instance too and take part as 
a process pool for the consumer group of the subscribed topics.

  was:
Allow users to subscribe to "topic-n*", so that the consumer automatically 
reads from "topic-n1", "topic-n2", ... and so on as they are added to Kafka.

I propose to implement this feature by the following description:

Since the overall list of partitions to read will change after job submission, 
the main big change required for this feature will be dynamic partition 
assignment to subtasks while the Kafka consumer is running. This will mainly be 
accomplished using Kafka 0.9.x API 
`KafkaConsumer#subscribe(java.util.regex.Pattern, ConsumerRebalanceListener)`. 
Each KafkaConsumers in each subtask will be added to the same consumer group 
when instantiated, and rely on Kafka to dynamically reassign partitions to them 
whenever a rebalance happens. The registered `ConsumerRebalanceListener` is a 
callback that is called right before and after rebalancing happens. We'll use 
this callback to let eac

[jira] [Commented] (FLINK-4002) [py] Improve testing infraestructure

2016-06-05 Thread ASF GitHub Bot (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-4002?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15315849#comment-15315849
 ] 

ASF GitHub Bot commented on FLINK-4002:
---

Github user zentol commented on the issue:

https://github.com/apache/flink/pull/2063
  
test files are never copied to the tmp folder regardless of whether HDFS is 
used or not. This is not a bug, but intended behaviour.

You can easily add the utils.py to the test invokation by modifying the 
call to the PythonPlanBinder.main in PythonPlanBinderTest. it should be as 
simple as appending "/utils.py" to the call.


> [py] Improve testing infraestructure
> 
>
> Key: FLINK-4002
> URL: https://issues.apache.org/jira/browse/FLINK-4002
> Project: Flink
>  Issue Type: Bug
>  Components: Python API
>Affects Versions: 1.0.3
>Reporter: Omar Alvarez
>Priority: Minor
>  Labels: Python, Testing
>   Original Estimate: 24h
>  Remaining Estimate: 24h
>
> The Verify() test function errors out when array elements are missing:
> {code}
> env.generate_sequence(1, 5)\
>  .map(Id()).map_partition(Verify([1,2,3,4], "Sequence")).output()
> {code}
> {quote}
> IndexError: list index out of range
> {quote}
> There should also be more documentation in test functions.
> I am already working on a pull request to fix this.



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


[GitHub] flink issue #2063: [FLINK-4002] [py] Improve testing infraestructure

2016-06-05 Thread zentol
Github user zentol commented on the issue:

https://github.com/apache/flink/pull/2063
  
test files are never copied to the tmp folder regardless of whether HDFS is 
used or not. This is not a bug, but intended behaviour.

You can easily add the utils.py to the test invokation by modifying the 
call to the PythonPlanBinder.main in PythonPlanBinderTest. it should be as 
simple as appending "/utils.py" to the call.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Updated] (FLINK-4020) Remove shard list querying from Kinesis consumer constructor

2016-06-05 Thread Tzu-Li (Gordon) Tai (JIRA)

 [ 
https://issues.apache.org/jira/browse/FLINK-4020?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Tzu-Li (Gordon) Tai updated FLINK-4020:
---
Description: Currently FlinkKinesisConsumer is querying for the whole list 
of shards in the constructor, forcing the client to be able to access Kinesis 
as well. This is also a drawback for handling Kinesis-side resharding, since 
we'd want all shard listing / shard-to-task assigning / shard end (result of 
resharding) handling logic to be capable of being independently done within 
task life cycle methods, with defined and definite results.  (was: Currently 
FlinkKinesisConsumer is querying for the whole list of shards in the 
constructor, forcing the client to be able to access Kinesis as well. This is 
also a drawback for handling Kinesis-side resharding, since we'd want all shard 
listing / shard-to-task assigning / shard end (result of resharding) handling 
logic to be capable of being independently done within task life cycle methods, 
with defined and definite results.

Main thing to overcome is coordination between parallel subtasks. All subtasks 
will need to retry (due to Amazon's operation rate limits) until all subtasks 
have succeeded. We could probably use either ZK or Amazon DynamoDB (user 
configurable) for coordinating subtask status.)

> Remove shard list querying from Kinesis consumer constructor
> 
>
> Key: FLINK-4020
> URL: https://issues.apache.org/jira/browse/FLINK-4020
> Project: Flink
>  Issue Type: Sub-task
>  Components: Streaming Connectors
>Reporter: Tzu-Li (Gordon) Tai
>
> Currently FlinkKinesisConsumer is querying for the whole list of shards in 
> the constructor, forcing the client to be able to access Kinesis as well. 
> This is also a drawback for handling Kinesis-side resharding, since we'd want 
> all shard listing / shard-to-task assigning / shard end (result of 
> resharding) handling logic to be capable of being independently done within 
> task life cycle methods, with defined and definite results.



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


[jira] [Assigned] (FLINK-4020) Remove shard list querying from Kinesis consumer constructor

2016-06-05 Thread Tzu-Li (Gordon) Tai (JIRA)

 [ 
https://issues.apache.org/jira/browse/FLINK-4020?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Tzu-Li (Gordon) Tai reassigned FLINK-4020:
--

Assignee: Tzu-Li (Gordon) Tai

> Remove shard list querying from Kinesis consumer constructor
> 
>
> Key: FLINK-4020
> URL: https://issues.apache.org/jira/browse/FLINK-4020
> Project: Flink
>  Issue Type: Sub-task
>  Components: Streaming Connectors
>Reporter: Tzu-Li (Gordon) Tai
>Assignee: Tzu-Li (Gordon) Tai
>
> Currently FlinkKinesisConsumer is querying for the whole list of shards in 
> the constructor, forcing the client to be able to access Kinesis as well. 
> This is also a drawback for handling Kinesis-side resharding, since we'd want 
> all shard listing / shard-to-task assigning / shard end (result of 
> resharding) handling logic to be capable of being independently done within 
> task life cycle methods, with defined and definite results.



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


[jira] [Created] (FLINK-4023) Move Kafka consumer partition discovery from constructor to open()

2016-06-05 Thread Michal Harish (JIRA)
Michal Harish created FLINK-4023:


 Summary: Move Kafka consumer partition discovery from constructor 
to open()
 Key: FLINK-4023
 URL: https://issues.apache.org/jira/browse/FLINK-4023
 Project: Flink
  Issue Type: Improvement
  Components: Kafka Connector
Reporter: Michal Harish
Priority: Minor






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


[jira] [Updated] (FLINK-4023) Move Kafka consumer partition discovery from constructor to open()

2016-06-05 Thread Michal Harish (JIRA)

 [ 
https://issues.apache.org/jira/browse/FLINK-4023?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Michal Harish updated FLINK-4023:
-
Description: 
Currently, Flink queries Kafka for partition information when creating the 
Kafka consumer. This is done on the client when submitting the Flink job, which 
requires the client to be able to fetch the partition data from Kafka.


> Move Kafka consumer partition discovery from constructor to open()
> --
>
> Key: FLINK-4023
> URL: https://issues.apache.org/jira/browse/FLINK-4023
> Project: Flink
>  Issue Type: Improvement
>  Components: Kafka Connector
>Reporter: Michal Harish
>Priority: Minor
>  Labels: kafka, kafka-0.8
>
> Currently, Flink queries Kafka for partition information when creating the 
> Kafka consumer. This is done on the client when submitting the Flink job, 
> which requires the client to be able to fetch the partition data from Kafka.



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


[jira] [Updated] (FLINK-4023) Move Kafka consumer partition discovery from constructor to open()

2016-06-05 Thread Michal Harish (JIRA)

 [ 
https://issues.apache.org/jira/browse/FLINK-4023?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Michal Harish updated FLINK-4023:
-
Description: 
Currently, Flink queries Kafka for partition information when creating the 
Kafka consumer. This is done on the client when submitting the Flink job, which 
requires the client to be able to fetch the partition data from Kafka which may 
only be accessible from the cluster environment where the tasks will be 
running. Moving the partition discovery to the open task should solve this 
problem.
 

  was:
Currently, Flink queries Kafka for partition information when creating the 
Kafka consumer. This is done on the client when submitting the Flink job, which 
requires the client to be able to fetch the partition data from Kafka.



> Move Kafka consumer partition discovery from constructor to open()
> --
>
> Key: FLINK-4023
> URL: https://issues.apache.org/jira/browse/FLINK-4023
> Project: Flink
>  Issue Type: Improvement
>  Components: Kafka Connector
>Reporter: Michal Harish
>Priority: Minor
>  Labels: kafka, kafka-0.8
>
> Currently, Flink queries Kafka for partition information when creating the 
> Kafka consumer. This is done on the client when submitting the Flink job, 
> which requires the client to be able to fetch the partition data from Kafka 
> which may only be accessible from the cluster environment where the tasks 
> will be running. Moving the partition discovery to the open task should solve 
> this problem.
>  



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


[jira] [Updated] (FLINK-4023) Move Kafka consumer partition discovery from constructor to open()

2016-06-05 Thread Michal Harish (JIRA)

 [ 
https://issues.apache.org/jira/browse/FLINK-4023?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Michal Harish updated FLINK-4023:
-
Description: 
Currently, Flink queries Kafka for partition information when creating the 
Kafka consumer. This is done on the client when submitting the Flink job, which 
requires the client to be able to fetch the partition data from Kafka which may 
only be accessible from the cluster environment where the tasks will be 
running. Moving the partition discovery to the open() method should solve this 
problem.
 

  was:
Currently, Flink queries Kafka for partition information when creating the 
Kafka consumer. This is done on the client when submitting the Flink job, which 
requires the client to be able to fetch the partition data from Kafka which may 
only be accessible from the cluster environment where the tasks will be 
running. Moving the partition discovery to the open task should solve this 
problem.
 


> Move Kafka consumer partition discovery from constructor to open()
> --
>
> Key: FLINK-4023
> URL: https://issues.apache.org/jira/browse/FLINK-4023
> Project: Flink
>  Issue Type: Improvement
>  Components: Kafka Connector
>Reporter: Michal Harish
>Priority: Minor
>  Labels: kafka, kafka-0.8
>
> Currently, Flink queries Kafka for partition information when creating the 
> Kafka consumer. This is done on the client when submitting the Flink job, 
> which requires the client to be able to fetch the partition data from Kafka 
> which may only be accessible from the cluster environment where the tasks 
> will be running. Moving the partition discovery to the open() method should 
> solve this problem.
>  



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


[jira] [Created] (FLINK-4024) FileSourceFunction not adjusted to new IF lifecycle

2016-06-05 Thread Chesnay Schepler (JIRA)
Chesnay Schepler created FLINK-4024:
---

 Summary: FileSourceFunction not adjusted to new IF lifecycle
 Key: FLINK-4024
 URL: https://issues.apache.org/jira/browse/FLINK-4024
 Project: Flink
  Issue Type: Bug
  Components: Streaming
Affects Versions: 1.1.0
Reporter: Chesnay Schepler
Assignee: Chesnay Schepler
Priority: Critical
 Fix For: 1.1.0


The InputFormat lifecycle was extended in 
ac2137cfa5e63bd4f53a4b7669dc591ab210093f, adding additional 
open-/closeInputFormat() methods.

The streaming FileSourceFunction was not adjusted for this change, and thus 
will fail for every InputFormat that leverages these new methods.



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


[jira] [Updated] (FLINK-4024) FileSourceFunction not adjusted to new IF lifecycle

2016-06-05 Thread Chesnay Schepler (JIRA)

 [ 
https://issues.apache.org/jira/browse/FLINK-4024?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Chesnay Schepler updated FLINK-4024:

Assignee: (was: Chesnay Schepler)

> FileSourceFunction not adjusted to new IF lifecycle
> ---
>
> Key: FLINK-4024
> URL: https://issues.apache.org/jira/browse/FLINK-4024
> Project: Flink
>  Issue Type: Bug
>  Components: Streaming
>Affects Versions: 1.1.0
>Reporter: Chesnay Schepler
>Priority: Critical
> Fix For: 1.1.0
>
>
> The InputFormat lifecycle was extended in 
> ac2137cfa5e63bd4f53a4b7669dc591ab210093f, adding additional 
> open-/closeInputFormat() methods.
> The streaming FileSourceFunction was not adjusted for this change, and thus 
> will fail for every InputFormat that leverages these new methods.



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


[GitHub] flink issue #2072: [FLINK-3948] Protect RocksDB cleanup by cleanup lock

2016-06-05 Thread StephanEwen
Github user StephanEwen commented on the issue:

https://github.com/apache/flink/pull/2072
  
Looks great, +1 to merge


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (FLINK-3948) EventTimeWindowCheckpointingITCase Fails with Core Dump

2016-06-05 Thread ASF GitHub Bot (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-3948?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316023#comment-15316023
 ] 

ASF GitHub Bot commented on FLINK-3948:
---

Github user StephanEwen commented on the issue:

https://github.com/apache/flink/pull/2072
  
Looks great, +1 to merge


> EventTimeWindowCheckpointingITCase Fails with Core Dump
> ---
>
> Key: FLINK-3948
> URL: https://issues.apache.org/jira/browse/FLINK-3948
> Project: Flink
>  Issue Type: Bug
>  Components: state backends
>Affects Versions: 1.1.0
>Reporter: Aljoscha Krettek
>Assignee: Aljoscha Krettek
>Priority: Critical
>
> It fails because of a core dump in RocksDB. 



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


[GitHub] flink issue #2065: [FLINK-4011] Keep UserCodeClassLoader in archived Executi...

2016-06-05 Thread StephanEwen
Github user StephanEwen commented on the issue:

https://github.com/apache/flink/pull/2065
  
You already outlined one of the drawbacks (holding on to the classes).
There are probably other parts in the execution graph that currently hold 
onto user defined classes as well.

A good (but more involved) longer term solution would be to create a 
"archived execution graph" that can be serialized and that does not contain any 
user classes, but only "stringified" versions of configs, accumulators, 
metrics, etc.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (FLINK-4011) Unable to access completed job in web frontend

2016-06-05 Thread ASF GitHub Bot (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-4011?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316024#comment-15316024
 ] 

ASF GitHub Bot commented on FLINK-4011:
---

Github user StephanEwen commented on the issue:

https://github.com/apache/flink/pull/2065
  
You already outlined one of the drawbacks (holding on to the classes).
There are probably other parts in the execution graph that currently hold 
onto user defined classes as well.

A good (but more involved) longer term solution would be to create a 
"archived execution graph" that can be serialized and that does not contain any 
user classes, but only "stringified" versions of configs, accumulators, 
metrics, etc.


> Unable to access completed job in web frontend
> --
>
> Key: FLINK-4011
> URL: https://issues.apache.org/jira/browse/FLINK-4011
> Project: Flink
>  Issue Type: Bug
>  Components: Webfrontend
>Affects Versions: 1.1.0
>Reporter: Robert Metzger
>Assignee: Robert Metzger
>Priority: Critical
>
> In the current master, I'm not able to access a finished job's detail page.
> The JobManager logs shows the following exception:
> {code}
> 2016-06-02 15:23:08,581 WARN  
> org.apache.flink.runtime.webmonitor.RuntimeMonitorHandler - Error while 
> handling request
> java.lang.RuntimeException: Couldn't deserialize ExecutionConfig.
> at 
> org.apache.flink.runtime.webmonitor.handlers.JobConfigHandler.handleRequest(JobConfigHandler.java:52)
> at 
> org.apache.flink.runtime.webmonitor.handlers.AbstractExecutionGraphRequestHandler.handleRequest(AbstractExecutionGraphRequestHandler.java:61)
> at 
> org.apache.flink.runtime.webmonitor.RuntimeMonitorHandler.respondAsLeader(RuntimeMonitorHandler.java:88)
> at 
> org.apache.flink.runtime.webmonitor.RuntimeMonitorHandlerBase.channelRead0(RuntimeMonitorHandlerBase.java:84)
> at 
> org.apache.flink.runtime.webmonitor.RuntimeMonitorHandlerBase.channelRead0(RuntimeMonitorHandlerBase.java:44)
> at 
> io.netty.channel.SimpleChannelInboundHandler.channelRead(SimpleChannelInboundHandler.java:105)
> at 
> io.netty.channel.AbstractChannelHandlerContext.invokeChannelRead(AbstractChannelHandlerContext.java:339)
> at 
> io.netty.channel.AbstractChannelHandlerContext.fireChannelRead(AbstractChannelHandlerContext.java:324)
> at io.netty.handler.codec.http.router.Handler.routed(Handler.java:62)
> at 
> io.netty.handler.codec.http.router.DualAbstractHandler.channelRead0(DualAbstractHandler.java:57)
> at 
> io.netty.handler.codec.http.router.DualAbstractHandler.channelRead0(DualAbstractHandler.java:20)
> at 
> io.netty.channel.SimpleChannelInboundHandler.channelRead(SimpleChannelInboundHandler.java:105)
> at 
> io.netty.channel.AbstractChannelHandlerContext.invokeChannelRead(AbstractChannelHandlerContext.java:339)
> at 
> io.netty.channel.AbstractChannelHandlerContext.fireChannelRead(AbstractChannelHandlerContext.java:324)
> at 
> org.apache.flink.runtime.webmonitor.HttpRequestHandler.channelRead0(HttpRequestHandler.java:105)
> at 
> org.apache.flink.runtime.webmonitor.HttpRequestHandler.channelRead0(HttpRequestHandler.java:65)
> at 
> io.netty.channel.SimpleChannelInboundHandler.channelRead(SimpleChannelInboundHandler.java:105)
> at 
> io.netty.channel.AbstractChannelHandlerContext.invokeChannelRead(AbstractChannelHandlerContext.java:339)
> at 
> io.netty.channel.AbstractChannelHandlerContext.fireChannelRead(AbstractChannelHandlerContext.java:324)
> at 
> io.netty.handler.codec.ByteToMessageDecoder.channelRead(ByteToMessageDecoder.java:242)
> at 
> io.netty.channel.CombinedChannelDuplexHandler.channelRead(CombinedChannelDuplexHandler.java:147)
> at 
> io.netty.channel.AbstractChannelHandlerContext.invokeChannelRead(AbstractChannelHandlerContext.java:339)
> at 
> io.netty.channel.AbstractChannelHandlerContext.fireChannelRead(AbstractChannelHandlerContext.java:324)
> at 
> io.netty.channel.DefaultChannelPipeline.fireChannelRead(DefaultChannelPipeline.java:847)
> at 
> io.netty.channel.nio.AbstractNioByteChannel$NioByteUnsafe.read(AbstractNioByteChannel.java:131)
> at 
> io.netty.channel.nio.NioEventLoop.processSelectedKey(NioEventLoop.java:511)
> at 
> io.netty.channel.nio.NioEventLoop.processSelectedKeysOptimized(NioEventLoop.java:468)
> at 
> io.netty.channel.nio.NioEventLoop.processSelectedKeys(NioEventLoop.java:382)
> at io.netty.channel.nio.NioEventLoop.run(NioEventLoop.java:354)
> at 
> io.netty.util.concurrent.SingleThreadEventExecutor$2.run(SingleThreadEventExecutor.java:111)
> at 
> io.netty.util.concurrent.DefaultThreadFa

[jira] [Created] (FLINK-4025) Add possiblity for the RMQ Streaming Source to customize the queue

2016-06-05 Thread Dominik Bruhn (JIRA)
Dominik Bruhn created FLINK-4025:


 Summary: Add possiblity for the RMQ Streaming Source to customize 
the queue
 Key: FLINK-4025
 URL: https://issues.apache.org/jira/browse/FLINK-4025
 Project: Flink
  Issue Type: Improvement
  Components: Streaming Connectors
Affects Versions: 1.0.2
Reporter: Dominik Bruhn


This patch adds the possibilty for the user of the RabbitMQ
Streaming Connector to customize the queue which is used. There
are use-cases in which you want to set custom parameters for the
queue (i.e. TTL of the messages if Flink reboots) or the
possibility to bind the queue to an exchange afterwards.

The commit doesn't change the actual behaviour but makes it
possible for users to override the newly create `setupQueue`
method and cutomize their implementation. This was not possible
before.



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


[GitHub] flink pull request #2073: [FLINK-4025] Add possiblity for the RMQ Streaming ...

2016-06-05 Thread theomega
GitHub user theomega opened a pull request:

https://github.com/apache/flink/pull/2073

[FLINK-4025] Add possiblity for the RMQ Streaming Source to customize the 
queue

This patch adds the possibilty for the user of the RabbitMQ
Streaming Connector to customize the queue which is used. There
are use-cases in which you want to set custom parameters for the
queue (i.e. TTL of the messages if Flink reboots) or the
possibility to bind the queue to an exchange afterwards.

The commit doesn't change the actual behaviour but makes it
possible for users to override the newly create `setupQueue`
method and cutomize their implementation. This was not possible
before.

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/theomega/flink 
rmq-streaming-connector_customize-queue

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/flink/pull/2073.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #2073


commit 811e88f17f8469f7d01a69158ab7c062325eeb34
Author: Dominik Bruhn 
Date:   2016-06-05T21:18:56Z

RMQ Streaming: Possibility to customize queue

This patch adds the possibilty for th user of the RabbitMQ
Streaming Connector to customize the queue which is used. There
are use-cases in which you want to set custom parameters for the
queue (i.e. TTL of the messages if Flink reboots) or the
possibility to bind the queue to an exchange afterwards.

The commit doesn't change the actual behaviour but makes it
possible for users to override the newly create `setupQueue`
method and cutomize their implementation. This was not possible
before.




---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (FLINK-4025) Add possiblity for the RMQ Streaming Source to customize the queue

2016-06-05 Thread ASF GitHub Bot (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-4025?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316028#comment-15316028
 ] 

ASF GitHub Bot commented on FLINK-4025:
---

GitHub user theomega opened a pull request:

https://github.com/apache/flink/pull/2073

[FLINK-4025] Add possiblity for the RMQ Streaming Source to customize the 
queue

This patch adds the possibilty for the user of the RabbitMQ
Streaming Connector to customize the queue which is used. There
are use-cases in which you want to set custom parameters for the
queue (i.e. TTL of the messages if Flink reboots) or the
possibility to bind the queue to an exchange afterwards.

The commit doesn't change the actual behaviour but makes it
possible for users to override the newly create `setupQueue`
method and cutomize their implementation. This was not possible
before.

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/theomega/flink 
rmq-streaming-connector_customize-queue

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/flink/pull/2073.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #2073


commit 811e88f17f8469f7d01a69158ab7c062325eeb34
Author: Dominik Bruhn 
Date:   2016-06-05T21:18:56Z

RMQ Streaming: Possibility to customize queue

This patch adds the possibilty for th user of the RabbitMQ
Streaming Connector to customize the queue which is used. There
are use-cases in which you want to set custom parameters for the
queue (i.e. TTL of the messages if Flink reboots) or the
possibility to bind the queue to an exchange afterwards.

The commit doesn't change the actual behaviour but makes it
possible for users to override the newly create `setupQueue`
method and cutomize their implementation. This was not possible
before.




> Add possiblity for the RMQ Streaming Source to customize the queue
> --
>
> Key: FLINK-4025
> URL: https://issues.apache.org/jira/browse/FLINK-4025
> Project: Flink
>  Issue Type: Improvement
>  Components: Streaming Connectors
>Affects Versions: 1.0.2
>Reporter: Dominik Bruhn
>
> This patch adds the possibilty for the user of the RabbitMQ
> Streaming Connector to customize the queue which is used. There
> are use-cases in which you want to set custom parameters for the
> queue (i.e. TTL of the messages if Flink reboots) or the
> possibility to bind the queue to an exchange afterwards.
> The commit doesn't change the actual behaviour but makes it
> possible for users to override the newly create `setupQueue`
> method and cutomize their implementation. This was not possible
> before.



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


[jira] [Commented] (FLINK-3777) Add open and close methods to manage IF lifecycle

2016-06-05 Thread Stephan Ewen (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-3777?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316032#comment-15316032
 ] 

Stephan Ewen commented on FLINK-3777:
-

Can I revive this thread quickly?
This is one more of these "try and keep the API simple" attempts from my side...

I was wondering how important these additional methods are? After all, 
initializing and releasing resource in "open" and "close" should work fine, and 
in the wort case just initialize and release a few times (but still quite 
coarse grained).
This addition makes the already complicated life cycle of input format even 
more complicated. We already found cases where the added methods were not 
handled correctly.

If the benefit of these additional methods is very small, I would be in favor 
to remove them again, and make the system simpler to maintain and more robust 
(less corner cases that can and will be overlooked).

Since this is not yet released in a stable release, we can still undo the 
change in the next few weeks.

> Add open and close methods to manage IF lifecycle
> -
>
> Key: FLINK-3777
> URL: https://issues.apache.org/jira/browse/FLINK-3777
> Project: Flink
>  Issue Type: Improvement
>  Components: Core
>Affects Versions: 1.0.1
>Reporter: Flavio Pompermaier
>Assignee: Flavio Pompermaier
>  Labels: inputformat, lifecycle
>
> At the moment the opening and closing of an inputFormat are not managed, 
> although open() could be (improperly IMHO) simulated by configure().
> This limits the possibility to reuse expensive resources (like database 
> connections) and manage their release. 
> Probably the best option would be to add 2 methods (i.e. openInputformat() 
> and closeInputFormat() ) to RichInputFormat*
> * NOTE: the best option from a "semantic" point of view would be to rename 
> the current open() and close() to openSplit() and closeSplit() respectively 
> while using open() and close() methods for the IF lifecycle management, but 
> this would cause a backward compatibility issue...



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


[GitHub] flink pull request #2066: Updated ssh configuration in base Dockerfile

2016-06-05 Thread StephanEwen
Github user StephanEwen commented on a diff in the pull request:

https://github.com/apache/flink/pull/2066#discussion_r65825673
  
--- Diff: flink-contrib/docker-flink/base/Dockerfile ---
@@ -38,12 +38,12 @@ ENV JAVA_HOME /usr/java/default/
 RUN echo 'root:secret' | chpasswd
 
 #SSH as root... probably needs to be revised for security!
-RUN sed -i 's/PermitRootLogin without-password/PermitRootLogin yes/' 
/etc/ssh/sshd_config
--- End diff --

+1 to Greg's suggestion. Otherwise users of old versions will have a 
problem.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (FLINK-3980) Remove ExecutionConfig.PARALLELISM_UNKNOWN

2016-06-05 Thread ASF GitHub Bot (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-3980?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316036#comment-15316036
 ] 

ASF GitHub Bot commented on FLINK-3980:
---

Github user StephanEwen commented on the issue:

https://github.com/apache/flink/pull/2064
  
Thanks, Greg, this looks good!

+1 to merge


> Remove ExecutionConfig.PARALLELISM_UNKNOWN
> --
>
> Key: FLINK-3980
> URL: https://issues.apache.org/jira/browse/FLINK-3980
> Project: Flink
>  Issue Type: Improvement
>  Components: Core
>Affects Versions: 1.1.0
>Reporter: Greg Hogan
>Assignee: Greg Hogan
>Priority: Minor
> Fix For: 1.1.0
>
>
> FLINK-3589 added {{ExecutionConfig.PARALLELISM_DEFAULT}} and 
> {{ExecutionConfig.PARALLELISM_UNKNOWN}}. The former gave a name to the 
> contant {{-1}}  and the latter was used as a default no-op when setting the 
> parallelism.
> It's nice to keep these intents separate but given the current implementation 
> of Operator parallelism users can get by using {{PARALLELISM_DEFAULT}}.



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


[GitHub] flink issue #2064: [FLINK-3980] [core] Remove ExecutionConfig.PARALLELISM_UN...

2016-06-05 Thread StephanEwen
Github user StephanEwen commented on the issue:

https://github.com/apache/flink/pull/2064
  
Thanks, Greg, this looks good!

+1 to merge


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (FLINK-4019) Expose approximateArrivalTimestamp through the KinesisDeserializationSchema interface

2016-06-05 Thread Stephan Ewen (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-4019?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316048#comment-15316048
 ] 

Stephan Ewen commented on FLINK-4019:
-

That would also be great to attach as a default time stamp. So, if you pick 
Kinesis and Event Time and do not manually assign timestamps and watermarks, 
you actually get "Kinesis Store Time".

> Expose approximateArrivalTimestamp through the KinesisDeserializationSchema 
> interface
> -
>
> Key: FLINK-4019
> URL: https://issues.apache.org/jira/browse/FLINK-4019
> Project: Flink
>  Issue Type: Sub-task
>  Components: Streaming Connectors
>Affects Versions: 1.1.0
>Reporter: Tzu-Li (Gordon) Tai
>
> Amazon's Record class also gives information about the timestamp of when 
> Kinesis successfully receives the record: 
> http://docs.aws.amazon.com/AWSJavaSDK/latest/javadoc/com/amazonaws/services/kinesis/model/Record.html#getApproximateArrivalTimestamp().
> This should be useful info for users and should be exposed through the 
> deserialization schema.



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


[GitHub] flink pull request #2068: [hotfix] [core] Fix scope format keys

2016-06-05 Thread asfgit
Github user asfgit closed the pull request at:

https://github.com/apache/flink/pull/2068


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (FLINK-4014) Jobs hang. Maybe NetworkBufferPool and LocalBufferPool has some problem

2016-06-05 Thread ZhengBowen (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-4014?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316110#comment-15316110
 ] 

ZhengBowen commented on FLINK-4014:
---

Hi,Ufuk,thank you for your help.Any good news these days?

> Jobs hang. Maybe NetworkBufferPool and LocalBufferPool has some problem
> ---
>
> Key: FLINK-4014
> URL: https://issues.apache.org/jira/browse/FLINK-4014
> Project: Flink
>  Issue Type: Bug
>Reporter: ZhengBowen
> Attachments: FlinkJob_20160603_174748_03.java, hang.stack
>
>
> I also run a five jobs, and all the slot filled.And I seem to find my five 
> jobs is all hanged.
> Following attachments are my jstack and job source code.
> The hang.stack display all thread related to the job is WAIT. Why?



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


[jira] [Commented] (FLINK-4019) Expose approximateArrivalTimestamp through the KinesisDeserializationSchema interface

2016-06-05 Thread Tzu-Li (Gordon) Tai (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-4019?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316225#comment-15316225
 ] 

Tzu-Li (Gordon) Tai commented on FLINK-4019:


Good suggestion! For now, the connector doesn't support users to give timestamp 
/ watermark assigners. Perhaps we should expand the scope of this JIRA to 
include this also?

> Expose approximateArrivalTimestamp through the KinesisDeserializationSchema 
> interface
> -
>
> Key: FLINK-4019
> URL: https://issues.apache.org/jira/browse/FLINK-4019
> Project: Flink
>  Issue Type: Sub-task
>  Components: Streaming Connectors
>Affects Versions: 1.1.0
>Reporter: Tzu-Li (Gordon) Tai
>
> Amazon's Record class also gives information about the timestamp of when 
> Kinesis successfully receives the record: 
> http://docs.aws.amazon.com/AWSJavaSDK/latest/javadoc/com/amazonaws/services/kinesis/model/Record.html#getApproximateArrivalTimestamp().
> This should be useful info for users and should be exposed through the 
> deserialization schema.



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


[GitHub] flink issue #2063: [FLINK-4002] [py] Improve testing infraestructure

2016-06-05 Thread omaralvarez
Github user omaralvarez commented on the issue:

https://github.com/apache/flink/pull/2063
  
Yes, I know that how python scripts are executed for the test is different. 
Let me elaborate:

Since running the tests are quite costly in my laptop, I normally test my 
changes executing them in a local instance of Flink 1.0.3, since this is less 
taxing. Once I complete the changes, I run `mvn verify`. The problem is that 
when I call `pyflink2.sh test_main.py utils.py, the module that I pass to the 
test script, is ignored unless I use HDFS, in which case, everything works fine.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (FLINK-4002) [py] Improve testing infraestructure

2016-06-05 Thread ASF GitHub Bot (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-4002?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316231#comment-15316231
 ] 

ASF GitHub Bot commented on FLINK-4002:
---

Github user omaralvarez commented on the issue:

https://github.com/apache/flink/pull/2063
  
Yes, I know that how python scripts are executed for the test is different. 
Let me elaborate:

Since running the tests are quite costly in my laptop, I normally test my 
changes executing them in a local instance of Flink 1.0.3, since this is less 
taxing. Once I complete the changes, I run `mvn verify`. The problem is that 
when I call `pyflink2.sh test_main.py utils.py, the module that I pass to the 
test script, is ignored unless I use HDFS, in which case, everything works fine.


> [py] Improve testing infraestructure
> 
>
> Key: FLINK-4002
> URL: https://issues.apache.org/jira/browse/FLINK-4002
> Project: Flink
>  Issue Type: Bug
>  Components: Python API
>Affects Versions: 1.0.3
>Reporter: Omar Alvarez
>Priority: Minor
>  Labels: Python, Testing
>   Original Estimate: 24h
>  Remaining Estimate: 24h
>
> The Verify() test function errors out when array elements are missing:
> {code}
> env.generate_sequence(1, 5)\
>  .map(Id()).map_partition(Verify([1,2,3,4], "Sequence")).output()
> {code}
> {quote}
> IndexError: list index out of range
> {quote}
> There should also be more documentation in test functions.
> I am already working on a pull request to fix this.



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


[jira] [Commented] (FLINK-3769) RabbitMQ Sink ability to publish to a different exchange

2016-06-05 Thread Subhankar Biswas (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-3769?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316243#comment-15316243
 ] 

Subhankar Biswas commented on FLINK-3769:
-

IMO user should be able to build different configuration for queue in both sink 
and source side. So if we can able to build some configuration for queue like 
this connection configuration:
https://github.com/subhankarb/flink/blob/670e92d0731652563fd58631107e0f19c5d5d9a1/flink-streaming-connectors/flink-connector-rabbitmq/src/main/java/org/apache/flink/streaming/connectors/rabbitmq/common/RMQConnectionConfig.java
and then pass to sink and source and declare queue based on this config, user 
will have more options to pass configurations.
I will start work on this once FLINK-3763 will get merged. 


> RabbitMQ Sink ability to publish to a different exchange
> 
>
> Key: FLINK-3769
> URL: https://issues.apache.org/jira/browse/FLINK-3769
> Project: Flink
>  Issue Type: Improvement
>  Components: Streaming Connectors
>Affects Versions: 1.0.1
>Reporter: Robert Batts
>Assignee: Subhankar Biswas
>  Labels: rabbitmq
>
> The RabbitMQ Sink can currently only publish to the "default" exchange. This 
> exchange is a direct exchange, so the routing key will route directly to the 
> queue name. Because of this, the current sink will only be 1-to-1-to-1 (1 job 
> to 1 exchange which routes to 1 queue). Additionally, I believe that if a 
> user decides to use a different exchange I think the following can be assumed:
> 1.) The provided exchange exists
> 2.) The user has declared the appropriate mapping and the appropriate queues 
> exist in RabbitMQ (therefore, nothing needs to be created)
> RabbitMQ currently provides four types of exchanges. Three of these will be 
> covered by just enabling exchanges (Direct, Fanout, Topic) because they use 
> the routingkey (or nothing). 
> The fourth exchange type relies on the message headers, which are currently 
> set to null by default on the publish. These headers may be on a per message 
> level, so the input of this stream will need to take this as input as well. 
> This forth exchange could very well be outside of the scope of this 
> Improvement and a "RabbitMQ Sink enable headers" Improvement might be the 
> better way to go with this.
> Exchange Types: https://www.rabbitmq.com/tutorials/amqp-concepts.html



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


[jira] [Commented] (FLINK-3769) RabbitMQ Sink ability to publish to a different exchange

2016-06-05 Thread Subhankar Biswas (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-3769?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316242#comment-15316242
 ] 

Subhankar Biswas commented on FLINK-3769:
-

IMO user should be able to build different configuration for queue in both sink 
and source side. So if we can able to build some configuration for queue like 
this connection configuration:
https://github.com/subhankarb/flink/blob/670e92d0731652563fd58631107e0f19c5d5d9a1/flink-streaming-connectors/flink-connector-rabbitmq/src/main/java/org/apache/flink/streaming/connectors/rabbitmq/common/RMQConnectionConfig.java
and then pass to sink and source and declare queue based on this config, user 
will have more options to pass configurations.
I will start work on this once FLINK-3763 will get merged. 


> RabbitMQ Sink ability to publish to a different exchange
> 
>
> Key: FLINK-3769
> URL: https://issues.apache.org/jira/browse/FLINK-3769
> Project: Flink
>  Issue Type: Improvement
>  Components: Streaming Connectors
>Affects Versions: 1.0.1
>Reporter: Robert Batts
>Assignee: Subhankar Biswas
>  Labels: rabbitmq
>
> The RabbitMQ Sink can currently only publish to the "default" exchange. This 
> exchange is a direct exchange, so the routing key will route directly to the 
> queue name. Because of this, the current sink will only be 1-to-1-to-1 (1 job 
> to 1 exchange which routes to 1 queue). Additionally, I believe that if a 
> user decides to use a different exchange I think the following can be assumed:
> 1.) The provided exchange exists
> 2.) The user has declared the appropriate mapping and the appropriate queues 
> exist in RabbitMQ (therefore, nothing needs to be created)
> RabbitMQ currently provides four types of exchanges. Three of these will be 
> covered by just enabling exchanges (Direct, Fanout, Topic) because they use 
> the routingkey (or nothing). 
> The fourth exchange type relies on the message headers, which are currently 
> set to null by default on the publish. These headers may be on a per message 
> level, so the input of this stream will need to take this as input as well. 
> This forth exchange could very well be outside of the scope of this 
> Improvement and a "RabbitMQ Sink enable headers" Improvement might be the 
> better way to go with this.
> Exchange Types: https://www.rabbitmq.com/tutorials/amqp-concepts.html



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


[jira] [Commented] (FLINK-3769) RabbitMQ Sink ability to publish to a different exchange

2016-06-05 Thread Subhankar Biswas (JIRA)

[ 
https://issues.apache.org/jira/browse/FLINK-3769?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15316250#comment-15316250
 ] 

Subhankar Biswas commented on FLINK-3769:
-

IMO user should be able to build different configuration for queue in both sink 
and source side. So if we can able to build some configuration for queue like 
this connection configuration:
https://github.com/subhankarb/flink/blob/670e92d0731652563fd58631107e0f19c5d5d9a1/flink-streaming-connectors/flink-connector-rabbitmq/src/main/java/org/apache/flink/streaming/connectors/rabbitmq/common/RMQConnectionConfig.java
and then pass to sink and source and declare queue based on this config, user 
will have more options to pass configurations.
I will start work on this once https://issues.apache.org/jira/browse/FLINK-3763 
will get merged.

> RabbitMQ Sink ability to publish to a different exchange
> 
>
> Key: FLINK-3769
> URL: https://issues.apache.org/jira/browse/FLINK-3769
> Project: Flink
>  Issue Type: Improvement
>  Components: Streaming Connectors
>Affects Versions: 1.0.1
>Reporter: Robert Batts
>Assignee: Subhankar Biswas
>  Labels: rabbitmq
>
> The RabbitMQ Sink can currently only publish to the "default" exchange. This 
> exchange is a direct exchange, so the routing key will route directly to the 
> queue name. Because of this, the current sink will only be 1-to-1-to-1 (1 job 
> to 1 exchange which routes to 1 queue). Additionally, I believe that if a 
> user decides to use a different exchange I think the following can be assumed:
> 1.) The provided exchange exists
> 2.) The user has declared the appropriate mapping and the appropriate queues 
> exist in RabbitMQ (therefore, nothing needs to be created)
> RabbitMQ currently provides four types of exchanges. Three of these will be 
> covered by just enabling exchanges (Direct, Fanout, Topic) because they use 
> the routingkey (or nothing). 
> The fourth exchange type relies on the message headers, which are currently 
> set to null by default on the publish. These headers may be on a per message 
> level, so the input of this stream will need to take this as input as well. 
> This forth exchange could very well be outside of the scope of this 
> Improvement and a "RabbitMQ Sink enable headers" Improvement might be the 
> better way to go with this.
> Exchange Types: https://www.rabbitmq.com/tutorials/amqp-concepts.html



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