This is an automated email from the ASF dual-hosted git repository.

jiabaosun pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/flink-connector-jdbc.git


The following commit(s) were added to refs/heads/main by this push:
     new 1fcb9dd  [FLINK-36051] Re-prepare invalid statement even if connection 
is valid (#138)
1fcb9dd is described below

commit 1fcb9dd43a496778ee9e717657fcccdd116078cd
Author: Sergei Morozov <[email protected]>
AuthorDate: Mon Sep 2 18:22:52 2024 -0700

    [FLINK-36051] Re-prepare invalid statement even if connection is valid 
(#138)
---
 .../apache/flink/connector/jdbc/internal/JdbcOutputFormat.java | 10 +++-------
 1 file changed, 3 insertions(+), 7 deletions(-)

diff --git 
a/flink-connector-jdbc-core/src/main/java/org/apache/flink/connector/jdbc/internal/JdbcOutputFormat.java
 
b/flink-connector-jdbc-core/src/main/java/org/apache/flink/connector/jdbc/internal/JdbcOutputFormat.java
index 93e8687..7668236 100644
--- 
a/flink-connector-jdbc-core/src/main/java/org/apache/flink/connector/jdbc/internal/JdbcOutputFormat.java
+++ 
b/flink-connector-jdbc-core/src/main/java/org/apache/flink/connector/jdbc/internal/JdbcOutputFormat.java
@@ -178,14 +178,10 @@ public class JdbcOutputFormat<In, JdbcIn, JdbcExec 
extends JdbcBatchStatementExe
                     throw new IOException(e);
                 }
                 try {
-                    if (!connectionProvider.isConnectionValid()) {
-                        updateExecutor(true);
-                    }
+                    updateExecutor(!connectionProvider.isConnectionValid());
                 } catch (Exception exception) {
-                    LOG.error(
-                            "JDBC connection is not valid, and reestablish 
connection failed.",
-                            exception);
-                    throw new IOException("Reestablish JDBC connection 
failed", exception);
+                    LOG.error("Attempt to update the JDBC statement executor 
failed.", exception);
+                    throw new IOException("Unable to update JDBC statement 
executor", exception);
                 }
                 try {
                     Thread.sleep(1000 * i);

Reply via email to