[jira] [Created] (FLINK-35215) The performance of serializerKryo and serializerKryoWithoutRegistration are regressed
Rui Fan created FLINK-35215: --- Summary: The performance of serializerKryo and serializerKryoWithoutRegistration are regressed Key: FLINK-35215 URL: https://issues.apache.org/jira/browse/FLINK-35215 Project: Flink Issue Type: Bug Components: API / Type Serialization System Affects Versions: 1.20.0 Reporter: Rui Fan The performance of serializerKryo and serializerKryoWithoutRegistration are regressed[1][2], I checked recent commits, and found FLINK-34954 changed related logic. [1] [http://flink-speed.xyz/timeline/#/?exe=1,6,12=serializerKryo=on=on=off=3=50] [2] http://flink-speed.xyz/timeline/#/?exe=1,6,12=serializerKryoWithoutRegistration=on=on=off=3=50 -- This message was sent by Atlassian Jira (v8.20.10#820010)
Re: [DISCUSS] FLIP-447: Upgrade FRocksDB from 6.20.3 to 8.10.0
Hi Yue, Thanks for driving this work. It has been three years since last major upgrade of FRocksDB. And it would be great improvement of Flink's state-backend with this upgrade. +1 for this work. Best Yun Tang From: Yanfei Lei Sent: Tuesday, April 23, 2024 12:50 To: dev@flink.apache.org Subject: Re: [DISCUSS] FLIP-447: Upgrade FRocksDB from 6.20.3 to 8.10.0 Hi Yue & Roman, Thanks for initiating this FLIP and all the efforts for the upgrade. 8.10.0 introduces some new features, making it possible for Flink to implement some new exciting features, and the upgrade also makes FRocksDB easier to maintain, +1 for upgrading. I read the FLIP and have a minor comment, it would be better to add some description about the environment/configuration of the nexmark's result. Roman Khachatryan 于2024年4月23日周二 12:07写道: > > Hi, > > Thanks for writing the proposal and preparing the upgrade. > > FRocksDB definitely needs to be kept in sync with the upstream and the new > APIs are necessary for faster rescaling. > We're already using a similar version internally. > > I reviewed the FLIP and it looks good to me (disclaimer: I took part in > some steps of this effort). > > > Regards, > Roman > > On Mon, Apr 22, 2024, 08:11 yue ma wrote: > > > Hi Flink devs, > > > > I would like to start a discussion on FLIP-447: Upgrade FRocksDB from > > 6.20.3 to 8.10.0 > > > > > > https://cwiki.apache.org/confluence/display/FLINK/FLIP-447%3A+Upgrade+FRocksDB+from+6.20.3++to+8.10.0 > > > > This FLIP proposes upgrading the version of FRocksDB in the Flink Project > > from 6.20.3 to 8.10.0. > > The FLIP mainly introduces the main benefits of upgrading FRocksDB, > > including the use of IngestDB which can improve Rescaling performance by > > more than 10 times in certain scenarios, as well as other potential > > optimization points such as async_io, blob db, and tiered storage.The > > FLIP also presented test results based on RocksDB 8.10, including > > StateBenchmark and Nexmark tests. > > Overall, upgrading FRocksDB may result in a small regression of write > > performance( which is a very small part of the overall overhead), but it > > can bring many important performance benefits. > > So we hope to upgrade the version of FRocksDB through this FLIP. > > > > Looking forward to everyone's feedback and suggestions. Thank you! > > -- > > Best regards, > > Yue > > -- Best, Yanfei
Re: [DISCUSS] FLIP-447: Upgrade FRocksDB from 6.20.3 to 8.10.0
Hi Yue & Roman, Thanks for initiating this FLIP and all the efforts for the upgrade. 8.10.0 introduces some new features, making it possible for Flink to implement some new exciting features, and the upgrade also makes FRocksDB easier to maintain, +1 for upgrading. I read the FLIP and have a minor comment, it would be better to add some description about the environment/configuration of the nexmark's result. Roman Khachatryan 于2024年4月23日周二 12:07写道: > > Hi, > > Thanks for writing the proposal and preparing the upgrade. > > FRocksDB definitely needs to be kept in sync with the upstream and the new > APIs are necessary for faster rescaling. > We're already using a similar version internally. > > I reviewed the FLIP and it looks good to me (disclaimer: I took part in > some steps of this effort). > > > Regards, > Roman > > On Mon, Apr 22, 2024, 08:11 yue ma wrote: > > > Hi Flink devs, > > > > I would like to start a discussion on FLIP-447: Upgrade FRocksDB from > > 6.20.3 to 8.10.0 > > > > > > https://cwiki.apache.org/confluence/display/FLINK/FLIP-447%3A+Upgrade+FRocksDB+from+6.20.3++to+8.10.0 > > > > This FLIP proposes upgrading the version of FRocksDB in the Flink Project > > from 6.20.3 to 8.10.0. > > The FLIP mainly introduces the main benefits of upgrading FRocksDB, > > including the use of IngestDB which can improve Rescaling performance by > > more than 10 times in certain scenarios, as well as other potential > > optimization points such as async_io, blob db, and tiered storage.The > > FLIP also presented test results based on RocksDB 8.10, including > > StateBenchmark and Nexmark tests. > > Overall, upgrading FRocksDB may result in a small regression of write > > performance( which is a very small part of the overall overhead), but it > > can bring many important performance benefits. > > So we hope to upgrade the version of FRocksDB through this FLIP. > > > > Looking forward to everyone's feedback and suggestions. Thank you! > > -- > > Best regards, > > Yue > > -- Best, Yanfei
Re: [DISCUSS] FLIP-447: Upgrade FRocksDB from 6.20.3 to 8.10.0
Thanks Yue for driving this proposal! This upgrade makes sense, +1 for that. Best, Rui On Tue, Apr 23, 2024 at 12:06 PM Roman Khachatryan wrote: > Hi, > > Thanks for writing the proposal and preparing the upgrade. > > FRocksDB definitely needs to be kept in sync with the upstream and the new > APIs are necessary for faster rescaling. > We're already using a similar version internally. > > I reviewed the FLIP and it looks good to me (disclaimer: I took part in > some steps of this effort). > > > Regards, > Roman > > On Mon, Apr 22, 2024, 08:11 yue ma wrote: > > > Hi Flink devs, > > > > I would like to start a discussion on FLIP-447: Upgrade FRocksDB from > > 6.20.3 to 8.10.0 > > > > > > > https://cwiki.apache.org/confluence/display/FLINK/FLIP-447%3A+Upgrade+FRocksDB+from+6.20.3++to+8.10.0 > > > > This FLIP proposes upgrading the version of FRocksDB in the Flink Project > > from 6.20.3 to 8.10.0. > > The FLIP mainly introduces the main benefits of upgrading FRocksDB, > > including the use of IngestDB which can improve Rescaling performance by > > more than 10 times in certain scenarios, as well as other potential > > optimization points such as async_io, blob db, and tiered storage.The > > FLIP also presented test results based on RocksDB 8.10, including > > StateBenchmark and Nexmark tests. > > Overall, upgrading FRocksDB may result in a small regression of write > > performance( which is a very small part of the overall overhead), but it > > can bring many important performance benefits. > > So we hope to upgrade the version of FRocksDB through this FLIP. > > > > Looking forward to everyone's feedback and suggestions. Thank you! > > -- > > Best regards, > > Yue > > >
[jira] [Created] (FLINK-35214) Update result partition id for remote input channel when unknown input channel is updated
Yuxin Tan created FLINK-35214: - Summary: Update result partition id for remote input channel when unknown input channel is updated Key: FLINK-35214 URL: https://issues.apache.org/jira/browse/FLINK-35214 Project: Flink Issue Type: Bug Components: Runtime / Network Affects Versions: 1.20.0 Reporter: Yuxin Tan In [FLINK-29768|https://issues.apache.org/jira/browse/FLINK-29768], the result partition in the local input channel has been updated to support speculation. It is necessary to similarly update the result partition ID in the remote input channel. -- This message was sent by Atlassian Jira (v8.20.10#820010)
Re: [DISCUSS] FLIP-447: Upgrade FRocksDB from 6.20.3 to 8.10.0
Hi, Thanks for writing the proposal and preparing the upgrade. FRocksDB definitely needs to be kept in sync with the upstream and the new APIs are necessary for faster rescaling. We're already using a similar version internally. I reviewed the FLIP and it looks good to me (disclaimer: I took part in some steps of this effort). Regards, Roman On Mon, Apr 22, 2024, 08:11 yue ma wrote: > Hi Flink devs, > > I would like to start a discussion on FLIP-447: Upgrade FRocksDB from > 6.20.3 to 8.10.0 > > > https://cwiki.apache.org/confluence/display/FLINK/FLIP-447%3A+Upgrade+FRocksDB+from+6.20.3++to+8.10.0 > > This FLIP proposes upgrading the version of FRocksDB in the Flink Project > from 6.20.3 to 8.10.0. > The FLIP mainly introduces the main benefits of upgrading FRocksDB, > including the use of IngestDB which can improve Rescaling performance by > more than 10 times in certain scenarios, as well as other potential > optimization points such as async_io, blob db, and tiered storage.The > FLIP also presented test results based on RocksDB 8.10, including > StateBenchmark and Nexmark tests. > Overall, upgrading FRocksDB may result in a small regression of write > performance( which is a very small part of the overall overhead), but it > can bring many important performance benefits. > So we hope to upgrade the version of FRocksDB through this FLIP. > > Looking forward to everyone's feedback and suggestions. Thank you! > -- > Best regards, > Yue >
[jira] [Created] (FLINK-35213) A potential inconsistent table structure issue
LvYanquan created FLINK-35213: - Summary: A potential inconsistent table structure issue Key: FLINK-35213 URL: https://issues.apache.org/jira/browse/FLINK-35213 Project: Flink Issue Type: Improvement Components: Flink CDC Affects Versions: 3.1.0 Reporter: LvYanquan Currently, DataSinkWriterOperator will [request CreateTableEvent|https://github.com/apache/flink-cdc/blob/313726b09690e82aa56fb5b42e89b535d24dadd7/flink-cdc-runtime/src/main/java/org/apache/flink/cdc/runtime/operators/sink/DataSinkWriterOperator.java#L149] from SchemaRegistry when restarted. However, If a SchemeChangeEvent is received during this process, SchemaOperator will 1. [request SchemaRegistry|https://github.com/apache/flink-cdc/blob/313726b09690e82aa56fb5b42e89b535d24dadd7/flink-cdc-runtime/src/main/java/org/apache/flink/cdc/runtime/operators/schema/SchemaOperator.java#L252] to update the schema, 2. and then send FlushEvent. As the network situation is quite complex, SchemaRegistry may update the schema first, and then send a CreateTableEvent with the new schema, which is incompatible with DatachangeEvent. -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35212) PyFlink thread mode process just can run once in standalonesession mode
Wei Yuan created FLINK-35212: Summary: PyFlink thread mode process just can run once in standalonesession mode Key: FLINK-35212 URL: https://issues.apache.org/jira/browse/FLINK-35212 Project: Flink Issue Type: Bug Components: API / Python Environment: Python 3.10.14 PyFlink==1.18.1 openjdk version "11.0.21" 2023-10-17 LTS OpenJDK Runtime Environment (Red_Hat-11.0.21.0.9-1.el7_9) (build 11.0.21+9-LTS) OpenJDK 64-Bit Server VM (Red_Hat-11.0.21.0.9-1.el7_9) (build 11.0.21+9-LTS, mixed mode, sharing) Reporter: Wei Yuan {code:java} from pyflink.common.types import Row from pyflink.datastream import StreamExecutionEnvironment from pyflink.common import Types, WatermarkStrategy, Configuration from pyflink.table import EnvironmentSettings, TableEnvironment from pyflink.table import StreamTableEnvironment, Schema from pyflink.datastream.functions import ProcessFunction, MapFunction from pyflink.common.time import Instant # init task env config = Configuration() config.set_string("python.execution-mode", "thread") # config.set_string("python.execution-mode", "process") config.set_string("python.client.executable", "/root/miniconda3/bin/python3") config.set_string("python.executable", "/root/miniconda3/bin/python3") env = StreamExecutionEnvironment.get_execution_environment(config) table_env = StreamTableEnvironment.create(env) # create a batch TableEnvironment table = table_env.from_elements([(1, 'Hi'), (2, 'Hello')]).alias("id", "content") table_env.create_temporary_view("test_table", table) result_table = table_env.sql_query("select *, NOW() as dt from test_table") result_ds = table_env.to_data_stream(result_table) # def test_func(row): # return row # result_ds.map(test_func).print() result_ds.print() env.execute() {code} Start a standalone session mode cluster by command: {code:java} /root/miniconda3/lib/python3.10/site-packages/pyflink/bin/bin/start-cluster.sh{code} Submit thread mode job for the first time, this job will success fnished. {code:java} /root/miniconda3/lib/python3.10/site-packages/pyflink/bin/flink run -py bug.py {code} Use above command to submit job for the second time, an error occured: {code:java} Job has been submitted with JobID a4f2728199277bba0500796f7925fa26 Traceback (most recent call last): File "/home/disk1/bug.py", line 34, in env.execute() File "/root/miniconda3/lib/python3.10/site-packages/pyflink/datastream/stream_execution_environment.py", line 773, in execute return JobExecutionResult(self._j_stream_execution_environment.execute(j_stream_graph)) File "/root/miniconda3/lib/python3.10/site-packages/py4j/java_gateway.py", line 1322, in __call__ return_value = get_return_value( File "/root/miniconda3/lib/python3.10/site-packages/pyflink/util/exceptions.py", line 146, in deco return f(*a, **kw) File "/root/miniconda3/lib/python3.10/site-packages/py4j/protocol.py", line 326, in get_return_value raise Py4JJavaError( py4j.protocol.Py4JJavaError: An error occurred while calling o7.execute. : java.util.concurrent.ExecutionException: org.apache.flink.client.program.ProgramInvocationException: Job failed (JobID: a4f2728199277bba0500796f7925fa26) at java.base/java.util.concurrent.CompletableFuture.reportGet(CompletableFuture.java:395) at java.base/java.util.concurrent.CompletableFuture.get(CompletableFuture.java:2005) at org.apache.flink.client.program.StreamContextEnvironment.getJobExecutionResult(StreamContextEnvironment.java:171) at org.apache.flink.client.program.StreamContextEnvironment.execute(StreamContextEnvironment.java:122) at java.base/jdk.internal.reflect.NativeMethodAccessorImpl.invoke0(Native Method) at java.base/jdk.internal.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62) at java.base/jdk.internal.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43) at java.base/java.lang.reflect.Method.invoke(Method.java:566) 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:374) 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.base/java.lang.Thread.run(Thread.java:829) Caused by: org.apache.flink.client.program.ProgramInvocationException: Job failed (JobID: a4f2728199277bba0500796f7925fa26) at
Re: [VOTE] Release flink-connector-cassandra v3.2.0, release candidate #1
Thanks! +1 (non-binding) - Checked 512 hash - Checked gpg signature - Reviewed web pr & release notes - Built the source with JDK 11 & 8 Best, Muhammet On 2024-04-22 13:04, Danny Cranmer wrote: Hi everyone, Please review and vote on release candidate #1 for flink-connector-cassandra v3.2.0, as follows: [ ] +1, Approve the release [ ] -1, Do not approve the release (please provide specific comments) This release supports Flink 1.18 and 1.19. The complete staging area is available for your review, which includes: * JIRA release notes [1], * the official Apache source release to be deployed to dist.apache.org [2], which are signed with the key with fingerprint 125FD8DB [3], * all artifacts to be deployed to the Maven Central Repository [4], * source code tag v3.2.0-rc1 [5], * website pull request listing the new release [6]. * CI build of the tag [7]. The vote will be open for at least 72 hours. It is adopted by majority approval, with at least 3 PMC affirmative votes. Thanks, Danny [1] https://issues.apache.org/jira/secure/ReleaseNote.jspa?projectId=12315522=12353148 [2] https://dist.apache.org/repos/dist/dev/flink/flink-connector-cassandra-3.2.0-rc1 [3] https://dist.apache.org/repos/dist/release/flink/KEYS [4] https://repository.apache.org/content/repositories/orgapacheflink-1722 [5] https://github.com/apache/flink-connector-cassandra/releases/tag/v3.2.0-rc1 [6] https://github.com/apache/flink-web/pull/737 [7] https://github.com/apache/flink-connector-cassandra/actions/runs/8784310241
Re: [VOTE] Release flink-connector-kafka v3.2.0, release candidate #1
Thanks Danny, +1 (non-binding) - Checked 512 hash - Checked gpg signature - Reviewed pr - Built the source with JDK 11 & 8 Best, Muhammet On 2024-04-22 13:55, Danny Cranmer wrote: Hi everyone, Please review and vote on release candidate #1 for flink-connector-kafka v3.2.0, as follows: [ ] +1, Approve the release [ ] -1, Do not approve the release (please provide specific comments) This release supports Flink 1.18 and 1.19. The complete staging area is available for your review, which includes: * JIRA release notes [1], * the official Apache source release to be deployed to dist.apache.org [2], which are signed with the key with fingerprint 125FD8DB [3], * all artifacts to be deployed to the Maven Central Repository [4], * source code tag v3.2.0-rc1 [5], * website pull request listing the new release [6]. * CI build of the tag [7]. The vote will be open for at least 72 hours. It is adopted by majority approval, with at least 3 PMC affirmative votes. Thanks, Danny [1] https://issues.apache.org/jira/secure/ReleaseNote.jspa?projectId=12315522=12354209 [2] https://dist.apache.org/repos/dist/dev/flink/flink-connector-kafka-3.2.0-rc1 [3] https://dist.apache.org/repos/dist/release/flink/KEYS [4] https://repository.apache.org/content/repositories/orgapacheflink-1723 [5] https://github.com/apache/flink-connector-kafka/releases/tag/v3.2.0-rc1 [6] https://github.com/apache/flink-web/pull/738 [7] https://github.com/apache/flink-connector-kafka
[jira] [Created] (FLINK-35211) when synchronize LOB fields using oracle cdc, an error occurs
wangsw created FLINK-35211: -- Summary: when synchronize LOB fields using oracle cdc, an error occurs Key: FLINK-35211 URL: https://issues.apache.org/jira/browse/FLINK-35211 Project: Flink Issue Type: Bug Components: Flink CDC Environment: Flink 18.1 Oracle cdc 3.0.1 Reporter: wangsw Attachments: cdcOracleToPrint.sql {code:java} Caused by: java.sql.SQLException: ORA-01291: 缺失日志文件 ORA-06512: 在 "SYS.DBMS_LOGMNR", line 58 ORA-06512: 在 line 1oracle.jdbc.driver.OracleStatement.doExecuteWithTimeout(OracleStatement.java:1205) oracle.jdbc.driver.OracleStatement.executeInternal(OracleStatement.java:1823) oracle.jdbc.driver.OracleStatement.execute(OracleStatement.java:1778) oracle.jdbc.driver.OracleStatementWrapper.execute(OracleStatementWrapper.java:303) io.debezium.jdbc.JdbcConnection.executeWithoutCommitting(JdbcConnection.java:1446) io.debezium.connector.oracle.logminer.LogMinerStreamingChangeEventSource.startMiningSession(LogMinerStreamingChangeEventSource.java:677) io.debezium.connector.oracle.logminer.LogMinerStreamingChangeEventSource.execute(LogMinerStreamingChangeEventSource.java:244) {code} when synchronize LOB fields using oracle cdc 3.0.1 , an error occurs. I checked that the debezium version of CDC is 1.9.7.Final. When I used the same version of debezium and the same configuration to synchronize LOB data, the error did not occur, so I am asking for help. Please view flink sql in the attachment。 Without adding parameter 'debezium.lob.enabled' = 'true', the error does not occur. -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35210) Give the option to set automatically the parallelism of the KafkaSource to the number of kafka partitions
Nicolas Perrin created FLINK-35210: -- Summary: Give the option to set automatically the parallelism of the KafkaSource to the number of kafka partitions Key: FLINK-35210 URL: https://issues.apache.org/jira/browse/FLINK-35210 Project: Flink Issue Type: Improvement Components: Connectors / Kafka Reporter: Nicolas Perrin Currently the setting of the `KafkaSource` Flink's operator parallelism needs to be manually chosen which can leads to highly skewed tasks if the developer doesn't do this job. To avoid this issue, I propose to: - retrieve dynamically the number of partitions of the topic using `KafkaConsumer. partitionsFor(topic).size()`, - set the parallelism of the stream built from the source based on this value. This way there won't be any idle tasks. -- This message was sent by Atlassian Jira (v8.20.10#820010)
[VOTE] Release flink-connector-kafka v3.2.0, release candidate #1
Hi everyone, Please review and vote on release candidate #1 for flink-connector-kafka v3.2.0, as follows: [ ] +1, Approve the release [ ] -1, Do not approve the release (please provide specific comments) This release supports Flink 1.18 and 1.19. The complete staging area is available for your review, which includes: * JIRA release notes [1], * the official Apache source release to be deployed to dist.apache.org [2], which are signed with the key with fingerprint 125FD8DB [3], * all artifacts to be deployed to the Maven Central Repository [4], * source code tag v3.2.0-rc1 [5], * website pull request listing the new release [6]. * CI build of the tag [7]. The vote will be open for at least 72 hours. It is adopted by majority approval, with at least 3 PMC affirmative votes. Thanks, Danny [1] https://issues.apache.org/jira/secure/ReleaseNote.jspa?projectId=12315522=12354209 [2] https://dist.apache.org/repos/dist/dev/flink/flink-connector-kafka-3.2.0-rc1 [3] https://dist.apache.org/repos/dist/release/flink/KEYS [4] https://repository.apache.org/content/repositories/orgapacheflink-1723 [5] https://github.com/apache/flink-connector-kafka/releases/tag/v3.2.0-rc1 [6] https://github.com/apache/flink-web/pull/738 [7] https://github.com/apache/flink-connector-kafka
[VOTE] Release flink-connector-cassandra v3.2.0, release candidate #1
Hi everyone, Please review and vote on release candidate #1 for flink-connector-cassandra v3.2.0, as follows: [ ] +1, Approve the release [ ] -1, Do not approve the release (please provide specific comments) This release supports Flink 1.18 and 1.19. The complete staging area is available for your review, which includes: * JIRA release notes [1], * the official Apache source release to be deployed to dist.apache.org [2], which are signed with the key with fingerprint 125FD8DB [3], * all artifacts to be deployed to the Maven Central Repository [4], * source code tag v3.2.0-rc1 [5], * website pull request listing the new release [6]. * CI build of the tag [7]. The vote will be open for at least 72 hours. It is adopted by majority approval, with at least 3 PMC affirmative votes. Thanks, Danny [1] https://issues.apache.org/jira/secure/ReleaseNote.jspa?projectId=12315522=12353148 [2] https://dist.apache.org/repos/dist/dev/flink/flink-connector-cassandra-3.2.0-rc1 [3] https://dist.apache.org/repos/dist/release/flink/KEYS [4] https://repository.apache.org/content/repositories/orgapacheflink-1722 [5] https://github.com/apache/flink-connector-cassandra/releases/tag/v3.2.0-rc1 [6] https://github.com/apache/flink-web/pull/737 [7] https://github.com/apache/flink-connector-cassandra/actions/runs/8784310241
[jira] [Created] (FLINK-35209) Add a DeserializationSchema decorator that counts deserialize errors
Nicolas Perrin created FLINK-35209: -- Summary: Add a DeserializationSchema decorator that counts deserialize errors Key: FLINK-35209 URL: https://issues.apache.org/jira/browse/FLINK-35209 Project: Flink Issue Type: New Feature Components: API / Core Reporter: Nicolas Perrin I would like to propose a PR that implements a decorator for `DeserializationSchema`. The decorator decorates an `DeserializationSchema` object. The purpose of this decorator is to catch any deserialization errors that could occur when deserializing messages. The decorator has a flag to decide to fail or not in this case. If it makes the error silent, then it would count them in a `flink.metrics.Counter` so the user can monitor the silent errors. This PR is ready to be created. This decorator could be improved by having a sink that would be used to sink all the messages causing deserialization errors. -- This message was sent by Atlassian Jira (v8.20.10#820010)
Re: [DISCUSS] FLIP-XXX Support currentFetchEventTimeLag and processingLag metrics
Sorry all, it seems bad formatting in my email message, now I send it again gently and hope it work. I would like to start a discussion about FLIP-XXX: SupportcurrentFetchEventTimeLag and processingLag metrics [1]. The main motivation for this change was that I had some difficulties inimplementing the currentFetchEventTimeLag metrics for KafkaSource [2]. So I proposed to let the SourceReaderMetricGroup provide an interface to capturethe FetchTime so that all the FLIP-27 [3] sources can easily implement thecurrentFetchEventTimeLag metrics. In addition, I propose to support the processingLag metric for the FLIP-27sources to measure the current processing latency of the source. See the FLIP [1] and Jira [2] for more details. Looking forward to your comments and opinions! Thanks, TanJiaLiang. [1] https://docs.google.com/document/d/1nPhh1A-v-a7zyQyl1A5-K5DeUqbfxNXdjr2TVBT-QMs/edit?usp=sharing [2] https://issues.apache.org/jira/browse/FLINK-33173 [3] https://cwiki.apache.org/confluence/display/FLINK/FLIP-27%3A+Refactor+Source+Interface >
[RESULT][VOTE] FLIP-442: General Improvement to Configuration for Flink 2.0
Hi devs, I'm happy to announce that FLIP-442: General Improvement to Configuration for Flink 2.0[1] has been accepted with 9 approving votes (4 bindings) [2]: - Rui Fan (binding) - Zakelly Lan (binding) - Xintong Song (binding) - Yuxin Tan (non-binding) - Zhu Zhu (binding) - Jeyhun Karimov (non-binding) - Ahmed Hamdy (non-binding) - Muhammet Orazov (non-binding) - Zhongqiang Gong (non-binding) There are no disapproving votes. Thanks again to everyone who participated in the discussion and voting. Best regards, Xuannan [1] https://cwiki.apache.org/confluence/display/FLINK/FLIP-442%3A+General+Improvement+to+Configuration+for+Flink+2.0 [2] https://lists.apache.org/thread/sx8v66w6rogj8v3sfm4wfmb6tj5dw89y
Re: [VOTE] FLIP-442: General Improvement to Configuration for Flink 2.0
Hi all, Thank you all! Closing the vote. The result will be announced in a separate email. Best regards, Xuannan On Fri, Apr 19, 2024 at 10:58 AM gongzhongqiang wrote: > > +1 (non-binding) > > > Best, > Zhongqiang Gong > > Xuannan Su 于2024年4月17日周三 13:02写道: > > > Hi everyone, > > > > Thanks for all the feedback about the FLIP-442: General Improvement to > > Configuration for Flink 2.0 [1] [2]. > > > > I'd like to start a vote for it. The vote will be open for at least 72 > > hours(excluding weekends,until APR 22, 12:00AM GMT) unless there is an > > objection or an insufficient number of votes. > > > > [1] > > https://cwiki.apache.org/confluence/display/FLINK/FLIP-442%3A+General+Improvement+to+Configuration+for+Flink+2.0 > > [2] https://lists.apache.org/thread/15k0stwyoknhxvd643ctwjw3fd17pqwk > > > > > > Best regards, > > Xuannan > >
[DISCUSS] FLIP-XXX Support currentFetchEventTimeLag and processingLag metrics
Hello devs, I would like to start a discussion about FLIP-XXX: Support currentFetchEventTimeLag and processingLag metrics [1]. The main motivation for this change was that I had some difficulties in implementing the currentFetchEventTimeLag metrics for KafkaSource [2]. So I proposed to let the SourceReaderMetricGroup provide an interface to capture the FetchTime so that all the FLIP-27 [3] sources can easily implement the currentFetchEventTimeLag metrics. In addition, I propose to support the processingLag metric for the FLIP-27 sources to measure the current processing latency of the source. See the FLIP [1] and Jira [2] for more details. Looking forward to your comments and opinions! Thanks, TanJiaLiang. [1] https://docs.google.com/document/d/1nPhh1A-v-a7zyQyl1A5-K5DeUqbfxNXdjr2TVBT-QMs/edit?usp=sharing [2] https://issues.apache.org/jira/browse/FLINK-33173 [3] https://cwiki.apache.org/confluence/display/FLINK/FLIP-27
[jira] [Created] (FLINK-35208) Respect pipeline.cached-files during processing Python dependencies
Dian Fu created FLINK-35208: --- Summary: Respect pipeline.cached-files during processing Python dependencies Key: FLINK-35208 URL: https://issues.apache.org/jira/browse/FLINK-35208 Project: Flink Issue Type: Bug Components: API / Python Reporter: Dian Fu Assignee: Dian Fu Currently, PyFlink will make use of distributed cache (update StreamExecutionEnvironment#cachedFiles) during handling the Python dependencies(See [https://github.com/apache/flink/blob/master/flink-python/src/main/java/org/apache/flink/python/util/PythonDependencyUtils.java#L339] for more details). However, if pipeline.cached-files is configured, it will clear StreamExecutionEnvironment#cachedFiles(see [https://github.com/apache/flink/blob/master/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/environment/StreamExecutionEnvironment.java#L1132] for more details) which may break the above functionalities. -- This message was sent by Atlassian Jira (v8.20.10#820010)
FW: RE: [DISCUSS] FLIP-XXX Apicurio-avro format
Hi Martijn, A gentle nudge, is this ok for you or one of the PMC or committers to create a Flip now? Kind regards, David. From: David Radley Date: Monday, 15 April 2024 at 12:29 To: dev@flink.apache.org Subject: Re: [EXTERNAL] RE: [DISCUSS] FLIP-XXX Apicurio-avro format Hi Martijn, Thanks for looking at this. I have used the template in a new Google Doc https://docs.google.com/document/d/1J1E-cE-X2H3-kw4rNjLn71OGPQk_Yl1iGX4-eCHWLgE/edit?usp=sharing. I have significantly reduced the content in the Flip, in line with what I see as the template and its usage. If this it too much or too little, I can amend, Kind regards, David. From: Martijn Visser Date: Friday, 12 April 2024 at 18:11 To: dev@flink.apache.org Subject: Re: [EXTERNAL] RE: [DISCUSS] FLIP-XXX Apicurio-avro format Hi David, I tried, but the format wasn't as the FLIP template expects, so I ended up needing to change the entire formatting and that was just too much work to be honest. If you could make sure that especially the headers match with the FLIP template, and that all of the contents from the FLIP template is there, that would make things much easier. Thanks, Martijn On Fri, Apr 12, 2024 at 6:08 PM David Radley wrote: > Hi, > A gentle nudge. Please could a committer/PMC member raise the Flip for > this, > Kind regards, David. > > > From: David Radley > Date: Monday, 8 April 2024 at 09:40 > To: dev@flink.apache.org > Subject: [EXTERNAL] RE: [DISCUSS] FLIP-XXX Apicurio-avro format > Hi, > I have posted a Google Doc [0] to the mailing list for a discussion thread > for a Flip proposal to introduce a Apicurio-avro format. The discussions > have been resolved, please could a committer/PMC member copy the contents > from the Google Doc, and create a FLIP number for this,. as per the process > [1], > Kind regards, David. > [0] > > https://docs.google.com/document/d/14LWZPVFQ7F9mryJPdKXb4l32n7B0iWYkcOdEd1xTC7w/edit?usp=sharing > > [1] > > https://cwiki.apache.org/confluence/display/FLINK/Flink+Improvement+Proposals#FlinkImprovementProposals-CreateyourOwnFLIP > > From: Jeyhun Karimov > Date: Friday, 22 March 2024 at 13:05 > To: dev@flink.apache.org > Subject: [EXTERNAL] Re: [DISCUSS] FLIP-XXX Apicurio-avro format > Hi David, > > Thanks a lot for clarification. > Sounds good to me. > > Regards, > Jeyhun > > On Fri, Mar 22, 2024 at 10:54 AM David Radley > wrote: > > > Hi Jeyhun, > > Thanks for your feedback. > > > > So for outbound messages, the message includes the global ID. We register > > the schema and match on the artifact id. So if the schema then evolved, > > adding a new version, the global ID would still be unique and the same > > version would be targeted. If you wanted to change the Flink table > > definition in line with a higher version, then you could do this – the > > artifact id would need to match for it to use the same schema and a > higher > > artifact version would need to be provided. I notice that Apicurio has > > rules around compatibility that you can configure, I suppose if we > attempt > > to create an artifact that breaks these rules , then the register schema > > will fail and the associated operation should fail (e.g. an insert). I > have > > not tried this. > > > > > > For inbound messages, using the global id in the header – this targets > one > > version of the schema. I can create different messages on the topic built > > with different schema versions, and I can create different tables in > Flink, > > as long as the reader and writer schemas are compatible as per the > > > https://github.com/apache/flink/blob/779459168c46b7b4c600ef52f99a5435f81b9048/flink-formats/flink-avro/src/main/java/org/apache/flink/formats/avro/RegistryAvroDeserializationSchema.java#L109 > > Then this should work. > > > > Does this address your question? > > Kind regards, David. > > > > > > From: Jeyhun Karimov > > Date: Thursday, 21 March 2024 at 21:06 > > To: dev@flink.apache.org > > Subject: [EXTERNAL] Re: [DISCUSS] FLIP-XXX Apicurio-avro format > > Hi David, > > > > Thanks for the FLIP. +1 for it. > > I have a minor comment. > > > > Can you please elaborate more on mechanisms in place to ensure data > > consistency and integrity, particularly in the event of schema conflicts? > > Since each message includes a schema ID for inbound and outbound > messages, > > can you elaborate more on message consistency in the context of schema > > evolution? > > > > Regards, > > Jeyhun > > > > > > > > > > > > On Wed, Mar 20, 2024 at 4:34 PM David Radley > wrote: > > > > > Thank you very much for your feedback Mark. I have made the changes in > > the > > > latest google document. On reflection I agree with you that the > > > globalIdPlacement format configuration should apply to the > > deserialization > > > as well, so it is declarative. I am also going to have a new > > configuration > > > option to work with content IDs as well as global IDs. In line with the > > > deser Apicurio IdHandler and
[jira] [Created] (FLINK-35207) Kubernetes session E2E test fails to fetch packages.
Ryan Skraba created FLINK-35207: --- Summary: Kubernetes session E2E test fails to fetch packages. Key: FLINK-35207 URL: https://issues.apache.org/jira/browse/FLINK-35207 Project: Flink Issue Type: Bug Affects Versions: 1.18.2 Reporter: Ryan Skraba 1.18 Default (Java 8) / E2E (group 1) https://github.com/apache/flink/commit/aacc735806acf1d63fa732706e079bc2ca1bb4fc/checks/24027142976/logs Looks like some flakiness when fetching packages to install (and just to track if this happens again) {code} 2024-04-19T14:28:15.9116531Z Apr 19 14:28:15 == 2024-04-19T14:28:15.9117204Z Apr 19 14:28:15 Running 'Run kubernetes session test (custom fs plugin)' 2024-04-19T14:28:15.9118209Z Apr 19 14:28:15 == 2024-04-19T14:28:15.9119866Z Apr 19 14:28:15 TEST_DATA_DIR: /home/runner/work/flink/flink/flink-end-to-end-tests/test-scripts/temp-test-directory-15907928199 2024-04-19T14:28:16.1477984Z Apr 19 14:28:16 Flink dist directory: /home/runner/work/flink/flink/flink-dist/target/flink-1.18-SNAPSHOT-bin/flink-1.18-SNAPSHOT 2024-04-19T14:28:16.1546131Z Apr 19 14:28:16 Flink dist directory: /home/runner/work/flink/flink/flink-dist/target/flink-1.18-SNAPSHOT-bin/flink-1.18-SNAPSHOT 2024-04-19T14:28:16.1670878Z Apr 19 14:28:16 Docker version 24.0.9, build 2936816 2024-04-19T14:28:16.5441575Z Apr 19 14:28:16 docker-compose version 1.29.2, build 5becea4c 2024-04-19T14:28:16.7073581Z Apr 19 14:28:16 Reading package lists... 2024-04-19T14:28:16.8529977Z Apr 19 14:28:16 Building dependency tree... 2024-04-19T14:28:16.8541118Z Apr 19 14:28:16 Reading state information... 2024-04-19T14:28:16.9872695Z Apr 19 14:28:16 conntrack is already the newest version (1:1.4.6-2build2). 2024-04-19T14:28:16.9873637Z Apr 19 14:28:16 0 upgraded, 0 newly installed, 0 to remove and 20 not upgraded. 2024-04-19T14:28:17.5567699Z 2024-04-19 14:28:17 URL:https://objects.githubusercontent.com/github-production-release-asset-2e65be/80172100/7186c302-3766-4ed5-920a-f85c9d6334ac?X-Amz-Algorithm=AWS4-HMAC-SHA256=AKIAVCODYLSA53PQK4ZA%2F20240419%2Fus-east-1%2Fs3%2Faws4_request=20240419T142817Z=300=fe759ee1ce1eb3ebeaee7d8e714aedcedbc9035c75bc656f3ecda57836820bdf=host_id=0_id=0_id=80172100=attachment%3B%20filename%3Dcrictl-v1.24.2-linux-amd64.tar.gz=application%2Foctet-stream [14553934/14553934] -> "crictl-v1.24.2-linux-amd64.tar.gz" [1] 2024-04-19T14:28:17.5668524Z Apr 19 14:28:17 crictl 2024-04-19T14:28:18.1236206Z 2024-04-19 14:28:18 URL:https://objects.githubusercontent.com/github-production-release-asset-2e65be/318491505/e304ee45-ccad-4438-bc2c-039c8f6755d1?X-Amz-Algorithm=AWS4-HMAC-SHA256=AKIAVCODYLSA53PQK4ZA%2F20240419%2Fus-east-1%2Fs3%2Faws4_request=20240419T142817Z=300=43ca222d979f2595126d94c344045c628c246597bfc9339d6b4dbf223e8b6be3=host_id=0_id=0_id=318491505=attachment%3B%20filename%3Dcri-dockerd-0.2.3.amd64.tgz=application%2Foctet-stream [23042323/23042323] -> "cri-dockerd-0.2.3.amd64.tgz.2" [1] 2024-04-19T14:28:18.1292589Z Apr 19 14:28:18 cri-dockerd/cri-dockerd 2024-04-19T14:28:18.6786614Z 2024-04-19 14:28:18 URL:https://raw.githubusercontent.com/Mirantis/cri-dockerd/v0.2.3/packaging/systemd/cri-docker.service [1337/1337] -> "cri-docker.service" [1] 2024-04-19T14:28:18.8479307Z 2024-04-19 14:28:18 URL:https://raw.githubusercontent.com/Mirantis/cri-dockerd/v0.2.3/packaging/systemd/cri-docker.socket [204/204] -> "cri-docker.socket" [1] 2024-04-19T14:28:19.5285798Z Apr 19 14:28:19 fs.protected_regular = 0 2024-04-19T14:28:19.6167026Z Apr 19 14:28:19 minikube 2024-04-19T14:28:19.6167602Z Apr 19 14:28:19 type: Control Plane 2024-04-19T14:28:19.6168146Z Apr 19 14:28:19 host: Stopped 2024-04-19T14:28:19.6170872Z Apr 19 14:28:19 kubelet: Stopped 2024-04-19T14:28:19.6175184Z Apr 19 14:28:19 apiserver: Stopped 2024-04-19T14:28:19.6179746Z Apr 19 14:28:19 kubeconfig: Stopped 2024-04-19T14:28:19.6180518Z Apr 19 14:28:19 2024-04-19T14:28:19.6211870Z Apr 19 14:28:19 Starting minikube ... 2024-04-19T14:28:19.6893918Z Apr 19 14:28:19 * minikube v1.28.0 on Ubuntu 22.04 2024-04-19T14:28:19.6934942Z Apr 19 14:28:19 * Using the none driver based on existing profile 2024-04-19T14:28:19.6951845Z Apr 19 14:28:19 * Starting control plane node minikube in cluster minikube 2024-04-19T14:28:19.7246076Z Apr 19 14:28:19 * Restarting existing none bare metal machine for "minikube" ... 2024-04-19T14:28:19.7365028Z Apr 19 14:28:19 * OS release is Ubuntu 22.04.4 LTS 2024-04-19T14:28:22.1596670Z Apr 19 14:28:22 * Preparing Kubernetes v1.25.3 on Docker 24.0.9 ... 2024-04-19T14:28:22.1618992Z Apr 19 14:28:22 - kubelet.image-gc-high-threshold=99 2024-04-19T14:28:22.1622618Z Apr 19 14:28:22 - kubelet.image-gc-low-threshold=98 2024-04-19T14:28:22.1626821Z Apr 19 14:28:22 -
Re: [VOTE] Release flink-connector-aws v4.3.0, release candidate #2
Thanks Danny, +1 (non-binding) - Verified Checksums - Verified Signatures - No binaries exists in source archive - Built source - Reviewed Web PR - Run basic Kinesis example Best Regards Ahmed Hamdy On Sun, 21 Apr 2024 at 14:25, Hang Ruan wrote: > +1 (non-binding) > > - Validated checksum hash > - Verified signature > - Verified that no binaries exist in the source archive > - Build the source with Maven and jdk8 > - Verified web PR > - Check that the jar is built by jdk8 > > Best, > Hang > > Danny Cranmer 于2024年4月19日周五 18:08写道: > > > Hi everyone, > > > > Please review and vote on release candidate #2 for flink-connector-aws > > v4.3.0, as follows: > > [ ] +1, Approve the release > > [ ] -1, Do not approve the release (please provide specific comments) > > > > This version supports Flink 1.18 and 1.19. > > > > The complete staging area is available for your review, which includes: > > * JIRA release notes [1], > > * the official Apache source release to be deployed to dist.apache.org > > [2], > > which are signed with the key with fingerprint 125FD8DB [3], > > * all artifacts to be deployed to the Maven Central Repository [4], > > * source code tag v4.3.0-rc2 [5], > > * website pull request listing the new release [6]. > > * CI build of the tag [7]. > > > > The vote will be open for at least 72 hours. It is adopted by majority > > approval, with at least 3 PMC affirmative votes. > > > > Thanks, > > Release Manager > > > > [1] > > > > > https://issues.apache.org/jira/secure/ReleaseNote.jspa?projectId=12315522=12353793 > > [2] > > > https://dist.apache.org/repos/dist/dev/flink/flink-connector-aws-4.3.0-rc2 > > [3] https://dist.apache.org/repos/dist/release/flink/KEYS > > [4] > > https://repository.apache.org/content/repositories/orgapacheflink-1721/ > > [5] > https://github.com/apache/flink-connector-aws/releases/tag/v4.3.0-rc2 > > [6] https://github.com/apache/flink-web/pull/733 > > [7] > https://github.com/apache/flink-connector-aws/actions/runs/8751694197 > > >
[jira] [Created] (FLINK-35206) Desc table syntax support materialized table
dalongliu created FLINK-35206: - Summary: Desc table syntax support materialized table Key: FLINK-35206 URL: https://issues.apache.org/jira/browse/FLINK-35206 Project: Flink Issue Type: Sub-task Components: Table SQL / API Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35205) Support the execution of alter materialized table reset options
dalongliu created FLINK-35205: - Summary: Support the execution of alter materialized table reset options Key: FLINK-35205 URL: https://issues.apache.org/jira/browse/FLINK-35205 Project: Flink Issue Type: Sub-task Components: Table SQL / API, Table SQL / Gateway Affects Versions: 1.20.0 Reporter: dalongliu -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35203) Support the execution of alter materialized table set refresh mode
dalongliu created FLINK-35203: - Summary: Support the execution of alter materialized table set refresh mode Key: FLINK-35203 URL: https://issues.apache.org/jira/browse/FLINK-35203 Project: Flink Issue Type: Sub-task Components: Table SQL / API, Table SQL / Gateway Affects Versions: 1.20.0 Reporter: dalongliu -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35204) Support the execution of alter materialized table set options
dalongliu created FLINK-35204: - Summary: Support the execution of alter materialized table set options Key: FLINK-35204 URL: https://issues.apache.org/jira/browse/FLINK-35204 Project: Flink Issue Type: Sub-task Components: Table SQL / API, Table SQL / Gateway Affects Versions: 1.20.0 Reporter: dalongliu -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35202) Support the execution of set freshness for materialized table
dalongliu created FLINK-35202: - Summary: Support the execution of set freshness for materialized table Key: FLINK-35202 URL: https://issues.apache.org/jira/browse/FLINK-35202 Project: Flink Issue Type: Sub-task Components: Table SQL / API, Table SQL / Gateway Affects Versions: 1.20.0 Reporter: dalongliu -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35201) Support the execution of drop materialized table in full refresh mode
dalongliu created FLINK-35201: - Summary: Support the execution of drop materialized table in full refresh mode Key: FLINK-35201 URL: https://issues.apache.org/jira/browse/FLINK-35201 Project: Flink Issue Type: Sub-task Components: Table SQL / API, Table SQL / Gateway Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35200) Support the execution of suspend, resume materialized table in full refresh mode
dalongliu created FLINK-35200: - Summary: Support the execution of suspend, resume materialized table in full refresh mode Key: FLINK-35200 URL: https://issues.apache.org/jira/browse/FLINK-35200 Project: Flink Issue Type: Sub-task Components: Table SQL / API, Table SQL / Gateway Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35199) Support the execution of create materialized table in full refreshmode
dalongliu created FLINK-35199: - Summary: Support the execution of create materialized table in full refreshmode Key: FLINK-35199 URL: https://issues.apache.org/jira/browse/FLINK-35199 Project: Flink Issue Type: Sub-task Components: Table SQL / API, Table SQL / Gateway Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35198) Support the execution of refresh materialized table
dalongliu created FLINK-35198: - Summary: Support the execution of refresh materialized table Key: FLINK-35198 URL: https://issues.apache.org/jira/browse/FLINK-35198 Project: Flink Issue Type: Sub-task Components: Table SQL / API Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35197) Support the execution of suspend, resume materialized table in continuous mode
dalongliu created FLINK-35197: - Summary: Support the execution of suspend, resume materialized table in continuous mode Key: FLINK-35197 URL: https://issues.apache.org/jira/browse/FLINK-35197 Project: Flink Issue Type: Sub-task Components: Table SQL / API, Table SQL / Gateway Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35196) bouncycastle class not found for flink pulsar connector
Wenbing Shen created FLINK-35196: Summary: bouncycastle class not found for flink pulsar connector Key: FLINK-35196 URL: https://issues.apache.org/jira/browse/FLINK-35196 Project: Flink Issue Type: Bug Components: Connectors / Pulsar Affects Versions: pulsar-4.1.0 Reporter: Wenbing Shen 2024-04-22 14:45:48 java.lang.NoClassDefFoundError: org/apache/pulsar/shade/org/bouncycastle/util/Arrays at org.apache.flink.connector.pulsar.table.sink.PulsarWritableMetadata.readMetadata(PulsarWritableMetadata.java:67) at org.apache.flink.connector.pulsar.table.sink.PulsarWritableMetadata.applyWritableMetadataInMessage(PulsarWritableMetadata.java:55) at org.apache.flink.connector.pulsar.table.sink.PulsarTableSerializationSchema.serialize(PulsarTableSerializationSchema.java:106) at org.apache.flink.connector.pulsar.table.sink.PulsarTableSerializationSchema.serialize(PulsarTableSerializationSchema.java:40) at org.apache.flink.connector.pulsar.sink.writer.PulsarWriter.write(PulsarWriter.java:143) at org.apache.flink.streaming.runtime.operators.sink.SinkWriterOperator.processElement(SinkWriterOperator.java:160) at org.apache.flink.streaming.runtime.tasks.OneInputStreamTask$StreamTaskNetworkOutput.emitRecord(OneInputStreamTask.java:233) at org.apache.flink.streaming.runtime.io.AbstractStreamTaskNetworkInput.processElement(AbstractStreamTaskNetworkInput.java:134) at org.apache.flink.streaming.runtime.io.AbstractStreamTaskNetworkInput.emitNext(AbstractStreamTaskNetworkInput.java:105) at org.apache.flink.streaming.runtime.io.StreamOneInputProcessor.processInput(StreamOneInputProcessor.java:65) at org.apache.flink.streaming.runtime.tasks.StreamTask.processInput(StreamTask.java:542) at org.apache.flink.streaming.runtime.tasks.mailbox.MailboxProcessor.runMailboxLoop(MailboxProcessor.java:231) at org.apache.flink.streaming.runtime.tasks.StreamTask.runMailboxLoop(StreamTask.java:831) at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:780) at org.apache.flink.runtime.taskmanager.Task.runWithSystemExitMonitoring(Task.java:935) at org.apache.flink.runtime.taskmanager.Task.restoreAndInvoke(Task.java:914) at org.apache.flink.runtime.taskmanager.Task.doRun(Task.java:728) at org.apache.flink.runtime.taskmanager.Task.run(Task.java:550) at java.lang.Thread.run(Thread.java:748) -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35195) Support the execution of materialized table in continuous mode
dalongliu created FLINK-35195: - Summary: Support the execution of materialized table in continuous mode Key: FLINK-35195 URL: https://issues.apache.org/jira/browse/FLINK-35195 Project: Flink Issue Type: Sub-task Components: Table SQL / API, Table SQL / Gateway Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35194) Support describe job syntax
dalongliu created FLINK-35194: - Summary: Support describe job syntax Key: FLINK-35194 URL: https://issues.apache.org/jira/browse/FLINK-35194 Project: Flink Issue Type: Sub-task Components: Table SQL / API Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35193) Support drop materialized table syntax and execution
dalongliu created FLINK-35193: - Summary: Support drop materialized table syntax and execution Key: FLINK-35193 URL: https://issues.apache.org/jira/browse/FLINK-35193 Project: Flink Issue Type: Sub-task Components: Table SQL / API Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35192) operator oom
chenyuzhi created FLINK-35192: - Summary: operator oom Key: FLINK-35192 URL: https://issues.apache.org/jira/browse/FLINK-35192 Project: Flink Issue Type: Bug Components: Kubernetes Operator Affects Versions: kubernetes-operator-1.6.1 Environment: jdk: openjdk11 operator version: 1.6.1 Reporter: chenyuzhi Attachments: image-2024-04-22-15-47-49-455.png, image-2024-04-22-15-52-51-600.png, image-2024-04-22-15-58-23-269.png, image-2024-04-22-15-58-42-850.png The kubernetest operator docker process was killed by kernel cause out of memory(the time is 2024.04.04: 18:16) !image-2024-04-22-15-47-49-455.png! metrics: And the pod memory (RSS) is increasing slowly in the past 7 days: !image-2024-04-22-15-52-51-600.png! However the jvm memory metrics of operator not shown obvious anomaly: !image-2024-04-22-15-58-23-269.png! !image-2024-04-22-15-58-42-850.png! -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35191) Support alter materialized table related syntaxes: suspend, resume, refresh, set and reset
dalongliu created FLINK-35191: - Summary: Support alter materialized table related syntaxes: suspend, resume, refresh, set and reset Key: FLINK-35191 URL: https://issues.apache.org/jira/browse/FLINK-35191 Project: Flink Issue Type: Sub-task Components: Table SQL / API Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35190) Support the syntaxes of Create, Suspend, Resume, Refresh and Drop materialized table
dalongliu created FLINK-35190: - Summary: Support the syntaxes of Create, Suspend, Resume, Refresh and Drop materialized table Key: FLINK-35190 URL: https://issues.apache.org/jira/browse/FLINK-35190 Project: Flink Issue Type: Sub-task Components: Table SQL / API Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35189) Introduce test-filesystem Catalog based on FileSystem Connector to support materialized table
dalongliu created FLINK-35189: - Summary: Introduce test-filesystem Catalog based on FileSystem Connector to support materialized table Key: FLINK-35189 URL: https://issues.apache.org/jira/browse/FLINK-35189 Project: Flink Issue Type: Sub-task Components: Table SQL / API Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35188) Introduce CatalogMaterializedTable and related interface to support materialized table
dalongliu created FLINK-35188: - Summary: Introduce CatalogMaterializedTable and related interface to support materialized table Key: FLINK-35188 URL: https://issues.apache.org/jira/browse/FLINK-35188 Project: Flink Issue Type: Sub-task Components: Table SQL / API Affects Versions: 1.20.0 Reporter: dalongliu Fix For: 1.20.0 -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35187) FLIP-435: Introduce a New Materialized Table for Simplifying Data Pipelines
dalongliu created FLINK-35187: - Summary: FLIP-435: Introduce a New Materialized Table for Simplifying Data Pipelines Key: FLINK-35187 URL: https://issues.apache.org/jira/browse/FLINK-35187 Project: Flink Issue Type: New Feature Components: Table SQL / API, Table SQL / Gateway Affects Versions: 1.20.0 Reporter: dalongliu This is an umbrella issue for FLIP-435: Introduce a New Materialized Table for Simplifying Data Pipelines, see FLIP design doc for more detail: https://cwiki.apache.org/confluence/display/FLINK/FLIP-435%3A+Introduce+a+New+Materialized+Table+for+Simplifying+Data+Pipelines -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (FLINK-35186) Create State V2 from new StateDescriptor
Zakelly Lan created FLINK-35186: --- Summary: Create State V2 from new StateDescriptor Key: FLINK-35186 URL: https://issues.apache.org/jira/browse/FLINK-35186 Project: Flink Issue Type: Sub-task Components: Runtime / State Backends, Runtime / Task Reporter: Zakelly Lan Assignee: Zakelly Lan -- This message was sent by Atlassian Jira (v8.20.10#820010)
[DISCUSS] FLIP-448: Introduce Pluggable Workflow Scheduler Interface for Materialized Table
Hi, Dev I would like to start a discussion about FLIP-448: Introduce Pluggable Workflow Scheduler Interface for Materialized Table. In FLIP-435[1], we proposed Materialized Table, which has two types of data refresh modes: Full Refresh & Continuous Refresh Mode. In Full Refresh mode, the Materialized Table relies on a workflow scheduler to perform periodic refresh operation to achieve the desired data freshness. There are numerous open-source workflow schedulers available, with popular ones including Airflow and DolphinScheduler. To enable Materialized Table to work with different workflow schedulers, we propose a pluggable workflow scheduler interface for Materialized Table in this FLIP. For more details, see FLIP-448 [2]. Looking forward to your feedback. [1] https://lists.apache.org/thread/c1gnn3bvbfs8v1trlf975t327s4rsffs [2] https://cwiki.apache.org/confluence/display/FLINK/FLIP-448%3A+Introduce+Pluggable+Workflow+Scheduler+Interface+for+Materialized+Table Best, Ron
[RESULT][VOTE] FLIP-435: Introduce a New Materialized Table for Simplifying Data Pipelines
Hi, Dev I'm happy to announce that FLIP-435: Introduce a New Materialized Table for Simplifying Data Pipelines[1] has been accepted with 13 approving votes (8 binding) [2] - Ron Liu(binding) - Feng Jin - Rui Fan(binding) - Yuepeng Pan - Ahmed Hamdy - Ferenc Csaky - Lincoln Lee(binding) - Leonard Xu(binding) - Jark Wu(binding) - Yun Tang(binding) - Jinsong Li(binding) - Zhongqiang Gong - Martijn Visser(binding) [1] https://cwiki.apache.org/confluence/display/FLINK/FLIP-435%3A+Introduce+a+New+Materialized+Table+for+Simplifying+Data+Pipelines [2] https://lists.apache.org/thread/woj27nsmx5xd7p87ryfo8h6gx37n3wlx Best, Ron
[DISCUSS] FLIP-447: Upgrade FRocksDB from 6.20.3 to 8.10.0
Hi Flink devs, I would like to start a discussion on FLIP-447: Upgrade FRocksDB from 6.20.3 to 8.10.0 https://cwiki.apache.org/confluence/display/FLINK/FLIP-447%3A+Upgrade+FRocksDB+from+6.20.3++to+8.10.0 This FLIP proposes upgrading the version of FRocksDB in the Flink Project from 6.20.3 to 8.10.0. The FLIP mainly introduces the main benefits of upgrading FRocksDB, including the use of IngestDB which can improve Rescaling performance by more than 10 times in certain scenarios, as well as other potential optimization points such as async_io, blob db, and tiered storage.The FLIP also presented test results based on RocksDB 8.10, including StateBenchmark and Nexmark tests. Overall, upgrading FRocksDB may result in a small regression of write performance( which is a very small part of the overall overhead), but it can bring many important performance benefits. So we hope to upgrade the version of FRocksDB through this FLIP. Looking forward to everyone's feedback and suggestions. Thank you! -- Best regards, Yue