This is an automated email from the ASF dual-hosted git repository.
apolovtsev pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/ignite-3.git
The following commit(s) were added to refs/heads/main by this push:
new 0d4dbce79c IGNITE-21896 Close RocksDB flush listener on stop (#3533)
0d4dbce79c is described below
commit 0d4dbce79cb4cb08eb90686e0b92449da5d669f4
Author: Roman Puchkovskiy <[email protected]>
AuthorDate: Tue Apr 2 11:17:04 2024 +0400
IGNITE-21896 Close RocksDB flush listener on stop (#3533)
---
.../org/apache/ignite/internal/rocksdb/flush/RocksDbFlusher.java | 7 +++++--
.../tx/storage/state/rocksdb/TxStateRocksDbSharedStorage.java | 1 +
2 files changed, 6 insertions(+), 2 deletions(-)
diff --git
a/modules/rocksdb-common/src/main/java/org/apache/ignite/internal/rocksdb/flush/RocksDbFlusher.java
b/modules/rocksdb-common/src/main/java/org/apache/ignite/internal/rocksdb/flush/RocksDbFlusher.java
index 62528bb05d..70ebe8fdc4 100644
---
a/modules/rocksdb-common/src/main/java/org/apache/ignite/internal/rocksdb/flush/RocksDbFlusher.java
+++
b/modules/rocksdb-common/src/main/java/org/apache/ignite/internal/rocksdb/flush/RocksDbFlusher.java
@@ -30,6 +30,7 @@ import java.util.concurrent.TimeUnit;
import java.util.function.IntSupplier;
import org.apache.ignite.internal.logger.IgniteLogger;
import org.apache.ignite.internal.logger.Loggers;
+import org.apache.ignite.internal.rocksdb.RocksUtils;
import org.apache.ignite.internal.util.IgniteSpinBusyLock;
import org.rocksdb.AbstractEventListener;
import org.rocksdb.ColumnFamilyHandle;
@@ -85,6 +86,8 @@ public class RocksDbFlusher {
/** Busy lock to stop synchronously. */
private final IgniteSpinBusyLock busyLock;
+ private final RocksDbFlushListener flushListener = new
RocksDbFlushListener(this);
+
/**
* Instance of the latest scheduled flush closure.
*
@@ -126,7 +129,7 @@ public class RocksDbFlusher {
* {@link Options#setListeners(List)} before database is started.
Otherwise, no events would occur.
*/
public AbstractEventListener listener() {
- return new RocksDbFlushListener(this);
+ return flushListener;
}
/**
@@ -262,7 +265,7 @@ public class RocksDbFlusher {
future.cancel(false);
}
- flushOptions.close();
+ RocksUtils.closeAll(flushListener, flushOptions);
}
/**
diff --git
a/modules/transactions/src/main/java/org/apache/ignite/internal/tx/storage/state/rocksdb/TxStateRocksDbSharedStorage.java
b/modules/transactions/src/main/java/org/apache/ignite/internal/tx/storage/state/rocksdb/TxStateRocksDbSharedStorage.java
index 0ca6392227..dbbb9cd5ef 100644
---
a/modules/transactions/src/main/java/org/apache/ignite/internal/tx/storage/state/rocksdb/TxStateRocksDbSharedStorage.java
+++
b/modules/transactions/src/main/java/org/apache/ignite/internal/tx/storage/state/rocksdb/TxStateRocksDbSharedStorage.java
@@ -180,6 +180,7 @@ public class TxStateRocksDbSharedStorage implements
ManuallyCloseable {
List<AutoCloseable> resources = new ArrayList<>();
+ resources.add(flusher::stop);
resources.add(readOptions);
resources.add(writeOptions);
resources.add(dbOptions);