[DISCUSS] Backpoint FLIP-126 (watermarks) integration with FLIP-27
Hi all! I want to discuss merging this PR to the 1.11 release branch: https://github.com/apache/flink/pull/12306 It contains the new FLIP-126 Watermarks, and per-partition watermarking to the FLIP-27 sources. In that sense it is partially a new feature after the feature freeze. Hence this discussion, and not just merging. The reasons why I suggest to back-port this to 1.11 are - It is API breaking. Without this patch, we would release a Source API and immediately break compatibility in the next release. - The FLIP-27 feature is experimental, but it should not be useless in the sense that users have to re-write all implemented sources in the next release. - It is a fairly isolated change, does not affect any existing feature in the system Please let me know if you have concerns about this. Best, Stephan
[jira] [Created] (FLINK-17938) Cannot run mvn clean verify flink-yarn-tests
Till Rohrmann created FLINK-17938: - Summary: Cannot run mvn clean verify flink-yarn-tests Key: FLINK-17938 URL: https://issues.apache.org/jira/browse/FLINK-17938 Project: Flink Issue Type: Bug Components: Deployment / YARN, Tests Affects Versions: 1.11.0 Reporter: Till Rohrmann Fix For: 1.11.0 As part of FLINK-11086, we introduced the setting of the yarn class path in a static initializer of {{YarnTestBase.java:199}}. The yarn class path file will be generated by the {{maven-dependency-plugin}} in the {{package}} phase. Due to this, the {{yarn.classpath}} file won't be accessible to all users of the {{YarnTestBase}} class which are run in a previous phase (e.g. {{UtilsTest.testUberjarLocator}}). -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-17939) Translate "Python Table API Installation" page into Chinese
Jark Wu created FLINK-17939: --- Summary: Translate "Python Table API Installation" page into Chinese Key: FLINK-17939 URL: https://issues.apache.org/jira/browse/FLINK-17939 Project: Flink Issue Type: Sub-task Components: API / Python, chinese-translation, Documentation Reporter: Jark Wu -- This message was sent by Atlassian Jira (v8.3.4#803005)
Re: [DISCUSS] Semantics of our JIRA fields
Hi, Sorry for a bit late response. I have two concerns: 1. Priority I would propose to stretch priorities that we are using to differentiate between things that must be fixed for given release: BLOCKER - drop anything you are doing, this issue must be fixed right now CRITICAL - release can not happen without fixing it, but can be fixed a bit later (for example without context switching and dropping whatever I’m doing right now) MAJOR - default, nice to have Anything below - meh We were already using this semantic for tracking test instabilities during the 1.11 release cycle. Good examples: BLOCKER - master branch not compiling, very frequent test failures (for example almost every build affected), … CRITICAL - performance regression/bug that we introduced in some feature, but which is not affecting other developers as much MAJOR - freshly discovered test instability with unknown impact/frequency (could be happening once a year), 2. Affects version If bug is only on the master branch, does it affect an unreleased version? So far I was assuming that it doesn’t - unreleased bugs would have empty “affects version” field. My reasoning was that this field should be used for Flink users, to check which RELEASED Flink versions are affected by some bug, that user is searching for. Otherwise it might be a bit confusing if there are lots of bugs with both affects version and fix version set to the same value. Piotrek > On 25 May 2020, at 16:40, Robert Metzger wrote: > > Hi all, > thanks a lot for the feedback. The majority of responses are very positive > to my proposal. > > I have put my proposal into our wiki: > https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=154995514 > > Regarding the comments so far: > @Jark: I clarified this in the wiki. > > @Israel: I have not considered build changing all 3000 resolved tickets to > closed yet, but after consideration I don't think it is necessary. If > others in the community would like to change them, please speak up in this > thread :) > > @Flavio: I agree that we can not ask new or infrequent users to fully > adhere to these definitions. I added a note in the Wiki. > Using the resolved state for indicating "PR available" is problematic > because there are plenty of cases where PRs are stale (and this ticket > would then appear as a "resolved"). The Apache tools are adding a link to > the PR, and some contributors are setting the ticket to "In Progress". I > don't see a problem that we need to solve here. > > @Yun: Thank you for your comment. I added an example clarifying how I would > handle such a case. It is slightly different from your proposal: You > suggested using x.y.0 versions, I used "the next supported, unreleased > version", because that's how we've done it so far (and I don't want to > change things, I just want to document how the majority of the core > contributors are using JIRA). > > Here are all the changes (in green, blue are just formatting changes) I > made compared to my initial proposal: > https://cwiki.apache.org/confluence/pages/diffpagesbyversion.action?pageId=154995514&selectedPageVersions=4&selectedPageVersions=1 > > > > On Mon, May 25, 2020 at 2:28 PM Congxian Qiu wrote: > >> @ches...@apache.org Thanks for the confirmation >> >> Best, >> Congxian >> >> >> Zhu Zhu 于2020年5月25日周一 下午4:13写道: >> >>> This is very helpful! >>> +1 >>> >>> Thanks, >>> Zhu Zhu >>> >>> Yang Wang 于2020年5月25日周一 下午4:04写道: >>> +1 from this useful proposal. This makes me clearer about "Resolve" and "Close" since I used to be confused by this two button. Best, Yang Jingsong Li 于2020年5月25日周一 下午3:10写道: > +1 for the proposal. > It makes me clearer. > > Best, > Jingsong Lee > > On Mon, May 25, 2020 at 2:51 PM Zhijiang .invalid> > wrote: > >> Thanks for launching this discussion and giving so detailed infos, >> Robert! +1 on my side for the proposal. >> >> For "Affects Version", I previously thought it was only for the >>> already >> released versions, so it can give a reminder that the fix should >> also > pick >> into the related released branches for future minor versions. >> I saw that Jark had somehow similar concerns for this field in >> below >> replies. Either way makes sense for me as long as we give a >>> determined >> rule in Wiki. >> >> Re Flavio' s comments, I agree that the Jira reporter can leave >> most >>> of >> the fields empty if not confirmed of them, then the respective component >> maintainer or committer can update them accordingly later. >> But the state of Jira should not be marked as "resolved" when the >> PR >>> is >> detected, that is not fitting into the resolved semantic I guess. >> If >> possible, the Jira can be updated as "in progress" automatically if >> the respective PR is ready, then it will save some time
Re: [DISCUSS] Backpoint FLIP-126 (watermarks) integration with FLIP-27
Usually we should avoid checking in patches other than bug fix after feature freeze. However, in this particular case, the code base is sort of in an incomplete state - an exposed known-to-change feature - due to missing this patch. Fixing forward seems the best option. Besides that, FLIP-27 has been highly anticipated by many users. So if one patch completes the story, personally speaking I am +1 to backport given the isolated impact and significant benefit of doing that. Thanks, Jiangjie (Becket) Qin On Tue, May 26, 2020 at 4:43 PM Stephan Ewen wrote: > Hi all! > > I want to discuss merging this PR to the 1.11 release branch: > https://github.com/apache/flink/pull/12306 > > It contains the new FLIP-126 Watermarks, and per-partition watermarking to > the FLIP-27 sources. In that sense it is partially a new feature after the > feature freeze. Hence this discussion, and not just merging. > > The reasons why I suggest to back-port this to 1.11 are > - It is API breaking. Without this patch, we would release a Source API > and immediately break compatibility in the next release. > - The FLIP-27 feature is experimental, but it should not be useless in > the sense that users have to re-write all implemented sources in the next > release. > - It is a fairly isolated change, does not affect any existing feature > in the system > > Please let me know if you have concerns about this. > > Best, > Stephan > >
Re: [DISCUSS] Backpoint FLIP-126 (watermarks) integration with FLIP-27
Hi, As we discussed this offline a bit, initially I was sceptical to merge it, as: - even it’s an isolated change, it can destabilise the builds and prolong release testing period - is distracting from solving release blockers etc However all in all I’m +0.5 to merge it because of this argument: > - It is API breaking. Without this patch, we would release a Source API and immediately break compatibility in the next release. And this: > - It is a fairly isolated change, does not affect any existing feature in the system Is limiting our risks, that we are not risking introducing bugs into the existing features. Piotrek wt., 26 maj 2020 o 12:43 Becket Qin napisał(a): > Usually we should avoid checking in patches other than bug fix after > feature freeze. However, in this particular case, the code base is sort of > in an incomplete state - an exposed known-to-change feature - due to > missing this patch. Fixing forward seems the best option. Besides that, > FLIP-27 has been highly anticipated by many users. So if one patch > completes the story, personally speaking I am +1 to backport given the > isolated impact and significant benefit of doing that. > > Thanks, > > Jiangjie (Becket) Qin > > > On Tue, May 26, 2020 at 4:43 PM Stephan Ewen wrote: > >> Hi all! >> >> I want to discuss merging this PR to the 1.11 release branch: >> https://github.com/apache/flink/pull/12306 >> >> It contains the new FLIP-126 Watermarks, and per-partition watermarking >> to the FLIP-27 sources. In that sense it is partially a new feature after >> the feature freeze. Hence this discussion, and not just merging. >> >> The reasons why I suggest to back-port this to 1.11 are >> - It is API breaking. Without this patch, we would release a Source API >> and immediately break compatibility in the next release. >> - The FLIP-27 feature is experimental, but it should not be useless in >> the sense that users have to re-write all implemented sources in the next >> release. >> - It is a fairly isolated change, does not affect any existing feature >> in the system >> >> Please let me know if you have concerns about this. >> >> Best, >> Stephan >> >>
[DISCUSS] (Document) Backwards Compatibility of Savepoints
Hi everyone, I recently stumbled across the fact that Savepoints created with Flink 1.11 can not be read by Flink 1.10. A use case for this might be when you want to rollback a framework upgrade (after some time) due to e.g. a performance or stability issue. >From the documentation [1] it seems as if the Savepoint format is generally only forward-compatible although in many cases it is actually also backwards compatible (e.g. Savepoint taken in Flink 1.10, restored with Flink 1.9). Was it a deliberate choice not to document any backwards compatibility? If not, should we add the missing entries in the compatibility table? Thanks, Konstantin [1] https://ci.apache.org/projects/flink/flink-docs-master/ops/upgrading.html#compatibility-table -- Konstantin Knauf https://twitter.com/snntrable https://github.com/knaufk
Re: [DISCUSS] (Document) Backwards Compatibility of Savepoints
Hi, It might have been implicit choice, but so far we were not supporting the scenario that you are asking for. It has never been tested and we have lot’s of state migration code sprinkled among our code base (for example upgrading state fields of the operators like [1]), that only supports upgrades, not downgrades. Also we do not have testing infrastructure for checking the downgrades. We would need to check if save points taken from master branch, are readable by previous releases (not release branch!). So all in all, I don’t think it can be easily done. It would require some effort to start maintaining backward compatibility. Piotrek [1] org.apache.flink.streaming.connectors.kafka.FlinkKafkaProducer011#migrateNextTransactionalIdHindState > On 26 May 2020, at 13:18, Konstantin Knauf wrote: > > Hi everyone, > > I recently stumbled across the fact that Savepoints created with Flink 1.11 > can not be read by Flink 1.10. A use case for this might be when you want > to rollback a framework upgrade (after some time) due to e.g. a performance > or stability issue. > > From the documentation [1] it seems as if the Savepoint format is generally > only forward-compatible although in many cases it is actually also > backwards compatible (e.g. Savepoint taken in Flink 1.10, restored with > Flink 1.9). > > Was it a deliberate choice not to document any backwards compatibility? If > not, should we add the missing entries in the compatibility table? > > Thanks, > > Konstantin > > [1] > https://ci.apache.org/projects/flink/flink-docs-master/ops/upgrading.html#compatibility-table > > -- > > Konstantin Knauf > > https://twitter.com/snntrable > > https://github.com/knaufk
[jira] [Created] (FLINK-17940) It will throw NullPointerException when write data with Avro format using new property key in SQL-Client
Shengkai Fang created FLINK-17940: - Summary: It will throw NullPointerException when write data with Avro format using new property key in SQL-Client Key: FLINK-17940 URL: https://issues.apache.org/jira/browse/FLINK-17940 Project: Flink Issue Type: Bug Components: Connectors / Kafka, Table SQL / Client Affects Versions: 1.11.0 Environment: Docker Environment: zookeeper: image: wurstmeister/zookeeper:3.4.6 ports: - "2181:2181" kafka: image: wurstmeister/kafka:2.12-2.2.1 ports: - "9092:9092" - "9094:9094" depends_on: - zookeeper environment: - KAFKA_ADVERTISED_LISTENERS=INSIDE://:9094,OUTSIDE://localhost:9092 - KAFKA_LISTENERS=INSIDE://:9094,OUTSIDE://:9092 - KAFKA_LISTENER_SECURITY_PROTOCOL_MAP=INSIDE:PLAINTEXT,OUTSIDE:PLAINTEXT - KAFKA_INTER_BROKER_LISTENER_NAME=INSIDE - KAFKA_ZOOKEEPER_CONNECT=zookeeper:2181 - KAFKA_CREATE_TOPICS:"order_cnt:1:1,orders:1:1,currency:1:1" volumes: - /var/run/docker.sock:/var/run/docker.sock Reporter: Shengkai Fang For the following job: {noformat} create table csv( user_name VARCHAR, is_new BOOLEAN, content VARCHAR ) with ( 'connector' = 'filesystem', 'path' = '/Users/ohmeatball/Work/flink-sql-etl/data- generator/src/main/resources/user.csv', 'format' = 'csv'); - CREATE TABLE AvroTest ( user_name VARCHAR, is_new BOOLEAN, content VARCHAR ) WITH ( 'connector' = 'kafka', 'topic' = 'avro_from_csv', 'properties.zookeeper.connect' = 'localhost:2181', 'properties.bootstrap.servers' = 'localhost:9092', 'properties.group.id' = 'testGroup3', 'scan.startup.mode' = 'earliest-offset', 'format' = 'avro'); - insert into AvroTest select user_name, is_new, content from csv; {noformat} The exception stack is following: {code:java} 2020-05-26 19:51:22,212 WARN org.apache.flink.runtime.taskmanager.Task [] - FileSystemTableSource(user_name, is_new, content) -> Sink: Sink(table=[default_catalog.default_database.AvroTest], fields=[user_name, is_new, content]) (1/1) (283a383f3ed93b051f56d4b5aca7dfb9) switched from RUNNING to FAILED.java.lang.RuntimeException: Failed to serialize row.at org.apache.flink.formats.avro.AvroRowDataSerializationSchema.serialize(AvroRowDataSerializationSchema.java:118) ~[flink-avro-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.formats.avro.AvroRowDataSerializationSchema.serialize(AvroRowDataSerializationSchema.java:63) ~[flink-avro-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.connectors.kafka.internals.KeyedSerializationSchemaWrapper.serializeValue(KeyedSerializationSchemaWrapper.java:51) ~[flink-sql-connector-kafka_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.connectors.kafka.FlinkKafkaProducer.invoke(FlinkKafkaProducer.java:775) ~[flink-sql-connector-kafka_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.connectors.kafka.FlinkKafkaProducer.invoke(FlinkKafkaProducer.java:98) ~[flink-sql-connector-kafka_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.api.functions.sink.TwoPhaseCommitSinkFunction.invoke(TwoPhaseCommitSinkFunction.java:235) ~[flink-dist_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.table.runtime.operators.sink.SinkOperator.processElement(SinkOperator.java:86) ~[flink-table-blink_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.pushToOperator(OperatorChain.java:717) ~[flink-dist_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.collect(OperatorChain.java:692) ~[flink-dist_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.collect(OperatorChain.java:672) ~[flink-dist_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.api.operators.CountingOutput.collect(CountingOutput.java:52) ~[flink-dist_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.api.operators.CountingOutput.collect(CountingOutput.java:30) ~[flink-dist_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.api.operators.StreamSourceContexts$ManualWatermarkContext.processAndCollect(StreamSourceContexts.java:305) ~[flink-dist_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.api.operators.StreamSourceContexts$WatermarkContext.collect(StreamSourceContexts.java:394) ~[flink-dist_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.streaming.api.functions.source.ContinuousFileReaderOperator.readAndCollectRecord(ContinuousFileReaderOperator.java:352) ~[flink-dist_2.11-1.11-SNAPSHOT.jar:1.11-SNAPSHOT]at org.apache.flink.s
[jira] [Created] (FLINK-17941) Switching database doesn't work from SQL CLI
Rui Li created FLINK-17941: -- Summary: Switching database doesn't work from SQL CLI Key: FLINK-17941 URL: https://issues.apache.org/jira/browse/FLINK-17941 Project: Flink Issue Type: Bug Components: Table SQL / Client Reporter: Rui Li Fix For: 1.11.0 -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-17942) Count distinct could not clean state in WindowOperator
Benchao Li created FLINK-17942: -- Summary: Count distinct could not clean state in WindowOperator Key: FLINK-17942 URL: https://issues.apache.org/jira/browse/FLINK-17942 Project: Flink Issue Type: Bug Affects Versions: 1.10.1, 1.9.3, 1.11.0 Reporter: Benchao Li MapView.clear() is generated in NamespaceAggsHandleFunction.cleanup, however it's never been called in WindowOperator in blink planner. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-17943) HiveFunctionWrapper#getUDFClass should use Thread.currentThread().getContextClassLoader()
Caizhi Weng created FLINK-17943: --- Summary: HiveFunctionWrapper#getUDFClass should use Thread.currentThread().getContextClassLoader() Key: FLINK-17943 URL: https://issues.apache.org/jira/browse/FLINK-17943 Project: Flink Issue Type: Bug Components: Connectors / Hive Affects Versions: 1.11.0 Reporter: Caizhi Weng {{HiveFunctionWrapper#getUDFClass}} currently uses {{Class.forName(className)}} to load Hive UDF classes, while {{HiveFunctionWrapper#createFunction}} uses {{Thread.currentThread().getContextClassLoader()}}. {{HiveFunctionWrapper#getUDFClass}} should also use {{Thread.currentThread().getContextClassLoader()}} as it is loading user classes. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-17944) Wrong output in sql client's table mode
Jeff Zhang created FLINK-17944: -- Summary: Wrong output in sql client's table mode Key: FLINK-17944 URL: https://issues.apache.org/jira/browse/FLINK-17944 Project: Flink Issue Type: Improvement Components: Table SQL / Client Affects Versions: 1.11.0 Reporter: Jeff Zhang When I run the following sql example, I get the wrong output {code:java} SELECT name, COUNT(*) AS cnt FROM (VALUES ('Bob'), ('Alice'), ('Greg'), ('Bob')) AS NameTable(name) GROUP BY name; {code} {code:java} Bob 1 Alice 1 Greg 1 Bob 2 {code} This is due to we add kind in Row, so the sematics of equals method changes -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-17945) Improve error reporting of Python CI tests
Robert Metzger created FLINK-17945: -- Summary: Improve error reporting of Python CI tests Key: FLINK-17945 URL: https://issues.apache.org/jira/browse/FLINK-17945 Project: Flink Issue Type: Improvement Components: API / Python, Tests Reporter: Robert Metzger -- This message was sent by Atlassian Jira (v8.3.4#803005)
Re: [DISCUSS] Semantics of our JIRA fields
Hi, 1. I'm okay with updating the definition of the priorities for the reason you've mentioned. 2. "Affects version" The reason why like to mark affects version on unreleased versions is to clearly indicate which branch is affected by a bug. Given the current Flink release status, if there's a bug only in "release-1.11", but not in "master", there is no way of figuring that out, if we only allow released versions for "affects version" (In my proposal, you would set "affects version" to '1.11.0', '1.12.0' to indicate that). What we could do is introduce "1.12-SNAPSHOT" as version to mark issues on unreleased versions. (But then people might accidentally set the "fix version" to a "-SNAPSHOT" version.) I'm still in favor of my proposal. I have never heard a report from a confused user about our Jira fields (I guess they usually check bugs for released versions only) On Tue, May 26, 2020 at 12:39 PM Piotr Nowojski wrote: > Hi, > > Sorry for a bit late response. I have two concerns: > > 1. Priority > > I would propose to stretch priorities that we are using to differentiate > between things that must be fixed for given release: > > BLOCKER - drop anything you are doing, this issue must be fixed right now > CRITICAL - release can not happen without fixing it, but can be fixed a > bit later (for example without context switching and dropping whatever I’m > doing right now) > MAJOR - default, nice to have > Anything below - meh > > We were already using this semantic for tracking test instabilities during > the 1.11 release cycle. Good examples: > > BLOCKER - master branch not compiling, very frequent test failures (for > example almost every build affected), … > CRITICAL - performance regression/bug that we introduced in some feature, > but which is not affecting other developers as much > MAJOR - freshly discovered test instability with unknown impact/frequency > (could be happening once a year), > > 2. Affects version > > If bug is only on the master branch, does it affect an unreleased version? > > So far I was assuming that it doesn’t - unreleased bugs would have empty > “affects version” field. My reasoning was that this field should be used > for Flink users, to check which RELEASED Flink versions are affected by > some bug, that user is searching for. Otherwise it might be a bit confusing > if there are lots of bugs with both affects version and fix version set to > the same value. > > Piotrek > > > On 25 May 2020, at 16:40, Robert Metzger wrote: > > > > Hi all, > > thanks a lot for the feedback. The majority of responses are very > positive > > to my proposal. > > > > I have put my proposal into our wiki: > > > https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=154995514 > > > > Regarding the comments so far: > > @Jark: I clarified this in the wiki. > > > > @Israel: I have not considered build changing all 3000 resolved tickets > to > > closed yet, but after consideration I don't think it is necessary. If > > others in the community would like to change them, please speak up in > this > > thread :) > > > > @Flavio: I agree that we can not ask new or infrequent users to fully > > adhere to these definitions. I added a note in the Wiki. > > Using the resolved state for indicating "PR available" is problematic > > because there are plenty of cases where PRs are stale (and this ticket > > would then appear as a "resolved"). The Apache tools are adding a link to > > the PR, and some contributors are setting the ticket to "In Progress". I > > don't see a problem that we need to solve here. > > > > @Yun: Thank you for your comment. I added an example clarifying how I > would > > handle such a case. It is slightly different from your proposal: You > > suggested using x.y.0 versions, I used "the next supported, unreleased > > version", because that's how we've done it so far (and I don't want to > > change things, I just want to document how the majority of the core > > contributors are using JIRA). > > > > Here are all the changes (in green, blue are just formatting changes) I > > made compared to my initial proposal: > > > https://cwiki.apache.org/confluence/pages/diffpagesbyversion.action?pageId=154995514&selectedPageVersions=4&selectedPageVersions=1 > > > > > > > > On Mon, May 25, 2020 at 2:28 PM Congxian Qiu > wrote: > > > >> @ches...@apache.org Thanks for the confirmation > >> > >> Best, > >> Congxian > >> > >> > >> Zhu Zhu 于2020年5月25日周一 下午4:13写道: > >> > >>> This is very helpful! > >>> +1 > >>> > >>> Thanks, > >>> Zhu Zhu > >>> > >>> Yang Wang 于2020年5月25日周一 下午4:04写道: > >>> > +1 from this useful proposal. > > This makes me clearer about "Resolve" and "Close" since I used to be > confused by this two button. > > Best, > Yang > > Jingsong Li 于2020年5月25日周一 下午3:10写道: > > > +1 for the proposal. > > It makes me clearer. > > > > Best, > > Jingsong Lee > > > > On Mon, May 25, 2020 at 2:51 PM Zhijiang > .i
[jira] [Created] (FLINK-17946) The config option "pipeline.jars" doesn't work if the job was executed via TableEnvironment.execute_sql and StatementSet.execute
Dian Fu created FLINK-17946: --- Summary: The config option "pipeline.jars" doesn't work if the job was executed via TableEnvironment.execute_sql and StatementSet.execute Key: FLINK-17946 URL: https://issues.apache.org/jira/browse/FLINK-17946 Project: Flink Issue Type: Bug Components: API / Python Reporter: Dian Fu Fix For: 1.11.0 For the following job: {code} import logging import sys import tempfile from pyflink.table import BatchTableEnvironment, EnvironmentSettings def word_count(): content = "line Licensed to the Apache Software Foundation ASF under one " \ "line or more contributor license agreements See the NOTICE file " \ "line distributed with this work for additional information " \ "line regarding copyright ownership The ASF licenses this file " \ "to you under the Apache License Version the " \ "License you may not use this file except in compliance " \ "with the License" environment_settings = EnvironmentSettings.new_instance().in_batch_mode().\ use_blink_planner().build() t_env = BatchTableEnvironment.create(environment_settings=environment_settings) t_env.get_config().get_configuration().set_string( "pipeline.jars", "file:///Users/dianfu/workspace/wordcount_python/flink-csv-1.11.0-sql-jar.jar") # register Results table in table environment tmp_dir = tempfile.gettempdir() result_path = tmp_dir + '/result' logging.info("Results directory: %s", result_path) sink_ddl = """ create table Results( word VARCHAR, `count` BIGINT ) with ( 'connector' = 'filesystem', 'format' = 'csv', 'path' = '{}' ) """.format(result_path) t_env.execute_sql(sink_ddl) elements = [(word, 1) for word in content.split(" ")] table = t_env.from_elements(elements, ["word", "count"]) \ .group_by("word") \ .select("word, count(1) as count") statement_set = t_env.create_statement_set() statement_set.add_insert("Results", table, overwrite=True) statement_set.execute() if __name__ == '__main__': logging.basicConfig(stream=sys.stdout, level=logging.INFO, format="%(message)s") word_count() {code} It will throw exceptions as following: {code} Caused by: java.lang.ClassCastException: cannot assign instance of java.lang.invoke.SerializedLambda to field org.apache.flink.table.filesystem.FileSystemOutputFormat.formatFactory of type org.apache.flink.table.filesystem.OutputFormatFactory in instance of org.apache.flink.table.filesystem.FileSystemOutputFormat at java.io.ObjectStreamClass$FieldReflector.setObjFieldValues(ObjectStreamClass.java:2133) at java.io.ObjectStreamClass.setObjFieldValues(ObjectStreamClass.java:1305) at java.io.ObjectInputStream.defaultReadFields(ObjectInputStream.java:2237) at java.io.ObjectInputStream.readSerialData(ObjectInputStream.java:2155) at java.io.ObjectInputStream.readOrdinaryObject(ObjectInputStream.java:2013) at java.io.ObjectInputStream.readObject0(ObjectInputStream.java:1535) at java.io.ObjectInputStream.defaultReadFields(ObjectInputStream.java:2231) at java.io.ObjectInputStream.readSerialData(ObjectInputStream.java:2155) at java.io.ObjectInputStream.readOrdinaryObject(ObjectInputStream.java:2013) at java.io.ObjectInputStream.readObject0(ObjectInputStream.java:1535) at java.io.ObjectInputStream.readObject(ObjectInputStream.java:422) at java.util.HashMap.readObject(HashMap.java:1404) at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method) at sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62) at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43) at java.lang.reflect.Method.invoke(Method.java:498) at java.io.ObjectStreamClass.invokeReadObject(ObjectStreamClass.java:1058) at java.io.ObjectInputStream.readSerialData(ObjectInputStream.java:2122) at java.io.ObjectInputStream.readOrdinaryObject(ObjectInputStream.java:2013) at java.io.ObjectInputStream.readObject0(ObjectInputStream.java:1535) at java.io.ObjectInputStream.defaultReadFields(ObjectInputStream.java:2231) at java.io.ObjectInputStream.readSerialData(ObjectInputStream.java:2155) at java.io.ObjectInputStream.readOrdinaryObject(ObjectInputStream.java:2013) at java.io.ObjectInputStream.readObject0(ObjectInputStream.java:1535) at java.io.ObjectInputStream.defaultReadFields(ObjectInputStream.java:2231) at java.io.ObjectInputStream.readSerialData(ObjectInputStream.java:2155) at java.io.ObjectInputStream.readOrdinaryObject(ObjectInpu
[jira] [Created] (FLINK-17947) Retry REST requests if RpcEndpoint died before responding to request
Till Rohrmann created FLINK-17947: - Summary: Retry REST requests if RpcEndpoint died before responding to request Key: FLINK-17947 URL: https://issues.apache.org/jira/browse/FLINK-17947 Project: Flink Issue Type: Improvement Components: Runtime / REST Affects Versions: 1.10.1, 1.11.0 Reporter: Till Rohrmann Fix For: 1.12.0 Currently, it can happen that a REST handler sends a request to a leader {{RpcEndpoint}} and before the {{RpcEndpoint}} has a chance to respond, it might shut down (e.g. due to losing the leadership). In this case, the {{ActorSystem}} will send an {{AskTimeoutException}} as the response with the message {{Recipient Actor[akka://flink/user/rpc/dispatcher_1#-1875884516] had already been terminated.}}. This exception will be treated as any other exception and forwarded to the REST client. There it will be treated as a normal timeout exception which causes the operation (e.g. requesting job details) to fail. I was wondering whether this case should not be handled slightly differently. If the REST handler would respond with a {{SERVICE_UNAVAILABLE}}} HTTP response code, then the {{RestClusterClient}} would retry the operation. One could think of it as if there wouldn't have been a leader available before. This is similar to the situation when there is no current leader and we are waiting for the leader election to finish. Alternatively, we could extend the {{RestClusterClient.isConnectionProblemOrServiceUnavailable}} predicate to also cover the case of special {{AskTimeoutExceptions}}. cc [~chesnay] -- This message was sent by Atlassian Jira (v8.3.4#803005)
Re: [DISCUSS] Semantics of our JIRA fields
If we change the meaning of the priority levels, then I would suggest to have a dedicated discussion for it. This would also be more visible than compared to being hidden in some lengthy discussion thread. I think the proposed definitions of priority levels differ slightly from how the community worked in the past. Cheers, Till On Tue, May 26, 2020 at 4:30 PM Robert Metzger wrote: > Hi, > > 1. I'm okay with updating the definition of the priorities for the reason > you've mentioned. > > 2. "Affects version" > > The reason why like to mark affects version on unreleased versions is to > clearly indicate which branch is affected by a bug. Given the current Flink > release status, if there's a bug only in "release-1.11", but not in > "master", there is no way of figuring that out, if we only allow released > versions for "affects version" (In my proposal, you would set "affects > version" to '1.11.0', '1.12.0' to indicate that). > > What we could do is introduce "1.12-SNAPSHOT" as version to mark issues on > unreleased versions. (But then people might accidentally set the "fix > version" to a "-SNAPSHOT" version.) > > I'm still in favor of my proposal. I have never heard a report from a > confused user about our Jira fields (I guess they usually check bugs for > released versions only) > > > On Tue, May 26, 2020 at 12:39 PM Piotr Nowojski > wrote: > > > Hi, > > > > Sorry for a bit late response. I have two concerns: > > > > 1. Priority > > > > I would propose to stretch priorities that we are using to differentiate > > between things that must be fixed for given release: > > > > BLOCKER - drop anything you are doing, this issue must be fixed right now > > CRITICAL - release can not happen without fixing it, but can be fixed a > > bit later (for example without context switching and dropping whatever > I’m > > doing right now) > > MAJOR - default, nice to have > > Anything below - meh > > > > We were already using this semantic for tracking test instabilities > during > > the 1.11 release cycle. Good examples: > > > > BLOCKER - master branch not compiling, very frequent test failures (for > > example almost every build affected), … > > CRITICAL - performance regression/bug that we introduced in some feature, > > but which is not affecting other developers as much > > MAJOR - freshly discovered test instability with unknown impact/frequency > > (could be happening once a year), > > > > 2. Affects version > > > > If bug is only on the master branch, does it affect an unreleased > version? > > > > So far I was assuming that it doesn’t - unreleased bugs would have empty > > “affects version” field. My reasoning was that this field should be used > > for Flink users, to check which RELEASED Flink versions are affected by > > some bug, that user is searching for. Otherwise it might be a bit > confusing > > if there are lots of bugs with both affects version and fix version set > to > > the same value. > > > > Piotrek > > > > > On 25 May 2020, at 16:40, Robert Metzger wrote: > > > > > > Hi all, > > > thanks a lot for the feedback. The majority of responses are very > > positive > > > to my proposal. > > > > > > I have put my proposal into our wiki: > > > > > > https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=154995514 > > > > > > Regarding the comments so far: > > > @Jark: I clarified this in the wiki. > > > > > > @Israel: I have not considered build changing all 3000 resolved tickets > > to > > > closed yet, but after consideration I don't think it is necessary. If > > > others in the community would like to change them, please speak up in > > this > > > thread :) > > > > > > @Flavio: I agree that we can not ask new or infrequent users to fully > > > adhere to these definitions. I added a note in the Wiki. > > > Using the resolved state for indicating "PR available" is problematic > > > because there are plenty of cases where PRs are stale (and this ticket > > > would then appear as a "resolved"). The Apache tools are adding a link > to > > > the PR, and some contributors are setting the ticket to "In Progress". > I > > > don't see a problem that we need to solve here. > > > > > > @Yun: Thank you for your comment. I added an example clarifying how I > > would > > > handle such a case. It is slightly different from your proposal: You > > > suggested using x.y.0 versions, I used "the next supported, unreleased > > > version", because that's how we've done it so far (and I don't want to > > > change things, I just want to document how the majority of the core > > > contributors are using JIRA). > > > > > > Here are all the changes (in green, blue are just formatting changes) I > > > made compared to my initial proposal: > > > > > > https://cwiki.apache.org/confluence/pages/diffpagesbyversion.action?pageId=154995514&selectedPageVersions=4&selectedPageVersions=1 > > > > > > > > > > > > On Mon, May 25, 2020 at 2:28 PM Congxian Qiu > > wrote: > > > > > >> @ches...@apache.org Thanks for the confirmat
[jira] [Created] (FLINK-17948) Strange precision performance of Timestamp and Decimal
Shengkai Fang created FLINK-17948: - Summary: Strange precision performance of Timestamp and Decimal Key: FLINK-17948 URL: https://issues.apache.org/jira/browse/FLINK-17948 Project: Flink Issue Type: Bug Components: Connectors / JDBC, Table SQL / Client Affects Versions: 1.11.0 Environment: mysql: image: mysql:8.0 volumes: - ./mysql/mktable.sql:/docker-entrypoint-initdb.d/mktable.sql environment: MYSQL_ROOT_PASSWORD: 123456 ports: - "3306:3306" Reporter: Shengkai Fang My job is following: {code:java} CREATE TABLE currency ( currency_id BIGINT, currency_name STRING, rate DOUBLE, currency_timestamp TIMESTAMP, country STRING, precise_timestamp TIMESTAMP(6), precise_time TIME(6), gdp DECIMAL(10, 6) ) WITH ( 'connector' = 'jdbc', 'url' = 'jdbc:mysql://localhost:3306/flink', 'username' = 'root', 'password' = '123456', 'table-name' = 'currency', 'driver' = 'com.mysql.jdbc.Driver', 'lookup.cache.max-rows' = '500', 'lookup.cache.ttl' = '10s', 'lookup.max-retries' = '3') {code} When select * from currency, the precision of results is not as same as expected. The reults of the precision of field precise_timestamp is 3 not 6, and the field gdp has many digit as expected. !image-2020-05-26-22-45-40-711.png! The data in mysql is following: !image-2020-05-26-22-52-02-661.png! -- This message was sent by Atlassian Jira (v8.3.4#803005)
Re: [DISCUSS] Backpoint FLIP-126 (watermarks) integration with FLIP-27
In the beginning, I have somehow similar concerns as Piotr mentioned below. After some offline discussions, also as explained by Stephan and Becket here, I am +1 to backport it to release-1.11. Best, Zhijiang -- From:Piotr Nowojski Send Time:2020年5月26日(星期二) 18:51 To:Becket Qin Cc:Stephan Ewen ; dev ; zhijiang Subject:Re: [DISCUSS] Backpoint FLIP-126 (watermarks) integration with FLIP-27 Hi, As we discussed this offline a bit, initially I was sceptical to merge it, as: - even it’s an isolated change, it can destabilise the builds and prolong release testing period - is distracting from solving release blockers etc However all in all I’m +0.5 to merge it because of this argument: > - It is API breaking. Without this patch, we would release a Source API and immediately break compatibility in the next release. And this: > - It is a fairly isolated change, does not affect any existing feature in the system Is limiting our risks, that we are not risking introducing bugs into the existing features. Piotrek wt., 26 maj 2020 o 12:43 Becket Qin napisał(a): > Usually we should avoid checking in patches other than bug fix after > feature freeze. However, in this particular case, the code base is sort of > in an incomplete state - an exposed known-to-change feature - due to > missing this patch. Fixing forward seems the best option. Besides that, > FLIP-27 has been highly anticipated by many users. So if one patch > completes the story, personally speaking I am +1 to backport given the > isolated impact and significant benefit of doing that. > > Thanks, > > Jiangjie (Becket) Qin > > > On Tue, May 26, 2020 at 4:43 PM Stephan Ewen wrote: > >> Hi all! >> >> I want to discuss merging this PR to the 1.11 release branch: >> https://github.com/apache/flink/pull/12306 >> >> It contains the new FLIP-126 Watermarks, and per-partition watermarking >> to the FLIP-27 sources. In that sense it is partially a new feature after >> the feature freeze. Hence this discussion, and not just merging. >> >> The reasons why I suggest to back-port this to 1.11 are >> - It is API breaking. Without this patch, we would release a Source API >> and immediately break compatibility in the next release. >> - The FLIP-27 feature is experimental, but it should not be useless in >> the sense that users have to re-write all implemented sources in the next >> release. >> - It is a fairly isolated change, does not affect any existing feature >> in the system >> >> Please let me know if you have concerns about this. >> >> Best, >> Stephan >> >>
[jira] [Created] (FLINK-17949) KafkaShuffleITCase.testSerDeIngestionTime:156->testRecordSerDe:388 expected:<310> but was:<0>
Robert Metzger created FLINK-17949: -- Summary: KafkaShuffleITCase.testSerDeIngestionTime:156->testRecordSerDe:388 expected:<310> but was:<0> Key: FLINK-17949 URL: https://issues.apache.org/jira/browse/FLINK-17949 Project: Flink Issue Type: Bug Components: Connectors / Kafka, Tests Affects Versions: 1.12.0 Reporter: Robert Metzger https://dev.azure.com/apache-flink/apache-flink/_build/results?buildId=2209&view=logs&j=c5f0071e-1851-543e-9a45-9ac140befc32&t=684b1416-4c17-504e-d5ab-97ee44e08a20 {code} 2020-05-26T13:35:19.4022562Z [ERROR] testSerDeIngestionTime(org.apache.flink.streaming.connectors.kafka.shuffle.KafkaShuffleITCase) Time elapsed: 5.786 s <<< FAILURE! 2020-05-26T13:35:19.4023185Z java.lang.AssertionError: expected:<310> but was:<0> 2020-05-26T13:35:19.4023498Zat org.junit.Assert.fail(Assert.java:88) 2020-05-26T13:35:19.4023825Zat org.junit.Assert.failNotEquals(Assert.java:834) 2020-05-26T13:35:19.4024461Zat org.junit.Assert.assertEquals(Assert.java:645) 2020-05-26T13:35:19.4024900Zat org.junit.Assert.assertEquals(Assert.java:631) 2020-05-26T13:35:19.4028546Zat org.apache.flink.streaming.connectors.kafka.shuffle.KafkaShuffleITCase.testRecordSerDe(KafkaShuffleITCase.java:388) 2020-05-26T13:35:19.4029629Zat org.apache.flink.streaming.connectors.kafka.shuffle.KafkaShuffleITCase.testSerDeIngestionTime(KafkaShuffleITCase.java:156) 2020-05-26T13:35:19.4030253Zat sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method) 2020-05-26T13:35:19.4030673Zat sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62) 2020-05-26T13:35:19.4031332Zat sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43) 2020-05-26T13:35:19.4031763Zat java.lang.reflect.Method.invoke(Method.java:498) 2020-05-26T13:35:19.4032155Zat org.junit.runners.model.FrameworkMethod$1.runReflectiveCall(FrameworkMethod.java:50) 2020-05-26T13:35:19.4032630Zat org.junit.internal.runners.model.ReflectiveCallable.run(ReflectiveCallable.java:12) 2020-05-26T13:35:19.4033188Zat org.junit.runners.model.FrameworkMethod.invokeExplosively(FrameworkMethod.java:47) 2020-05-26T13:35:19.4033638Zat org.junit.internal.runners.statements.InvokeMethod.evaluate(InvokeMethod.java:17) 2020-05-26T13:35:19.4034103Zat org.junit.internal.runners.statements.RunBefores.evaluate(RunBefores.java:26) 2020-05-26T13:35:19.4034593Zat org.junit.internal.runners.statements.FailOnTimeout$CallableStatement.call(FailOnTimeout.java:298) 2020-05-26T13:35:19.4035118Zat org.junit.internal.runners.statements.FailOnTimeout$CallableStatement.call(FailOnTimeout.java:292) 2020-05-26T13:35:19.4035570Zat java.util.concurrent.FutureTask.run(FutureTask.java:266) 2020-05-26T13:35:19.4035888Zat java.lang.Thread.run(Thread.java:748) {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
Re: [DISCUSS] (Document) Backwards Compatibility of Savepoints
> A use case for this might be when you want to rollback a framework upgrade (after some time) due to e.g. a performance or stability issue. Downgrade (that Konstantin called out) is an important and realistic scenario. It will be great to support backward compatibility for savepoint or at least document any breaking change. On Tue, May 26, 2020 at 4:39 AM Piotr Nowojski wrote: > Hi, > > It might have been implicit choice, but so far we were not supporting the > scenario that you are asking for. It has never been tested and we have > lot’s of state migration code sprinkled among our code base (for example > upgrading state fields of the operators like [1]), that only supports > upgrades, not downgrades. > > Also we do not have testing infrastructure for checking the downgrades. We > would need to check if save points taken from master branch, are readable > by previous releases (not release branch!). > > So all in all, I don’t think it can be easily done. It would require some > effort to start maintaining backward compatibility. > > Piotrek > > [1] > org.apache.flink.streaming.connectors.kafka.FlinkKafkaProducer011#migrateNextTransactionalIdHindState > > > On 26 May 2020, at 13:18, Konstantin Knauf wrote: > > > > Hi everyone, > > > > I recently stumbled across the fact that Savepoints created with Flink > 1.11 > > can not be read by Flink 1.10. A use case for this might be when you want > > to rollback a framework upgrade (after some time) due to e.g. a > performance > > or stability issue. > > > > From the documentation [1] it seems as if the Savepoint format is > generally > > only forward-compatible although in many cases it is actually also > > backwards compatible (e.g. Savepoint taken in Flink 1.10, restored with > > Flink 1.9). > > > > Was it a deliberate choice not to document any backwards compatibility? > If > > not, should we add the missing entries in the compatibility table? > > > > Thanks, > > > > Konstantin > > > > [1] > > > https://ci.apache.org/projects/flink/flink-docs-master/ops/upgrading.html#compatibility-table > > > > -- > > > > Konstantin Knauf > > > > https://twitter.com/snntrable > > > > https://github.com/knaufk > >
[jira] [Created] (FLINK-17950) Broken Scala env.countinuousSource method
Stephan Ewen created FLINK-17950: Summary: Broken Scala env.countinuousSource method Key: FLINK-17950 URL: https://issues.apache.org/jira/browse/FLINK-17950 Project: Flink Issue Type: Bug Components: API / Scala Reporter: Stephan Ewen Assignee: Stephan Ewen Fix For: 1.11.0 The Scala {{StreamExecutionEnvironment.countinuousSource(...)}} method has two critical problems: - Its return type is {{Unit}} instead of {{DataStream}}, so that no one can use the created stream - It does not forward the TypeInformation identified by the ScalaCompiler but relies on the Java TypeExtraction stack, which cannot handle most Scala types. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-17951) Improve the console message about executing a PyFlink job
Dian Fu created FLINK-17951: --- Summary: Improve the console message about executing a PyFlink job Key: FLINK-17951 URL: https://issues.apache.org/jira/browse/FLINK-17951 Project: Flink Issue Type: Improvement Components: API / Python Affects Versions: 1.10.0, 1.9.0, 1.11.0 Reporter: Dian Fu Thanks for the feedback from [~rmetzger]: {code:java} I run python ./flink-tutorial.py, it waits for a few seconds, and then it returns, that’s it What I would expect is something like $ python ./flink-tutorial.py Running PyFlink job … PyFlink job completed after 14 seconds $ {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-17952) Improve the error message when old planner and batch mode is used via EnvironmentSettings
Dian Fu created FLINK-17952: --- Summary: Improve the error message when old planner and batch mode is used via EnvironmentSettings Key: FLINK-17952 URL: https://issues.apache.org/jira/browse/FLINK-17952 Project: Flink Issue Type: Improvement Components: API / Python Affects Versions: 1.10.0, 1.9.0, 1.11.0 Reporter: Dian Fu Fix For: 1.11.0 Currently it doesn't support to use batch mode of the old planner via EnvironmentSettings. The following message will be thrown in that case: {code} : org.apache.flink.table.api.NoMatchingTableFactoryException: Could not find a suitable table factory for 'org.apache.flink.table.delegation.ExecutorFactory' in: org.apache.flink.table.api.NoMatchingTableFactoryException: Could not find a suitable table factory for 'org.apache.flink.table.delegation.ExecutorFactory' inthe classpath. Reason: No factory supports the additional filters. The following properties are requested:class-name=org.apache.flink.table.executor.StreamExecutorFactorystreaming-mode=false The following factories have been considered:org.apache.flink.table.planner.delegation.BlinkExecutorFactory at org.apache.flink.table.factories.ComponentFactoryService.find(ComponentFactoryService.java:71) at org.apache.flink.table.api.internal.TableEnvironmentImpl.create(TableEnvironmentImpl.java:253) at org.apache.flink.table.api.TableEnvironment.create(TableEnvironment.java:91) at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method) at sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62) at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43) at java.lang.reflect.Method.invoke(Method.java:498) at org.apache.flink.api.python.shaded.py4j.reflection.MethodInvoker.invoke(MethodInvoker.java:244) at org.apache.flink.api.python.shaded.py4j.reflection.ReflectionEngine.invoke(ReflectionEngine.java:357) at org.apache.flink.api.python.shaded.py4j.Gateway.invoke(Gateway.java:282) at org.apache.flink.api.python.shaded.py4j.commands.AbstractCommand.invokeMethod(AbstractCommand.java:132) at org.apache.flink.api.python.shaded.py4j.commands.CallCommand.execute(CallCommand.java:79) at org.apache.flink.api.python.shaded.py4j.GatewayConnection.run(GatewayConnection.java:238) at java.lang.Thread.run(Thread.java:745) {code} This exception message is confusing for a Python users and we should improve it. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-17953) OverWindow doesn't support to order by non-time attribute in batch mode for Table API program
Dian Fu created FLINK-17953: --- Summary: OverWindow doesn't support to order by non-time attribute in batch mode for Table API program Key: FLINK-17953 URL: https://issues.apache.org/jira/browse/FLINK-17953 Project: Flink Issue Type: Bug Components: Table SQL / Planner Affects Versions: 1.10.0, 1.9.0, 1.11.0 Reporter: Dian Fu For a simple batch job: {code} INSERT INTO results SELECT id, sum(sales) OVER (PARTITION BY id ORDER BY ts ROWS BETWEEN 2 PRECEDING AND 0 FOLLOWING) FROM input {code} It could pass in blink planner. However, if we rewrite it in Table API, it will throw the following exception: {code} py4j.protocol.Py4JJavaError: An error occurred while calling o85.select.py4j.protocol.Py4JJavaError: An error occurred while calling o85.select.: org.apache.flink.table.api.ValidationException: Ordering must be defined on a time attribute. at org.apache.flink.table.planner.expressions.PlannerTypeInferenceUtilImpl.validateArguments(PlannerTypeInferenceUtilImpl.java:112) at org.apache.flink.table.planner.expressions.PlannerTypeInferenceUtilImpl.runTypeInference(PlannerTypeInferenceUtilImpl.java:71) at org.apache.flink.table.expressions.resolver.rules.ResolveCallByArgumentsRule$ResolvingCallVisitor.runLegacyTypeInference(ResolveCallByArgumentsRule.java:218) at org.apache.flink.table.expressions.resolver.rules.ResolveCallByArgumentsRule$ResolvingCallVisitor.lambda$visit$2(ResolveCallByArgumentsRule.java:134) at java.util.Optional.orElseGet(Optional.java:267) at org.apache.flink.table.expressions.resolver.rules.ResolveCallByArgumentsRule$ResolvingCallVisitor.visit(ResolveCallByArgumentsRule.java:134) at org.apache.flink.table.expressions.resolver.rules.ResolveCallByArgumentsRule$ResolvingCallVisitor.visit(ResolveCallByArgumentsRule.java:89) at org.apache.flink.table.expressions.ApiExpressionVisitor.visit(ApiExpressionVisitor.java:39) at org.apache.flink.table.expressions.UnresolvedCallExpression.accept(UnresolvedCallExpression.java:132) at org.apache.flink.table.expressions.resolver.rules.ResolveCallByArgumentsRule$ResolvingCallVisitor.visit(ResolveCallByArgumentsRule.java:124) at org.apache.flink.table.expressions.resolver.rules.ResolveCallByArgumentsRule$ResolvingCallVisitor.visit(ResolveCallByArgumentsRule.java:89) at org.apache.flink.table.expressions.ApiExpressionVisitor.visit(ApiExpressionVisitor.java:39) at org.apache.flink.table.expressions.UnresolvedCallExpression.accept(UnresolvedCallExpression.java:132) at org.apache.flink.table.expressions.resolver.rules.ResolveCallByArgumentsRule.lambda$apply$0(ResolveCallByArgumentsRule.java:83) at java.util.stream.ReferencePipeline$7$1.accept(ReferencePipeline.java:267) at java.util.ArrayList$ArrayListSpliterator.forEachRemaining(ArrayList.java:1374) at java.util.stream.AbstractPipeline.copyInto(AbstractPipeline.java:481) at java.util.stream.AbstractPipeline.wrapAndCopyInto(AbstractPipeline.java:471) at java.util.stream.ReduceOps$ReduceOp.evaluateSequential(ReduceOps.java:708) at java.util.stream.AbstractPipeline.evaluate(AbstractPipeline.java:234) at java.util.stream.ReferencePipeline.collect(ReferencePipeline.java:499) at org.apache.flink.table.expressions.resolver.rules.ResolveCallByArgumentsRule.apply(ResolveCallByArgumentsRule.java:84) at org.apache.flink.table.expressions.resolver.ExpressionResolver.lambda$null$1(ExpressionResolver.java:211) at java.util.function.Function.lambda$andThen$1(Function.java:88) at org.apache.flink.table.expressions.resolver.ExpressionResolver.resolve(ExpressionResolver.java:178) at org.apache.flink.table.operations.utils.OperationTreeBuilder.projectInternal(OperationTreeBuilder.java:191) at org.apache.flink.table.operations.utils.OperationTreeBuilder.project(OperationTreeBuilder.java:170) at org.apache.flink.table.api.internal.TableImpl$OverWindowedTableImpl.select(TableImpl.java:953) at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method) at sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62) at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43) at java.lang.reflect.Method.invoke(Method.java:498) at org.apache.flink.api.python.shaded.py4j.reflection.MethodInvoker.invoke(MethodInvoker.java:244) at org.apache.flink.api.python.shaded.py4j.reflection.ReflectionEngine.invoke(ReflectionEngine.java:357) at org.apache.flink.api.python.shaded.py4j.Gateway.invoke(Gateway.java:282) at org.apache.flink.api.python.shaded.py4j.commands.AbstractCommand.invokeMethod(AbstractCommand.java:132) at org.apache.flink.api.python.shaded.py4j.commands.CallCommand.execute(CallCommand.java:79) at org.apache.flink.api.python.shaded.py4j.GatewayConnection.run(GatewayConnection.java:238) at java.lang.Thread.run(Thread.java:745) {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-17954) Do not multiplex remote function state into single PersistedTable
Tzu-Li (Gordon) Tai created FLINK-17954: --- Summary: Do not multiplex remote function state into single PersistedTable Key: FLINK-17954 URL: https://issues.apache.org/jira/browse/FLINK-17954 Project: Flink Issue Type: Task Components: Stateful Functions Affects Versions: statefun-2.0.1, statefun-2.1.0 Reporter: Tzu-Li (Gordon) Tai We are currently multiplexing multiple remote function's user value states into a single {{PersistedTable}}, using the state name as the table key. This is not nice since: - It does not allow individual states to have different properties, such as TTL expiration. - We are restricted to only value states for remote functions -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-17955) BucketLifeCycleListener should just in Buckets
Jingsong Lee created FLINK-17955: Summary: BucketLifeCycleListener should just in Buckets Key: FLINK-17955 URL: https://issues.apache.org/jira/browse/FLINK-17955 Project: Flink Issue Type: Bug Components: Connectors / FileSystem Reporter: Jingsong Lee Assignee: Jingsong Lee Fix For: 1.11.0 We should keep BucketLifeCycleListener just in runtime. -- This message was sent by Atlassian Jira (v8.3.4#803005)
[jira] [Created] (FLINK-17956) Add Flink 1.11 MigrationVersion
Aljoscha Krettek created FLINK-17956: Summary: Add Flink 1.11 MigrationVersion Key: FLINK-17956 URL: https://issues.apache.org/jira/browse/FLINK-17956 Project: Flink Issue Type: Task Components: API / Core Reporter: Aljoscha Krettek Assignee: Aljoscha Krettek Fix For: 1.11.0 -- This message was sent by Atlassian Jira (v8.3.4#803005)