Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Kostas Kloudas
Hi Tison, The changes that this FLIP propose are: - the introduction of the Executor interface - the fact that everything in the current state of job submission in Flink can be defined through configuration parameters - implementation of Executors that do not change any of the semantics of the cur

Testing DataStreams

2019-10-02 Thread Dominik Wosiński
Hello, I have a question, since I am observing quite weird behavior. In the documentation[1] the example of FlinkMiniCluster usage, shows that we can expect the results to appear in the same order as they were injected to the stream by use of *fromElements(). *I mean that Java version of the code i

Re: Checkpointing under backpressure

2019-10-02 Thread Arvid Heise
Sry incorrect link, please follow [1]. [1] https://mail-archives.apache.org/mod_mbox/flink-dev/201909.mbox/%3CCAGZNd0FgVL0oDQJHpBwJ1Ha8QevsVG0FHixdet11tLhW2p-2hg%40mail.gmail.com%3E On Wed, Oct 2, 2019 at 3:44 PM Arvid Heise wrote: > FYI, we published FLIP-76 to address the issue and discussion

Re: [DISCUSS] FLIP-74: Flink JobClient API

2019-10-02 Thread Zili Chen
Hi all, Narrow the scope to FLIP-74 we aimed at introduce a useful and extensible user-facing public interface JobClient. Let me reemphasize two major works under this thread. 1. standard interface As in FLIP-74 we introduce an interface JobClient with its methods, we'd like to make it a standar

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Zili Chen
- for Preview/OptimizedPlanEnv: I think they are orthogonal to the Executors work, as they are using the exexute() method because this is the only "entry" to the user program. To this regard, I believe we should just see the fact that they have their dedicated environment as an "implementation det

Re: [VOTE] FLIP-57: Rework FunctionCatalog

2019-10-02 Thread Bowen Li
Introducing a new term "path" to APIs like "getShortPath(Identifier)/getLongPath(Identifier)" would be confusing to users, thus I feel "getSimpleName/getIdentifier" is fine. To summarize the discussion result. - categorize functions into 2 dimensions - system v.s. catalog, non-temp v.s. tem

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Zili Chen
BTW, correct me if I misunderstand, now I learn more about our community way. Since FLIP-73 aimed at introducing an interface with community consensus the discussion is more about the interface in order to properly define a useful and extensible API. The integration story could be a follow up since

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Zili Chen
Hi Kostas, It seems does no harm we have a configuration parameter of Executor#execute since we can merge this one with the one configured on Executor created and let this one overwhelm that one. I can see it is useful that conceptually we can create an Executor for a series jobs to the same clus

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Kostas Kloudas
Hi again, I did not include this to my previous email, as this is related to the proposal on the FLIP itself. In the existing proposal, the Executor interface is the following. public interface Executor { JobExecutionResult execute(Pipeline pipeline) throws Exception; } This implies that al

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Kostas Kloudas
Hi all, I agree with Tison that we should disentangle threads so that people can work independently. For FLIP-73: - for Preview/OptimizedPlanEnv: I think they are orthogonal to the Executors work, as they are using the exexute() method because this is the only "entry" to the user program. To thi

[jira] [Created] (FLINK-14315) NPE with JobMaster.disconnectTaskManager

2019-10-02 Thread Steven Zhen Wu (Jira)
Steven Zhen Wu created FLINK-14315: -- Summary: NPE with JobMaster.disconnectTaskManager Key: FLINK-14315 URL: https://issues.apache.org/jira/browse/FLINK-14315 Project: Flink Issue Type: Bug

[SURVEY] Dropping non Credit-based Flow Control

2019-10-02 Thread Piotr Nowojski
Hi, In Flink 1.5 we have introduced Credit-based Flow Control [1] in the network stack. Back then we were aware about potential downsides of it [2] and we decided to keep the old model in the code base ( configurable by setting `taskmanager.network.credit-model: false` ). Now, that we are abou

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Zili Chen
Thanks for your thoughts Aljoscha. Another question since FLIP-73 might contains refactors on Environemnt: shall we support something like PreviewPlanEnvironment? If so, how? From a user perspective preview plan is useful, by give visual view, to modify topos and configure without submit it. Best

Re: [DISCUSS] FLIP-67: Global partitions lifecycle

2019-10-02 Thread Till Rohrmann
Thanks for addressing our comments Chesnay. See some comments inline. On Wed, Oct 2, 2019 at 4:07 PM Chesnay Schepler wrote: > Thank you for your comments; I've aggregated them a bit and added > comments to each of them. > > 1) Concept name (proposal: persistent) > > I agree that "global" is rat

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Aljoscha Krettek
I agree with Till that we should not change the semantics of per-job mode. In my opinion per-job mode means that the cluster (JobManager) is brought up with one job and it only executes that one job. There should be no open ports/anything that would allow submitting further jobs. This is very im

Re: [DISCUSS] FLIP-67: Global partitions lifecycle

2019-10-02 Thread Chesnay Schepler
Thank you for your comments; I've aggregated them a bit and added comments to each of them. 1) Concept name (proposal: persistent) I agree that "global" is rather undescriptive, particularly so since we never had a notion of "local" partitions. I'm not a fan of "persistent"; as to me this alwa

Re: Checkpointing under backpressure

2019-10-02 Thread Arvid Heise
FYI, we published FLIP-76 to address the issue and discussion has been opened in [1]. Looking forward to your feedback, Arvid [1] https://mail-archives.apache.org/mod_mbox/flink-dev/201909.mbox/browser On Thu, Aug 15, 2019 at 9:43 AM Yun Gao wrote: > Hi, > Very thanks for the great points

Re: [VOTE] FLIP-57: Rework FunctionCatalog

2019-10-02 Thread Dawid Wysakowicz
Hi, I very much agree with Xuefu's summary of the two points, especially on the "functionIdentifier doesn't need to reflect the categories". For the factory methods I think methods of should be enough: // for temporary/non-temporary system function public FunctionIdentifier of(String name)

Re: [CODE STYLE] Parameters of method are always final

2019-10-02 Thread Piotr Nowojski
+1 from my side. > On 2 Oct 2019, at 13:07, Zili Chen wrote: > > Yes exactly. > > > Piotr Nowojski 于2019年10月2日周三 下午7:03写道: > >> Hi Tison, >> >> To clarify your proposal. You are proposing to actually drop the >> `final` keyword from the parameters and we should implicilty assume that >

Re: [CODE STYLE] Parameters of method are always final

2019-10-02 Thread Zili Chen
Yes exactly. Piotr Nowojski 于2019年10月2日周三 下午7:03写道: > Hi Tison, > > To clarify your proposal. You are proposing to actually drop the > `final` keyword from the parameters and we should implicilty assume that > it’s always there (in other words, we shouldn’t be modifying the > parameters).

Re: [CODE STYLE] Parameters of method are always final

2019-10-02 Thread Piotr Nowojski
Hi Tison, To clarify your proposal. You are proposing to actually drop the `final` keyword from the parameters and we should implicilty assume that it’s always there (in other words, we shouldn’t be modifying the parameters). Did I understand this correctly? Piotrek > On 1 Oct 2019, at

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Zili Chen
Thanks for your clarification Till. I agree with the current semantics of the per-job mode, one should deploy a new cluster for each part of the job. Apart from the performance concern it also means that PerJobExecutor knows how to deploy a cluster actually, which is different from the description

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Till Rohrmann
I'm not saying that one cannot change the semantics of the per-job mode, I'm mainly wondering whether this needs to happen as part of this FLIP. In my understanding this has lot of overlap with the driver mode effort and maybe not so much the executors. However, I agree that it is helpful to have a

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Zili Chen
Hi Till, The purpose to post thoughts above here is because FLIP-73 is unclear on how to achieve PerJobExecutor. In order to discuss this topic it is necessary to clarify how per-job mode runs regardless what it is now. With PerJobExecutor called in Environment I don't think we still keep current

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Zili Chen
To be honest I formerly want to firstly start a thread discuss about what per-job mode means because things gets quite different whether or not per-job mode contains exactly one JobGraph or allow to have multiple part. Plus the complexity that whether or not we support post-execution logic it becom

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Till Rohrmann
I'm not sure whether removing the current per-job mode semantics all together is a good idea. It has some nice properties, for example the JobGraph stays constant. With your proposal which I would coin the driver mode, the JobGraph would be regenerated in case of a failover. Depending on the user c

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Zili Chen
Thomas, For your requirement on jar run REST endpoint we can follow a similar way as job cluster way described above. That is, we submit the jar and runs user program on cluster side. Also cc Aljoscha, There is no JobManager.runJar in mind. All logic that handles jar run is in WebMonitor, we pre

Re: Checkpoint metrics.

2019-10-02 Thread Stephan Ewen
Hi Jamie! (and adding Klou) I think the Streaming FIle Sink has a limit on the number of concurrent uploads. Could it be that too many uploads enqueue and at some point, the checkpoint blocks for a long time until that queue is worked off? Klou, do you have more insights here? Best, Stephan On

[jira] [Created] (FLINK-14314) Allocate shared slot resources respecting the resources of all vertices in the group

2019-10-02 Thread Zhu Zhu (Jira)
Zhu Zhu created FLINK-14314: --- Summary: Allocate shared slot resources respecting the resources of all vertices in the group Key: FLINK-14314 URL: https://issues.apache.org/jira/browse/FLINK-14314 Project: F

Re: [DISCUSS] FLIP-73: Introducing Executors for job submission

2019-10-02 Thread Zili Chen
Thanks for your thoughts Kostas! I agree Executor to be a concept on clients now. And sincerely second the description Now the Executor simply uses a client, e.g. a ClusterClient, to submit the job (JobGraph) that it will create from the user program. In that sense, the Executor is one level of a

[jira] [Created] (FLINK-14313) Add Gojek to Chinese Powered By page

2019-10-02 Thread Fabian Hueske (Jira)
Fabian Hueske created FLINK-14313: - Summary: Add Gojek to Chinese Powered By page Key: FLINK-14313 URL: https://issues.apache.org/jira/browse/FLINK-14313 Project: Flink Issue Type: Task

[jira] [Created] (FLINK-14312) Support logical pipelined region partitioning of JobGraph

2019-10-02 Thread Zhu Zhu (Jira)
Zhu Zhu created FLINK-14312: --- Summary: Support logical pipelined region partitioning of JobGraph Key: FLINK-14312 URL: https://issues.apache.org/jira/browse/FLINK-14312 Project: Flink Issue Type: S

[jira] [Created] (FLINK-14311) Streaming File Sink end-to-end test failed on Travis

2019-10-02 Thread Till Rohrmann (Jira)
Till Rohrmann created FLINK-14311: - Summary: Streaming File Sink end-to-end test failed on Travis Key: FLINK-14311 URL: https://issues.apache.org/jira/browse/FLINK-14311 Project: Flink Issue

Re: [DISCUSS] FLIP-74: Flink JobClient API

2019-10-02 Thread Zili Chen
Hi Konstantin, * should we add "cancelWithSavepeoint" to a new public API, when we have deprecated the corresponding REST API/CLI methods? In my understanding there is no reason to use it anymore. Good point. We can exclude "cancelWithSavepoint" from public API at least for now, since it is depre