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

russellspitzer pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/iceberg.git


The following commit(s) were added to refs/heads/main by this push:
     new b7a519a149 JDBC: JDBC Catalog should handle Postgres exception for 
duplicate keys (#15434)
b7a519a149 is described below

commit b7a519a149961672274c3954afeddb717477d853
Author: Han You <[email protected]>
AuthorDate: Thu Feb 26 09:42:45 2026 -0600

    JDBC: JDBC Catalog should handle Postgres exception for duplicate keys 
(#15434)
---
 .../java/org/apache/iceberg/jdbc/JdbcCatalog.java  | 10 +--
 .../apache/iceberg/jdbc/JdbcTableOperations.java   | 18 ++---
 .../java/org/apache/iceberg/jdbc/JdbcUtil.java     |  8 ++
 .../apache/iceberg/jdbc/JdbcViewOperations.java    | 17 ++---
 .../org/apache/iceberg/jdbc/TestJdbcCatalog.java   | 67 +++++++++++++++-
 .../java/org/apache/iceberg/jdbc/TestJdbcUtil.java | 24 ++++++
 .../apache/iceberg/jdbc/TestJdbcViewCatalog.java   | 88 +++++++++++++++++++++-
 7 files changed, 199 insertions(+), 33 deletions(-)

diff --git a/core/src/main/java/org/apache/iceberg/jdbc/JdbcCatalog.java 
b/core/src/main/java/org/apache/iceberg/jdbc/JdbcCatalog.java
index 0c8fbe41df..a3f40512a0 100644
--- a/core/src/main/java/org/apache/iceberg/jdbc/JdbcCatalog.java
+++ b/core/src/main/java/org/apache/iceberg/jdbc/JdbcCatalog.java
@@ -24,7 +24,6 @@ import java.sql.DatabaseMetaData;
 import java.sql.PreparedStatement;
 import java.sql.ResultSet;
 import java.sql.SQLException;
-import java.sql.SQLIntegrityConstraintViolationException;
 import java.sql.SQLNonTransientConnectionException;
 import java.sql.SQLTimeoutException;
 import java.sql.SQLTransientConnectionException;
@@ -337,7 +336,6 @@ public class JdbcCatalog extends BaseMetastoreViewCatalog
         JdbcUtil.namespaceToString(namespace));
   }
 
-  @SuppressWarnings("checkstyle:CyclomaticComplexity")
   @Override
   public void renameTable(TableIdentifier from, TableIdentifier to) {
     if (from.equals(to)) {
@@ -363,9 +361,7 @@ public class JdbcCatalog extends BaseMetastoreViewCatalog
     int updatedRecords =
         execute(
             err -> {
-              // SQLite doesn't set SQLState or throw 
SQLIntegrityConstraintViolationException
-              if (err instanceof SQLIntegrityConstraintViolationException
-                  || (err.getMessage() != null && 
err.getMessage().contains("constraint failed"))) {
+              if (JdbcUtil.isConstraintViolation(err)) {
                 throw new AlreadyExistsException("Table already exists: %s", 
to);
               }
             },
@@ -713,9 +709,7 @@ public class JdbcCatalog extends BaseMetastoreViewCatalog
     int updatedRecords =
         execute(
             err -> {
-              // SQLite doesn't set SQLState or throw 
SQLIntegrityConstraintViolationException
-              if (err instanceof SQLIntegrityConstraintViolationException
-                  || (err.getMessage() != null && 
err.getMessage().contains("constraint failed"))) {
+              if (JdbcUtil.isConstraintViolation(err)) {
                 throw new AlreadyExistsException(
                     "Cannot rename %s to %s. View already exists", from, to);
               }
diff --git 
a/core/src/main/java/org/apache/iceberg/jdbc/JdbcTableOperations.java 
b/core/src/main/java/org/apache/iceberg/jdbc/JdbcTableOperations.java
index 619296ad33..079faf1c55 100644
--- a/core/src/main/java/org/apache/iceberg/jdbc/JdbcTableOperations.java
+++ b/core/src/main/java/org/apache/iceberg/jdbc/JdbcTableOperations.java
@@ -20,7 +20,6 @@ package org.apache.iceberg.jdbc;
 
 import java.sql.DataTruncation;
 import java.sql.SQLException;
-import java.sql.SQLIntegrityConstraintViolationException;
 import java.sql.SQLNonTransientConnectionException;
 import java.sql.SQLTimeoutException;
 import java.sql.SQLTransientConnectionException;
@@ -120,14 +119,6 @@ class JdbcTableOperations extends 
BaseMetastoreTableOperations {
         createTable(newMetadataLocation);
       }
 
-    } catch (SQLIntegrityConstraintViolationException e) {
-
-      if (currentMetadataLocation() == null) {
-        throw new AlreadyExistsException(e, "Table already exists: %s", 
tableIdentifier);
-      } else {
-        throw new UncheckedSQLException(e, "Table already exists: %s", 
tableIdentifier);
-      }
-
     } catch (SQLTimeoutException e) {
       throw new UncheckedSQLException(e, "Database Connection timeout");
     } catch (SQLTransientConnectionException | 
SQLNonTransientConnectionException e) {
@@ -137,9 +128,12 @@ class JdbcTableOperations extends 
BaseMetastoreTableOperations {
     } catch (SQLWarning e) {
       throw new UncheckedSQLException(e, "Database warning");
     } catch (SQLException e) {
-      // SQLite doesn't set SQLState or throw 
SQLIntegrityConstraintViolationException
-      if (e.getMessage() != null && e.getMessage().contains("constraint 
failed")) {
-        throw new AlreadyExistsException("Table already exists: %s", 
tableIdentifier);
+      if (JdbcUtil.isConstraintViolation(e)) {
+        if (currentMetadataLocation() == null) {
+          throw new AlreadyExistsException(e, "Table already exists: %s", 
tableIdentifier);
+        } else {
+          throw new UncheckedSQLException(e, "Table already exists: %s", 
tableIdentifier);
+        }
       }
 
       throw new UncheckedSQLException(e, "Unknown failure");
diff --git a/core/src/main/java/org/apache/iceberg/jdbc/JdbcUtil.java 
b/core/src/main/java/org/apache/iceberg/jdbc/JdbcUtil.java
index 85e59328db..d59da3ad04 100644
--- a/core/src/main/java/org/apache/iceberg/jdbc/JdbcUtil.java
+++ b/core/src/main/java/org/apache/iceberg/jdbc/JdbcUtil.java
@@ -21,6 +21,7 @@ package org.apache.iceberg.jdbc;
 import java.sql.PreparedStatement;
 import java.sql.ResultSet;
 import java.sql.SQLException;
+import java.sql.SQLIntegrityConstraintViolationException;
 import java.util.Collections;
 import java.util.Map;
 import java.util.Properties;
@@ -45,6 +46,7 @@ final class JdbcUtil {
       JdbcCatalog.PROPERTY_PREFIX + "init-catalog-tables";
 
   static final String RETRYABLE_STATUS_CODES = "retryable_status_codes";
+  private static final String POSTGRES_UNIQUE_VIOLATION_SQLSTATE = "23505";
 
   enum SchemaVersion {
     V0,
@@ -522,6 +524,12 @@ final class JdbcUtil {
     return result;
   }
 
+  static boolean isConstraintViolation(SQLException ex) {
+    return ex instanceof SQLIntegrityConstraintViolationException
+        || POSTGRES_UNIQUE_VIOLATION_SQLSTATE.equals(ex.getSQLState())
+        || (ex.getMessage() != null && ex.getMessage().contains("constraint 
failed"));
+  }
+
   private static int update(
       boolean isTable,
       SchemaVersion schemaVersion,
diff --git a/core/src/main/java/org/apache/iceberg/jdbc/JdbcViewOperations.java 
b/core/src/main/java/org/apache/iceberg/jdbc/JdbcViewOperations.java
index 10f46941d6..646e5b8606 100644
--- a/core/src/main/java/org/apache/iceberg/jdbc/JdbcViewOperations.java
+++ b/core/src/main/java/org/apache/iceberg/jdbc/JdbcViewOperations.java
@@ -20,7 +20,6 @@ package org.apache.iceberg.jdbc;
 
 import java.sql.DataTruncation;
 import java.sql.SQLException;
-import java.sql.SQLIntegrityConstraintViolationException;
 import java.sql.SQLNonTransientConnectionException;
 import java.sql.SQLTimeoutException;
 import java.sql.SQLTransientConnectionException;
@@ -112,13 +111,6 @@ public class JdbcViewOperations extends BaseViewOperations 
{
         createView(newMetadataLocation);
       }
 
-    } catch (SQLIntegrityConstraintViolationException e) {
-      if (currentMetadataLocation() == null) {
-        throw new AlreadyExistsException(e, "View already exists: %s", 
viewIdentifier);
-      } else {
-        throw new UncheckedSQLException(e, "View already exists: %s", 
viewIdentifier);
-      }
-
     } catch (SQLTimeoutException e) {
       throw new UncheckedSQLException(e, "Database Connection timeout");
     } catch (SQLTransientConnectionException | 
SQLNonTransientConnectionException e) {
@@ -128,9 +120,12 @@ public class JdbcViewOperations extends BaseViewOperations 
{
     } catch (SQLWarning e) {
       throw new UncheckedSQLException(e, "Database warning");
     } catch (SQLException e) {
-      // SQLite doesn't set SQLState or throw 
SQLIntegrityConstraintViolationException
-      if (e.getMessage() != null && e.getMessage().contains("constraint 
failed")) {
-        throw new AlreadyExistsException("View already exists: %s", 
viewIdentifier);
+      if (JdbcUtil.isConstraintViolation(e)) {
+        if (currentMetadataLocation() == null) {
+          throw new AlreadyExistsException(e, "View already exists: %s", 
viewIdentifier);
+        } else {
+          throw new UncheckedSQLException(e, "View already exists: %s", 
viewIdentifier);
+        }
       }
 
       throw new UncheckedSQLException(e, "Unknown failure");
diff --git a/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcCatalog.java 
b/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcCatalog.java
index 978c1d9698..310d918849 100644
--- a/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcCatalog.java
+++ b/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcCatalog.java
@@ -1083,6 +1083,7 @@ public class TestJdbcCatalog extends 
CatalogTests<JdbcCatalog> {
       mockedStatic
           .when(() -> JdbcUtil.loadTable(any(), any(), any(), any()))
           .thenThrow(new SQLException());
+      mockedStatic.when(() -> 
JdbcUtil.isConstraintViolation(any())).thenCallRealMethod();
       assertThatThrownBy(() -> ops.commit(ops.current(), metadataV1))
           .isInstanceOf(UncheckedSQLException.class)
           .hasMessageStartingWith("Unknown failure");
@@ -1103,12 +1104,76 @@ public class TestJdbcCatalog extends 
CatalogTests<JdbcCatalog> {
       mockedStatic
           .when(() -> JdbcUtil.loadTable(any(), any(), any(), any()))
           .thenThrow(new SQLException("constraint failed"));
+      mockedStatic.when(() -> 
JdbcUtil.isConstraintViolation(any())).thenCallRealMethod();
       assertThatThrownBy(() -> ops.commit(ops.current(), metadataV1))
-          .isInstanceOf(AlreadyExistsException.class)
+          .isInstanceOf(UncheckedSQLException.class)
           .hasMessageStartingWith("Table already exists: " + tableIdent);
     }
   }
 
+  @Test
+  public void testCommitExceptionWithPostgresUniqueViolation() {
+    TableIdentifier tableIdent = TableIdentifier.of("db", "tbl");
+    BaseTable table = (BaseTable) catalog.buildTable(tableIdent, 
SCHEMA).create();
+    TableOperations ops = table.operations();
+    TableMetadata metadataV1 = ops.current();
+
+    table.updateSchema().addColumn("n", Types.IntegerType.get()).commit();
+    ops.refresh();
+
+    try (MockedStatic<JdbcUtil> mockedStatic = 
Mockito.mockStatic(JdbcUtil.class)) {
+      mockedStatic
+          .when(() -> JdbcUtil.loadTable(any(), any(), any(), any()))
+          .thenThrow(new SQLException("unique violation", "23505"));
+      mockedStatic.when(() -> 
JdbcUtil.isConstraintViolation(any())).thenCallRealMethod();
+      assertThatThrownBy(() -> ops.commit(ops.current(), metadataV1))
+          .isInstanceOf(UncheckedSQLException.class)
+          .hasMessageStartingWith("Table already exists: " + tableIdent);
+    }
+  }
+
+  @Test
+  public void testCreateTableConstraintExceptionWithMessage() {
+    TableIdentifier existingIdent = TableIdentifier.of("db", "existing_tbl");
+    BaseTable existing = (BaseTable) catalog.buildTable(existingIdent, 
SCHEMA).create();
+    TableMetadata metadata = existing.operations().current();
+
+    TableIdentifier newIdent = TableIdentifier.of("db", "new_tbl");
+    TableOperations ops = catalog.newTableOps(newIdent);
+
+    try (MockedStatic<JdbcUtil> mockedStatic = 
Mockito.mockStatic(JdbcUtil.class)) {
+      mockedStatic
+          .when(() -> JdbcUtil.loadTable(any(), any(), any(), any()))
+          .thenReturn(Maps.newHashMap())
+          .thenThrow(new SQLException("constraint failed"));
+      mockedStatic.when(() -> 
JdbcUtil.isConstraintViolation(any())).thenCallRealMethod();
+      assertThatThrownBy(() -> ops.commit(null, metadata))
+          .isInstanceOf(AlreadyExistsException.class)
+          .hasMessageStartingWith("Table already exists: " + newIdent);
+    }
+  }
+
+  @Test
+  public void testCreateTableConstraintExceptionWithPostgresUniqueViolation() {
+    TableIdentifier existingIdent = TableIdentifier.of("db", "existing_tbl2");
+    BaseTable existing = (BaseTable) catalog.buildTable(existingIdent, 
SCHEMA).create();
+    TableMetadata metadata = existing.operations().current();
+
+    TableIdentifier newIdent = TableIdentifier.of("db", "new_tbl2");
+    TableOperations ops = catalog.newTableOps(newIdent);
+
+    try (MockedStatic<JdbcUtil> mockedStatic = 
Mockito.mockStatic(JdbcUtil.class)) {
+      mockedStatic
+          .when(() -> JdbcUtil.loadTable(any(), any(), any(), any()))
+          .thenReturn(Maps.newHashMap())
+          .thenThrow(new SQLException("unique violation", "23505"));
+      mockedStatic.when(() -> 
JdbcUtil.isConstraintViolation(any())).thenCallRealMethod();
+      assertThatThrownBy(() -> ops.commit(null, metadata))
+          .isInstanceOf(AlreadyExistsException.class)
+          .hasMessageStartingWith("Table already exists: " + newIdent);
+    }
+  }
+
   private String createMetadataLocationViaJdbcCatalog(TableIdentifier 
identifier)
       throws SQLException {
     // temporary connection just to actually create a concrete metadata 
location
diff --git a/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcUtil.java 
b/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcUtil.java
index 4ac3a9301b..8128139da0 100644
--- a/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcUtil.java
+++ b/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcUtil.java
@@ -23,6 +23,8 @@ import static org.assertj.core.api.Assertions.assertThat;
 import java.nio.file.Files;
 import java.sql.PreparedStatement;
 import java.sql.ResultSet;
+import java.sql.SQLException;
+import java.sql.SQLIntegrityConstraintViolationException;
 import java.util.Map;
 import java.util.Properties;
 import org.apache.iceberg.catalog.Namespace;
@@ -144,6 +146,28 @@ public class TestJdbcUtil {
     }
   }
 
+  @Test
+  public void 
testIsConstraintViolationWithSQLIntegrityConstraintViolationException() {
+    assertThat(JdbcUtil.isConstraintViolation(new 
SQLIntegrityConstraintViolationException()))
+        .isTrue();
+  }
+
+  @Test
+  public void testIsConstraintViolationWithPostgresSQLState() {
+    assertThat(JdbcUtil.isConstraintViolation(new SQLException("unique 
violation", "23505")))
+        .isTrue();
+  }
+
+  @Test
+  public void testIsConstraintViolationWithConstraintFailedMessage() {
+    assertThat(JdbcUtil.isConstraintViolation(new SQLException("constraint 
failed"))).isTrue();
+  }
+
+  @Test
+  public void testIsConstraintViolationWithPlainSQLException() {
+    assertThat(JdbcUtil.isConstraintViolation(new SQLException("some other 
error"))).isFalse();
+  }
+
   @Test
   public void emptyNamespaceInIdentifier() {
     assertThat(JdbcUtil.stringToTableIdentifier("", "tblName"))
diff --git 
a/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcViewCatalog.java 
b/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcViewCatalog.java
index 38c908a0c0..b9f51dc5d9 100644
--- a/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcViewCatalog.java
+++ b/core/src/test/java/org/apache/iceberg/jdbc/TestJdbcViewCatalog.java
@@ -112,6 +112,7 @@ public class TestJdbcViewCatalog extends 
ViewCatalogTests<JdbcCatalog> {
       mockedStatic
           .when(() -> JdbcUtil.loadView(any(), any(), any(), any()))
           .thenThrow(new SQLException());
+      mockedStatic.when(() -> 
JdbcUtil.isConstraintViolation(any())).thenCallRealMethod();
       assertThatThrownBy(() -> ops.commit(ops.current(), metadataV1))
           .isInstanceOf(UncheckedSQLException.class)
           .hasMessageStartingWith("Unknown failure");
@@ -139,12 +140,97 @@ public class TestJdbcViewCatalog extends 
ViewCatalogTests<JdbcCatalog> {
       mockedStatic
           .when(() -> JdbcUtil.loadView(any(), any(), any(), any()))
           .thenThrow(new SQLException("constraint failed"));
+      mockedStatic.when(() -> 
JdbcUtil.isConstraintViolation(any())).thenCallRealMethod();
       assertThatThrownBy(() -> ops.commit(ops.current(), metadataV1))
-          .isInstanceOf(AlreadyExistsException.class)
+          .isInstanceOf(UncheckedSQLException.class)
+          .hasMessageStartingWith("View already exists: " + identifier);
+    }
+  }
+
+  @Test
+  public void testCommitExceptionWithPostgresUniqueViolation() {
+    TableIdentifier identifier = TableIdentifier.of("namespace1", "view");
+    BaseView view =
+        (BaseView)
+            catalog
+                .buildView(identifier)
+                .withQuery("spark", "select * from tbl")
+                .withSchema(SCHEMA)
+                .withDefaultNamespace(Namespace.of("namespace1"))
+                .create();
+    ViewOperations ops = view.operations();
+    ViewMetadata metadataV1 = ops.current();
+
+    view.updateProperties().set("k1", "v1").commit();
+    ops.refresh();
+
+    try (MockedStatic<JdbcUtil> mockedStatic = 
Mockito.mockStatic(JdbcUtil.class)) {
+      mockedStatic
+          .when(() -> JdbcUtil.loadView(any(), any(), any(), any()))
+          .thenThrow(new SQLException("unique violation", "23505"));
+      mockedStatic.when(() -> 
JdbcUtil.isConstraintViolation(any())).thenCallRealMethod();
+      assertThatThrownBy(() -> ops.commit(ops.current(), metadataV1))
+          .isInstanceOf(UncheckedSQLException.class)
           .hasMessageStartingWith("View already exists: " + identifier);
     }
   }
 
+  @Test
+  public void testCreateViewConstraintExceptionWithMessage() {
+    TableIdentifier existingIdent = TableIdentifier.of("namespace1", 
"existing_view");
+    BaseView existing =
+        (BaseView)
+            catalog
+                .buildView(existingIdent)
+                .withQuery("spark", "select * from tbl")
+                .withSchema(SCHEMA)
+                .withDefaultNamespace(Namespace.of("namespace1"))
+                .create();
+    ViewMetadata metadata = existing.operations().current();
+
+    TableIdentifier newIdent = TableIdentifier.of("namespace1", "new_view");
+    ViewOperations ops = catalog.newViewOps(newIdent);
+
+    try (MockedStatic<JdbcUtil> mockedStatic = 
Mockito.mockStatic(JdbcUtil.class)) {
+      mockedStatic
+          .when(() -> JdbcUtil.loadView(any(), any(), any(), any()))
+          .thenReturn(Maps.newHashMap())
+          .thenThrow(new SQLException("constraint failed"));
+      mockedStatic.when(() -> 
JdbcUtil.isConstraintViolation(any())).thenCallRealMethod();
+      assertThatThrownBy(() -> ops.commit(null, metadata))
+          .isInstanceOf(AlreadyExistsException.class)
+          .hasMessageStartingWith("View already exists: " + newIdent);
+    }
+  }
+
+  @Test
+  public void testCreateViewConstraintExceptionWithPostgresUniqueViolation() {
+    TableIdentifier existingIdent = TableIdentifier.of("namespace1", 
"existing_view2");
+    BaseView existing =
+        (BaseView)
+            catalog
+                .buildView(existingIdent)
+                .withQuery("spark", "select * from tbl")
+                .withSchema(SCHEMA)
+                .withDefaultNamespace(Namespace.of("namespace1"))
+                .create();
+    ViewMetadata metadata = existing.operations().current();
+
+    TableIdentifier newIdent = TableIdentifier.of("namespace1", "new_view2");
+    ViewOperations ops = catalog.newViewOps(newIdent);
+
+    try (MockedStatic<JdbcUtil> mockedStatic = 
Mockito.mockStatic(JdbcUtil.class)) {
+      mockedStatic
+          .when(() -> JdbcUtil.loadView(any(), any(), any(), any()))
+          .thenReturn(Maps.newHashMap())
+          .thenThrow(new SQLException("unique violation", "23505"));
+      mockedStatic.when(() -> 
JdbcUtil.isConstraintViolation(any())).thenCallRealMethod();
+      assertThatThrownBy(() -> ops.commit(null, metadata))
+          .isInstanceOf(AlreadyExistsException.class)
+          .hasMessageStartingWith("View already exists: " + newIdent);
+    }
+  }
+
   @Test
   public void dropViewShouldNotDropMetadataFileIfGcNotEnabled() {
     TableIdentifier identifier = TableIdentifier.of("namespace1", "view");

Reply via email to