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

pnowojski pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/flink.git

commit 16e5fbd3c1d91978f0d6029e21227aed3f306868
Author: Piotr Nowojski <[email protected]>
AuthorDate: Thu May 15 16:33:59 2025 +0200

    [FLINK-37807][rocksdb] Do not log CancelTaskException during RocksDB 
building as errors
---
 .../flink/state/rocksdb/RocksDBKeyedStateBackendBuilder.java  | 11 ++++++++++-
 1 file changed, 10 insertions(+), 1 deletion(-)

diff --git 
a/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/state/rocksdb/RocksDBKeyedStateBackendBuilder.java
 
b/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/state/rocksdb/RocksDBKeyedStateBackendBuilder.java
index 163d6891ea8..dd5a391f184 100644
--- 
a/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/state/rocksdb/RocksDBKeyedStateBackendBuilder.java
+++ 
b/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/state/rocksdb/RocksDBKeyedStateBackendBuilder.java
@@ -23,6 +23,7 @@ import org.apache.flink.api.common.ExecutionConfig;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.core.fs.CloseableRegistry;
 import org.apache.flink.metrics.MetricGroup;
+import org.apache.flink.runtime.execution.CancelTaskException;
 import org.apache.flink.runtime.query.TaskKvStateRegistry;
 import org.apache.flink.runtime.state.AbstractKeyedStateBackendBuilder;
 import org.apache.flink.runtime.state.BackendBuildingException;
@@ -440,8 +441,13 @@ public class RocksDBKeyedStateBackendBuilder<K> extends 
AbstractKeyedStateBacken
                             nativeMetricMonitor,
                             manualCompactionManager);
         } catch (Throwable e) {
+            boolean cancelled = cancelStreamRegistry.isClosed();
             // log ASAP because close can block or fail too
-            logger.warn("Failed to build RocksDB state backend", e);
+            if (cancelled) {
+                logger.info("RocksDB state backend build cancelled");
+            } else {
+                logger.warn("Failed to build RocksDB state backend", e);
+            }
             // Do clean up
             List<ColumnFamilyOptions> columnFamilyOptions =
                     new ArrayList<>(kvStateInformation.values().size());
@@ -474,6 +480,9 @@ public class RocksDBKeyedStateBackendBuilder<K> extends 
AbstractKeyedStateBacken
             // Log and rethrow
             if (e instanceof BackendBuildingException) {
                 throw (BackendBuildingException) e;
+            } else if (cancelled) {
+                throw new BackendBuildingException(
+                        "Task was cancelled", new CancelTaskException(e));
             } else {
                 String errMsg = "Caught unexpected exception.";
                 logger.error(errMsg, e);

Reply via email to