[jira] [Comment Edited] (FLINK-29128) Uncatch IllegalStateException found when log split changes handling result in KafkaPartitionSplitReader
[ https://issues.apache.org/jira/browse/FLINK-29128?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17598157#comment-17598157 ] Leo zhang edited comment on FLINK-29128 at 8/31/22 4:10 AM: Actually, for unbounded case, we do assign the empty splits since it may eventually contain data. The "empty split" is only for bounded case, for unbounded case, this "empty split" is never considered as empty . Following I would show some evidence: 1、By default, if no stopping offset is set in the constructor, NO_STOPPING_OFFSET which will be assigned as Long.MIN_VALUE, a negative number. public KafkaPartitionSplit(TopicPartition tp, long startingOffset) { this(tp, startingOffset, NO_STOPPING_OFFSET); } 2、In KafkaPartitionSplit#getStoppingOffset, any negative number except LATEST_OFFSET (-1),COMMITTED_OFFSET (-3), will be considered as no stopping offset, so we return Optional.empty(). public Optional getStoppingOffset() { return stoppingOffset >= 0 || stoppingOffset == LATEST_OFFSET // -1 || stoppingOffset == COMMITTED_OFFSET //-3 ? Optional.of(stoppingOffset) : Optional.empty(); } 3、In KafkaPartitionSplitReader#parseStoppingOffsets, only when split.getStoppingOffset().ifPresent, we do parse the stopping offset.This means if the stopping offset is empty, no stopping offset will be parse. private void parseStoppingOffsets( KafkaPartitionSplit split, List partitionsStoppingAtLatest, Set partitionsStoppingAtCommitted) { TopicPartition tp = split.getTopicPartition(); split.getStoppingOffset() .ifPresent( stoppingOffset -> { if (stoppingOffset >= 0) { stoppingOffsets.put(tp, stoppingOffset); } else if (stoppingOffset == KafkaPartitionSplit.LATEST_OFFSET) { partitionsStoppingAtLatest.add(tp); } else if (stoppingOffset == KafkaPartitionSplit.COMMITTED_OFFSET) { partitionsStoppingAtCommitted.add(tp); } else { // This should not happen. throw new FlinkRuntimeException( String.format( "Invalid stopping offset %d for partition %s", stoppingOffset, tp)); } }); } 4、If stopping offset is LATEST_OFFSET, COMMITTED_OFFSET, it will be set to actual offset by KafkaPartitionSplitReader#acquireAndSetStoppingOffsets. 5、In KafkaPartitionSplitReader#getStoppingOffset, the default stopping offset is set to Long.MAX_VALUE. This means if no stopping offset is set or parsed, the stream's stopping offset will be considered as Long.MAX_VALUE, just like the stream is unbounded, and will be run until the offset is up to Long.MAX_VALUE. private long getStoppingOffset(TopicPartition tp) { return stoppingOffsets.getOrDefault(tp, Long.MAX_VALUE); } I make a summary here: 1、Only when stoppingOffset >=0, or equal to LATEST_OFFSET ,COMMITTED_OFFSET , a stopping offset will be parse and set. In this case, it's bounded.If the starting offset is equal to or less than the stopping offset, an empty split is found, and this partition will be unassign. 2、When the stopping offset is not set, or the stopping offset is set to a wrong negative number, this stream is considered as unbounded, the stopping offset will be considered as Long.MAX_VALUE.In this case, no empty split will be found in unbounded mode. Since a change split maybe found as a empty split, and relative topic partition will by unsigned, this situation should be considered in KafkaPartitionSplitReader#maybeLogSplitChangesHandlingResult. That how I solve this bug in the pull request. was (Author: JIRAUSER285575): Actually, for unbounded case, we do assign the empty splits since it may eventually contain data. The "empty split" is only for bounded case, for unbounded case, this "empty split" is never considered as empty . Following I would show some evidence: 1、By default, if no stopping offset is set in the constructor, NO_STOPPING_OFFSET which will be assigned as Long.MIN_VALUE, a negative number. public KafkaPartitionSplit(TopicPartition tp, long startingOffset) { this(tp, startingOffset, NO_STOPPING_OFFSET); } 2、In KafkaPartitionSplit#getStoppingOffset, any negative number except LATEST_OFFSET (-1),COMMITTED_OFFSET (-3), will be considered as no stopping offset, so we return Optional.empty(). public Optional getStoppingOffset() { return stoppingOffset >= 0 || stoppingOffset == LATEST_OFFSET // -1 || stoppingOffset == COMMITTED_OFFSET //-3 ? Optional.of(stoppingOffset) : Optional.empty(); } 3、In KafkaPartitionSplitReader#parseStoppingOffsets, only when split.getStoppingOffset().ifPresent, we do parse the stopping offset.This means if the stopping offset is empty, no stopping offset will be parse. private void parseStoppingOffsets( KafkaPartitionSplit split, List partitionsStoppingAtLatest, Set partitionsStoppingAtCommitted) { TopicPartition tp = split.getTopicPartition(); split.getStoppingOffset() .ifPresent( stoppingOffset -> { if (stoppingOffset >= 0) { stoppingOffsets.put(tp, stoppingOffset); } else if (stoppingOffset ==
[jira] [Comment Edited] (FLINK-29128) Uncatch IllegalStateException found when log split changes handling result in KafkaPartitionSplitReader
[ https://issues.apache.org/jira/browse/FLINK-29128?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17598157#comment-17598157 ] Leo zhang edited comment on FLINK-29128 at 8/31/22 4:05 AM: Actually, for unbounded case, we do assign the empty splits since it may eventually contain data. The "empty split" is only for bounded case, for unbounded case, this "empty split" is never considered as empty . Following I would show some evidence: 1、By default, if no stopping offset is set in the constructor, NO_STOPPING_OFFSET which will be assigned as Long.MIN_VALUE, a negative number. public KafkaPartitionSplit(TopicPartition tp, long startingOffset) { this(tp, startingOffset, NO_STOPPING_OFFSET); } 2、In KafkaPartitionSplit#getStoppingOffset, any negative number except LATEST_OFFSET (-1),COMMITTED_OFFSET (-3), will be considered as no stopping offset, so we return Optional.empty(). public Optional getStoppingOffset() { return stoppingOffset >= 0 || stoppingOffset == LATEST_OFFSET // -1 || stoppingOffset == COMMITTED_OFFSET //-3 ? Optional.of(stoppingOffset) : Optional.empty(); } 3、In KafkaPartitionSplitReader#parseStoppingOffsets, only when split.getStoppingOffset().ifPresent, we do parse the stopping offset.This means if the stopping offset is empty, no stopping offset will be parse. private void parseStoppingOffsets( KafkaPartitionSplit split, List partitionsStoppingAtLatest, Set partitionsStoppingAtCommitted) { TopicPartition tp = split.getTopicPartition(); split.getStoppingOffset() .ifPresent( stoppingOffset -> { if (stoppingOffset >= 0) { stoppingOffsets.put(tp, stoppingOffset); } else if (stoppingOffset == KafkaPartitionSplit.LATEST_OFFSET) { partitionsStoppingAtLatest.add(tp); } else if (stoppingOffset == KafkaPartitionSplit.COMMITTED_OFFSET) { partitionsStoppingAtCommitted.add(tp); } else { // This should not happen. throw new FlinkRuntimeException( String.format( "Invalid stopping offset %d for partition %s", stoppingOffset, tp)); } }); } 4、If stopping offset is LATEST_OFFSET, COMMITTED_OFFSET, it will be set to actual offset by KafkaPartitionSplitReader#acquireAndSetStoppingOffsets. 5、In KafkaPartitionSplitReader#getStoppingOffset, the default stopping offset is set to Long.MAX_VALUE. This means if no stopping offset is set or parsed, the stream's stopping offset will be considered as Long.MAX_VALUE, just like the stream is unbounded, and will be run until the offset is up to Long.MAX_VALUE. private long getStoppingOffset(TopicPartition tp) { return stoppingOffsets.getOrDefault(tp, Long.MAX_VALUE); } I make a summary here: 1、Only when stoppingOffset >=0, or equal to LATEST_OFFSET ,COMMITTED_OFFSET , a stopping offset will be parse and set. In this case, it's bounded.If the starting offset is equal to or less than the stopping offset, an empty split is found, and this partition will be unassign. 2、When the stopping offset is not set, or the stopping offset is set to a wrong negative number, this stream is considered as unbounded, the stopping offset will be considered as Long.MAX_VALUE.In this case, no empty split will be found in unbounded mode. was (Author: JIRAUSER285575): Actually, for unbounded case, we do assign the empty splits since it may eventually contain data. The "empty split" is only for bounded case, for unbounded case, this "empty case" is not considered as empty case. Following I would show some evidence: 1、By default, if no stopping offset is set in the constructor, NO_STOPPING_OFFSET which is assigned as Long.MIN_VALUE, a negative number, will be set. public KafkaPartitionSplit(TopicPartition tp, long startingOffset) { this(tp, startingOffset, NO_STOPPING_OFFSET); } 2、In KafkaPartitionSplit#getStoppingOffset, any negative number except LATEST_OFFSET (-1),COMMITTED_OFFSET (-3), will be considered as no stopping offset, so we return Optional.empty(). public Optional getStoppingOffset() { return stoppingOffset >= 0 || stoppingOffset == LATEST_OFFSET // -1 || stoppingOffset == COMMITTED_OFFSET //-3 ? Optional.of(stoppingOffset) : Optional.empty(); } 3、In KafkaPartitionSplitReader#parseStoppingOffsets, only when split.getStoppingOffset().ifPresent, we do parse the stopping offset.This means if no stopping offset is set, or the stopping offset is set to a wrong negative number, no stopping offset will be parse. private void parseStoppingOffsets( KafkaPartitionSplit split, List partitionsStoppingAtLatest, Set partitionsStoppingAtCommitted) { TopicPartition tp = split.getTopicPartition(); split.getStoppingOffset() .ifPresent( stoppingOffset -> { if (stoppingOffset >= 0) { stoppingOffsets.put(tp, stoppingOffset); } else if (stoppingOffset == KafkaPartitionSplit.LATEST_OFFSET) { partitionsStoppingAtLatest.add(tp); } else if (stoppingOffset == KafkaPartitionSplit.COMMITTED_OFFSET) { partitionsStoppingAtCommitted.add(tp); } else { //
[jira] [Commented] (FLINK-29128) Uncatch IllegalStateException found when log split changes handling result in KafkaPartitionSplitReader
[ https://issues.apache.org/jira/browse/FLINK-29128?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17598157#comment-17598157 ] Leo zhang commented on FLINK-29128: --- Actually, for unbounded case, we do assign the empty splits since it may eventually contain data. The "empty split" is only for bounded case, for unbounded case, this "empty case" is not considered as empty case. Following I would show some evidence: 1、By default, if no stopping offset is set in the constructor, NO_STOPPING_OFFSET which is assigned as Long.MIN_VALUE, a negative number, will be set. public KafkaPartitionSplit(TopicPartition tp, long startingOffset) { this(tp, startingOffset, NO_STOPPING_OFFSET); } 2、In KafkaPartitionSplit#getStoppingOffset, any negative number except LATEST_OFFSET (-1),COMMITTED_OFFSET (-3), will be considered as no stopping offset, so we return Optional.empty(). public Optional getStoppingOffset() { return stoppingOffset >= 0 || stoppingOffset == LATEST_OFFSET // -1 || stoppingOffset == COMMITTED_OFFSET //-3 ? Optional.of(stoppingOffset) : Optional.empty(); } 3、In KafkaPartitionSplitReader#parseStoppingOffsets, only when split.getStoppingOffset().ifPresent, we do parse the stopping offset.This means if no stopping offset is set, or the stopping offset is set to a wrong negative number, no stopping offset will be parse. private void parseStoppingOffsets( KafkaPartitionSplit split, List partitionsStoppingAtLatest, Set partitionsStoppingAtCommitted) { TopicPartition tp = split.getTopicPartition(); split.getStoppingOffset() .ifPresent( stoppingOffset -> { if (stoppingOffset >= 0) { stoppingOffsets.put(tp, stoppingOffset); } else if (stoppingOffset == KafkaPartitionSplit.LATEST_OFFSET) { partitionsStoppingAtLatest.add(tp); } else if (stoppingOffset == KafkaPartitionSplit.COMMITTED_OFFSET) { partitionsStoppingAtCommitted.add(tp); } else { // This should not happen. throw new FlinkRuntimeException( String.format( "Invalid stopping offset %d for partition %s", stoppingOffset, tp)); } }); } 4、If stopping offset is LATEST_OFFSET, COMMITTED_OFFSET, we will set to actual offset by KafkaPartitionSplitReader#acquireAndSetStoppingOffsets. 5、In KafkaPartitionSplitReader#getStoppingOffset, the default stopping offset is set to Long.MAX_VALUE. This means if no stopping offset is set, the stream's stopping offset will be set to Long.MAX_VALUE, just like the stream is unbounded, and will be run until the offset is up to Long.MAX_VALUE. private long getStoppingOffset(TopicPartition tp) { return stoppingOffsets.getOrDefault(tp, Long.MAX_VALUE); } I make a summary here: 1、Only when stoppingOffset >=0, or equal to LATEST_OFFSET ,COMMITTED_OFFSET , a stopping offset will be parse and set. In this case, it's bounded.If the starting offset is equal to the stopping offset, an empty split is found, and this partition will by unassign. 2、When the stopping offset is not set, or the stopping offset is set to a wrong negative number, this stream is considered as unbounded, the stopping offset will be considered as Long.MAX_VALUE.In this case, no empty split will be found in unbounded mode. > Uncatch IllegalStateException found when log split changes handling result in > KafkaPartitionSplitReader > --- > > Key: FLINK-29128 > URL: https://issues.apache.org/jira/browse/FLINK-29128 > Project: Flink > Issue Type: Bug > Components: Connectors / Kafka >Affects Versions: 1.14.5, 1.15.2 >Reporter: Leo zhang >Assignee: Leo zhang >Priority: Minor > Labels: pull-request-available > > When logger is set to debug mode, > KafkaPartitionSplitReader#maybeLogSplitChangesHandingResult log the handing > result of all SplitsChange, and the handling result > include the kafka partition's starting offset, which is get from kafka > api(consumer.position). > When a SplitsChange is a empty split,it will be > removed(unassign partition), IllegalStateException will be thrown by > consumer.position, since we can only check the position for partitions > assigned to the consumer.And this exception has not been catch, and is > rethrown as RuntimeExption, which lead to a failure of the application's > execution. -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-29128) Uncatch IllegalStateException found when log split changes handling result in KafkaPartitionSplitReader
Leo zhang created FLINK-29128: - Summary: Uncatch IllegalStateException found when log split changes handling result in KafkaPartitionSplitReader Key: FLINK-29128 URL: https://issues.apache.org/jira/browse/FLINK-29128 Project: Flink Issue Type: Bug Components: Connectors / Kafka Affects Versions: 1.15.2, 1.14.5 Reporter: Leo zhang When logger is set to debug mode, KafkaPartitionSplitReader#maybeLogSplitChangesHandingResult log the handing result of all SplitsChange, and the handling result include the kafka partition's starting offset, which is get from kafka api(consumer.position). When a SplitsChange is a empty split,it will be removed(unassign partition), IllegalStateException will be thrown by consumer.position, since we can only check the position for partitions assigned to the consumer.And this exception has not been catch, and is rethrown as RuntimeExption, which lead to a failure of the application's execution. -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-28475) kafka connector won't stop when the stopping offset is zero
Leo zhang created FLINK-28475: - Summary: kafka connector won't stop when the stopping offset is zero Key: FLINK-28475 URL: https://issues.apache.org/jira/browse/FLINK-28475 Project: Flink Issue Type: Bug Components: Connectors / Kafka Affects Versions: 1.15.0 Reporter: Leo zhang when use kafka connector in bounded mode,and the stopping offset hapends to be 0,the kafka connector won't stop,which is not expected. I had traced the code, and found the stopping offset will be set to empty when it is zero, and an empty stopping offset means no stopping offset when serialized. This leads to a wrong execution. I had fixed this in my personal branch,now I am logging this issue in Jira so that I can make merge request. -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Commented] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16993116#comment-16993116 ] Leo Zhang commented on FLINK-14055: --- The committer requires a discussion first. So please join the discussion on dev mailing list: [DISCUSS] Register user jar files in \{Stream}ExecutionEnvironment > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Commented] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16993113#comment-16993113 ] Leo Zhang commented on FLINK-14319: --- Please join the discussion on dev mailing list: [DISCUSS] Register user jar files in \{Stream}ExecutionEnvironment > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Time Spent: 20m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement > applications based on loading external jars for now, not just SQL but also > streaming ones. And the related API proposals have been issued in the > sub-task FLINK-14055 under task FLINK-10232 Add a SQL DDL. > To support this sub-task FLINK-14055 , we need the other new task for > \{Stream}ExecutionEnvironment::registerUserJarFile() interface which will be > addressed in this issue. > Here is the plan. > *Design* > * Add interface > {code:java} > void registerUserJarFile(String jarFile) > {code} > into _StreamExecutionEnvironment_ ( in module flink-streaming-java). The > affected classes are _StreamGraph_, _StreamGraphGenerator_, > _StreamingJobGraphGenerator_ to support getting and setting a list of user > jars. And all they are in module flink-streaming-java. > * Add interface > {code:java} > void registerUserJarFile(String jarFile) > {code} > into _ExecutionEnvironment_ (in module flink-java). The affected classes is > _Plan_, in module flink-core, to support getting and setting a list of user > jars. > * Add interface > {code:java} > void addUserJars(List userJars, JobGraph jobGraph) > {code} > into _JobGraphGenerator_ and add the user jars within the method _JobGraph_ > _compileJobGraph(OptimizedPlan program, JobID jobId)_ so that user jars can > be shipped with user's program and submitted to cluster. _JobGraphGenerator_ > is in module flink-optimizer. > * Add interface > {code:java} > void registerUserJarFile(String jarFile) > {code} > into \{Stream}ExecutionEnvironment (in module flink-scala and > flink-streaming-scala) and just use the wrapped _javaEnv_ to achieve > registration. > *Testing* > * One test case for adding local user jars both in the streaming and batch > jobs. We need to process test classes into a jar before testing. For this > purpose, we can add a goal in process-test-classes for this testing case in > the pom file. The affected module is flink-tests. > * Another test case for adding use jars in HDFS. The same idea with the > previous one. The affected module is flink-fs-tests. > * Note that python API is not included in this issue just as registering > cached files. But we still need to modify some python test cases in order to > avoid building error as lacking some methods declared in java. The affected > files are > _flink-python/pyflink/dataset/tests/test_execution_environment_completeness.py_ > and > _flink-python/pyflink/datastream/tests/test_stream_execution_environment_completeness.py_. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Updated] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Leo Zhang updated FLINK-14319: -- Description: I see that there are some use cases in which people want to implement applications based on loading external jars for now, not just SQL but also streaming ones. And the related API proposals have been issued in the sub-task FLINK-14055 under task FLINK-10232 Add a SQL DDL. To support this sub-task FLINK-14055 , we need the other new task for \{Stream}ExecutionEnvironment::registerUserJarFile() interface which will be addressed in this issue. Here is the plan. *Design* * Add interface {code:java} void registerUserJarFile(String jarFile) {code} into _StreamExecutionEnvironment_ ( in module flink-streaming-java). The affected classes are _StreamGraph_, _StreamGraphGenerator_, _StreamingJobGraphGenerator_ to support getting and setting a list of user jars. And all they are in module flink-streaming-java. * Add interface {code:java} void registerUserJarFile(String jarFile) {code} into _ExecutionEnvironment_ (in module flink-java). The affected classes is _Plan_, in module flink-core, to support getting and setting a list of user jars. * Add interface {code:java} void addUserJars(List userJars, JobGraph jobGraph) {code} into _JobGraphGenerator_ and add the user jars within the method _JobGraph_ _compileJobGraph(OptimizedPlan program, JobID jobId)_ so that user jars can be shipped with user's program and submitted to cluster. _JobGraphGenerator_ is in module flink-optimizer. * Add interface {code:java} void registerUserJarFile(String jarFile) {code} into \{Stream}ExecutionEnvironment (in module flink-scala and flink-streaming-scala) and just use the wrapped _javaEnv_ to achieve registration. *Testing* * One test case for adding local user jars both in the streaming and batch jobs. We need to process test classes into a jar before testing. For this purpose, we can add a goal in process-test-classes for this testing case in the pom file. The affected module is flink-tests. * Another test case for adding use jars in HDFS. The same idea with the previous one. The affected module is flink-fs-tests. * Note that python API is not included in this issue just as registering cached files. But we still need to modify some python test cases in order to avoid building error as lacking some methods declared in java. The affected files are _flink-python/pyflink/dataset/tests/test_execution_environment_completeness.py_ and _flink-python/pyflink/datastream/tests/test_stream_execution_environment_completeness.py_. was: I see that there are some use cases in which people want to implement applications based on loading external jars for now, not just SQL but also streaming ones. And the related API proposals have been issued in the sub-task FLINK-14055 under task FLINK-10232 Add a SQL DDL. To support this sub-task FLINK-14055 , we need the other new task for \{Stream}ExecutionEnvironment::registerUserJarFile() interface which will be addressed in this issue. Here is the plan. *Design* * Add interface {code:java} void registerUserJarFile(String jarFile) {code} into _StreamExecutionEnvironment_ ( in module flink-streaming-java). The affected classes are _StreamGraph_, _StreamGraphGenerator_, _StreamingJobGraphGenerator_ to support getting and setting a list of user jars. And all they are in module flink-streaming-java. * Add _void_ _registerUserJarFile(String jarFile)_ into _ExecutionEnvironment_ (in module flink-java). The affected classes is _Plan_, in module flink-core, to support getting and setting a list of user jars. {code:java} void registerUserJarFile(String jarFile){code} * Add interface {code:java} void addUserJars(List userJars, JobGraph jobGraph) {code} into _JobGraphGenerator_ and add the user jars within the method _JobGraph_ _compileJobGraph(OptimizedPlan program, JobID jobId)_ so that user jars can be shipped with user's program and submitted to cluster. _JobGraphGenerator_ is in module flink-optimizer. * Add interface {code:java} void registerUserJarFile(String jarFile) {code} into \{Stream}ExecutionEnvironment (in module flink-scala and flink-streaming-scala) and just use the wrapped _javaEnv_ to achieve registration. *Testing* * One test case for adding local user jars both in the streaming and batch jobs. We need to process test classes into a jar before testing. For this purpose, we can add a goal in process-test-classes for this testing case in the pom file. The affected module is flink-tests. * Another test case for adding use jars in HDFS. The same idea with the previous one. The affected module is flink-fs-tests. * Note that python API is not included in this issue just as registering cached files. But we still need to modify some python test cases in order to avoid building error
[jira] [Updated] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Leo Zhang updated FLINK-14319: -- Description: I see that there are some use cases in which people want to implement applications based on loading external jars for now, not just SQL but also streaming ones. And the related API proposals have been issued in the sub-task FLINK-14055 under task FLINK-10232 Add a SQL DDL. To support this sub-task FLINK-14055 , we need the other new task for \{Stream}ExecutionEnvironment::registerUserJarFile() interface which will be addressed in this issue. Here is the plan. *Design* * Add interface {code:java} void registerUserJarFile(String jarFile) {code} into _StreamExecutionEnvironment_ ( in module flink-streaming-java). The affected classes are _StreamGraph_, _StreamGraphGenerator_, _StreamingJobGraphGenerator_ to support getting and setting a list of user jars. And all they are in module flink-streaming-java. * Add _void_ _registerUserJarFile(String jarFile)_ into _ExecutionEnvironment_ (in module flink-java). The affected classes is _Plan_, in module flink-core, to support getting and setting a list of user jars. {code:java} void registerUserJarFile(String jarFile){code} * Add interface {code:java} void addUserJars(List userJars, JobGraph jobGraph) {code} into _JobGraphGenerator_ and add the user jars within the method _JobGraph_ _compileJobGraph(OptimizedPlan program, JobID jobId)_ so that user jars can be shipped with user's program and submitted to cluster. _JobGraphGenerator_ is in module flink-optimizer. * Add interface {code:java} void registerUserJarFile(String jarFile) {code} into \{Stream}ExecutionEnvironment (in module flink-scala and flink-streaming-scala) and just use the wrapped _javaEnv_ to achieve registration. *Testing* * One test case for adding local user jars both in the streaming and batch jobs. We need to process test classes into a jar before testing. For this purpose, we can add a goal in process-test-classes for this testing case in the pom file. The affected module is flink-tests. * Another test case for adding use jars in HDFS. The same idea with the previous one. The affected module is flink-fs-tests. * Note that python API is not included in this issue just as registering cached files. But we still need to modify some python test cases in order to avoid building error as lacking some methods declared in java. The affected files are _flink-python/pyflink/dataset/tests/test_execution_environment_completeness.py_ and _flink-python/pyflink/datastream/tests/test_stream_execution_environment_completeness.py_. was: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel like it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for \{Stream}ExecutionEnvironment::registerUserJarFile() interface which will be addressed in this issue. Here is the plan. *Design* * Add _void_ _registerUserJarFile(String jarFile)_ into _StreamExecutionEnvironment_ ( in module flink-streaming-java). The affected classes are _StreamGraph_, _StreamGraphGenerator_, _StreamingJobGraphGenerator_ to support getting and setting a list of user jars. And all they are in module flink-streaming-java. * Add _void_ _registerUserJarFile(String jarFile)_ into _ExecutionEnvironment_ (in module flink-java). The affected classes is _Plan_, in module flink-core, to support getting and setting a list of user jars. * Add _void addUserJars(List userJars, JobGraph jobGraph)_ into _JobGraphGenerator_ and add the user jars within the method _JobGraph_ _compileHobGraph(OptimizedPlan program, JobID jobId)_ so that user jars can be shipped with user's program and submitted to cluster. _JobGraphGenerator_ is in module flink-optimizer. * Add _void_ _registerUserJarFile(String jarFile)_ into \{Stream}ExecutionEnvironment (in module flink-scala and flink-streaming-scala) and just use the wrapped _javaEnv_ to achieve registration. *Testing* * One test case for adding local user jars both in the streaming and batch jobs. We need to process test classes into a jar before testing. For this purpose, we can add a goal in process-test-classes for this testing case in the pom file. The affected module is flink-tests. * Another test case for adding use jars in HDFS. The same idea with the previous one. The affected module is flink-fs-tests. * Note that python API is not included in this issue just as registering cached files. But we still need to modify some python test cases in order to avoid building error as lacking some methods declared in java. The
[jira] [Updated] (FLINK-14729) Multi-topics consuming from KafkaTableSource
[ https://issues.apache.org/jira/browse/FLINK-14729?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Leo Zhang updated FLINK-14729: -- Fix Version/s: (was: 1.10.0) > Multi-topics consuming from KafkaTableSource > > > Key: FLINK-14729 > URL: https://issues.apache.org/jira/browse/FLINK-14729 > Project: Flink > Issue Type: New Feature > Components: Connectors / Kafka >Reporter: Leo Zhang >Priority: Major > Labels: features > > Hi, all. I propose a new functionality of KafkaTableSource which can consume > multiple topics at the same time. > *Design plan* > * Add a new constructor in KafkaTableSource which accepts topics with List > type as one parameter. > * Modify the existed one which only accepts one topic as string type to call > the proposed one to finish the instantiation. That is to say, wrap this topic > in a list and pass it to the multi-topics-consuming constructor. > * Modify the overridden method createKafkaConsumer in KafkaTableSource to > pass topics as List instead of String. > * Replace the field topic with topics as List type in KafkaTableSourceBase > and modify every place using topic with topics. So we just need to modify the > constructor KafkaTableSourceBase, method getDataStream, and equals and > hashCode. > *Test plan* > There is less to do as KafkaTableSource is based on FlinkKafkaConsumer which > already supports consuming multiple topics and is tested well. Of course, we > can easily add further more tests if needed. > > So what's your opinion? -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Updated] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Leo Zhang updated FLINK-14319: -- Fix Version/s: (was: 1.10.0) > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Time Spent: 20m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > \{Stream}ExecutionEnvironment::registerUserJarFile() interface which will be > addressed in this issue. > Here is the plan. > *Design* > * Add _void_ _registerUserJarFile(String jarFile)_ into > _StreamExecutionEnvironment_ ( in module flink-streaming-java). The affected > classes are _StreamGraph_, _StreamGraphGenerator_, > _StreamingJobGraphGenerator_ to support getting and setting a list of user > jars. And all they are in module flink-streaming-java. > * Add _void_ _registerUserJarFile(String jarFile)_ into > _ExecutionEnvironment_ (in module flink-java). The affected classes is > _Plan_, in module flink-core, to support getting and setting a list of user > jars. > * Add _void addUserJars(List userJars, JobGraph jobGraph)_ into > _JobGraphGenerator_ and add the user jars within the method _JobGraph_ > _compileHobGraph(OptimizedPlan program, JobID jobId)_ so that user jars can > be shipped with user's program and submitted to cluster. _JobGraphGenerator_ > is in module flink-optimizer. > * Add _void_ _registerUserJarFile(String jarFile)_ into > \{Stream}ExecutionEnvironment (in module flink-scala and > flink-streaming-scala) and just use the wrapped _javaEnv_ to achieve > registration. > *Testing* > * One test case for adding local user jars both in the streaming and batch > jobs. We need to process test classes into a jar before testing. For this > purpose, we can add a goal in process-test-classes for this testing case in > the pom file. The affected module is flink-tests. > * Another test case for adding use jars in HDFS. The same idea with the > previous one. The affected module is flink-fs-tests. > * Note that python API is not included in this issue just as registering > cached files. But we still need to modify some python test cases in order to > avoid building error as lacking some methods declared in java. The affected > files are > _flink-python/pyflink/dataset/tests/test_execution_environment_completeness.py_ > and > _flink-python/pyflink/datastream/tests/test_stream_execution_environment_completeness.py_. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-14729) Multi-topics consuming from KafkaTableSource
Leo Zhang created FLINK-14729: - Summary: Multi-topics consuming from KafkaTableSource Key: FLINK-14729 URL: https://issues.apache.org/jira/browse/FLINK-14729 Project: Flink Issue Type: New Feature Components: Connectors / Kafka Reporter: Leo Zhang Fix For: 1.10.0 Hi, all. I propose a new functionality of KafkaTableSource which can consume multiple topics at the same time. *Design plan* * Add a new constructor in KafkaTableSource which accepts topics with List type as one parameter. * Modify the existed one which only accepts one topic as string type to call the proposed one to finish the instantiation. That is to say, wrap this topic in a list and pass it to the multi-topics-consuming constructor. * Modify the overridden method createKafkaConsumer in KafkaTableSource to pass topics as List instead of String. * Replace the field topic with topics as List type in KafkaTableSourceBase and modify every place using topic with topics. So we just need to modify the constructor KafkaTableSourceBase, method getDataStream, and equals and hashCode. *Test plan* There is less to do as KafkaTableSource is based on FlinkKafkaConsumer which already supports consuming multiple topics and is tested well. Of course, we can easily add further more tests if needed. So what's your opinion? -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Commented] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16965265#comment-16965265 ] Leo Zhang commented on FLINK-14319: --- Once this issue gets enough votes and the community reaches an agreement on it, I will open that PR again. Please vote for this if it looks good to you. [~simonss] > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Fix For: 1.10.0 > > Time Spent: 20m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > \{Stream}ExecutionEnvironment::registerUserJarFile() interface which will be > addressed in this issue. > Here is the plan. > *Design* > * Add _void_ _registerUserJarFile(String jarFile)_ into > _StreamExecutionEnvironment_ ( in module flink-streaming-java). The affected > classes are _StreamGraph_, _StreamGraphGenerator_, > _StreamingJobGraphGenerator_ to support getting and setting a list of user > jars. And all they are in module flink-streaming-java. > * Add _void_ _registerUserJarFile(String jarFile)_ into > _ExecutionEnvironment_ (in module flink-java). The affected classes is > _Plan_, in module flink-core, to support getting and setting a list of user > jars. > * Add _void addUserJars(List userJars, JobGraph jobGraph)_ into > _JobGraphGenerator_ and add the user jars within the method _JobGraph_ > _compileHobGraph(OptimizedPlan program, JobID jobId)_ so that user jars can > be shipped with user's program and submitted to cluster. _JobGraphGenerator_ > is in module flink-optimizer. > * Add _void_ _registerUserJarFile(String jarFile)_ into > \{Stream}ExecutionEnvironment (in module flink-scala and > flink-streaming-scala) and just use the wrapped _javaEnv_ to achieve > registration. > *Testing* > * One test case for adding local user jars both in the streaming and batch > jobs. We need to process test classes into a jar before testing. For this > purpose, we can add a goal in process-test-classes for this testing case in > the pom file. The affected module is flink-tests. > * Another test case for adding use jars in HDFS. The same idea with the > previous one. The affected module is flink-fs-tests. > * Note that python API is not included in this issue just as registering > cached files. But we still need to modify some python test cases in order to > avoid building error as lacking some methods declared in java. The affected > files are > _flink-python/pyflink/dataset/tests/test_execution_environment_completeness.py_ > and > _flink-python/pyflink/datastream/tests/test_stream_execution_environment_completeness.py_. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Commented] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16947284#comment-16947284 ] Leo Zhang commented on FLINK-14319: --- Thanks. The PR is closed for now. And I proposed a plan for design and testing. Please review this and vote for this issue if it's reasonable. :) > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Fix For: 1.10.0 > > Time Spent: 20m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > \{Stream}ExecutionEnvironment::registerUserJarFile() interface which will be > addressed in this issue. > Here is the plan. > *Design* > * Add _void_ _registerUserJarFile(String jarFile)_ into > _StreamExecutionEnvironment_ ( in module flink-streaming-java). The affected > classes are _StreamGraph_, _StreamGraphGenerator_, > _StreamingJobGraphGenerator_ to support getting and setting a list of user > jars. And all they are in module flink-streaming-java. > * Add _void_ _registerUserJarFile(String jarFile)_ into > _ExecutionEnvironment_ (in module flink-java). The affected classes is > _Plan_, in module flink-core, to support getting and setting a list of user > jars. > * Add _void addUserJars(List userJars, JobGraph jobGraph)_ into > _JobGraphGenerator_ and add the user jars within the method _JobGraph_ > _compileHobGraph(OptimizedPlan program, JobID jobId)_ so that user jars can > be shipped with user's program and submitted to cluster. _JobGraphGenerator_ > is in module flink-optimizer. > * Add _void_ _registerUserJarFile(String jarFile)_ into > \{Stream}ExecutionEnvironment (in module flink-scala and > flink-streaming-scala) and just use the wrapped _javaEnv_ to achieve > registration. > *Testing* > * One test case for adding local user jars both in the streaming and batch > jobs. We need to process test classes into a jar before testing. For this > purpose, we can add a goal in process-test-classes for this testing case in > the pom file. The affected module is flink-tests. > * Another test case for adding use jars in HDFS. The same idea with the > previous one. The affected module is flink-fs-tests. > * Note that python API is not included in this issue just as registering > cached files. But we still need to modify some python test cases in order to > avoid building error as lacking some methods declared in java. The affected > files are > _flink-python/pyflink/dataset/tests/test_execution_environment_completeness.py_ > and > _flink-python/pyflink/datastream/tests/test_stream_execution_environment_completeness.py_. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Updated] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Leo Zhang updated FLINK-14319: -- Description: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel like it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for \{Stream}ExecutionEnvironment::registerUserJarFile() interface which will be addressed in this issue. Here is the plan. *Design* * Add _void_ _registerUserJarFile(String jarFile)_ into _StreamExecutionEnvironment_ ( in module flink-streaming-java). The affected classes are _StreamGraph_, _StreamGraphGenerator_, _StreamingJobGraphGenerator_ to support getting and setting a list of user jars. And all they are in module flink-streaming-java. * Add _void_ _registerUserJarFile(String jarFile)_ into _ExecutionEnvironment_ (in module flink-java). The affected classes is _Plan_, in module flink-core, to support getting and setting a list of user jars. * Add _void addUserJars(List userJars, JobGraph jobGraph)_ into _JobGraphGenerator_ and add the user jars within the method _JobGraph_ _compileHobGraph(OptimizedPlan program, JobID jobId)_ so that user jars can be shipped with user's program and submitted to cluster. _JobGraphGenerator_ is in module flink-optimizer. * Add _void_ _registerUserJarFile(String jarFile)_ into \{Stream}ExecutionEnvironment (in module flink-scala and flink-streaming-scala) and just use the wrapped _javaEnv_ to achieve registration. *Testing* * One test case for adding local user jars both in the streaming and batch jobs. We need to process test classes into a jar before testing. For this purpose, we can add a goal in process-test-classes for this testing case in the pom file. The affected module is flink-tests. * Another test case for adding use jars in HDFS. The same idea with the previous one. The affected module is flink-fs-tests. * Note that python API is not included in this issue just as registering cached files. But we still need to modify some python test cases in order to avoid building error as lacking some methods declared in java. The affected files are _flink-python/pyflink/dataset/tests/test_execution_environment_completeness.py_ and _flink-python/pyflink/datastream/tests/test_stream_execution_environment_completeness.py_. was: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel like it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for _{Stream}ExecutionEnvironment::registerUserJarFile()_ interface which will be addressed in this issue. Here is the plan. *Design* * Add _void_ _registerUserJarFile(String jarFile)_ into StreamExecutionEnvironment ( in module flink-streaming-java). The affected classes are StreamGraph, StreamGraphGenerator, StreamingJobGraphGenerator to support getting and setting a list of user jars. And all they are in module flink-streaming-java. * Add _void_ _registerUserJarFile(String jarFile)_ into ExecutionEnvironment (in module flink-java). The affected classes is Plan, in module flink-core, to support getting and setting a list of user jars. * Add void addUserJars(List userJars, JobGraph jobGraph) into JobGraphGenerator and add the user jars within the method compileHobGraph(OptimizedPlan program, JobID jobId) so that user jars can be shipped with user's program and submitted to cluster. JobGraphGenerator is in module flink-optimizer. * Add _void_ _registerUserJarFile(String jarFile)_ into \{Stream}ExecutionEnvironment.scala (in module flink-scala and flink-streaming-scala) and just use the wrapped javaEnv to achieve registration. *Testing* * One test case for adding local user jars both in the streaming and batch jobs. We need to process test classes into a jar before testing. For this purpose, we can add a goal in process-test-classes for this testing case in pom file. The affected module is flink-tests. * Another test case for adding use jars in HDFS. The same idea with the previous one. The affected module is flink-fs-tests. * Note that python API is not included in this issue just as registering cached files. But we still need to modify some python test cases in order to avoid building error as lacking some methods declared in java. The affected files are flink-python/pyflink/dataset/tests/test_execution_environment_completeness.py and
[jira] [Updated] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Leo Zhang updated FLINK-14319: -- Docs Text: (was: Design - 1 - 2 - 3) Description: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel like it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for _{Stream}ExecutionEnvironment::registerUserJarFile()_ interface which will be addressed in this issue. Here is the plan. *Design* * Add _void_ _registerUserJarFile(String jarFile)_ into StreamExecutionEnvironment ( in module flink-streaming-java). The affected classes are StreamGraph, StreamGraphGenerator, StreamingJobGraphGenerator to support getting and setting a list of user jars. And all they are in module flink-streaming-java. * Add _void_ _registerUserJarFile(String jarFile)_ into ExecutionEnvironment (in module flink-java). The affected classes is Plan, in module flink-core, to support getting and setting a list of user jars. * Add void addUserJars(List userJars, JobGraph jobGraph) into JobGraphGenerator and add the user jars within the method compileHobGraph(OptimizedPlan program, JobID jobId) so that user jars can be shipped with user's program and submitted to cluster. JobGraphGenerator is in module flink-optimizer. * Add _void_ _registerUserJarFile(String jarFile)_ into \{Stream}ExecutionEnvironment.scala (in module flink-scala and flink-streaming-scala) and just use the wrapped javaEnv to achieve registration. *Testing* * One test case for adding local user jars both in the streaming and batch jobs. We need to process test classes into a jar before testing. For this purpose, we can add a goal in process-test-classes for this testing case in pom file. The affected module is flink-tests. * Another test case for adding use jars in HDFS. The same idea with the previous one. The affected module is flink-fs-tests. * Note that python API is not included in this issue just as registering cached files. But we still need to modify some python test cases in order to avoid building error as lacking some methods declared in java. The affected files are flink-python/pyflink/dataset/tests/test_execution_environment_completeness.py and flink-python/pyflink/datastream/tests/test_stream_execution_environment_completeness.py. was: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel like it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. I have implemented the interfaces both for java and scala API, and they are tested well. And my implementation exactly obeys to the design doc of the FLINK-10232 and chooses the first option from design alternatives. So I wanna share my codes if it's ok. > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Fix For: 1.10.0 > > Time Spent: 20m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > _{Stream}ExecutionEnvironment::registerUserJarFile()_ interface which will be > addressed in this issue. > Here is the plan. > *Design* > * Add _void_ _registerUserJarFile(String jarFile)_ into > StreamExecutionEnvironment ( in module flink-streaming-java). The affected > classes are StreamGraph, StreamGraphGenerator, StreamingJobGraphGenerator to > support getting and setting a list of user jars. And all they are in module > flink-streaming-java. > * Add _void_ _registerUserJarFile(String jarFile)_ into ExecutionEnvironment > (in module flink-java). The affected classes is Plan, in module flink-core, > to
[jira] [Updated] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Leo Zhang updated FLINK-14319: -- Docs Text: Design - 1 - 2 - 3 Labels: (was: pull-request-available) > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Fix For: 1.10.0 > > Time Spent: 20m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. > I have implemented the interfaces both for java and scala API, and they are > tested well. And my implementation exactly obeys to the design doc of the > FLINK-10232 and chooses the first option from design alternatives. So I > wanna share my codes if it's ok. > -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944911#comment-16944911 ] Leo Zhang edited comment on FLINK-14319 at 10/9/19 12:08 AM: - PR on github [https://github.com/apache/flink/pull/9841 (CLOSED)|https://github.com/apache/flink/pull/9841-] was (Author: 50man): PR on github [https://github.com/apache/flink/pull/9841|https://github.com/apache/flink/pull/9841-] > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Labels: pull-request-available > Fix For: 1.10.0 > > Time Spent: 20m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. > I have implemented the interfaces both for java and scala API, and they are > tested well. And my implementation exactly obeys to the design doc of the > FLINK-10232 and chooses the first option from design alternatives. So I > wanna share my codes if it's ok. > -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944911#comment-16944911 ] Leo Zhang edited comment on FLINK-14319 at 10/9/19 12:08 AM: - PR on github [https://github.com/apache/flink/pull/9841|https://github.com/apache/flink/pull/9841-] was (Author: 50man): PR on github -https://github.com/apache/flink/pull/9841- > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Labels: pull-request-available > Fix For: 1.10.0 > > Time Spent: 20m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. > I have implemented the interfaces both for java and scala API, and they are > tested well. And my implementation exactly obeys to the design doc of the > FLINK-10232 and chooses the first option from design alternatives. So I > wanna share my codes if it's ok. > -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944911#comment-16944911 ] Leo Zhang edited comment on FLINK-14319 at 10/9/19 12:07 AM: - PR on github -https://github.com/apache/flink/pull/9841- was (Author: 50man): PR on github [-https://github.com/apache/flink/pull/9841-] > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Labels: pull-request-available > Fix For: 1.10.0 > > Time Spent: 20m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. > I have implemented the interfaces both for java and scala API, and they are > tested well. And my implementation exactly obeys to the design doc of the > FLINK-10232 and chooses the first option from design alternatives. So I > wanna share my codes if it's ok. > -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944911#comment-16944911 ] Leo Zhang edited comment on FLINK-14319 at 10/9/19 12:06 AM: - PR on github [-https://github.com/apache/flink/pull/9841-] was (Author: 50man): -PR on github https://github.com/apache/flink/pull/9841- > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Labels: pull-request-available > Fix For: 1.10.0 > > Time Spent: 20m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. > I have implemented the interfaces both for java and scala API, and they are > tested well. And my implementation exactly obeys to the design doc of the > FLINK-10232 and chooses the first option from design alternatives. So I > wanna share my codes if it's ok. > -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944911#comment-16944911 ] Leo Zhang edited comment on FLINK-14319 at 10/9/19 12:05 AM: - -PR on github https://github.com/apache/flink/pull/9841- was (Author: 50man): PR on github https://github.com/apache/flink/pull/9841 > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Labels: pull-request-available > Fix For: 1.10.0 > > Time Spent: 20m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. > I have implemented the interfaces both for java and scala API, and they are > tested well. And my implementation exactly obeys to the design doc of the > FLINK-10232 and chooses the first option from design alternatives. So I > wanna share my codes if it's ok. > -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-10232) Add a SQL DDL
[ https://issues.apache.org/jira/browse/FLINK-10232?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944458#comment-16944458 ] Leo Zhang edited comment on FLINK-10232 at 10/5/19 10:05 AM: - I feel like it's better to split sub-task 10 FLINK-14055 into two goals, one for DDL and the other new task for _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. And I posed an issue, please review FLINK-14319 [~twalthr] [~danny0405] was (Author: 50man): I feel like it's better to split sub-task 10 FLINK-14055 into two goals, one for DDL and the other new task for _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. And I posed an issue, please review this [~twalthr] [~danny0405] > Add a SQL DDL > - > > Key: FLINK-10232 > URL: https://issues.apache.org/jira/browse/FLINK-10232 > Project: Flink > Issue Type: New Feature > Components: Table SQL / API >Reporter: Timo Walther >Assignee: Danny Chen >Priority: Major > > This is an umbrella issue for all efforts related to supporting a SQL Data > Definition Language (DDL) in Flink's Table & SQL API. > Such a DDL includes creating, deleting, replacing: > - tables > - views > - functions > - types > - libraries > - catalogs > If possible, the parsing/validating/logical part should be done using > Calcite. Related issues are CALCITE-707, CALCITE-2045, CALCITE-2046, > CALCITE-2214, and others. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944912#comment-16944912 ] Leo Zhang edited comment on FLINK-14055 at 10/4/19 11:40 PM: - PR on github for FLINK-14319 https://github.com/apache/flink/pull/9841 [~hpeter][~twalthr] was (Author: 50man): PR on github https://github.com/apache/flink/pull/9841 [~hpeter][~twalthr] > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Commented] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944912#comment-16944912 ] Leo Zhang commented on FLINK-14055: --- PR on github https://github.com/apache/flink/pull/9841 [~hpeter][~twalthr] > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Commented] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944911#comment-16944911 ] Leo Zhang commented on FLINK-14319: --- PR on github https://github.com/apache/flink/pull/9841 > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Labels: pull-request-available > Fix For: 1.10.0 > > Time Spent: 10m > Remaining Estimate: 0h > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. > I have implemented the interfaces both for java and scala API, and they are > tested well. And my implementation exactly obeys to the design doc of the > FLINK-10232 and chooses the first option from design alternatives. So I > wanna share my codes if it's ok. > -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944595#comment-16944595 ] Leo Zhang edited comment on FLINK-14055 at 10/4/19 3:28 PM: Yeah. I totally agree with you [~twalthr] that this sub-task may be oversized. And it should be reasonable to fire a new issue to support loading external user jars. It will be useful and handy for the end-users in many cases besides Table API I set up this goal in FLINK-14319 and please discuss it there sir. [~twalthr]. was (Author: 50man): Yeah. I totally agree with you [~twalthr] that this sub-task may be oversized. And it should be reasonable to fire a new issue to support loading external user jars. It will be useful and handy for the end-users in many cases besides Table API I set up this goal in FLINK-14319 and please discuss it there sir. [~twalthr]. > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Commented] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944595#comment-16944595 ] Leo Zhang commented on FLINK-14055: --- Yeah. I totally agree with you [~twalthr] that this sub-task may be oversized. And it should be reasonable to fire a new issue to support loading external user jars. It will be useful and handy for the end-users in many cases besides Table API I set up this goal in FLINK-14319 and please discuss it there sir. [~twalthr]. > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-10232) Add a SQL DDL
[ https://issues.apache.org/jira/browse/FLINK-10232?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944458#comment-16944458 ] Leo Zhang edited comment on FLINK-10232 at 10/4/19 12:29 PM: - I feel like it's better to split sub-task 10 FLINK-14055 into two goals, one for DDL and the other new task for _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. And I posed an issue, please review this [~twalthr] [~danny0405] was (Author: 50man): I feel like it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. And I posed an issue, please review this [~twalthr] [~danny0405] > Add a SQL DDL > - > > Key: FLINK-10232 > URL: https://issues.apache.org/jira/browse/FLINK-10232 > Project: Flink > Issue Type: New Feature > Components: Table SQL / API >Reporter: Timo Walther >Assignee: Danny Chen >Priority: Major > > This is an umbrella issue for all efforts related to supporting a SQL Data > Definition Language (DDL) in Flink's Table & SQL API. > Such a DDL includes creating, deleting, replacing: > - tables > - views > - functions > - types > - libraries > - catalogs > If possible, the parsing/validating/logical part should be done using > Calcite. Related issues are CALCITE-707, CALCITE-2045, CALCITE-2046, > CALCITE-2214, and others. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Commented] (FLINK-10232) Add a SQL DDL
[ https://issues.apache.org/jira/browse/FLINK-10232?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16944458#comment-16944458 ] Leo Zhang commented on FLINK-10232: --- I feel like it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. And I posed an issue, please review this [~twalthr] [~danny0405] > Add a SQL DDL > - > > Key: FLINK-10232 > URL: https://issues.apache.org/jira/browse/FLINK-10232 > Project: Flink > Issue Type: New Feature > Components: Table SQL / API >Reporter: Timo Walther >Assignee: Danny Chen >Priority: Major > > This is an umbrella issue for all efforts related to supporting a SQL Data > Definition Language (DDL) in Flink's Table & SQL API. > Such a DDL includes creating, deleting, replacing: > - tables > - views > - functions > - types > - libraries > - catalogs > If possible, the parsing/validating/logical part should be done using > Calcite. Related issues are CALCITE-707, CALCITE-2045, CALCITE-2046, > CALCITE-2214, and others. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Updated] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Leo Zhang updated FLINK-14319: -- Description: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel like it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. I have implemented the interfaces both for java and scala API, and they are tested well. And my implementation exactly obeys to the design doc of the FLINK-10232 and chooses the first option from design alternatives. So I wanna share my codes if it's ok. was: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel like it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. I have implemented the interfaces both for java and scala API, and they are tested well. And my implementation exactly obeys to the design doc of the FLINK-10232 and chooses the first option from design alternatives. So I wanna share my codes if it's ok. > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Fix For: 1.10.0 > > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And > the related sub-task FLINK-14055 is unresolved and its status is still open. > I feel like it's better to split this task FLINK-14055 into two goals, one > for DDL and the other new task for > _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. > I have implemented the interfaces both for java and scala API, and they are > tested well. And my implementation exactly obeys to the design doc of the > FLINK-10232 and chooses the first option from design alternatives. So I > wanna share my codes if it's ok. > -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Updated] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Leo Zhang updated FLINK-14319: -- Description: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel like it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. I have implemented the interfaces both for java and scala API, and they are tested well. And my implementation exactly obeys to the design doc of the FLINK-10232 and chooses the first option from design alternatives. So I wanna share my codes if it's ok. was: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel that it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for \{Stream\}ExecutionEnvironment::registerUserJarFile() interface. I have implemented the interfaces both for java and scala API, and they are tested well. And my implementation exactly obeys to the design doc of the FLINK-10232 and chooses the first option from design alternatives. So I wanna share my codes if it's ok. > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Fix For: 1.10.0 > > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . > And the related sub-task FLINK-14055 is unresolved and its status is still > open. I feel like it's better to split this task FLINK-14055 into two goals, > one for DDL and the other new task for > _\{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. > I have implemented the interfaces both for java and scala API, and they are > tested well. And my implementation exactly obeys to the design doc of the > FLINK-10232 and chooses the first option from design alternatives. So I > wanna share my codes if it's ok. > -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Updated] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Leo Zhang updated FLINK-14319: -- Description: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel that it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for \{Stream\}ExecutionEnvironment::registerUserJarFile() interface. I have implemented the interfaces both for java and scala API, and they are tested well. And my implementation exactly obeys to the design doc of the FLINK-10232 and chooses the first option from design alternatives. So I wanna share my codes if it's ok. was: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel that it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for _{Stream}ExecutionEnvironment::registerUserJarFile()_ interface. I have implemented the interfaces both for java and scala API, and they are tested well. And my implementation exactly obeys to the design doc of the FLINK-10232 and chooses the first option from design alternatives. So I wanna share my codes if it's ok. > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Fix For: 1.10.0 > > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . > And the related sub-task FLINK-14055 is unresolved and its status is still > open. I feel that it's better to split this task FLINK-14055 into two goals, > one for DDL and the other new task for > \{Stream\}ExecutionEnvironment::registerUserJarFile() interface. > I have implemented the interfaces both for java and scala API, and they are > tested well. And my implementation exactly obeys to the design doc of the > FLINK-10232 and chooses the first option from design alternatives. So I > wanna share my codes if it's ok. > -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Updated] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
[ https://issues.apache.org/jira/browse/FLINK-14319?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Leo Zhang updated FLINK-14319: -- Description: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel that it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for _{Stream}ExecutionEnvironment::registerUserJarFile()_ interface. I have implemented the interfaces both for java and scala API, and they are tested well. And my implementation exactly obeys to the design doc of the FLINK-10232 and chooses the first option from design alternatives. So I wanna share my codes if it's ok. was: I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel that it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for \_{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. I have implemented the interfaces both for java and scala API, and they are tested well. And my implementation exactly obeys to the design doc of the FLINK-10232 and chooses the first option from design alternatives. So I wanna share my codes if it's ok. > Register user jar files in {Stream}ExecutionEnvironment > > > Key: FLINK-14319 > URL: https://issues.apache.org/jira/browse/FLINK-14319 > Project: Flink > Issue Type: New Feature > Components: API / DataSet, API / DataStream >Reporter: Leo Zhang >Priority: Major > Fix For: 1.10.0 > > > I see that there are some use cases in which people want to implement their > own SQL application based on loading external jars for now. And the related > API proposals have been issued in the task FLINK-10232 Add a SQL DDL . > And the related sub-task FLINK-14055 is unresolved and its status is still > open. I feel that it's better to split this task FLINK-14055 into two goals, > one for DDL and the other new task for > _{Stream}ExecutionEnvironment::registerUserJarFile()_ interface. > I have implemented the interfaces both for java and scala API, and they are > tested well. And my implementation exactly obeys to the design doc of the > FLINK-10232 and chooses the first option from design alternatives. So I > wanna share my codes if it's ok. > -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-14319) Register user jar files in {Stream}ExecutionEnvironment
Leo Zhang created FLINK-14319: - Summary: Register user jar files in {Stream}ExecutionEnvironment Key: FLINK-14319 URL: https://issues.apache.org/jira/browse/FLINK-14319 Project: Flink Issue Type: New Feature Components: API / DataSet, API / DataStream Reporter: Leo Zhang Fix For: 1.10.0 I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And the related API proposals have been issued in the task FLINK-10232 Add a SQL DDL . And the related sub-task FLINK-14055 is unresolved and its status is still open. I feel that it's better to split this task FLINK-14055 into two goals, one for DDL and the other new task for \_{Stream\}ExecutionEnvironment::registerUserJarFile()_ interface. I have implemented the interfaces both for java and scala API, and they are tested well. And my implementation exactly obeys to the design doc of the FLINK-10232 and chooses the first option from design alternatives. So I wanna share my codes if it's ok. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16943304#comment-16943304 ] Leo Zhang edited comment on FLINK-14055 at 10/3/19 4:23 AM: Hi, everyone. I have implemented *\\{Stream\}ExecutionEnvironment::registerUserJarFile()* interfaces both for java and scala API, and they are tested well. Do you think it is better to split this task FLINK-14055 into two goals, one for DDL and the other for *registerUserJarFile()* interface? I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And I'd like to help here and could you please assign the latter goal to me so that I can propose a PR on Github as soon as possible? [~hpeter] [~phoenixjiangnan] was (Author: 50man): Hi, everyone. I have implemented *\\{Stream\}ExecutionEnvironment::registerUserJarFile()* interfaces both for java and scala API, and they are tested well. Do you think it is better to split this task into two goals, one for DDL and the other for *registerUserJarFile()* interface? I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And I'd like to help here and could you please assign the latter goal to me so that I can propose a PR on Github as soon as possible? [~hpeter] [~phoenixjiangnan] > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Commented] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16943311#comment-16943311 ] Leo Zhang commented on FLINK-14055: --- You're right that we should follow the design doc FLINK-10232. And my implementation exactly obeys to the design doc and chooses the first option from design alternatives. Should we pose an issue on Github to discuss it further? > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16943304#comment-16943304 ] Leo Zhang edited comment on FLINK-14055 at 10/3/19 3:50 AM: Hi, everyone. I have implemented *\\{Stream\}ExecutionEnvironment::registerUserJarFile()* interfaces both for java and scala API, and they are tested well. Do you think it is better to split this task into two goals, one for DDL and the other for *registerUserJarFile()* interface? I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And I'd like to help here and could you please assign the latter goal to me so that I can propose a PR on Github as soon as possible? [~hpeter] [~phoenixjiangnan] was (Author: 50man): Hi, everyone. I have implemented *{Stream}ExecutionEnvironment::registerUserJarFile()* interfaces both for java and scala API, and they are tested well. Do you think it is better to split this task into two goals, one for DDL and the other for *registerUserJarFile()* interface? I see that there are some use cases in which people want to implement their own SQL application based on loading external jars for now. And I'd like to help here and could you please assign the latter goal to me so that I can propose a PR on Github as soon as possible? [~hpeter] [~phoenixjiangnan] > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16943304#comment-16943304 ] Leo Zhang edited comment on FLINK-14055 at 10/3/19 3:47 AM: Hi, everyone. I have implemented *\\{Stream\}ExecutionEnvironment::registerUserJarFile()* interfaces both for java and scala API, and they are tested well. Do you think it is better to split this task into two goals, one for DDL and the other for *registerUserJarFile()* interface? And I'd like to help here and could you please assign the latter goal to me so that I can propose a PR on Github as soon as possible? [~hpeter] [~phoenixjiangnan] was (Author: 50man): Hi, everyone. I have implemented *{Stream}ExecutionEnvironment::registerUserJarFile()* interfaces both for java and scala API, and they are tested well. Do you think it is better to split this task into two goals, one for DDL and the other for *registerUserJarFile()* interface? And I'd like to help here and could you please assign the latter goal to me so that I can propose a PR on Github as soon as possible? [~hpeter] [~phoenixjiangnan] > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16943304#comment-16943304 ] Leo Zhang edited comment on FLINK-14055 at 10/3/19 3:46 AM: Hi, everyone. I have implemented *{Stream}ExecutionEnvironment::registerUserJarFile()* interfaces both for java and scala API, and they are tested well. Do you think it is better to split this task into two goals, one for DDL and the other for *registerUserJarFile()* interface? And I'd like to help here and could you please assign the latter goal to me so that I can propose a PR on Github as soon as possible? [~hpeter] [~phoenixjiangnan] was (Author: 50man): Hi, everyone. I have implemented _{Stream}ExecutionEnvironment::registerUserJarFile()_ interfaces both for java and scala API, and they are tested well. Do you think it is better to split this task into two goals, one for DDL and the other for _registerUserJarFile()_ interface? And I'd like to help here and could you please assign the latter goal to me so that I can propose a PR on Github as soon as possible? [~hpeter] [~phoenixjiangnan] > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Comment Edited] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16943304#comment-16943304 ] Leo Zhang edited comment on FLINK-14055 at 10/3/19 3:45 AM: Hi, everyone. I have implemented _{Stream}ExecutionEnvironment::registerUserJarFile()_ interfaces both for java and scala API, and they are tested well. Do you think it is better to split this task into two goals, one for DDL and the other for _registerUserJarFile()_ interface? And I'd like to help here and could you please assign the latter goal to me so that I can propose a PR on Github as soon as possible? [~hpeter] [~phoenixjiangnan] was (Author: 50man): Hi, everyone. I have implemented \{Stream}ExecutionEnvironment::registerUserJarFile() interfaces both for java and scala API, and they are tested well. Do you think its better to split this task into to goals one for DDL and the other for registerUserJarFile() interfaces. And I'd like to help here and could you please assign the latter goal to me so that I can propose a PR on Github as soon as possible? > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Commented] (FLINK-14055) Add advanced function DDL syntax "USING JAR/FILE/ACHIVE"
[ https://issues.apache.org/jira/browse/FLINK-14055?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=16943304#comment-16943304 ] Leo Zhang commented on FLINK-14055: --- Hi, everyone. I have implemented \{Stream}ExecutionEnvironment::registerUserJarFile() interfaces both for java and scala API, and they are tested well. Do you think its better to split this task into to goals one for DDL and the other for registerUserJarFile() interfaces. And I'd like to help here and could you please assign the latter goal to me so that I can propose a PR on Github as soon as possible? > Add advanced function DDL syntax "USING JAR/FILE/ACHIVE" > > > Key: FLINK-14055 > URL: https://issues.apache.org/jira/browse/FLINK-14055 > Project: Flink > Issue Type: Sub-task > Components: Table SQL / API >Reporter: Bowen Li >Assignee: Zhenqiu Huang >Priority: Major > > As FLINK-7151 adds basic function DDL to Flink, this ticket is to support > dynamically loading functions from external source in function DDL with > advanced syntax like > > {code:java} > CREATE FUNCTION func_name as class_name USING JAR/FILE/ACHIEVE 'xxx' [, > JAR/FILE/ACHIEVE 'yyy'] ; > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)