From 3e8a57428840ad0c396f17b281ec0c1b88cef0c9 Mon Sep 17 00:00:00 2001 From: jellymlg Date: Wed, 17 Jan 2024 21:48:39 +0100 Subject: [PATCH 01/15] Migrate to RocksDB --- avldb/build.sbt | 2 +- .../main/scala/scorex/db/KVStoreReader.scala | 29 +++-- .../src/main/scala/scorex/db/LDBFactory.scala | 109 +++++------------- .../src/main/scala/scorex/db/LDBKVStore.scala | 28 +---- .../scala/scorex/db/LDBVersionedStore.scala | 82 ++++++------- .../VersionedLDBAVLStorageSpecification.scala | 2 +- build.sbt | 8 +- .../network/peer/PeerDatabase.scala | 5 +- .../history/storage/HistoryStorage.scala | 6 +- .../nodeView/state/SnapshotsDb.scala | 3 +- .../wallet/persistence/WalletStorage.scala | 8 +- .../scala/org/ergoplatform/db/DBSpec.scala | 17 ++- .../org/ergoplatform/db/LDBKVStoreSpec.scala | 21 ---- 13 files changed, 104 insertions(+), 216 deletions(-) diff --git a/avldb/build.sbt b/avldb/build.sbt index 7add06c7d9..25efabc0d7 100644 --- a/avldb/build.sbt +++ b/avldb/build.sbt @@ -17,7 +17,7 @@ libraryDependencies ++= Seq( ) libraryDependencies ++= Seq( - "org.ethereum" % "leveldbjni-all" % "1.18.3", + "org.rocksdb" % "rocksdbjni" % "8.9.1", "org.typelevel" %% "spire" % "0.14.1" ) diff --git a/avldb/src/main/scala/scorex/db/KVStoreReader.scala b/avldb/src/main/scala/scorex/db/KVStoreReader.scala index 9147aeb358..d285c39c97 100644 --- a/avldb/src/main/scala/scorex/db/KVStoreReader.scala +++ b/avldb/src/main/scala/scorex/db/KVStoreReader.scala @@ -1,8 +1,7 @@ package scorex.db import java.util.concurrent.locks.ReentrantReadWriteLock - -import org.iq80.leveldb.{DB, ReadOptions} +import org.rocksdb.{ReadOptions, RocksDB} import scala.collection.mutable @@ -15,7 +14,7 @@ trait KVStoreReader extends AutoCloseable { type K = Array[Byte] type V = Array[Byte] - protected val db: DB + protected val db: RocksDB protected val lock = new ReentrantReadWriteLock() @@ -41,16 +40,16 @@ trait KVStoreReader extends AutoCloseable { */ def getWithFilter(cond: (K, V) => Boolean): Iterator[(K, V)] = { val ro = new ReadOptions() - ro.snapshot(db.getSnapshot) - val iter = db.iterator(ro) + ro.setSnapshot(db.getSnapshot) + val iter = db.newIterator(ro) try { iter.seekToFirst() val bf = mutable.ArrayBuffer.empty[(K, V)] - while (iter.hasNext) { - val next = iter.next() - val key = next.getKey - val value = next.getValue + while (iter.isValid) { + val key = iter.key() + val value = iter.value() if (cond(key, value)) bf += (key -> value) + iter.next() } bf.toIterator } finally { @@ -97,18 +96,18 @@ trait KVStoreReader extends AutoCloseable { */ def getRange(start: K, end: K, limit: Int = Int.MaxValue): Array[(K, V)] = { val ro = new ReadOptions() - ro.snapshot(db.getSnapshot) - val iter = db.iterator(ro) + ro.setSnapshot(db.getSnapshot) + val iter = db.newIterator(ro) try { iter.seek(start) val bf = mutable.ArrayBuffer.empty[(K, V)] var elemCounter = 0 - while (iter.hasNext && elemCounter < limit) { - val next = iter.next() - if(ByteArrayUtils.compare(next.getKey, end) <= 0) { + while (iter.isValid && elemCounter < limit) { + if(ByteArrayUtils.compare(iter.key(), end) <= 0) { elemCounter += 1 - bf += (next.getKey -> next.getValue) + bf += (iter.key() -> iter.value()) } else elemCounter = limit // break + iter.next() } bf.toArray[(K,V)] } finally { diff --git a/avldb/src/main/scala/scorex/db/LDBFactory.scala b/avldb/src/main/scala/scorex/db/LDBFactory.scala index 65fca5a397..9b3c93811b 100644 --- a/avldb/src/main/scala/scorex/db/LDBFactory.scala +++ b/avldb/src/main/scala/scorex/db/LDBFactory.scala @@ -1,8 +1,10 @@ package scorex.db +import org.rocksdb.util.SizeUnit + import java.io.File import java.util.concurrent.locks.ReentrantReadWriteLock -import org.iq80.leveldb.{DB, DBFactory, DBIterator, Options, Range, ReadOptions, Snapshot, WriteBatch, WriteOptions} +import org.rocksdb.{CompactionStyle, CompressionType, Options, ReadOptions, RocksDB, RocksIterator, Snapshot, WriteBatch, WriteOptions} import scorex.util.ScorexLogging import scala.collection.mutable @@ -13,7 +15,7 @@ import scala.collection.mutable * And ergo application (mostly tests) quite frequently doesn't not explicitly close * database and tries to reopen it. */ -case class StoreRegistry(factory: DBFactory) extends DBFactory with ScorexLogging { +case class StoreRegistry() extends ScorexLogging { val lock = new ReentrantReadWriteLock() val map = new mutable.HashMap[File, RegisteredDB] @@ -23,49 +25,43 @@ case class StoreRegistry(factory: DBFactory) extends DBFactory with ScorexLoggin * So if database was not explicitly closed, then next attempt to open database with the same path will * return existed instance instead of creating new one. */ - case class RegisteredDB(impl: DB, path: File) extends DB { + case class RegisteredDB(impl: RocksDB, path: File) { def get(key: Array[Byte]): Array[Byte] = impl.get(key) - def get(key: Array[Byte], options: ReadOptions): Array[Byte] = impl.get(key, options) + def get(key: Array[Byte], options: ReadOptions): Array[Byte] = impl.get(options, key) - def iterator: DBIterator = impl.iterator + def iterator: RocksIterator = impl.newIterator() - def iterator(options: ReadOptions): DBIterator = impl.iterator(options) + def iterator(options: ReadOptions): RocksIterator = impl.newIterator(options) def put(key: Array[Byte], value: Array[Byte]): Unit = impl.put(key, value) def delete(key: Array[Byte]): Unit = impl.delete(key) - def write(batch: WriteBatch): Unit = impl.write(batch) + def write(batch: WriteBatch): Unit = impl.write(new WriteOptions(), batch) - def write(batch: WriteBatch, options: WriteOptions): Snapshot = impl.write(batch, options) + def write(batch: WriteBatch, options: WriteOptions): Unit = impl.write(options, batch) - def createWriteBatch: WriteBatch = impl.createWriteBatch() + def createWriteBatch: WriteBatch = new WriteBatch() - def put(key: Array[Byte], value: Array[Byte], options: WriteOptions): Snapshot = impl.put(key, value, options) + def put(key: Array[Byte], value: Array[Byte], options: WriteOptions): Unit = impl.put(options, key, value) - def delete(key: Array[Byte], options: WriteOptions): Snapshot = impl.delete(key, options) + def delete(key: Array[Byte], options: WriteOptions): Unit = impl.delete(options, key) def getSnapshot: Snapshot = impl.getSnapshot - def getApproximateSizes(ranges: Range*): Array[Long] = impl.getApproximateSizes(ranges: _*) - def getProperty(name: String): String = impl.getProperty(name) - def suspendCompactions(): Unit = impl.suspendCompactions() - - def resumeCompactions(): Unit = impl.resumeCompactions() - def compactRange(begin: Array[Byte], end: Array[Byte]): Unit = impl.compactRange(begin, end) - override def close(): Unit = { + def close(): Unit = { remove(path) impl.close() } } - private def add(file: File, create: => DB): DB = { + private def add(file: File, create: => RocksDB): RegisteredDB = { lock.writeLock().lock() try { map.getOrElseUpdate(file, RegisteredDB(create, file)) @@ -83,10 +79,10 @@ case class StoreRegistry(factory: DBFactory) extends DBFactory with ScorexLoggin } } - def open(path: File, options: Options): DB = { + def open(path: File, options: Options): RegisteredDB = { lock.writeLock().lock() try { - add(path, factory.open(path, options)) + add(path, LDBFactory.openDb(path, options)) } catch { case x: Throwable => log.error(s"Failed to initialize storage: $x. Please check that directory $path exists and is not used by some other active node") @@ -97,34 +93,24 @@ case class StoreRegistry(factory: DBFactory) extends DBFactory with ScorexLoggin } } - def destroy(path: File, options: Options): Unit = { - factory.destroy(path, options) - } - - def repair(path: File, options: Options): Unit = { - factory.repair(path, options) - } } object LDBFactory extends ScorexLogging { - private val nativeFactory = "org.fusesource.leveldbjni.JniDBFactory" - private val javaFactory = "org.iq80.leveldb.impl.Iq80DBFactory" - private val memoryPoolSize = 512 * 1024 - - def setLevelDBParams(factory: DBFactory): AnyRef = { - val pushMemoryPool = factory.getClass.getDeclaredMethod("pushMemoryPool", classOf[Int]) - pushMemoryPool.invoke(null, Integer.valueOf(memoryPoolSize)) + def openDb(path: File, options: Options = new Options()): RocksDB = { + path.mkdirs() + options.setCreateIfMissing(true) + .setWriteBufferSize(128 * SizeUnit.MB) + .setMaxWriteBufferNumber(3) + .setMaxBackgroundJobs(10) + .setCompressionType(CompressionType.NO_COMPRESSION) + .setCompactionStyle(CompactionStyle.LEVEL) + RocksDB.open(options, path.toString) } - def createKvDb(path: String): LDBKVStore = { - val dir = new File(path) - dir.mkdirs() - val options = new Options() - options.createIfMissing(true) + def createKvDb(path: File): LDBKVStore = { try { - val db = factory.open(dir, options) - new LDBKVStore(db) + new LDBKVStore(openDb(path)) } catch { case x: Throwable => log.error(s"Failed to initialize storage: $x. Please check that directory $path could be accessed " + @@ -134,43 +120,4 @@ object LDBFactory extends ScorexLogging { } } - - lazy val factory: DBFactory = { - val loaders = List(ClassLoader.getSystemClassLoader, this.getClass.getClassLoader) - - // As LevelDB-JNI has problems on Mac (see https://github.com/ergoplatform/ergo/issues/1067), - // we are using only pure-Java LevelDB on Mac - val isMac = System.getProperty("os.name").toLowerCase().indexOf("mac") >= 0 - val factories = if(isMac) { - List(javaFactory) - } else { - List(nativeFactory, javaFactory) - } - - val pairs = loaders.view - .zip(factories) - .flatMap { case (loader, factoryName) => - loadFactory(loader, factoryName).map(factoryName -> _) - } - - val (name, factory) = pairs.headOption.getOrElse { - throw new RuntimeException(s"Could not load any of the factory classes: $factories") - } - - if (name == javaFactory) { - log.warn("Using the pure java LevelDB implementation which is still experimental") - } else { - log.info(s"Loaded $name with $factory") - setLevelDBParams(factory) - } - StoreRegistry(factory) - } - - private def loadFactory(loader: ClassLoader, factoryName: String): Option[DBFactory] = - try Some(loader.loadClass(factoryName).getConstructor().newInstance().asInstanceOf[DBFactory]) - catch { - case e: Throwable => - log.warn(s"Failed to load database factory $factoryName due to: $e") - None - } } diff --git a/avldb/src/main/scala/scorex/db/LDBKVStore.scala b/avldb/src/main/scala/scorex/db/LDBKVStore.scala index 035e43510a..a63d0829c9 100644 --- a/avldb/src/main/scala/scorex/db/LDBKVStore.scala +++ b/avldb/src/main/scala/scorex/db/LDBKVStore.scala @@ -1,6 +1,6 @@ package scorex.db -import org.iq80.leveldb.DB +import org.rocksdb.{RocksDB, WriteBatch, WriteOptions} import scorex.util.ScorexLogging import scala.util.{Failure, Success, Try} @@ -12,7 +12,7 @@ import spire.syntax.all.cfor * * Both keys and values are var-sized byte arrays. */ -class LDBKVStore(protected val db: DB) extends KVStoreReader with ScorexLogging { +class LDBKVStore(protected val db: RocksDB) extends KVStoreReader with ScorexLogging { /** Immutable empty array can be shared to avoid allocations. */ private val emptyArrayOfByteArray = Array.empty[Array[Byte]] @@ -25,12 +25,12 @@ class LDBKVStore(protected val db: DB) extends KVStoreReader with ScorexLogging * @return - error if it happens, or success status */ def update(toInsertKeys: Array[K], toInsertValues: Array[V], toRemove: Array[K]): Try[Unit] = { - val batch = db.createWriteBatch() + val batch = new WriteBatch() try { require(toInsertKeys.length == toInsertValues.length) cfor(0)(_ < toInsertKeys.length, _ + 1) { i => batch.put(toInsertKeys(i), toInsertValues(i))} cfor(0)(_ < toRemove.length, _ + 1) { i => batch.delete(toRemove(i))} - db.write(batch) + db.write(new WriteOptions(), batch) Success(()) } catch { case t: Throwable => Failure(t) @@ -68,24 +68,4 @@ class LDBKVStore(protected val db: DB) extends KVStoreReader with ScorexLogging update(emptyArrayOfByteArray, emptyArrayOfByteArray, keys) } - /** - * Get last key within some range (inclusive) by used comparator. - * Could be useful for applications with sequential ids. - * The method iterates over all the keys so could be slow if there are many keys in the range. - */ - def lastKeyInRange(first: Array[Byte], last: Array[Byte]): Option[K] = { - import util.control.Breaks._ - - val i = db.iterator() - var res: Option[K] = None - i.seek(first) - breakable { - while (i.hasNext) { - val key = i.next().getKey - if (ByteArrayUtils.compare(key, last) <= 0) res = Some(key) else break - } - } - res - } - } diff --git a/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala b/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala index 51340723f9..213a04115b 100644 --- a/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala +++ b/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala @@ -1,8 +1,7 @@ package scorex.db import java.io.File -import scorex.db.LDBFactory.factory -import org.iq80.leveldb._ +import org.rocksdb.{Options, ReadOptions, RocksDB, WriteBatch, WriteOptions} import java.nio.ByteBuffer import scala.collection.mutable.ArrayBuffer @@ -37,10 +36,10 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) private var keepVersions: Int = initialKeepVersions - override val db: DB = createDB(dir, "ldb_main") // storage for main data + override val db: RocksDB = createDB(dir, "ldb_main") // storage for main data override val lock = new ReentrantReadWriteLock() - private val undo: DB = createDB(dir, "ldb_undo") // storage for undo data + private val undo: RocksDB = createDB(dir, "ldb_undo") // storage for undo data private var lsn: LSN = getLastLSN // last assigned logical serial number private var versionLsn = ArrayBuffer.empty[LSN] // LSNs of versions (var because we need to invert this array) @@ -52,11 +51,11 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) //default write options, no sync! private val writeOptions = new WriteOptions() - private def createDB(dir: File, storeName: String): DB = { + private def createDB(dir: File, storeName: String): RocksDB = { val op = new Options() - op.createIfMissing(true) - op.paranoidChecks(true) - factory.open(new File(dir, storeName), op) + op.setCreateIfMissing(true) + op.setParanoidChecks(true) + LDBFactory.openDb(new File(dir, storeName), op) } /** Set new keep versions threshold, remove not needed versions and return old value of keep versions */ @@ -102,12 +101,12 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) def processAll(consumer: (K, V) => Unit): Unit = { lock.readLock().lock() - val iterator = db.iterator() + val iterator = db.newIterator() try { iterator.seekToFirst() - while (iterator.hasNext) { - val n = iterator.next() - consumer(n.getKey, n.getValue) + while (iterator.isValid) { + consumer(iterator.key(), iterator.value()) + iterator.next() } } finally { iterator.close() @@ -135,11 +134,11 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) } private def getLastLSN: LSN = { - val iterator = undo.iterator + val iterator = undo.newIterator() try { iterator.seekToFirst() - if (iterator.hasNext) { - decodeLSN(iterator.peekNext().getKey) + if (iterator.isValid) { + decodeLSN(iterator.key()) } else { 0 } @@ -166,17 +165,17 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) var lastVersion: Option[VersionID] = None var lastLsn: LSN = 0 // We iterate in LSN descending order - val iterator = undo.iterator() + val iterator = undo.newIterator() iterator.seekToFirst() - while (iterator.hasNext) { - val entry = iterator.next - val currVersion = deserializeUndo(entry.getValue).versionID - lastLsn = decodeLSN(entry.getKey) + while (iterator.isValid) { + val currVersion = deserializeUndo(iterator.value()).versionID + lastLsn = decodeLSN(iterator.key()) if (!lastVersion.exists(_.sameElements(currVersion))) { versionLsn += lastLsn + 1 // this is first LSN of successor version versions += currVersion lastVersion = Some(currVersion) } + iterator.next() } iterator.close() // As far as org.iq80.leveldb doesn't support iteration in reverse order, we have to iterate in the order @@ -241,8 +240,8 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) toUpdate: TraversableOnce[(Array[Byte], Array[Byte])]): Try[Unit] = Try { lock.writeLock().lock() val lastLsn = lsn // remember current LSN value - val batch = db.createWriteBatch() - val undoBatch = undo.createWriteBatch() + val batch = new WriteBatch() + val undoBatch = new WriteBatch() try { toRemove.foreach(key => { batch.delete(key) @@ -266,7 +265,7 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) if (lsn == lastLsn) { // no records were written for this version: generate dummy record undoBatch.put(newLSN(), serializeUndo(versionID, new Array[Byte](0), null)) } - undo.write(undoBatch, writeOptions) + undo.write(writeOptions, undoBatch) if (lastVersion.isEmpty || !versionID.sameElements(lastVersion.get)) { versions += versionID versionLsn += lastLsn + 1 // first LSN for this version @@ -284,7 +283,7 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) } } - db.write(batch, writeOptions) + db.write(writeOptions, batch) lastVersion = Some(versionID) } finally { // Make sure you close the batch to avoid resource leaks. @@ -305,12 +304,12 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) if (deteriorated >= 0) { val fromLsn = versionLsn(0) val tillLsn = if (deteriorated+1 < versions.size) versionLsn(deteriorated+1) else lsn+1 - val batch = undo.createWriteBatch() + val batch = new WriteBatch() try { for (lsn <- fromLsn until tillLsn) { batch.delete(encodeLSN(lsn)) } - undo.write(batch, writeOptions) + undo.write(writeOptions, batch) } finally { batch.close() } @@ -330,13 +329,6 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) } finally { lock.writeLock().unlock() } - undo.resumeCompactions() - db.resumeCompactions() - } - - def cleanStop(): Unit = { - undo.suspendCompactions() - db.suspendCompactions() } override def close(): Unit = { @@ -356,22 +348,21 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) val versionIndex = versions.indexWhere(_.sameElements(versionID)) if (versionIndex >= 0) { if (versionIndex != versions.size-1) { - val batch = db.createWriteBatch() - val undoBatch = undo.createWriteBatch() + val batch = new WriteBatch() + val undoBatch = new WriteBatch() var nUndoRecords: Long = 0 - val iterator = undo.iterator() + val iterator = undo.newIterator() var lastLsn: LSN = 0 try { var undoing = true iterator.seekToFirst() - while (undoing && iterator.hasNext) { - val entry = iterator.next() - val undo = deserializeUndo(entry.getValue) + while (undoing && iterator.isValid) { + val undo = deserializeUndo(iterator.value()) if (undo.versionID.sameElements(versionID)) { undoing = false - lastLsn = decodeLSN(entry.getKey) + lastLsn = decodeLSN(iterator.key()) } else { - undoBatch.delete(entry.getKey) + undoBatch.delete(iterator.key()) nUndoRecords += 1 if (undo.value == null) { if (undo.key.length != 0) { // dummy record @@ -381,9 +372,10 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) batch.put(undo.key, undo.value) } } + iterator.next() } - db.write(batch, writeOptions) - undo.write(undoBatch, writeOptions) + db.write(writeOptions, batch) + undo.write(writeOptions, undoBatch) } finally { // Make sure you close the batch to avoid resource leaks. iterator.close() @@ -424,11 +416,11 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) val ro = new ReadOptions() try { lock.writeLock().lock() - ro.snapshot(db.getSnapshot) + ro.setSnapshot(db.getSnapshot) lock.writeLock().unlock() object readInterface extends SnapshotReadInterface { - def get(key: Array[Byte]): Array[Byte] = db.get(key, ro) + def get(key: Array[Byte]): Array[Byte] = db.get(ro, key) } Success(logic(readInterface)) } catch { diff --git a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageSpecification.scala b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageSpecification.scala index 810e986b00..5cd577b491 100644 --- a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageSpecification.scala +++ b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageSpecification.scala @@ -350,7 +350,7 @@ class VersionedLDBAVLStorageSpecification blockchainWorkflowTest(prover) val storage = prover.storage.asInstanceOf[VersionedLDBAVLStorage] - val store = LDBFactory.createKvDb(getRandomTempDir.getAbsolutePath) + val store = LDBFactory.createKvDb(getRandomTempDir) val rootNodeLabel = storage.dumpSnapshot(store, manifestDepth, prover.digest.dropRight(1)).get rootNodeLabel.sameElements(prover.digest.dropRight(1)) shouldBe true diff --git a/build.sbt b/build.sbt index dccd41e4b1..dacdc1502d 100644 --- a/build.sbt +++ b/build.sbt @@ -128,7 +128,7 @@ assemblyMergeStrategy in assembly := { case x if x.endsWith("module-info.class") => MergeStrategy.discard case "reference.conf" => CustomMergeStrategy.concatReversed case PathList("org", "bouncycastle", xs @ _*) => MergeStrategy.first - case PathList("org", "iq80", "leveldb", xs @ _*) => MergeStrategy.first + case PathList("org", "rocksdb", xs @ _*) => MergeStrategy.first case PathList("org", "bouncycastle", xs @ _*) => MergeStrategy.first case PathList("javax", "activation", xs @ _*) => MergeStrategy.last case PathList("javax", "annotation", xs @ _*) => MergeStrategy.last @@ -219,11 +219,7 @@ lazy val avldb = (project in file("avldb")) javacOptions in(Compile, compile) ++= javacReleaseOption, libraryDependencies ++= Seq( // database dependencies - "org.ethereum" % "leveldbjni-all" % "1.18.3", - //the following pure-java leveldb implementation is needed only on specific platforms, such as 32-bit Raspberry Pi - //in future, it could be reasonable to have special builds with this Java db only, and for most of platforms use - //jni wrapper over native library included in leveldbjni-all - "org.iq80.leveldb" % "leveldb" % "0.12" + "org.rocksdb" % "rocksdbjni" % "8.9.1" ) ) diff --git a/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala b/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala index 5040fbe561..1c3e833e96 100644 --- a/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala +++ b/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala @@ -1,7 +1,8 @@ package org.ergoplatform.network.peer import org.ergoplatform.nodeView.history.ErgoHistoryUtils._ -import java.io.{ByteArrayInputStream, ByteArrayOutputStream, ObjectInputStream, ObjectOutputStream} + +import java.io.{ByteArrayInputStream, ByteArrayOutputStream, File, ObjectInputStream, ObjectOutputStream} import java.net.{InetAddress, InetSocketAddress} import org.ergoplatform.settings.ErgoSettings import scorex.db.LDBFactory @@ -15,7 +16,7 @@ import scala.util.{Failure, Success, Try} */ final class PeerDatabase(settings: ErgoSettings) extends ScorexLogging { - private val persistentStore = LDBFactory.createKvDb(s"${settings.directory}/peers") + private val persistentStore = LDBFactory.createKvDb(new File(s"${settings.directory}/peers")) private var peers = loadPeers match { diff --git a/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala b/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala index 53e06b3738..feac7eb031 100644 --- a/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala +++ b/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala @@ -226,9 +226,9 @@ class HistoryStorage private(indexStore: LDBKVStore, objectsStore: LDBKVStore, e object HistoryStorage { def apply(ergoSettings: ErgoSettings): HistoryStorage = { - val indexStore = LDBFactory.createKvDb(s"${ergoSettings.directory}/history/index") - val objectsStore = LDBFactory.createKvDb(s"${ergoSettings.directory}/history/objects") - val extraStore = LDBFactory.createKvDb(s"${ergoSettings.directory}/history/extra") + val indexStore = LDBFactory.createKvDb(new File(s"${ergoSettings.directory}/history/index")) + val objectsStore = LDBFactory.createKvDb(new File(s"${ergoSettings.directory}/history/objects")) + val extraStore = LDBFactory.createKvDb(new File(s"${ergoSettings.directory}/history/extra")) new HistoryStorage(indexStore, objectsStore, extraStore, ergoSettings.cacheSettings) } } diff --git a/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala b/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala index 5889c9458f..839d66a644 100644 --- a/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala +++ b/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala @@ -10,6 +10,7 @@ import scorex.db.{LDBFactory, LDBKVStore} import scorex.util.ScorexLogging import scorex.util.encode.Base16 +import java.io.File import scala.collection.mutable import scala.collection.mutable.ArrayBuffer import scala.util.{Failure, Success, Try} @@ -141,7 +142,7 @@ object SnapshotsDb { // internal method to open or init snapshots database in given folder // private[nodeView] to use it in tests also private[nodeView] def create(dir: String): SnapshotsDb = { - val store = LDBFactory.createKvDb(dir) + val store = LDBFactory.createKvDb(new File(dir)) new SnapshotsDb(store) } diff --git a/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala b/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala index 2f52c218f0..5263a039aa 100644 --- a/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala +++ b/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala @@ -183,14 +183,8 @@ final class WalletStorage(store: LDBKVStore, settings: ErgoSettings) extends Sco * @return identifier of last inserted scan */ def lastUsedScanId: Short = { - // pre-3.3.7 method to get last used scan id, now useful to read pre-3.3.7 databases - def oldScanId: Option[Short] = - store.lastKeyInRange(SmallestPossibleScanId, BiggestPossibleScanId) - .map(bs => Shorts.fromByteArray(bs.takeRight(2))) - store.get(lastUsedScanIdKey) .map(bs => Shorts.fromByteArray(bs)) - .orElse(oldScanId) .getOrElse(PaymentsScanId) } @@ -251,7 +245,7 @@ object WalletStorage { def storageFolder(settings: ErgoSettings): File = new File(s"${settings.directory}/wallet/storage") def readOrCreate(settings: ErgoSettings): WalletStorage = { - val db = LDBFactory.createKvDb(storageFolder(settings).getPath) + val db = LDBFactory.createKvDb(storageFolder(settings)) new WalletStorage(db, settings) } diff --git a/src/test/scala/org/ergoplatform/db/DBSpec.scala b/src/test/scala/org/ergoplatform/db/DBSpec.scala index 58f908be48..e8df7f0db0 100644 --- a/src/test/scala/org/ergoplatform/db/DBSpec.scala +++ b/src/test/scala/org/ergoplatform/db/DBSpec.scala @@ -3,9 +3,8 @@ package org.ergoplatform.db import akka.util.ByteString import org.ergoplatform.settings.Algos import org.ergoplatform.wallet.utils.TestFileUtils -import org.iq80.leveldb.{DB, Options} -import scorex.db.LDBFactory.factory -import scorex.db.{LDBKVStore, LDBVersionedStore} +import org.rocksdb.{Options, RocksDB} +import scorex.db.{LDBFactory, LDBKVStore, LDBVersionedStore} trait DBSpec extends TestFileUtils { @@ -21,19 +20,19 @@ trait DBSpec extends TestFileUtils { protected def byteString32(s: String): Array[Byte] = Algos.hash(byteString(s)) - protected def withDb[T](body: DB => T): T = { + protected def withDb[T](body: RocksDB => T): T = { val options = new Options() - options.createIfMissing(true) - options.verifyChecksums(true) - options.maxOpenFiles(2000) - val db = factory.open(createTempDir, options) + options.setCreateIfMissing(true) + options.setParanoidChecks(true) + options.setMaxOpenFiles(2000) + val db = LDBFactory.openDb(createTempDir, options) try body(db) finally db.close() } protected def versionId(s: String): Array[Byte] = byteString32(s) protected def withStore[T](body: LDBKVStore => T): T = - withDb { db: DB => body(new LDBKVStore(db)) } + withDb { db: RocksDB => body(new LDBKVStore(db)) } protected def withVersionedStore[T](keepVersions: Int)(body: LDBVersionedStore => T): T = { val db = new LDBVersionedStore(createTempDir, keepVersions) diff --git a/src/test/scala/org/ergoplatform/db/LDBKVStoreSpec.scala b/src/test/scala/org/ergoplatform/db/LDBKVStoreSpec.scala index cd32b21c29..f43452160a 100644 --- a/src/test/scala/org/ergoplatform/db/LDBKVStoreSpec.scala +++ b/src/test/scala/org/ergoplatform/db/LDBKVStoreSpec.scala @@ -40,25 +40,4 @@ class LDBKVStoreSpec extends AnyPropSpec with Matchers with DBSpec { } } - property("last key in range") { - withStore { store => - val valueA = (byteString("A"), byteString("1")) - val valueB = (byteString("B"), byteString("2")) - val valueC = (byteString("C"), byteString("1")) - val valueD = (byteString("D"), byteString("2")) - val valueE = (byteString("E"), byteString("3")) - val valueF = (byteString("F"), byteString("4")) - - val values = Array(valueA, valueB, valueC, valueD, valueE, valueF) - store.insert(values.map(_._1), values.map(_._2)).get - - store.lastKeyInRange(valueA._1, valueC._1).get.toSeq shouldBe valueC._1.toSeq - store.lastKeyInRange(valueD._1, valueF._1).get.toSeq shouldBe valueF._1.toSeq - store.lastKeyInRange(valueF._1, byteString32("Z")).get.toSeq shouldBe valueF._1.toSeq - store.lastKeyInRange(Array(10: Byte), valueA._1).get.toSeq shouldBe valueA._1.toSeq - - store.lastKeyInRange(Array(10: Byte), Array(11: Byte)).isDefined shouldBe false - } - } - } From 378afbbb01fd959c591b66575063b9d8caa0fe86 Mon Sep 17 00:00:00 2001 From: jellymlg Date: Thu, 18 Jan 2024 05:38:09 +0100 Subject: [PATCH 02/15] Fixed tests, refactored db factory, added missing release for snapshots --- .../main/scala/scorex/db/KVStoreReader.scala | 35 +++--- .../src/main/scala/scorex/db/LDBFactory.scala | 106 ++++++------------ .../src/main/scala/scorex/db/LDBKVStore.scala | 12 +- .../scala/scorex/db/LDBVersionedStore.scala | 35 +++--- .../scorex/db/LDBVersionedStoreSpec.scala | 6 +- .../scala/org/ergoplatform/db/DBSpec.scala | 9 +- 6 files changed, 81 insertions(+), 122 deletions(-) diff --git a/avldb/src/main/scala/scorex/db/KVStoreReader.scala b/avldb/src/main/scala/scorex/db/KVStoreReader.scala index d285c39c97..7d8f1a1236 100644 --- a/avldb/src/main/scala/scorex/db/KVStoreReader.scala +++ b/avldb/src/main/scala/scorex/db/KVStoreReader.scala @@ -1,8 +1,9 @@ package scorex.db -import java.util.concurrent.locks.ReentrantReadWriteLock -import org.rocksdb.{ReadOptions, RocksDB} +import org.rocksdb.ReadOptions +import scorex.db.LDBFactory.RegisteredDB +import java.util.concurrent.locks.ReentrantReadWriteLock import scala.collection.mutable /** @@ -14,7 +15,7 @@ trait KVStoreReader extends AutoCloseable { type K = Array[Byte] type V = Array[Byte] - protected val db: RocksDB + protected val db: RegisteredDB protected val lock = new ReentrantReadWriteLock() @@ -32,38 +33,29 @@ trait KVStoreReader extends AutoCloseable { } } - /** - * Iterate through the database to read elements according to a filter function. - * @param cond - the filter function - * @return iterator over elements satisfying the filter function + * Read all the database elements. + * @return iterator over database contents */ - def getWithFilter(cond: (K, V) => Boolean): Iterator[(K, V)] = { + def getAll: Iterator[(K, V)] = { val ro = new ReadOptions() ro.setSnapshot(db.getSnapshot) - val iter = db.newIterator(ro) + val iter = db.iterator(ro) try { iter.seekToFirst() val bf = mutable.ArrayBuffer.empty[(K, V)] while (iter.isValid) { - val key = iter.key() - val value = iter.value() - if (cond(key, value)) bf += (key -> value) + bf += (iter.key() -> iter.value()) iter.next() } bf.toIterator } finally { iter.close() - ro.snapshot().close() + db.releaseSnapshot(ro.snapshot()) + ro.close() } } - /** - * Read all the database elements. - * @return iterator over database contents - */ - def getAll: Iterator[(K, V)] = getWithFilter((_, _) => true) - /** Returns value associated with the key, or default value from user */ def getOrElse(key: K, default: => V): V = @@ -97,7 +89,7 @@ trait KVStoreReader extends AutoCloseable { def getRange(start: K, end: K, limit: Int = Int.MaxValue): Array[(K, V)] = { val ro = new ReadOptions() ro.setSnapshot(db.getSnapshot) - val iter = db.newIterator(ro) + val iter = db.iterator(ro) try { iter.seek(start) val bf = mutable.ArrayBuffer.empty[(K, V)] @@ -112,7 +104,8 @@ trait KVStoreReader extends AutoCloseable { bf.toArray[(K,V)] } finally { iter.close() - ro.snapshot().close() + db.releaseSnapshot(ro.snapshot()) + ro.close() } } diff --git a/avldb/src/main/scala/scorex/db/LDBFactory.scala b/avldb/src/main/scala/scorex/db/LDBFactory.scala index 9b3c93811b..adf255313e 100644 --- a/avldb/src/main/scala/scorex/db/LDBFactory.scala +++ b/avldb/src/main/scala/scorex/db/LDBFactory.scala @@ -1,88 +1,63 @@ package scorex.db +import org.rocksdb._ import org.rocksdb.util.SizeUnit +import scorex.util.ScorexLogging import java.io.File import java.util.concurrent.locks.ReentrantReadWriteLock -import org.rocksdb.{CompactionStyle, CompressionType, Options, ReadOptions, RocksDB, RocksIterator, Snapshot, WriteBatch, WriteOptions} -import scorex.util.ScorexLogging - import scala.collection.mutable /** - * Registry of opened LevelDB instances. - * LevelDB prohibit access to the same storage file from more than one DB instance. + * Registry of opened RocksDB instances. + * RocksDB prohibit access to the same storage file from more than one DB instance. * And ergo application (mostly tests) quite frequently doesn't not explicitly close * database and tries to reopen it. */ -case class StoreRegistry() extends ScorexLogging { +object LDBFactory extends ScorexLogging { + + RocksDB.loadLibrary() - val lock = new ReentrantReadWriteLock() - val map = new mutable.HashMap[File, RegisteredDB] + private val lock = new ReentrantReadWriteLock() + private val map = new mutable.HashMap[File, RegisteredDB] /** - * Decorator of LevelDB DB class which overrides close() methods and unlinks database from registry on close. - * So if database was not explicitly closed, then next attempt to open database with the same path will - * return existed instance instead of creating new one. - */ + * Decorator of RocksDB class which overrides close() methods and unlinks database from registry on close. + * So if database was not explicitly closed, then next attempt to open database with the same path will + * return existed instance instead of creating new one. + */ case class RegisteredDB(impl: RocksDB, path: File) { - def get(key: Array[Byte]): Array[Byte] = impl.get(key) - - def get(key: Array[Byte], options: ReadOptions): Array[Byte] = impl.get(options, key) - + def get(options: ReadOptions, key: Array[Byte]): Array[Byte] = impl.get(options, key) def iterator: RocksIterator = impl.newIterator() - def iterator(options: ReadOptions): RocksIterator = impl.newIterator(options) - def put(key: Array[Byte], value: Array[Byte]): Unit = impl.put(key, value) - - def delete(key: Array[Byte]): Unit = impl.delete(key) - - def write(batch: WriteBatch): Unit = impl.write(new WriteOptions(), batch) - - def write(batch: WriteBatch, options: WriteOptions): Unit = impl.write(options, batch) - - def createWriteBatch: WriteBatch = new WriteBatch() - - def put(key: Array[Byte], value: Array[Byte], options: WriteOptions): Unit = impl.put(options, key, value) - - def delete(key: Array[Byte], options: WriteOptions): Unit = impl.delete(options, key) - + def write(options: WriteOptions, batch: WriteBatch): Unit = impl.write(options, batch) def getSnapshot: Snapshot = impl.getSnapshot - - def getProperty(name: String): String = impl.getProperty(name) - - def compactRange(begin: Array[Byte], end: Array[Byte]): Unit = impl.compactRange(begin, end) - + def releaseSnapshot(snapshot: Snapshot): Unit = impl.releaseSnapshot(snapshot) def close(): Unit = { - remove(path) - impl.close() - } - } - - private def add(file: File, create: => RocksDB): RegisteredDB = { - lock.writeLock().lock() - try { - map.getOrElseUpdate(file, RegisteredDB(create, file)) - } finally { - lock.writeLock().unlock() - } - } - - private def remove(path: File): Option[RegisteredDB] = { - lock.writeLock().lock() - try { - map.remove(path) - } finally { - lock.writeLock().unlock() + lock.writeLock().lock() + try { + map.remove(path) + impl.syncWal() + impl.close() + } finally { + lock.writeLock().unlock() + } } } def open(path: File, options: Options): RegisteredDB = { lock.writeLock().lock() try { - add(path, LDBFactory.openDb(path, options)) + path.mkdirs() + options.setCreateIfMissing(true) + .setWriteBufferSize(32 * SizeUnit.MB) + .setAllowMmapReads(true) + .setIncreaseParallelism(4) + .setCompressionType(CompressionType.LZ4_COMPRESSION) + .setCompactionStyle(CompactionStyle.UNIVERSAL) + map.getOrElseUpdate(path, RegisteredDB(RocksDB.open(options, path.toString), path)) } catch { case x: Throwable => log.error(s"Failed to initialize storage: $x. Please check that directory $path exists and is not used by some other active node") @@ -93,24 +68,9 @@ case class StoreRegistry() extends ScorexLogging { } } -} - -object LDBFactory extends ScorexLogging { - - def openDb(path: File, options: Options = new Options()): RocksDB = { - path.mkdirs() - options.setCreateIfMissing(true) - .setWriteBufferSize(128 * SizeUnit.MB) - .setMaxWriteBufferNumber(3) - .setMaxBackgroundJobs(10) - .setCompressionType(CompressionType.NO_COMPRESSION) - .setCompactionStyle(CompactionStyle.LEVEL) - RocksDB.open(options, path.toString) - } - def createKvDb(path: File): LDBKVStore = { try { - new LDBKVStore(openDb(path)) + new LDBKVStore(open(path, new Options())) } catch { case x: Throwable => log.error(s"Failed to initialize storage: $x. Please check that directory $path could be accessed " + diff --git a/avldb/src/main/scala/scorex/db/LDBKVStore.scala b/avldb/src/main/scala/scorex/db/LDBKVStore.scala index a63d0829c9..af50bb3150 100644 --- a/avldb/src/main/scala/scorex/db/LDBKVStore.scala +++ b/avldb/src/main/scala/scorex/db/LDBKVStore.scala @@ -1,10 +1,11 @@ package scorex.db -import org.rocksdb.{RocksDB, WriteBatch, WriteOptions} +import org.rocksdb.{WriteBatch, WriteOptions} +import scorex.db.LDBFactory.RegisteredDB import scorex.util.ScorexLogging +import spire.syntax.all.cfor import scala.util.{Failure, Success, Try} -import spire.syntax.all.cfor /** @@ -12,10 +13,13 @@ import spire.syntax.all.cfor * * Both keys and values are var-sized byte arrays. */ -class LDBKVStore(protected val db: RocksDB) extends KVStoreReader with ScorexLogging { +class LDBKVStore(protected val db: RegisteredDB) extends KVStoreReader with ScorexLogging { /** Immutable empty array can be shared to avoid allocations. */ private val emptyArrayOfByteArray = Array.empty[Array[Byte]] + /** default write options, snyc enabled */ + private val wo: WriteOptions = new WriteOptions().setSync(true) + /** * Update this database atomically with a batch of insertion and removal operations * @@ -30,7 +34,7 @@ class LDBKVStore(protected val db: RocksDB) extends KVStoreReader with ScorexLog require(toInsertKeys.length == toInsertValues.length) cfor(0)(_ < toInsertKeys.length, _ + 1) { i => batch.put(toInsertKeys(i), toInsertValues(i))} cfor(0)(_ < toRemove.length, _ + 1) { i => batch.delete(toRemove(i))} - db.write(new WriteOptions(), batch) + db.write(wo, batch) Success(()) } catch { case t: Throwable => Failure(t) diff --git a/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala b/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala index 213a04115b..0a6639637a 100644 --- a/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala +++ b/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala @@ -1,15 +1,15 @@ package scorex.db -import java.io.File -import org.rocksdb.{Options, ReadOptions, RocksDB, WriteBatch, WriteOptions} - -import java.nio.ByteBuffer -import scala.collection.mutable.ArrayBuffer -import java.util.concurrent.locks.ReentrantReadWriteLock +import org.rocksdb.{Options, ReadOptions, WriteBatch, WriteOptions} import scorex.crypto.hash.Blake2b256 +import scorex.db.LDBFactory.RegisteredDB import scorex.db.LDBVersionedStore.SnapshotReadInterface import scorex.util.ScorexLogging +import java.io.File +import java.nio.ByteBuffer +import java.util.concurrent.locks.ReentrantReadWriteLock +import scala.collection.mutable.ArrayBuffer import scala.util.{Failure, Success, Try} @@ -36,10 +36,10 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) private var keepVersions: Int = initialKeepVersions - override val db: RocksDB = createDB(dir, "ldb_main") // storage for main data + override val db: RegisteredDB = createDB(dir, "ldb_main") // storage for main data override val lock = new ReentrantReadWriteLock() - private val undo: RocksDB = createDB(dir, "ldb_undo") // storage for undo data + private val undo: RegisteredDB = createDB(dir, "ldb_undo") // storage for undo data private var lsn: LSN = getLastLSN // last assigned logical serial number private var versionLsn = ArrayBuffer.empty[LSN] // LSNs of versions (var because we need to invert this array) @@ -51,11 +51,11 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) //default write options, no sync! private val writeOptions = new WriteOptions() - private def createDB(dir: File, storeName: String): RocksDB = { + private def createDB(dir: File, storeName: String): RegisteredDB = { val op = new Options() - op.setCreateIfMissing(true) - op.setParanoidChecks(true) - LDBFactory.openDb(new File(dir, storeName), op) + .setCreateIfMissing(true) + .setParanoidChecks(true) + LDBFactory.open(new File(dir, storeName), op) } /** Set new keep versions threshold, remove not needed versions and return old value of keep versions */ @@ -101,7 +101,7 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) def processAll(consumer: (K, V) => Unit): Unit = { lock.readLock().lock() - val iterator = db.newIterator() + val iterator = db.iterator try { iterator.seekToFirst() while (iterator.isValid) { @@ -134,7 +134,7 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) } private def getLastLSN: LSN = { - val iterator = undo.newIterator() + val iterator = undo.iterator try { iterator.seekToFirst() if (iterator.isValid) { @@ -165,7 +165,7 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) var lastVersion: Option[VersionID] = None var lastLsn: LSN = 0 // We iterate in LSN descending order - val iterator = undo.newIterator() + val iterator = undo.iterator iterator.seekToFirst() while (iterator.isValid) { val currVersion = deserializeUndo(iterator.value()).versionID @@ -351,7 +351,7 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) val batch = new WriteBatch() val undoBatch = new WriteBatch() var nUndoRecords: Long = 0 - val iterator = undo.newIterator() + val iterator = undo.iterator var lastLsn: LSN = 0 try { var undoing = true @@ -429,7 +429,8 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) Failure(t) } finally { // Close the snapshot to avoid resource leaks - ro.snapshot().close() + db.releaseSnapshot(ro.snapshot()) + ro.close() } } diff --git a/avldb/src/test/scala/scorex/db/LDBVersionedStoreSpec.scala b/avldb/src/test/scala/scorex/db/LDBVersionedStoreSpec.scala index 59863452ab..9a7bfa4f70 100644 --- a/avldb/src/test/scala/scorex/db/LDBVersionedStoreSpec.scala +++ b/avldb/src/test/scala/scorex/db/LDBVersionedStoreSpec.scala @@ -44,7 +44,7 @@ class LDBVersionedStoreSpec extends AnyPropSpec with Matchers { store.rollbackVersions().toSeq.map(_.toSeq) shouldBe Seq(versionId3.toSeq, versionId2.toSeq, versionId.toSeq) } - property("processAll && getWithFilter") { + property("processAll && getAll") { val version = Longs.toByteArray(Long.MaxValue) val k1 = Longs.toByteArray(Int.MaxValue + 1) val k2 = Longs.toByteArray(Int.MaxValue + 2) @@ -53,7 +53,7 @@ class LDBVersionedStoreSpec extends AnyPropSpec with Matchers { store.update(version, Seq.empty, Seq(k1 -> v1, k2 -> v2)).get //read all keys - val keys = store.getWithFilter((_, _) => true).toSeq.map(_._1) + val keys = store.getAll.toSeq.map(_._1) val ks = keys.map(_.toSeq) ks.contains(k1.toSeq) shouldBe true @@ -74,7 +74,7 @@ class LDBVersionedStoreSpec extends AnyPropSpec with Matchers { store.update(Longs.toByteArray(Long.MinValue), keys, Seq.empty).get - store.getWithFilter((_, _) => true).toSeq.length shouldBe 0 + store.getAll.toSeq.length shouldBe 0 var cnt = 0 store.processAll({ case (_, _) => diff --git a/src/test/scala/org/ergoplatform/db/DBSpec.scala b/src/test/scala/org/ergoplatform/db/DBSpec.scala index e8df7f0db0..c36d244858 100644 --- a/src/test/scala/org/ergoplatform/db/DBSpec.scala +++ b/src/test/scala/org/ergoplatform/db/DBSpec.scala @@ -3,7 +3,8 @@ package org.ergoplatform.db import akka.util.ByteString import org.ergoplatform.settings.Algos import org.ergoplatform.wallet.utils.TestFileUtils -import org.rocksdb.{Options, RocksDB} +import org.rocksdb.Options +import scorex.db.LDBFactory.RegisteredDB import scorex.db.{LDBFactory, LDBKVStore, LDBVersionedStore} trait DBSpec extends TestFileUtils { @@ -20,19 +21,19 @@ trait DBSpec extends TestFileUtils { protected def byteString32(s: String): Array[Byte] = Algos.hash(byteString(s)) - protected def withDb[T](body: RocksDB => T): T = { + protected def withDb[T](body: RegisteredDB => T): T = { val options = new Options() options.setCreateIfMissing(true) options.setParanoidChecks(true) options.setMaxOpenFiles(2000) - val db = LDBFactory.openDb(createTempDir, options) + val db = LDBFactory.open(createTempDir, options) try body(db) finally db.close() } protected def versionId(s: String): Array[Byte] = byteString32(s) protected def withStore[T](body: LDBKVStore => T): T = - withDb { db: RocksDB => body(new LDBKVStore(db)) } + withDb { db: RegisteredDB => body(new LDBKVStore(db)) } protected def withVersionedStore[T](keepVersions: Int)(body: LDBVersionedStore => T): T = { val db = new LDBVersionedStore(createTempDir, keepVersions) From 22e878d764f8d3dc064ceb68b8920f377cbee569 Mon Sep 17 00:00:00 2001 From: jellymlg Date: Fri, 19 Jan 2024 05:22:14 +0100 Subject: [PATCH 03/15] Fixed tests db size problem --- .../main/scala/scorex/db/KVStoreReader.scala | 14 +++++++ .../src/main/scala/scorex/db/LDBFactory.scala | 40 +++++++++---------- .../scala/scorex/db/LDBVersionedStore.scala | 10 ++--- .../VersionedLDBAVLStorageSpecification.scala | 4 +- build.sbt | 1 + .../wallet/utils/TestFileUtils.scala | 11 +---- .../network/peer/PeerDatabase.scala | 4 +- .../history/storage/HistoryStorage.scala | 13 +++--- .../nodeView/state/SnapshotsDb.scala | 2 +- .../wallet/persistence/WalletStorage.scala | 2 +- .../scala/org/ergoplatform/db/DBSpec.scala | 7 +--- 11 files changed, 54 insertions(+), 54 deletions(-) diff --git a/avldb/src/main/scala/scorex/db/KVStoreReader.scala b/avldb/src/main/scala/scorex/db/KVStoreReader.scala index 7d8f1a1236..cc37af3f8b 100644 --- a/avldb/src/main/scala/scorex/db/KVStoreReader.scala +++ b/avldb/src/main/scala/scorex/db/KVStoreReader.scala @@ -33,6 +33,20 @@ trait KVStoreReader extends AutoCloseable { } } + /** + * Query if database contains key + * @param key - key + * @return true if key exists, false otherwise + */ + def contains(key: K): Boolean = { + lock.readLock().lock() + try { + db.contains(key) + } finally { + lock.readLock().unlock() + } + } + /** * Read all the database elements. * @return iterator over database contents diff --git a/avldb/src/main/scala/scorex/db/LDBFactory.scala b/avldb/src/main/scala/scorex/db/LDBFactory.scala index adf255313e..a0cec503ef 100644 --- a/avldb/src/main/scala/scorex/db/LDBFactory.scala +++ b/avldb/src/main/scala/scorex/db/LDBFactory.scala @@ -29,6 +29,7 @@ object LDBFactory extends ScorexLogging { case class RegisteredDB(impl: RocksDB, path: File) { def get(key: Array[Byte]): Array[Byte] = impl.get(key) def get(options: ReadOptions, key: Array[Byte]): Array[Byte] = impl.get(options, key) + def contains(key: Array[Byte]): Boolean = impl.keyExists(key) def iterator: RocksIterator = impl.newIterator() def iterator(options: ReadOptions): RocksIterator = impl.newIterator(options) def put(key: Array[Byte], value: Array[Byte]): Unit = impl.put(key, value) @@ -39,7 +40,8 @@ object LDBFactory extends ScorexLogging { lock.writeLock().lock() try { map.remove(path) - impl.syncWal() + impl.flushWal(true) + impl.cancelAllBackgroundWork(true) impl.close() } finally { lock.writeLock().unlock() @@ -47,16 +49,26 @@ object LDBFactory extends ScorexLogging { } } - def open(path: File, options: Options): RegisteredDB = { + private val normalOptions: Options = new Options() + .setCreateIfMissing(true) + .setWriteBufferSize(32 * SizeUnit.MB) + .setAllowMmapReads(true) + .setIncreaseParallelism(4) + .setCompressionType(CompressionType.LZ4_COMPRESSION) + .setCompactionStyle(CompactionStyle.LEVEL) + + private val testOptions: Options = new Options() + .setCreateIfMissing(true) + .setWriteBufferSize(64 * SizeUnit.KB) + .setManifestPreallocationSize(32 * SizeUnit.KB) + .setCompressionType(CompressionType.LZ4_COMPRESSION) + .setCompactionStyle(CompactionStyle.LEVEL) + + def open(path: File): RegisteredDB = { lock.writeLock().lock() try { path.mkdirs() - options.setCreateIfMissing(true) - .setWriteBufferSize(32 * SizeUnit.MB) - .setAllowMmapReads(true) - .setIncreaseParallelism(4) - .setCompressionType(CompressionType.LZ4_COMPRESSION) - .setCompactionStyle(CompactionStyle.UNIVERSAL) + val options = if(System.getProperty("dbTest") == null) normalOptions else testOptions map.getOrElseUpdate(path, RegisteredDB(RocksDB.open(options, path.toString), path)) } catch { case x: Throwable => @@ -68,16 +80,4 @@ object LDBFactory extends ScorexLogging { } } - def createKvDb(path: File): LDBKVStore = { - try { - new LDBKVStore(open(path, new Options())) - } catch { - case x: Throwable => - log.error(s"Failed to initialize storage: $x. Please check that directory $path could be accessed " + - s"and is not used by some other active node") - java.lang.System.exit(2) - null - } - } - } diff --git a/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala b/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala index 0a6639637a..36f0cdaff1 100644 --- a/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala +++ b/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala @@ -1,6 +1,6 @@ package scorex.db -import org.rocksdb.{Options, ReadOptions, WriteBatch, WriteOptions} +import org.rocksdb.{ReadOptions, WriteBatch, WriteOptions} import scorex.crypto.hash.Blake2b256 import scorex.db.LDBFactory.RegisteredDB import scorex.db.LDBVersionedStore.SnapshotReadInterface @@ -51,12 +51,8 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) //default write options, no sync! private val writeOptions = new WriteOptions() - private def createDB(dir: File, storeName: String): RegisteredDB = { - val op = new Options() - .setCreateIfMissing(true) - .setParanoidChecks(true) - LDBFactory.open(new File(dir, storeName), op) - } + private def createDB(dir: File, storeName: String): RegisteredDB = + LDBFactory.open(new File(dir, storeName)) /** Set new keep versions threshold, remove not needed versions and return old value of keep versions */ def setKeepVersions(newKeepVersions: Int): Int = { diff --git a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageSpecification.scala b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageSpecification.scala index 5cd577b491..605609f108 100644 --- a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageSpecification.scala +++ b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageSpecification.scala @@ -11,7 +11,7 @@ import scorex.crypto.authds.avltree.batch.helpers.TestHelper import scorex.crypto.authds.{ADDigest, ADKey, ADValue, SerializedAdProof} import scorex.util.encode.Base16 import scorex.crypto.hash.{Blake2b256, Digest32} -import scorex.db.{LDBFactory, LDBVersionedStore} +import scorex.db.{LDBFactory, LDBKVStore, LDBVersionedStore} import scorex.util.ByteArrayBuilder import scorex.util.serialization.VLQByteBufferWriter import scorex.utils.{Random => RandomBytes} @@ -350,7 +350,7 @@ class VersionedLDBAVLStorageSpecification blockchainWorkflowTest(prover) val storage = prover.storage.asInstanceOf[VersionedLDBAVLStorage] - val store = LDBFactory.createKvDb(getRandomTempDir) + val store = new LDBKVStore(LDBFactory.open(getRandomTempDir)) val rootNodeLabel = storage.dumpSnapshot(store, manifestDepth, prover.digest.dropRight(1)).get rootNodeLabel.sameElements(prover.digest.dropRight(1)) shouldBe true diff --git a/build.sbt b/build.sbt index 3db46e6646..8436431b93 100644 --- a/build.sbt +++ b/build.sbt @@ -200,6 +200,7 @@ scapegoatVersion in ThisBuild := "1.3.3" scapegoatDisabledInspections := Seq("FinalModifierOnCaseClass") Test / testOptions := Seq(Tests.Filter(s => !s.endsWith("Bench"))) +Test / javaOptions := Seq("-DdbTest=true") lazy val avldb = (project in file("avldb")) .disablePlugins(ScapegoatSbtPlugin) // not compatible with crossScalaVersions diff --git a/ergo-wallet/src/test/scala/org/ergoplatform/wallet/utils/TestFileUtils.scala b/ergo-wallet/src/test/scala/org/ergoplatform/wallet/utils/TestFileUtils.scala index 3d29330cd6..bcd959fbf4 100644 --- a/ergo-wallet/src/test/scala/org/ergoplatform/wallet/utils/TestFileUtils.scala +++ b/ergo-wallet/src/test/scala/org/ergoplatform/wallet/utils/TestFileUtils.scala @@ -1,12 +1,3 @@ package org.ergoplatform.wallet.utils -import java.nio.file.Path - -trait TestFileUtils extends FileUtils { - - val basePath: Path = java.nio.file.Files.createTempDirectory(s"scorex-${System.nanoTime()}") - - sys.addShutdownHook { - deleteRecursive(basePath.toFile) - } -} +trait TestFileUtils extends FileUtils diff --git a/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala b/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala index 1c3e833e96..ef99cdbd2a 100644 --- a/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala +++ b/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala @@ -5,7 +5,7 @@ import org.ergoplatform.nodeView.history.ErgoHistoryUtils._ import java.io.{ByteArrayInputStream, ByteArrayOutputStream, File, ObjectInputStream, ObjectOutputStream} import java.net.{InetAddress, InetSocketAddress} import org.ergoplatform.settings.ErgoSettings -import scorex.db.LDBFactory +import scorex.db.{LDBFactory, LDBKVStore} import scorex.util.ScorexLogging import scala.concurrent.duration._ @@ -16,7 +16,7 @@ import scala.util.{Failure, Success, Try} */ final class PeerDatabase(settings: ErgoSettings) extends ScorexLogging { - private val persistentStore = LDBFactory.createKvDb(new File(s"${settings.directory}/peers")) + private val persistentStore = new LDBKVStore(LDBFactory.open(new File(s"${settings.directory}/peers"))) private var peers = loadPeers match { diff --git a/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala b/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala index feac7eb031..bf11d3d23a 100644 --- a/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala +++ b/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala @@ -125,8 +125,11 @@ class HistoryStorage private(indexStore: LDBKVStore, objectsStore: LDBKVStore, e /** * @return if object with `id` is in the objects database */ - def contains(id: Array[Byte]): Boolean = get(id).isDefined - def contains(id: ModifierId): Boolean = get(id).isDefined + def contains(id: Array[Byte]): Boolean = objectsStore.contains(id) || extraStore.contains(id) + def contains(id: ModifierId): Boolean = { + val idBytes = idToBytes(id) + objectsStore.contains(idBytes) || extraStore.contains(idBytes) + } def insert(indexesToInsert: Array[(ByteArrayWrapper, Array[Byte])], objectsToInsert: Array[BlockSection]): Try[Unit] = { @@ -226,9 +229,9 @@ class HistoryStorage private(indexStore: LDBKVStore, objectsStore: LDBKVStore, e object HistoryStorage { def apply(ergoSettings: ErgoSettings): HistoryStorage = { - val indexStore = LDBFactory.createKvDb(new File(s"${ergoSettings.directory}/history/index")) - val objectsStore = LDBFactory.createKvDb(new File(s"${ergoSettings.directory}/history/objects")) - val extraStore = LDBFactory.createKvDb(new File(s"${ergoSettings.directory}/history/extra")) + val indexStore = new LDBKVStore(LDBFactory.open(new File(s"${ergoSettings.directory}/history/index"))) + val objectsStore = new LDBKVStore(LDBFactory.open(new File(s"${ergoSettings.directory}/history/objects"))) + val extraStore = new LDBKVStore(LDBFactory.open(new File(s"${ergoSettings.directory}/history/extra"))) new HistoryStorage(indexStore, objectsStore, extraStore, ergoSettings.cacheSettings) } } diff --git a/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala b/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala index 839d66a644..bf28fe47db 100644 --- a/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala +++ b/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala @@ -142,7 +142,7 @@ object SnapshotsDb { // internal method to open or init snapshots database in given folder // private[nodeView] to use it in tests also private[nodeView] def create(dir: String): SnapshotsDb = { - val store = LDBFactory.createKvDb(new File(dir)) + val store = new LDBKVStore(LDBFactory.open(new File(dir))) new SnapshotsDb(store) } diff --git a/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala b/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala index 5263a039aa..21e46ff2c6 100644 --- a/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala +++ b/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala @@ -245,7 +245,7 @@ object WalletStorage { def storageFolder(settings: ErgoSettings): File = new File(s"${settings.directory}/wallet/storage") def readOrCreate(settings: ErgoSettings): WalletStorage = { - val db = LDBFactory.createKvDb(storageFolder(settings)) + val db = new LDBKVStore(LDBFactory.open(storageFolder(settings))) new WalletStorage(db, settings) } diff --git a/src/test/scala/org/ergoplatform/db/DBSpec.scala b/src/test/scala/org/ergoplatform/db/DBSpec.scala index c36d244858..eac623f878 100644 --- a/src/test/scala/org/ergoplatform/db/DBSpec.scala +++ b/src/test/scala/org/ergoplatform/db/DBSpec.scala @@ -3,7 +3,6 @@ package org.ergoplatform.db import akka.util.ByteString import org.ergoplatform.settings.Algos import org.ergoplatform.wallet.utils.TestFileUtils -import org.rocksdb.Options import scorex.db.LDBFactory.RegisteredDB import scorex.db.{LDBFactory, LDBKVStore, LDBVersionedStore} @@ -22,11 +21,7 @@ trait DBSpec extends TestFileUtils { protected def byteString32(s: String): Array[Byte] = Algos.hash(byteString(s)) protected def withDb[T](body: RegisteredDB => T): T = { - val options = new Options() - options.setCreateIfMissing(true) - options.setParanoidChecks(true) - options.setMaxOpenFiles(2000) - val db = LDBFactory.open(createTempDir, options) + val db = LDBFactory.open(createTempDir) try body(db) finally db.close() } From 25586a4ce286e581b4782125f11e7172231e8a3b Mon Sep 17 00:00:00 2001 From: jellymlg Date: Fri, 19 Jan 2024 16:07:53 +0100 Subject: [PATCH 04/15] Refactored tests to use temp folder --- avldb/src/main/scala/scorex/db/LDBFactory.scala | 4 ++-- build.sbt | 2 +- .../org/ergoplatform/wallet/utils/FileUtils.scala | 13 ++----------- .../wallet/secrets/JsonSecretStorageSpec.scala | 8 ++++---- src/main/scala/org/ergoplatform/ErgoApp.scala | 2 +- .../reemission/ReemissionRulesUtils.scala | 2 +- .../ergoplatform/settings/ErgoSettingsReader.scala | 11 ++++++++--- src/test/scala/org/ergoplatform/db/DBSpec.scala | 4 ++-- .../http/routes/EmissionApiRouteSpec.scala | 2 ++ .../org/ergoplatform/mining/ErgoMinerSpec.scala | 12 ++++++------ .../ErgoNodeViewSynchronizerSpecification.scala | 2 +- .../nodeView/NodeViewSynchronizerTests.scala | 2 +- .../UtxoSetSnapshotProcessorSpecification.scala | 2 +- .../history/extra/ExtraIndexerTestActor.scala | 2 +- .../ergoplatform/nodeView/mempool/ScriptsSpec.scala | 2 +- .../nodeView/state/DigestStateSpecification.scala | 2 +- .../nodeView/state/ErgoStateSpecification.scala | 2 +- .../nodeView/state/SnapshotsDbSpecification.scala | 2 +- .../org/ergoplatform/sanity/ErgoSanityDigest.scala | 4 ++-- .../org/ergoplatform/sanity/ErgoSanityUTXO.scala | 2 +- .../settings/ErgoSettingsSpecification.scala | 5 ----- .../tools/DifficultyControlSimulator.scala | 3 ++- .../org/ergoplatform/utils/ErgoTestConstants.scala | 3 ++- .../org/ergoplatform/utils/HistoryTestHelpers.scala | 2 +- .../org/ergoplatform/utils/NodeViewTestConfig.scala | 3 ++- src/test/scala/org/ergoplatform/utils/Stubs.scala | 8 ++++---- .../utils/fixtures/NodeViewFixture.scala | 2 +- .../utils/generators/ValidBlocksGenerators.scala | 6 +++--- 28 files changed, 55 insertions(+), 59 deletions(-) diff --git a/avldb/src/main/scala/scorex/db/LDBFactory.scala b/avldb/src/main/scala/scorex/db/LDBFactory.scala index a0cec503ef..143b01b4d6 100644 --- a/avldb/src/main/scala/scorex/db/LDBFactory.scala +++ b/avldb/src/main/scala/scorex/db/LDBFactory.scala @@ -40,7 +40,7 @@ object LDBFactory extends ScorexLogging { lock.writeLock().lock() try { map.remove(path) - impl.flushWal(true) + impl.syncWal() impl.cancelAllBackgroundWork(true) impl.close() } finally { @@ -68,7 +68,7 @@ object LDBFactory extends ScorexLogging { lock.writeLock().lock() try { path.mkdirs() - val options = if(System.getProperty("dbTest") == null) normalOptions else testOptions + val options = if(System.getProperty("env") == "test") testOptions else normalOptions map.getOrElseUpdate(path, RegisteredDB(RocksDB.open(options, path.toString), path)) } catch { case x: Throwable => diff --git a/build.sbt b/build.sbt index 8436431b93..afa3255ebf 100644 --- a/build.sbt +++ b/build.sbt @@ -200,7 +200,7 @@ scapegoatVersion in ThisBuild := "1.3.3" scapegoatDisabledInspections := Seq("FinalModifierOnCaseClass") Test / testOptions := Seq(Tests.Filter(s => !s.endsWith("Bench"))) -Test / javaOptions := Seq("-DdbTest=true") +Test / javaOptions := Seq("-Denv=test") lazy val avldb = (project in file("avldb")) .disablePlugins(ScapegoatSbtPlugin) // not compatible with crossScalaVersions diff --git a/ergo-wallet/src/main/scala/org/ergoplatform/wallet/utils/FileUtils.scala b/ergo-wallet/src/main/scala/org/ergoplatform/wallet/utils/FileUtils.scala index 898116dcd2..335262860a 100644 --- a/ergo-wallet/src/main/scala/org/ergoplatform/wallet/utils/FileUtils.scala +++ b/ergo-wallet/src/main/scala/org/ergoplatform/wallet/utils/FileUtils.scala @@ -1,7 +1,7 @@ package org.ergoplatform.wallet.utils import java.io.File -import java.nio.file.{Files, Path} +import java.nio.file.Files import scala.collection.JavaConverters._ import scala.util.Try @@ -18,16 +18,7 @@ trait FileUtils { } } - def createTempFile: java.io.File = { - val dir = createTempDir - val prefix = scala.util.Random.alphanumeric.take(randomPrefixLength).mkString - val suffix = scala.util.Random.alphanumeric.take(randomPrefixLength).mkString - val file = java.nio.file.Files.createTempFile(dir.toPath, prefix, suffix).toFile - file.deleteOnExit() - file - } - - def createTempDir: java.io.File = { + implicit def createTempDir(nothing: Int = 0): java.io.File = { val rndString = scala.util.Random.alphanumeric.take(randomPrefixLength).mkString createTempDirForPrefix(rndString) } diff --git a/ergo-wallet/src/test/scala/org/ergoplatform/wallet/secrets/JsonSecretStorageSpec.scala b/ergo-wallet/src/test/scala/org/ergoplatform/wallet/secrets/JsonSecretStorageSpec.scala index 39317270a6..e58efd56c6 100644 --- a/ergo-wallet/src/test/scala/org/ergoplatform/wallet/secrets/JsonSecretStorageSpec.scala +++ b/ergo-wallet/src/test/scala/org/ergoplatform/wallet/secrets/JsonSecretStorageSpec.scala @@ -22,7 +22,7 @@ class JsonSecretStorageSpec property("initialization and unlock") { forAll(entropyGen, passwordGen, encryptionSettingsGen, Arbitrary.arbBool.arbitrary) { (seed, pass, cryptoSettings, usePre1627KeyDerivation) => - val dir = createTempDir + val dir = createTempDir() val settings = SecretStorageSettings(dir.getAbsolutePath, cryptoSettings) val storage = JsonSecretStorage.init(seed, SecretString.create(pass), usePre1627KeyDerivation)(settings) @@ -41,7 +41,7 @@ class JsonSecretStorageSpec property("secrets erasure on lock") { forAll(entropyGen, passwordGen, encryptionSettingsGen) { (seed, pass, cryptoSettings) => - val dir = createTempDir + val dir = createTempDir() val settings = SecretStorageSettings(dir.getAbsolutePath, cryptoSettings) val storage = JsonSecretStorage.init(seed, SecretString.create(pass), usePre1627KeyDerivation = false)(settings) @@ -62,7 +62,7 @@ class JsonSecretStorageSpec property("restore from mnemonic") { forAll(mnemonicGen, passwordGen, encryptionSettingsGen, Arbitrary.arbBool.arbitrary) { (mnemonic, pass, cryptoSettings, usePre1627KeyDerivation) => - val dir = createTempDir + val dir = createTempDir() val settings = SecretStorageSettings(dir.getAbsolutePath, cryptoSettings) val storage = JsonSecretStorage.restore(SecretString.create(mnemonic.toString()), mnemonicPassOpt = None, @@ -78,7 +78,7 @@ class JsonSecretStorageSpec } property("pre 1627 key derivation is used for loaded storage with missing usePre1627KeyDerivation") { - val dir = createTempDir + val dir = createTempDir() // mock JSON file without usePre1627KeyDerivation property set, simulating pre 1627 wallet file val jsonFileRaw = """ {"cipherText":"e134f488c52a87ccb0287fdd164bdb3b67f04c33ba94eab169e802df7a082addd434e1f36dccbf5362f97a9e57ef97879807bdc632072fb2b3ae9a9b08a6caf6","salt":"988e9d31c675bf6012c235e4c238f22649285140544b17600e2887f655b74ae7","iv":"dff8c6b120cdfaac4192e9c1","authTag":"de1c6443808263b749f4c29f146208ba","cipherParams":{"prf":"HmacSHA1","c":7,"dkLen":256}} diff --git a/src/main/scala/org/ergoplatform/ErgoApp.scala b/src/main/scala/org/ergoplatform/ErgoApp.scala index 323bd5f227..b70764f46d 100644 --- a/src/main/scala/org/ergoplatform/ErgoApp.scala +++ b/src/main/scala/org/ergoplatform/ErgoApp.scala @@ -35,7 +35,7 @@ class ErgoApp(args: Args) extends ScorexLogging { log.info(s"Running with args: $args") - private val ergoSettings: ErgoSettings = ErgoSettingsReader.read(args) + private val ergoSettings: ErgoSettings = ErgoSettingsReader.read(args)(null) require( ergoSettings.scorexSettings.restApi.apiKeyHash.isDefined, diff --git a/src/main/scala/org/ergoplatform/reemission/ReemissionRulesUtils.scala b/src/main/scala/org/ergoplatform/reemission/ReemissionRulesUtils.scala index f149097489..b80d3bc0af 100644 --- a/src/main/scala/org/ergoplatform/reemission/ReemissionRulesUtils.scala +++ b/src/main/scala/org/ergoplatform/reemission/ReemissionRulesUtils.scala @@ -44,7 +44,7 @@ object ReemissionRulesUtils { println(new ErgoAddressEncoder(ErgoAddressEncoder.TestnetNetworkPrefix).fromString(p2sAddress.toString())) println("injectioon box p2s: " + p2sAddress) - val settings = ErgoSettingsReader.read() + val settings = ErgoSettingsReader.read()(null) println("Monetary settings: " + settings.chainSettings.monetary) println("Reemission settings: " + settings.chainSettings.reemission) diff --git a/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala b/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala index 4cacd4be02..02d0328a58 100644 --- a/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala +++ b/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala @@ -23,12 +23,17 @@ object ErgoSettingsReader extends ScorexLogging with NodeConfigurationReaders with SettingsReaders { - def read(args: Args = Args.empty): ErgoSettings = { + def read(args: Args = Args.empty)(implicit tmp: Int => File): ErgoSettings = { fromConfig(readConfig(args), args.networkTypeOpt) } - def fromConfig(config: Config, desiredNetworkTypeOpt: Option[NetworkType] = None): ErgoSettings = { - val directory = config.as[String](s"$configPath.directory") + def fromConfig(config: Config, desiredNetworkTypeOpt: Option[NetworkType] = None)(implicit tmp: Int => File): ErgoSettings = { + val directory = if(System.getProperty("env") == "test") { + val dir = tmp(0) + dir.mkdirs() + dir.getAbsolutePath + } else + config.as[String](s"$configPath.directory") val networkTypeName = config.as[String](s"$configPath.networkType") val networkType = NetworkType.fromString(networkTypeName) .getOrElse(throw new Error(s"Unknown `networkType = $networkTypeName`")) diff --git a/src/test/scala/org/ergoplatform/db/DBSpec.scala b/src/test/scala/org/ergoplatform/db/DBSpec.scala index eac623f878..0c7a88de14 100644 --- a/src/test/scala/org/ergoplatform/db/DBSpec.scala +++ b/src/test/scala/org/ergoplatform/db/DBSpec.scala @@ -21,7 +21,7 @@ trait DBSpec extends TestFileUtils { protected def byteString32(s: String): Array[Byte] = Algos.hash(byteString(s)) protected def withDb[T](body: RegisteredDB => T): T = { - val db = LDBFactory.open(createTempDir) + val db = LDBFactory.open(createTempDir()) try body(db) finally db.close() } @@ -31,7 +31,7 @@ trait DBSpec extends TestFileUtils { withDb { db: RegisteredDB => body(new LDBKVStore(db)) } protected def withVersionedStore[T](keepVersions: Int)(body: LDBVersionedStore => T): T = { - val db = new LDBVersionedStore(createTempDir, keepVersions) + val db = new LDBVersionedStore(createTempDir(), keepVersions) try body(db) finally db.close() } diff --git a/src/test/scala/org/ergoplatform/http/routes/EmissionApiRouteSpec.scala b/src/test/scala/org/ergoplatform/http/routes/EmissionApiRouteSpec.scala index c48d4a6dbe..135d6cf8dd 100644 --- a/src/test/scala/org/ergoplatform/http/routes/EmissionApiRouteSpec.scala +++ b/src/test/scala/org/ergoplatform/http/routes/EmissionApiRouteSpec.scala @@ -10,12 +10,14 @@ import io.circe.syntax._ import org.ergoplatform.http.api.EmissionApiRoute import org.ergoplatform.mining.emission.EmissionRules import org.ergoplatform.settings.{ErgoSettings, ErgoSettingsReader, ReemissionSettings} +import org.ergoplatform.wallet.utils.FileUtils import org.scalatest.flatspec.AnyFlatSpec import org.scalatest.matchers.should.Matchers import scala.concurrent.duration._ class EmissionApiRouteSpec extends AnyFlatSpec + with FileUtils with Matchers with ScalatestRouteTest with FailFastCirceSupport { diff --git a/src/test/scala/org/ergoplatform/mining/ErgoMinerSpec.scala b/src/test/scala/org/ergoplatform/mining/ErgoMinerSpec.scala index c86eea5209..c003edaeb4 100644 --- a/src/test/scala/org/ergoplatform/mining/ErgoMinerSpec.scala +++ b/src/test/scala/org/ergoplatform/mining/ErgoMinerSpec.scala @@ -62,7 +62,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen it should "not include too costly transactions" in new TestKit(ActorSystem()) { val testProbe = new TestProbe(system) system.eventStream.subscribe(testProbe.ref, newBlockSignal) - val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir.getAbsolutePath) + val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir().getAbsolutePath) val complexScript: ErgoTree = (0 until 100).foldLeft(SigmaAnd(SigmaPropConstant(defaultMinerPk), SigmaPropConstant(defaultMinerPk))) { (l, _) => SigmaAnd(SigmaPropConstant(defaultMinerPk), l) } @@ -139,7 +139,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen it should "not freeze while mempool is full" in new TestKit(ActorSystem()) { // generate amount of transactions, twice more than can fit in one block val desiredSize: Int = Math.ceil((parameters.maxBlockCost / ErgoInterpreter.interpreterInitCost) * 1.2).toInt - val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir.getAbsolutePath) + val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir().getAbsolutePath) val testProbe = new TestProbe(system) system.eventStream.subscribe(testProbe.ref, newBlockSignal) @@ -216,7 +216,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen it should "include only one transaction from 2 spending the same box" in new TestKit(ActorSystem()) { val testProbe = new TestProbe(system) system.eventStream.subscribe(testProbe.ref, newBlockSignal) - val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir.getAbsolutePath) + val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir().getAbsolutePath) val nodeViewHolderRef: ActorRef = ErgoNodeViewRef(ergoSettings) val readersHolderRef: ActorRef = ErgoReadersHolderRef(nodeViewHolderRef) @@ -291,7 +291,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen } it should "prepare external candidate" in new TestKit(ActorSystem()) { - val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir.getAbsolutePath) + val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir().getAbsolutePath) val nodeViewHolderRef: ActorRef = ErgoNodeViewRef(ergoSettings) val readersHolderRef: ActorRef = ErgoReadersHolderRef(nodeViewHolderRef) @@ -314,7 +314,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen it should "include mandatory transactions" in new TestKit(ActorSystem()) { val testProbe = new TestProbe(system) system.eventStream.subscribe(testProbe.ref, newBlockSignal) - val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir.getAbsolutePath) + val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir().getAbsolutePath) val nodeViewHolderRef: ActorRef = ErgoNodeViewRef(ergoSettings) val readersHolderRef: ActorRef = ErgoReadersHolderRef(nodeViewHolderRef) @@ -391,7 +391,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen version2ActivationDifficultyHex = "10", votingLength = forkHeight) ) - empty.copy(nodeSettings = nodeSettings, chainSettings = chainSettings, directory = createTempDir.getAbsolutePath) + empty.copy(nodeSettings = nodeSettings, chainSettings = chainSettings, directory = createTempDir().getAbsolutePath) } val nodeViewHolderRef: ActorRef = ErgoNodeViewRef(forkSettings) diff --git a/src/test/scala/org/ergoplatform/network/ErgoNodeViewSynchronizerSpecification.scala b/src/test/scala/org/ergoplatform/network/ErgoNodeViewSynchronizerSpecification.scala index 6a84b4f8f6..4a1ab050d4 100644 --- a/src/test/scala/org/ergoplatform/network/ErgoNodeViewSynchronizerSpecification.scala +++ b/src/test/scala/org/ergoplatform/network/ErgoNodeViewSynchronizerSpecification.scala @@ -86,7 +86,7 @@ class ErgoNodeViewSynchronizerSpecification extends HistoryTestHelpers with Matc } val localStateGen: Gen[WrappedUtxoState] = - boxesHolderGen.map(WrappedUtxoState(_, createTempDir, None, parameters, settings)) + boxesHolderGen.map(WrappedUtxoState(_, createTempDir(), None, parameters, settings)) def semanticallyValidModifier(state: UTXO_ST): PM = { statefulyValidFullBlock(state.asInstanceOf[WrappedUtxoState]) diff --git a/src/test/scala/org/ergoplatform/nodeView/NodeViewSynchronizerTests.scala b/src/test/scala/org/ergoplatform/nodeView/NodeViewSynchronizerTests.scala index 1c383c2dd5..3d62396ca9 100644 --- a/src/test/scala/org/ergoplatform/nodeView/NodeViewSynchronizerTests.scala +++ b/src/test/scala/org/ergoplatform/nodeView/NodeViewSynchronizerTests.scala @@ -281,7 +281,7 @@ trait NodeViewSynchronizerTests[ST <: ErgoState[ST]] extends AnyPropSpec Random.nextInt(1000000) -> (Digest32 @@ Algos.decode(mod.id).get) }.toMap val si = new SnapshotsInfo(m) - val db = SnapshotsDb.create(createTempDir.getPath) + val db = SnapshotsDb.create(createTempDir().getPath) db.writeSnapshotsInfo(si) // Then send message to request it diff --git a/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala b/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala index def8600a21..476c56f80c 100644 --- a/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala +++ b/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala @@ -78,7 +78,7 @@ class UtxoSetSnapshotProcessorSpecification extends HistoryTestHelpers { val s = utxoSetSnapshotProcessor.downloadedChunksIterator().map(s => ModifierId @@ Algos.encode(s.id)).toSeq s shouldBe subtreeIdsEncoded - val dir = createTempDir + val dir = createTempDir() val store = new LDBVersionedStore(dir, initialKeepVersions = 100) val restoredProver = utxoSetSnapshotProcessor.createPersistentProver(store, history, snapshotHeight, blockId).get bh.sortedBoxes.foreach { box => diff --git a/src/test/scala/org/ergoplatform/nodeView/history/extra/ExtraIndexerTestActor.scala b/src/test/scala/org/ergoplatform/nodeView/history/extra/ExtraIndexerTestActor.scala index e32c3c647f..4a147a4bd1 100644 --- a/src/test/scala/org/ergoplatform/nodeView/history/extra/ExtraIndexerTestActor.scala +++ b/src/test/scala/org/ergoplatform/nodeView/history/extra/ExtraIndexerTestActor.scala @@ -32,7 +32,7 @@ class ExtraIndexerTestActor(test: ExtraIndexerSpecification) extends ExtraIndexe 1000000, 100, adProofsSuffixLength = 112 * 1024, extraIndex = false) def createDB(): Unit = { - val dir: File = createTempDir + val dir: File = createTempDir() dir.mkdirs() val fullHistorySettings: ErgoSettings = ErgoSettings(dir.getAbsolutePath, NetworkType.TestNet, test.initSettings.chainSettings, diff --git a/src/test/scala/org/ergoplatform/nodeView/mempool/ScriptsSpec.scala b/src/test/scala/org/ergoplatform/nodeView/mempool/ScriptsSpec.scala index 7f2f6b7cca..a51d27c478 100644 --- a/src/test/scala/org/ergoplatform/nodeView/mempool/ScriptsSpec.scala +++ b/src/test/scala/org/ergoplatform/nodeView/mempool/ScriptsSpec.scala @@ -67,7 +67,7 @@ class ScriptsSpec extends ErgoPropertyTest { private def applyBlockSpendingScript(script: ErgoTree): Try[UtxoState] = { val scriptBox = ergoBoxGen(script, heightGen = 0).sample.get val bh = BoxHolder(Seq(fixedBox, scriptBox)) - val us = UtxoState.fromBoxHolder(bh, None, createTempDir, settings, parameters) + val us = UtxoState.fromBoxHolder(bh, None, createTempDir(), settings, parameters) bh.boxes.map(b => us.boxById(b._2.id) shouldBe Some(b._2)) val tx = validTransactionsFromBoxHolder(bh, new RandomWrapper(Some(1)), 201)._1 tx.size shouldBe 1 diff --git a/src/test/scala/org/ergoplatform/nodeView/state/DigestStateSpecification.scala b/src/test/scala/org/ergoplatform/nodeView/state/DigestStateSpecification.scala index e9db1e9e72..b595e7d0d1 100644 --- a/src/test/scala/org/ergoplatform/nodeView/state/DigestStateSpecification.scala +++ b/src/test/scala/org/ergoplatform/nodeView/state/DigestStateSpecification.scala @@ -20,7 +20,7 @@ class DigestStateSpecification extends ErgoPropertyTest { bh.sortedBoxes.foreach(box => us.boxById(box.id) should not be None) val fb = validFullBlock(parentOpt = None, us, bh) - val dir2 = createTempDir + val dir2 = createTempDir() val ds = DigestState.create(Some(us.version), Some(us.rootDigest), dir2, settings) ds.applyModifier(fb, None)(_ => ()) shouldBe 'success ds.close() diff --git a/src/test/scala/org/ergoplatform/nodeView/state/ErgoStateSpecification.scala b/src/test/scala/org/ergoplatform/nodeView/state/ErgoStateSpecification.scala index 2260b25314..0ce29871ab 100644 --- a/src/test/scala/org/ergoplatform/nodeView/state/ErgoStateSpecification.scala +++ b/src/test/scala/org/ergoplatform/nodeView/state/ErgoStateSpecification.scala @@ -67,7 +67,7 @@ class ErgoStateSpecification extends ErgoPropertyTest { property("generateGenesisUtxoState & generateGenesisDigestState are compliant") { val settings = ErgoSettingsReader.read(Args.empty) - val dir = createTempDir + val dir = createTempDir() val rootHash = createUtxoState(settings)._1.rootDigest val expectedRootHash = ErgoState.generateGenesisDigestState(dir, settings).rootDigest rootHash shouldBe expectedRootHash diff --git a/src/test/scala/org/ergoplatform/nodeView/state/SnapshotsDbSpecification.scala b/src/test/scala/org/ergoplatform/nodeView/state/SnapshotsDbSpecification.scala index f273749ca7..583c93c441 100644 --- a/src/test/scala/org/ergoplatform/nodeView/state/SnapshotsDbSpecification.scala +++ b/src/test/scala/org/ergoplatform/nodeView/state/SnapshotsDbSpecification.scala @@ -14,7 +14,7 @@ class SnapshotsDbSpecification extends ErgoPropertyTest { Random.nextInt(1000000) -> (Digest32 @@ idToBytes(mid)) }.toMap val si = new SnapshotsInfo(m) - val dir = createTempDir.getAbsolutePath + val dir = createTempDir().getAbsolutePath val db = SnapshotsDb.create(dir) db.writeSnapshotsInfo(si) si -> db diff --git a/src/test/scala/org/ergoplatform/sanity/ErgoSanityDigest.scala b/src/test/scala/org/ergoplatform/sanity/ErgoSanityDigest.scala index 0b7d947f44..63a8bdff13 100644 --- a/src/test/scala/org/ergoplatform/sanity/ErgoSanityDigest.scala +++ b/src/test/scala/org/ergoplatform/sanity/ErgoSanityDigest.scala @@ -27,8 +27,8 @@ class ErgoSanityDigest extends ErgoSanity[DIGEST_ST] { generateHistory(verifyTransactions = true, StateType.Digest, PoPoWBootstrap = false, -1) override val stateGen: Gen[WrappedDigestState] = { - boxesHolderGen.map(WrappedUtxoState(_, createTempDir, None, parameters, settings)).map { wus => - val digestState = DigestState.create(Some(wus.version), Some(wus.rootDigest), createTempDir, settings) + boxesHolderGen.map(WrappedUtxoState(_, createTempDir(), None, parameters, settings)).map { wus => + val digestState = DigestState.create(Some(wus.version), Some(wus.rootDigest), createTempDir(), settings) new WrappedDigestState(digestState, wus, settings) } } diff --git a/src/test/scala/org/ergoplatform/sanity/ErgoSanityUTXO.scala b/src/test/scala/org/ergoplatform/sanity/ErgoSanityUTXO.scala index a7e55e929e..34e2c76381 100644 --- a/src/test/scala/org/ergoplatform/sanity/ErgoSanityUTXO.scala +++ b/src/test/scala/org/ergoplatform/sanity/ErgoSanityUTXO.scala @@ -27,7 +27,7 @@ class ErgoSanityUTXO extends ErgoSanity[UTXO_ST] with ErgoTestHelpers { generateHistory(verifyTransactions = true, StateType.Utxo, PoPoWBootstrap = false, blocksToKeep = -1) override val stateGen: Gen[WrappedUtxoState] = - boxesHolderGen.map(WrappedUtxoState(_, createTempDir, None, parameters, settings)) + boxesHolderGen.map(WrappedUtxoState(_, createTempDir(), None, parameters, settings)) override def semanticallyValidModifier(state: UTXO_ST): PM = { statefulyValidFullBlock(state.asInstanceOf[WrappedUtxoState]) diff --git a/src/test/scala/org/ergoplatform/settings/ErgoSettingsSpecification.scala b/src/test/scala/org/ergoplatform/settings/ErgoSettingsSpecification.scala index 7d52cc5f71..a781e304d0 100644 --- a/src/test/scala/org/ergoplatform/settings/ErgoSettingsSpecification.scala +++ b/src/test/scala/org/ergoplatform/settings/ErgoSettingsSpecification.scala @@ -12,11 +12,6 @@ class ErgoSettingsSpecification extends ErgoPropertyTest { private val txCostLimit = initSettings.nodeSettings.maxTransactionCost private val txSizeLimit = initSettings.nodeSettings.maxTransactionSize - property("should keep data user home by default") { - val settings = ErgoSettingsReader.read() - settings.directory shouldBe System.getProperty("user.dir") + "/.ergo_test/data" - } - property("should read default settings") { val settings = ErgoSettingsReader.read() settings.nodeSettings shouldBe NodeConfigurationSettings( diff --git a/src/test/scala/org/ergoplatform/tools/DifficultyControlSimulator.scala b/src/test/scala/org/ergoplatform/tools/DifficultyControlSimulator.scala index dd743deb3d..0802cdae0f 100644 --- a/src/test/scala/org/ergoplatform/tools/DifficultyControlSimulator.scala +++ b/src/test/scala/org/ergoplatform/tools/DifficultyControlSimulator.scala @@ -5,6 +5,7 @@ import org.ergoplatform.modifiers.history.header.Header import org.ergoplatform.nodeView.history.ErgoHistoryUtils.Difficulty import org.ergoplatform.settings.ErgoSettingsReader import org.ergoplatform.utils.generators.ErgoGenerators +import org.ergoplatform.wallet.utils.FileUtils import scala.annotation.tailrec import scala.collection.mutable @@ -16,7 +17,7 @@ import scala.util.Random * Object that allows to simulate blockchain with the specified difficulty control function and estimate deviation * of block interval from the desired one */ -object DifficultyControlSimulator extends App with ErgoGenerators { +object DifficultyControlSimulator extends App with ErgoGenerators with FileUtils { val baseHeader = defaultHeaderGen.sample.get // val difficultyControl = new LinearDifficultyControl(1.minute, useLastEpochs = 100, epochLength = 1) diff --git a/src/test/scala/org/ergoplatform/utils/ErgoTestConstants.scala b/src/test/scala/org/ergoplatform/utils/ErgoTestConstants.scala index 4cd5bc4311..ab3d98810c 100644 --- a/src/test/scala/org/ergoplatform/utils/ErgoTestConstants.scala +++ b/src/test/scala/org/ergoplatform/utils/ErgoTestConstants.scala @@ -16,6 +16,7 @@ import org.ergoplatform.settings._ import org.ergoplatform.wallet.interface4j.SecretString import org.ergoplatform.wallet.interpreter.{ErgoInterpreter, ErgoProvingInterpreter} import org.ergoplatform.wallet.mnemonic.Mnemonic +import org.ergoplatform.wallet.utils.FileUtils import org.ergoplatform.{DataInput, ErgoBox, ErgoTreePredef} import scorex.crypto.authds.ADDigest import scorex.crypto.hash.Digest32 @@ -27,7 +28,7 @@ import sigmastate.interpreter.{ContextExtension, ProverResult} import scala.concurrent.duration._ -trait ErgoTestConstants extends ScorexLogging { +trait ErgoTestConstants extends ScorexLogging with FileUtils { implicit val votingSettings: VotingSettings = VotingSettings(1024, 32, 128, 32 * 1024, "01") val validationSettings: ErgoValidationSettings = ErgoValidationSettings.initial diff --git a/src/test/scala/org/ergoplatform/utils/HistoryTestHelpers.scala b/src/test/scala/org/ergoplatform/utils/HistoryTestHelpers.scala index c6b1c00233..6028697ddf 100644 --- a/src/test/scala/org/ergoplatform/utils/HistoryTestHelpers.scala +++ b/src/test/scala/org/ergoplatform/utils/HistoryTestHelpers.scala @@ -62,7 +62,7 @@ trait HistoryTestHelpers extends ErgoPropertyTest { settings.chainSettings.copy(epochLength = epochLength, useLastEpochs = useLastEpochs, genesisId = genesisIdOpt) } - val dir = createTempDir + val dir = createTempDir() val fullHistorySettings: ErgoSettings = ErgoSettings(dir.getAbsolutePath, NetworkType.TestNet, chainSettings, nodeSettings, scorexSettings, walletSettings, settings.cacheSettings) diff --git a/src/test/scala/org/ergoplatform/utils/NodeViewTestConfig.scala b/src/test/scala/org/ergoplatform/utils/NodeViewTestConfig.scala index 3b708d9d06..6f7e3b6f05 100644 --- a/src/test/scala/org/ergoplatform/utils/NodeViewTestConfig.scala +++ b/src/test/scala/org/ergoplatform/utils/NodeViewTestConfig.scala @@ -3,11 +3,12 @@ package org.ergoplatform.utils import org.ergoplatform.mining.DefaultFakePowScheme import org.ergoplatform.nodeView.state.StateType import org.ergoplatform.settings.{ErgoSettings, ErgoSettingsReader, NipopowSettings} +import org.ergoplatform.wallet.utils.FileUtils case class NodeViewTestConfig(stateType: StateType, verifyTransactions: Boolean, - popowBootstrap: Boolean) { + popowBootstrap: Boolean) extends FileUtils { def toSettings: ErgoSettings = { val defaultSettings = ErgoSettingsReader.read() diff --git a/src/test/scala/org/ergoplatform/utils/Stubs.scala b/src/test/scala/org/ergoplatform/utils/Stubs.scala index 53186f5a77..abae63689f 100644 --- a/src/test/scala/org/ergoplatform/utils/Stubs.scala +++ b/src/test/scala/org/ergoplatform/utils/Stubs.scala @@ -56,15 +56,15 @@ trait Stubs extends ErgoGenerators with ErgoTestHelpers with ChainGenerator with val history: HT = applyChain(generateHistory(), chain) val digestState: DigestState = { - boxesHolderGen.map(WrappedUtxoState(_, createTempDir, None, parameters, settings)).map { wus => - DigestState.create(Some(wus.version), Some(wus.rootDigest), createTempDir, settings) + boxesHolderGen.map(WrappedUtxoState(_, createTempDir(), None, parameters, settings)).map { wus => + DigestState.create(Some(wus.version), Some(wus.rootDigest), createTempDir(), settings) } }.sample.value val utxoSettings: ErgoSettings = settings.copy(nodeSettings = settings.nodeSettings.copy(stateType = StateType.Utxo)) val utxoState: WrappedUtxoState = - boxesHolderGen.map(WrappedUtxoState(_, createTempDir, None, parameters, utxoSettings)).sample.value + boxesHolderGen.map(WrappedUtxoState(_, createTempDir(), None, parameters, utxoSettings)).sample.value lazy val wallet = new WalletStub @@ -377,7 +377,7 @@ trait Stubs extends ErgoGenerators with ErgoTestHelpers with ChainGenerator with val walletSettings: WalletSettings = null val chainSettings = settings.chainSettings.copy(epochLength = epochLength, useLastEpochs = useLastEpochs) - val dir = createTempDir + val dir = createTempDir() val fullHistorySettings: ErgoSettings = ErgoSettings(dir.getAbsolutePath, NetworkType.TestNet, chainSettings, nodeSettings, scorexSettings, walletSettings, settings.cacheSettings) diff --git a/src/test/scala/org/ergoplatform/utils/fixtures/NodeViewFixture.scala b/src/test/scala/org/ergoplatform/utils/fixtures/NodeViewFixture.scala index 5127620226..79bc3291e7 100644 --- a/src/test/scala/org/ergoplatform/utils/fixtures/NodeViewFixture.scala +++ b/src/test/scala/org/ergoplatform/utils/fixtures/NodeViewFixture.scala @@ -19,7 +19,7 @@ class NodeViewFixture(protoSettings: ErgoSettings, parameters: Parameters) exten implicit val executionContext: ExecutionContext = actorSystem.dispatchers.lookup("scorex.executionContext") implicit def ctx: NodeViewTestContext = this - val nodeViewDir: java.io.File = createTempDir + val nodeViewDir: java.io.File = createTempDir() @volatile var settings: ErgoSettings = protoSettings.copy(directory = nodeViewDir.getAbsolutePath) val emission: EmissionRules = new EmissionRules(settings.chainSettings.monetary) @volatile var nodeViewHolderRef: ActorRef = ErgoNodeViewRef(settings) diff --git a/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala b/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala index e24688951b..f5e4d5226a 100644 --- a/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala +++ b/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala @@ -27,14 +27,14 @@ trait ValidBlocksGenerators extends TestkitHelpers with TestFileUtils with Matchers with ChainGenerator with ErgoTransactionGenerators { def createUtxoState(settings: ErgoSettings): (UtxoState, BoxHolder) = { - ErgoState.generateGenesisUtxoState(createTempDir, settings) + ErgoState.generateGenesisUtxoState(createTempDir(), settings) } def createUtxoState(bh: BoxHolder, parameters: Parameters): UtxoState = - UtxoState.fromBoxHolder(bh, None, createTempDir, settings, parameters) + UtxoState.fromBoxHolder(bh, None, createTempDir(), settings, parameters) def createDigestState(version: VersionTag, digest: ADDigest): DigestState = - DigestState.create(Some(version), Some(digest), createTempDir, settings) + DigestState.create(Some(version), Some(digest), createTempDir(), settings) def validTransactionsFromBoxHolder(boxHolder: BoxHolder): (Seq[ErgoTransaction], BoxHolder) = validTransactionsFromBoxHolder(boxHolder, new RandomWrapper) From e46fc2d6f9477572c58c55de37243021dea46f27 Mon Sep 17 00:00:00 2001 From: Alexander Chepurnoy Date: Fri, 16 Feb 2024 23:43:02 +0300 Subject: [PATCH 05/15] 5.1.0 version set --- src/main/resources/api/openapi-ai.yaml | 2 +- src/main/resources/api/openapi.yaml | 2 +- src/main/resources/application.conf | 2 +- 3 files changed, 3 insertions(+), 3 deletions(-) diff --git a/src/main/resources/api/openapi-ai.yaml b/src/main/resources/api/openapi-ai.yaml index 1e562ff0f5..9379f4cd3b 100644 --- a/src/main/resources/api/openapi-ai.yaml +++ b/src/main/resources/api/openapi-ai.yaml @@ -1,7 +1,7 @@ openapi: "3.0.2" info: - version: "5.0.20" + version: "5.1.0" title: Ergo Node API description: Specification of Ergo Node API for ChatGPT plugin. The following endpoints supported diff --git a/src/main/resources/api/openapi.yaml b/src/main/resources/api/openapi.yaml index 24062ad56d..5624fba3ef 100644 --- a/src/main/resources/api/openapi.yaml +++ b/src/main/resources/api/openapi.yaml @@ -1,7 +1,7 @@ openapi: "3.0.2" info: - version: "5.0.20" + version: "5.1.0" title: Ergo Node API description: API docs for Ergo Node. Models are shared between all Ergo products contact: diff --git a/src/main/resources/application.conf b/src/main/resources/application.conf index 14fccf5ea2..6c3fa08caa 100644 --- a/src/main/resources/application.conf +++ b/src/main/resources/application.conf @@ -436,7 +436,7 @@ scorex { nodeName = "ergo-node" # Network protocol version to be sent in handshakes - appVersion = 5.0.20 + appVersion = 5.1.0 # Network agent name. May contain information about client code # stack, starting from core code-base up to the end graphical interface. From 4370bfb55a43d306e0e053df6318490720e39d4b Mon Sep 17 00:00:00 2001 From: jellymlg Date: Sat, 17 Feb 2024 22:07:26 +0100 Subject: [PATCH 06/15] Fixed tests crash --- avldb/src/main/scala/scorex/db/LDBFactory.scala | 2 -- 1 file changed, 2 deletions(-) diff --git a/avldb/src/main/scala/scorex/db/LDBFactory.scala b/avldb/src/main/scala/scorex/db/LDBFactory.scala index 143b01b4d6..da474ba1e6 100644 --- a/avldb/src/main/scala/scorex/db/LDBFactory.scala +++ b/avldb/src/main/scala/scorex/db/LDBFactory.scala @@ -40,8 +40,6 @@ object LDBFactory extends ScorexLogging { lock.writeLock().lock() try { map.remove(path) - impl.syncWal() - impl.cancelAllBackgroundWork(true) impl.close() } finally { lock.writeLock().unlock() From a6473f35810cde0579f3015d1f1162802551c372 Mon Sep 17 00:00:00 2001 From: jellymlg Date: Sun, 18 Feb 2024 10:54:13 +0100 Subject: [PATCH 07/15] Added logs to find problems in tests --- avldb/src/main/scala/scorex/db/LDBFactory.scala | 15 ++++++++++++--- 1 file changed, 12 insertions(+), 3 deletions(-) diff --git a/avldb/src/main/scala/scorex/db/LDBFactory.scala b/avldb/src/main/scala/scorex/db/LDBFactory.scala index da474ba1e6..4a4991be75 100644 --- a/avldb/src/main/scala/scorex/db/LDBFactory.scala +++ b/avldb/src/main/scala/scorex/db/LDBFactory.scala @@ -27,8 +27,14 @@ object LDBFactory extends ScorexLogging { * return existed instance instead of creating new one. */ case class RegisteredDB(impl: RocksDB, path: File) { - def get(key: Array[Byte]): Array[Byte] = impl.get(key) - def get(options: ReadOptions, key: Array[Byte]): Array[Byte] = impl.get(options, key) + def get(key: Array[Byte]): Array[Byte] = { + log.warn(s"Read key from DB at $path") + impl.get(key) + } + def get(options: ReadOptions, key: Array[Byte]): Array[Byte] = { + log.warn(s"Read key with options from DB at $path") + impl.get(options, key) + } def contains(key: Array[Byte]): Boolean = impl.keyExists(key) def iterator: RocksIterator = impl.newIterator() def iterator(options: ReadOptions): RocksIterator = impl.newIterator(options) @@ -40,8 +46,11 @@ object LDBFactory extends ScorexLogging { lock.writeLock().lock() try { map.remove(path) - impl.close() + impl.closeE() + } catch { + case e: Throwable => log.warn(s"Error closing DB at $path: $e") } finally { + log.warn(s"Closed DB at $path") lock.writeLock().unlock() } } From e8269ac013b828b7118622c81e4e92a023b9d68c Mon Sep 17 00:00:00 2001 From: jellymlg Date: Sun, 18 Feb 2024 12:13:59 +0100 Subject: [PATCH 08/15] Try AtomicBoolean --- .../src/main/scala/scorex/db/LDBFactory.scala | 19 +++++++++---------- 1 file changed, 9 insertions(+), 10 deletions(-) diff --git a/avldb/src/main/scala/scorex/db/LDBFactory.scala b/avldb/src/main/scala/scorex/db/LDBFactory.scala index 4a4991be75..d6afa891bc 100644 --- a/avldb/src/main/scala/scorex/db/LDBFactory.scala +++ b/avldb/src/main/scala/scorex/db/LDBFactory.scala @@ -5,6 +5,7 @@ import org.rocksdb.util.SizeUnit import scorex.util.ScorexLogging import java.io.File +import java.util.concurrent.atomic.AtomicBoolean import java.util.concurrent.locks.ReentrantReadWriteLock import scala.collection.mutable @@ -27,14 +28,14 @@ object LDBFactory extends ScorexLogging { * return existed instance instead of creating new one. */ case class RegisteredDB(impl: RocksDB, path: File) { + val open: AtomicBoolean = new AtomicBoolean(true) def get(key: Array[Byte]): Array[Byte] = { - log.warn(s"Read key from DB at $path") - impl.get(key) - } - def get(options: ReadOptions, key: Array[Byte]): Array[Byte] = { - log.warn(s"Read key with options from DB at $path") - impl.get(options, key) + if(open.get()) + impl.get(key) + else + null } + def get(options: ReadOptions, key: Array[Byte]): Array[Byte] = impl.get(options, key) def contains(key: Array[Byte]): Boolean = impl.keyExists(key) def iterator: RocksIterator = impl.newIterator() def iterator(options: ReadOptions): RocksIterator = impl.newIterator(options) @@ -46,11 +47,9 @@ object LDBFactory extends ScorexLogging { lock.writeLock().lock() try { map.remove(path) - impl.closeE() - } catch { - case e: Throwable => log.warn(s"Error closing DB at $path: $e") + impl.close() + open.set(false) } finally { - log.warn(s"Closed DB at $path") lock.writeLock().unlock() } } From 0122529d0e7526d6e247c6e49f3910337649d79e Mon Sep 17 00:00:00 2001 From: jellymlg Date: Sun, 18 Feb 2024 13:13:31 +0100 Subject: [PATCH 09/15] Removed TestFileUtils --- .../ergoplatform/wallet/utils/FileUtils.scala | 3 ++- .../secrets/JsonSecretStorageSpec.scala | 8 ++++---- .../wallet/utils/TestFileUtils.scala | 3 --- src/main/scala/org/ergoplatform/ErgoApp.scala | 2 +- .../reemission/ReemissionRulesUtils.scala | 2 +- .../settings/ErgoSettingsReader.scala | 19 ++++++++++--------- .../scala/org/ergoplatform/db/DBSpec.scala | 8 ++++---- .../ergoplatform/mining/ErgoMinerSpec.scala | 12 ++++++------ ...rgoNodeViewSynchronizerSpecification.scala | 2 +- .../nodeView/NodeViewSynchronizerTests.scala | 6 +++--- ...txoSetSnapshotProcessorSpecification.scala | 2 +- .../history/extra/ExtraIndexerTestActor.scala | 3 +-- .../nodeView/mempool/ScriptsSpec.scala | 2 +- .../state/DigestStateSpecification.scala | 2 +- .../state/ErgoStateSpecification.scala | 2 +- .../state/SnapshotsDbSpecification.scala | 2 +- .../sanity/ErgoSanityDigest.scala | 4 ++-- .../ergoplatform/sanity/ErgoSanityUTXO.scala | 2 +- .../utils/HistoryTestHelpers.scala | 2 +- .../scala/org/ergoplatform/utils/Stubs.scala | 12 ++++++------ .../utils/fixtures/NodeViewFixture.scala | 6 +++--- .../generators/ValidBlocksGenerators.scala | 10 +++++----- 22 files changed, 56 insertions(+), 58 deletions(-) delete mode 100644 ergo-wallet/src/test/scala/org/ergoplatform/wallet/utils/TestFileUtils.scala diff --git a/ergo-wallet/src/main/scala/org/ergoplatform/wallet/utils/FileUtils.scala b/ergo-wallet/src/main/scala/org/ergoplatform/wallet/utils/FileUtils.scala index 335262860a..c3658263a6 100644 --- a/ergo-wallet/src/main/scala/org/ergoplatform/wallet/utils/FileUtils.scala +++ b/ergo-wallet/src/main/scala/org/ergoplatform/wallet/utils/FileUtils.scala @@ -18,13 +18,14 @@ trait FileUtils { } } - implicit def createTempDir(nothing: Int = 0): java.io.File = { + implicit def createTempDir: java.io.File = { val rndString = scala.util.Random.alphanumeric.take(randomPrefixLength).mkString createTempDirForPrefix(rndString) } private def createTempDirForPrefix(prefix: String): java.io.File = { val file = java.nio.file.Files.createTempDirectory(prefix).toFile + file.mkdirs() file.deleteOnExit() file } diff --git a/ergo-wallet/src/test/scala/org/ergoplatform/wallet/secrets/JsonSecretStorageSpec.scala b/ergo-wallet/src/test/scala/org/ergoplatform/wallet/secrets/JsonSecretStorageSpec.scala index e58efd56c6..39317270a6 100644 --- a/ergo-wallet/src/test/scala/org/ergoplatform/wallet/secrets/JsonSecretStorageSpec.scala +++ b/ergo-wallet/src/test/scala/org/ergoplatform/wallet/secrets/JsonSecretStorageSpec.scala @@ -22,7 +22,7 @@ class JsonSecretStorageSpec property("initialization and unlock") { forAll(entropyGen, passwordGen, encryptionSettingsGen, Arbitrary.arbBool.arbitrary) { (seed, pass, cryptoSettings, usePre1627KeyDerivation) => - val dir = createTempDir() + val dir = createTempDir val settings = SecretStorageSettings(dir.getAbsolutePath, cryptoSettings) val storage = JsonSecretStorage.init(seed, SecretString.create(pass), usePre1627KeyDerivation)(settings) @@ -41,7 +41,7 @@ class JsonSecretStorageSpec property("secrets erasure on lock") { forAll(entropyGen, passwordGen, encryptionSettingsGen) { (seed, pass, cryptoSettings) => - val dir = createTempDir() + val dir = createTempDir val settings = SecretStorageSettings(dir.getAbsolutePath, cryptoSettings) val storage = JsonSecretStorage.init(seed, SecretString.create(pass), usePre1627KeyDerivation = false)(settings) @@ -62,7 +62,7 @@ class JsonSecretStorageSpec property("restore from mnemonic") { forAll(mnemonicGen, passwordGen, encryptionSettingsGen, Arbitrary.arbBool.arbitrary) { (mnemonic, pass, cryptoSettings, usePre1627KeyDerivation) => - val dir = createTempDir() + val dir = createTempDir val settings = SecretStorageSettings(dir.getAbsolutePath, cryptoSettings) val storage = JsonSecretStorage.restore(SecretString.create(mnemonic.toString()), mnemonicPassOpt = None, @@ -78,7 +78,7 @@ class JsonSecretStorageSpec } property("pre 1627 key derivation is used for loaded storage with missing usePre1627KeyDerivation") { - val dir = createTempDir() + val dir = createTempDir // mock JSON file without usePre1627KeyDerivation property set, simulating pre 1627 wallet file val jsonFileRaw = """ {"cipherText":"e134f488c52a87ccb0287fdd164bdb3b67f04c33ba94eab169e802df7a082addd434e1f36dccbf5362f97a9e57ef97879807bdc632072fb2b3ae9a9b08a6caf6","salt":"988e9d31c675bf6012c235e4c238f22649285140544b17600e2887f655b74ae7","iv":"dff8c6b120cdfaac4192e9c1","authTag":"de1c6443808263b749f4c29f146208ba","cipherParams":{"prf":"HmacSHA1","c":7,"dkLen":256}} diff --git a/ergo-wallet/src/test/scala/org/ergoplatform/wallet/utils/TestFileUtils.scala b/ergo-wallet/src/test/scala/org/ergoplatform/wallet/utils/TestFileUtils.scala deleted file mode 100644 index bcd959fbf4..0000000000 --- a/ergo-wallet/src/test/scala/org/ergoplatform/wallet/utils/TestFileUtils.scala +++ /dev/null @@ -1,3 +0,0 @@ -package org.ergoplatform.wallet.utils - -trait TestFileUtils extends FileUtils diff --git a/src/main/scala/org/ergoplatform/ErgoApp.scala b/src/main/scala/org/ergoplatform/ErgoApp.scala index b70764f46d..323bd5f227 100644 --- a/src/main/scala/org/ergoplatform/ErgoApp.scala +++ b/src/main/scala/org/ergoplatform/ErgoApp.scala @@ -35,7 +35,7 @@ class ErgoApp(args: Args) extends ScorexLogging { log.info(s"Running with args: $args") - private val ergoSettings: ErgoSettings = ErgoSettingsReader.read(args)(null) + private val ergoSettings: ErgoSettings = ErgoSettingsReader.read(args) require( ergoSettings.scorexSettings.restApi.apiKeyHash.isDefined, diff --git a/src/main/scala/org/ergoplatform/reemission/ReemissionRulesUtils.scala b/src/main/scala/org/ergoplatform/reemission/ReemissionRulesUtils.scala index b80d3bc0af..f149097489 100644 --- a/src/main/scala/org/ergoplatform/reemission/ReemissionRulesUtils.scala +++ b/src/main/scala/org/ergoplatform/reemission/ReemissionRulesUtils.scala @@ -44,7 +44,7 @@ object ReemissionRulesUtils { println(new ErgoAddressEncoder(ErgoAddressEncoder.TestnetNetworkPrefix).fromString(p2sAddress.toString())) println("injectioon box p2s: " + p2sAddress) - val settings = ErgoSettingsReader.read()(null) + val settings = ErgoSettingsReader.read() println("Monetary settings: " + settings.chainSettings.monetary) println("Reemission settings: " + settings.chainSettings.reemission) diff --git a/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala b/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala index 02d0328a58..bd83826f90 100644 --- a/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala +++ b/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala @@ -8,6 +8,7 @@ import com.typesafe.config.{Config, ConfigFactory, ConfigValueFactory} import net.ceedubs.ficus.Ficus._ import net.ceedubs.ficus.readers.ArbitraryTypeReader._ import org.ergoplatform.nodeView.state.StateType.Digest +import org.ergoplatform.wallet.utils.FileUtils import org.ergoplatform.ErgoApp import scorex.util.ScorexLogging import org.ergoplatform.settings.ErgoSettings.{configPath, scorexConfigPath} @@ -21,19 +22,19 @@ import scala.util.Try object ErgoSettingsReader extends ScorexLogging with PowSchemeReaders with NodeConfigurationReaders - with SettingsReaders { + with SettingsReaders + with FileUtils { - def read(args: Args = Args.empty)(implicit tmp: Int => File): ErgoSettings = { + def read(args: Args = Args.empty): ErgoSettings = { fromConfig(readConfig(args), args.networkTypeOpt) } - def fromConfig(config: Config, desiredNetworkTypeOpt: Option[NetworkType] = None)(implicit tmp: Int => File): ErgoSettings = { - val directory = if(System.getProperty("env") == "test") { - val dir = tmp(0) - dir.mkdirs() - dir.getAbsolutePath - } else - config.as[String](s"$configPath.directory") + def fromConfig(config: Config, desiredNetworkTypeOpt: Option[NetworkType] = None): ErgoSettings = { + val directory = + if(System.getProperty("env") == "test") + createTempDir.getAbsolutePath + else + config.as[String](s"$configPath.directory") val networkTypeName = config.as[String](s"$configPath.networkType") val networkType = NetworkType.fromString(networkTypeName) .getOrElse(throw new Error(s"Unknown `networkType = $networkTypeName`")) diff --git a/src/test/scala/org/ergoplatform/db/DBSpec.scala b/src/test/scala/org/ergoplatform/db/DBSpec.scala index 0c7a88de14..c899372833 100644 --- a/src/test/scala/org/ergoplatform/db/DBSpec.scala +++ b/src/test/scala/org/ergoplatform/db/DBSpec.scala @@ -2,11 +2,11 @@ package org.ergoplatform.db import akka.util.ByteString import org.ergoplatform.settings.Algos -import org.ergoplatform.wallet.utils.TestFileUtils +import org.ergoplatform.wallet.utils.FileUtils import scorex.db.LDBFactory.RegisteredDB import scorex.db.{LDBFactory, LDBKVStore, LDBVersionedStore} -trait DBSpec extends TestFileUtils { +trait DBSpec extends FileUtils { implicit class ValueOps(x: Option[Array[Byte]]) { def toBs: Option[ByteString] = x.map(ByteString.apply) @@ -21,7 +21,7 @@ trait DBSpec extends TestFileUtils { protected def byteString32(s: String): Array[Byte] = Algos.hash(byteString(s)) protected def withDb[T](body: RegisteredDB => T): T = { - val db = LDBFactory.open(createTempDir()) + val db = LDBFactory.open(createTempDir) try body(db) finally db.close() } @@ -31,7 +31,7 @@ trait DBSpec extends TestFileUtils { withDb { db: RegisteredDB => body(new LDBKVStore(db)) } protected def withVersionedStore[T](keepVersions: Int)(body: LDBVersionedStore => T): T = { - val db = new LDBVersionedStore(createTempDir(), keepVersions) + val db = new LDBVersionedStore(createTempDir, keepVersions) try body(db) finally db.close() } diff --git a/src/test/scala/org/ergoplatform/mining/ErgoMinerSpec.scala b/src/test/scala/org/ergoplatform/mining/ErgoMinerSpec.scala index c003edaeb4..c86eea5209 100644 --- a/src/test/scala/org/ergoplatform/mining/ErgoMinerSpec.scala +++ b/src/test/scala/org/ergoplatform/mining/ErgoMinerSpec.scala @@ -62,7 +62,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen it should "not include too costly transactions" in new TestKit(ActorSystem()) { val testProbe = new TestProbe(system) system.eventStream.subscribe(testProbe.ref, newBlockSignal) - val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir().getAbsolutePath) + val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir.getAbsolutePath) val complexScript: ErgoTree = (0 until 100).foldLeft(SigmaAnd(SigmaPropConstant(defaultMinerPk), SigmaPropConstant(defaultMinerPk))) { (l, _) => SigmaAnd(SigmaPropConstant(defaultMinerPk), l) } @@ -139,7 +139,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen it should "not freeze while mempool is full" in new TestKit(ActorSystem()) { // generate amount of transactions, twice more than can fit in one block val desiredSize: Int = Math.ceil((parameters.maxBlockCost / ErgoInterpreter.interpreterInitCost) * 1.2).toInt - val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir().getAbsolutePath) + val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir.getAbsolutePath) val testProbe = new TestProbe(system) system.eventStream.subscribe(testProbe.ref, newBlockSignal) @@ -216,7 +216,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen it should "include only one transaction from 2 spending the same box" in new TestKit(ActorSystem()) { val testProbe = new TestProbe(system) system.eventStream.subscribe(testProbe.ref, newBlockSignal) - val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir().getAbsolutePath) + val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir.getAbsolutePath) val nodeViewHolderRef: ActorRef = ErgoNodeViewRef(ergoSettings) val readersHolderRef: ActorRef = ErgoReadersHolderRef(nodeViewHolderRef) @@ -291,7 +291,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen } it should "prepare external candidate" in new TestKit(ActorSystem()) { - val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir().getAbsolutePath) + val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir.getAbsolutePath) val nodeViewHolderRef: ActorRef = ErgoNodeViewRef(ergoSettings) val readersHolderRef: ActorRef = ErgoReadersHolderRef(nodeViewHolderRef) @@ -314,7 +314,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen it should "include mandatory transactions" in new TestKit(ActorSystem()) { val testProbe = new TestProbe(system) system.eventStream.subscribe(testProbe.ref, newBlockSignal) - val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir().getAbsolutePath) + val ergoSettings: ErgoSettings = defaultSettings.copy(directory = createTempDir.getAbsolutePath) val nodeViewHolderRef: ActorRef = ErgoNodeViewRef(ergoSettings) val readersHolderRef: ActorRef = ErgoReadersHolderRef(nodeViewHolderRef) @@ -391,7 +391,7 @@ class ErgoMinerSpec extends AnyFlatSpec with ErgoTestHelpers with ValidBlocksGen version2ActivationDifficultyHex = "10", votingLength = forkHeight) ) - empty.copy(nodeSettings = nodeSettings, chainSettings = chainSettings, directory = createTempDir().getAbsolutePath) + empty.copy(nodeSettings = nodeSettings, chainSettings = chainSettings, directory = createTempDir.getAbsolutePath) } val nodeViewHolderRef: ActorRef = ErgoNodeViewRef(forkSettings) diff --git a/src/test/scala/org/ergoplatform/network/ErgoNodeViewSynchronizerSpecification.scala b/src/test/scala/org/ergoplatform/network/ErgoNodeViewSynchronizerSpecification.scala index 4a1ab050d4..6a84b4f8f6 100644 --- a/src/test/scala/org/ergoplatform/network/ErgoNodeViewSynchronizerSpecification.scala +++ b/src/test/scala/org/ergoplatform/network/ErgoNodeViewSynchronizerSpecification.scala @@ -86,7 +86,7 @@ class ErgoNodeViewSynchronizerSpecification extends HistoryTestHelpers with Matc } val localStateGen: Gen[WrappedUtxoState] = - boxesHolderGen.map(WrappedUtxoState(_, createTempDir(), None, parameters, settings)) + boxesHolderGen.map(WrappedUtxoState(_, createTempDir, None, parameters, settings)) def semanticallyValidModifier(state: UTXO_ST): PM = { statefulyValidFullBlock(state.asInstanceOf[WrappedUtxoState]) diff --git a/src/test/scala/org/ergoplatform/nodeView/NodeViewSynchronizerTests.scala b/src/test/scala/org/ergoplatform/nodeView/NodeViewSynchronizerTests.scala index 3d62396ca9..6d9a4d9cbb 100644 --- a/src/test/scala/org/ergoplatform/nodeView/NodeViewSynchronizerTests.scala +++ b/src/test/scala/org/ergoplatform/nodeView/NodeViewSynchronizerTests.scala @@ -14,7 +14,7 @@ import org.ergoplatform.nodeView.mempool.ErgoMemPool import org.ergoplatform.nodeView.state.UtxoState.ManifestId import org.ergoplatform.nodeView.state._ import org.ergoplatform.settings.Algos -import org.ergoplatform.wallet.utils.TestFileUtils +import org.ergoplatform.wallet.utils.FileUtils import org.scalacheck.Gen import org.scalatest.matchers.should.Matchers import org.scalatest.propspec.AnyPropSpec @@ -42,7 +42,7 @@ trait NodeViewSynchronizerTests[ST <: ErgoState[ST]] extends AnyPropSpec with SyntacticallyTargetedModifierProducer with TotallyValidModifierProducer[ST] with ChainGenerator - with TestFileUtils { + with FileUtils { implicit val ec: scala.concurrent.ExecutionContext = scala.concurrent.ExecutionContext.global @@ -281,7 +281,7 @@ trait NodeViewSynchronizerTests[ST <: ErgoState[ST]] extends AnyPropSpec Random.nextInt(1000000) -> (Digest32 @@ Algos.decode(mod.id).get) }.toMap val si = new SnapshotsInfo(m) - val db = SnapshotsDb.create(createTempDir().getPath) + val db = SnapshotsDb.create(createTempDir.getPath) db.writeSnapshotsInfo(si) // Then send message to request it diff --git a/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala b/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala index 476c56f80c..def8600a21 100644 --- a/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala +++ b/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala @@ -78,7 +78,7 @@ class UtxoSetSnapshotProcessorSpecification extends HistoryTestHelpers { val s = utxoSetSnapshotProcessor.downloadedChunksIterator().map(s => ModifierId @@ Algos.encode(s.id)).toSeq s shouldBe subtreeIdsEncoded - val dir = createTempDir() + val dir = createTempDir val store = new LDBVersionedStore(dir, initialKeepVersions = 100) val restoredProver = utxoSetSnapshotProcessor.createPersistentProver(store, history, snapshotHeight, blockId).get bh.sortedBoxes.foreach { box => diff --git a/src/test/scala/org/ergoplatform/nodeView/history/extra/ExtraIndexerTestActor.scala b/src/test/scala/org/ergoplatform/nodeView/history/extra/ExtraIndexerTestActor.scala index 4a147a4bd1..4f69555edb 100644 --- a/src/test/scala/org/ergoplatform/nodeView/history/extra/ExtraIndexerTestActor.scala +++ b/src/test/scala/org/ergoplatform/nodeView/history/extra/ExtraIndexerTestActor.scala @@ -32,8 +32,7 @@ class ExtraIndexerTestActor(test: ExtraIndexerSpecification) extends ExtraIndexe 1000000, 100, adProofsSuffixLength = 112 * 1024, extraIndex = false) def createDB(): Unit = { - val dir: File = createTempDir() - dir.mkdirs() + val dir: File = createTempDir val fullHistorySettings: ErgoSettings = ErgoSettings(dir.getAbsolutePath, NetworkType.TestNet, test.initSettings.chainSettings, nodeSettings, test.initSettings.scorexSettings, test.initSettings.walletSettings, test.initSettings.cacheSettings) diff --git a/src/test/scala/org/ergoplatform/nodeView/mempool/ScriptsSpec.scala b/src/test/scala/org/ergoplatform/nodeView/mempool/ScriptsSpec.scala index a51d27c478..7f2f6b7cca 100644 --- a/src/test/scala/org/ergoplatform/nodeView/mempool/ScriptsSpec.scala +++ b/src/test/scala/org/ergoplatform/nodeView/mempool/ScriptsSpec.scala @@ -67,7 +67,7 @@ class ScriptsSpec extends ErgoPropertyTest { private def applyBlockSpendingScript(script: ErgoTree): Try[UtxoState] = { val scriptBox = ergoBoxGen(script, heightGen = 0).sample.get val bh = BoxHolder(Seq(fixedBox, scriptBox)) - val us = UtxoState.fromBoxHolder(bh, None, createTempDir(), settings, parameters) + val us = UtxoState.fromBoxHolder(bh, None, createTempDir, settings, parameters) bh.boxes.map(b => us.boxById(b._2.id) shouldBe Some(b._2)) val tx = validTransactionsFromBoxHolder(bh, new RandomWrapper(Some(1)), 201)._1 tx.size shouldBe 1 diff --git a/src/test/scala/org/ergoplatform/nodeView/state/DigestStateSpecification.scala b/src/test/scala/org/ergoplatform/nodeView/state/DigestStateSpecification.scala index b595e7d0d1..e9db1e9e72 100644 --- a/src/test/scala/org/ergoplatform/nodeView/state/DigestStateSpecification.scala +++ b/src/test/scala/org/ergoplatform/nodeView/state/DigestStateSpecification.scala @@ -20,7 +20,7 @@ class DigestStateSpecification extends ErgoPropertyTest { bh.sortedBoxes.foreach(box => us.boxById(box.id) should not be None) val fb = validFullBlock(parentOpt = None, us, bh) - val dir2 = createTempDir() + val dir2 = createTempDir val ds = DigestState.create(Some(us.version), Some(us.rootDigest), dir2, settings) ds.applyModifier(fb, None)(_ => ()) shouldBe 'success ds.close() diff --git a/src/test/scala/org/ergoplatform/nodeView/state/ErgoStateSpecification.scala b/src/test/scala/org/ergoplatform/nodeView/state/ErgoStateSpecification.scala index 0ce29871ab..2260b25314 100644 --- a/src/test/scala/org/ergoplatform/nodeView/state/ErgoStateSpecification.scala +++ b/src/test/scala/org/ergoplatform/nodeView/state/ErgoStateSpecification.scala @@ -67,7 +67,7 @@ class ErgoStateSpecification extends ErgoPropertyTest { property("generateGenesisUtxoState & generateGenesisDigestState are compliant") { val settings = ErgoSettingsReader.read(Args.empty) - val dir = createTempDir() + val dir = createTempDir val rootHash = createUtxoState(settings)._1.rootDigest val expectedRootHash = ErgoState.generateGenesisDigestState(dir, settings).rootDigest rootHash shouldBe expectedRootHash diff --git a/src/test/scala/org/ergoplatform/nodeView/state/SnapshotsDbSpecification.scala b/src/test/scala/org/ergoplatform/nodeView/state/SnapshotsDbSpecification.scala index 583c93c441..f273749ca7 100644 --- a/src/test/scala/org/ergoplatform/nodeView/state/SnapshotsDbSpecification.scala +++ b/src/test/scala/org/ergoplatform/nodeView/state/SnapshotsDbSpecification.scala @@ -14,7 +14,7 @@ class SnapshotsDbSpecification extends ErgoPropertyTest { Random.nextInt(1000000) -> (Digest32 @@ idToBytes(mid)) }.toMap val si = new SnapshotsInfo(m) - val dir = createTempDir().getAbsolutePath + val dir = createTempDir.getAbsolutePath val db = SnapshotsDb.create(dir) db.writeSnapshotsInfo(si) si -> db diff --git a/src/test/scala/org/ergoplatform/sanity/ErgoSanityDigest.scala b/src/test/scala/org/ergoplatform/sanity/ErgoSanityDigest.scala index 63a8bdff13..0b7d947f44 100644 --- a/src/test/scala/org/ergoplatform/sanity/ErgoSanityDigest.scala +++ b/src/test/scala/org/ergoplatform/sanity/ErgoSanityDigest.scala @@ -27,8 +27,8 @@ class ErgoSanityDigest extends ErgoSanity[DIGEST_ST] { generateHistory(verifyTransactions = true, StateType.Digest, PoPoWBootstrap = false, -1) override val stateGen: Gen[WrappedDigestState] = { - boxesHolderGen.map(WrappedUtxoState(_, createTempDir(), None, parameters, settings)).map { wus => - val digestState = DigestState.create(Some(wus.version), Some(wus.rootDigest), createTempDir(), settings) + boxesHolderGen.map(WrappedUtxoState(_, createTempDir, None, parameters, settings)).map { wus => + val digestState = DigestState.create(Some(wus.version), Some(wus.rootDigest), createTempDir, settings) new WrappedDigestState(digestState, wus, settings) } } diff --git a/src/test/scala/org/ergoplatform/sanity/ErgoSanityUTXO.scala b/src/test/scala/org/ergoplatform/sanity/ErgoSanityUTXO.scala index 34e2c76381..a7e55e929e 100644 --- a/src/test/scala/org/ergoplatform/sanity/ErgoSanityUTXO.scala +++ b/src/test/scala/org/ergoplatform/sanity/ErgoSanityUTXO.scala @@ -27,7 +27,7 @@ class ErgoSanityUTXO extends ErgoSanity[UTXO_ST] with ErgoTestHelpers { generateHistory(verifyTransactions = true, StateType.Utxo, PoPoWBootstrap = false, blocksToKeep = -1) override val stateGen: Gen[WrappedUtxoState] = - boxesHolderGen.map(WrappedUtxoState(_, createTempDir(), None, parameters, settings)) + boxesHolderGen.map(WrappedUtxoState(_, createTempDir, None, parameters, settings)) override def semanticallyValidModifier(state: UTXO_ST): PM = { statefulyValidFullBlock(state.asInstanceOf[WrappedUtxoState]) diff --git a/src/test/scala/org/ergoplatform/utils/HistoryTestHelpers.scala b/src/test/scala/org/ergoplatform/utils/HistoryTestHelpers.scala index 6028697ddf..c6b1c00233 100644 --- a/src/test/scala/org/ergoplatform/utils/HistoryTestHelpers.scala +++ b/src/test/scala/org/ergoplatform/utils/HistoryTestHelpers.scala @@ -62,7 +62,7 @@ trait HistoryTestHelpers extends ErgoPropertyTest { settings.chainSettings.copy(epochLength = epochLength, useLastEpochs = useLastEpochs, genesisId = genesisIdOpt) } - val dir = createTempDir() + val dir = createTempDir val fullHistorySettings: ErgoSettings = ErgoSettings(dir.getAbsolutePath, NetworkType.TestNet, chainSettings, nodeSettings, scorexSettings, walletSettings, settings.cacheSettings) diff --git a/src/test/scala/org/ergoplatform/utils/Stubs.scala b/src/test/scala/org/ergoplatform/utils/Stubs.scala index abae63689f..081364cdd9 100644 --- a/src/test/scala/org/ergoplatform/utils/Stubs.scala +++ b/src/test/scala/org/ergoplatform/utils/Stubs.scala @@ -32,7 +32,7 @@ import org.ergoplatform.wallet.boxes.{ChainStatus, TrackedBox} import org.ergoplatform.wallet.interface4j.SecretString import org.ergoplatform.wallet.interpreter.ErgoProvingInterpreter import org.ergoplatform.wallet.mnemonic.Mnemonic -import org.ergoplatform.wallet.utils.TestFileUtils +import org.ergoplatform.wallet.utils.FileUtils import org.scalacheck.Gen import scorex.core.network.NetworkController.ReceivableMessages.GetConnectedPeers import org.ergoplatform.network.peer.PeerManager.ReceivableMessages.{GetAllPeers, GetBlacklistedPeers} @@ -47,7 +47,7 @@ import scala.collection.mutable import scala.concurrent.duration._ import scala.util.{Failure, Success, Try} -trait Stubs extends ErgoGenerators with ErgoTestHelpers with ChainGenerator with TestFileUtils { +trait Stubs extends ErgoGenerators with ErgoTestHelpers with ChainGenerator with FileUtils { implicit val system: ActorSystem @@ -56,15 +56,15 @@ trait Stubs extends ErgoGenerators with ErgoTestHelpers with ChainGenerator with val history: HT = applyChain(generateHistory(), chain) val digestState: DigestState = { - boxesHolderGen.map(WrappedUtxoState(_, createTempDir(), None, parameters, settings)).map { wus => - DigestState.create(Some(wus.version), Some(wus.rootDigest), createTempDir(), settings) + boxesHolderGen.map(WrappedUtxoState(_, createTempDir, None, parameters, settings)).map { wus => + DigestState.create(Some(wus.version), Some(wus.rootDigest), createTempDir, settings) } }.sample.value val utxoSettings: ErgoSettings = settings.copy(nodeSettings = settings.nodeSettings.copy(stateType = StateType.Utxo)) val utxoState: WrappedUtxoState = - boxesHolderGen.map(WrappedUtxoState(_, createTempDir(), None, parameters, utxoSettings)).sample.value + boxesHolderGen.map(WrappedUtxoState(_, createTempDir, None, parameters, utxoSettings)).sample.value lazy val wallet = new WalletStub @@ -377,7 +377,7 @@ trait Stubs extends ErgoGenerators with ErgoTestHelpers with ChainGenerator with val walletSettings: WalletSettings = null val chainSettings = settings.chainSettings.copy(epochLength = epochLength, useLastEpochs = useLastEpochs) - val dir = createTempDir() + val dir = createTempDir val fullHistorySettings: ErgoSettings = ErgoSettings(dir.getAbsolutePath, NetworkType.TestNet, chainSettings, nodeSettings, scorexSettings, walletSettings, settings.cacheSettings) diff --git a/src/test/scala/org/ergoplatform/utils/fixtures/NodeViewFixture.scala b/src/test/scala/org/ergoplatform/utils/fixtures/NodeViewFixture.scala index 79bc3291e7..3ee6fe3d9b 100644 --- a/src/test/scala/org/ergoplatform/utils/fixtures/NodeViewFixture.scala +++ b/src/test/scala/org/ergoplatform/utils/fixtures/NodeViewFixture.scala @@ -6,20 +6,20 @@ import org.ergoplatform.mining.emission.EmissionRules import org.ergoplatform.nodeView.ErgoNodeViewRef import org.ergoplatform.settings.{ErgoSettings, Parameters} import org.ergoplatform.utils.NodeViewTestContext -import org.ergoplatform.wallet.utils.TestFileUtils +import org.ergoplatform.wallet.utils.FileUtils import scala.concurrent.ExecutionContext /** This uses TestProbe to receive messages from actor. * To make TestProbe work `defaultSender` implicit should be imported */ -class NodeViewFixture(protoSettings: ErgoSettings, parameters: Parameters) extends NodeViewTestContext with TestFileUtils { self => +class NodeViewFixture(protoSettings: ErgoSettings, parameters: Parameters) extends NodeViewTestContext with FileUtils { self => implicit val actorSystem: ActorSystem = ActorSystem() implicit val executionContext: ExecutionContext = actorSystem.dispatchers.lookup("scorex.executionContext") implicit def ctx: NodeViewTestContext = this - val nodeViewDir: java.io.File = createTempDir() + val nodeViewDir: java.io.File = createTempDir @volatile var settings: ErgoSettings = protoSettings.copy(directory = nodeViewDir.getAbsolutePath) val emission: EmissionRules = new EmissionRules(settings.chainSettings.monetary) @volatile var nodeViewHolderRef: ActorRef = ErgoNodeViewRef(settings) diff --git a/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala b/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala index f5e4d5226a..90ccc72f47 100644 --- a/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala +++ b/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala @@ -12,7 +12,7 @@ import org.ergoplatform.nodeView.state.wrapped.WrappedUtxoState import org.ergoplatform.settings.{Algos, Constants, ErgoSettings, Parameters} import org.ergoplatform.testkit.TestkitHelpers import org.ergoplatform.utils.{LoggingUtil, RandomLike, RandomWrapper} -import org.ergoplatform.wallet.utils.TestFileUtils +import org.ergoplatform.wallet.utils.FileUtils import org.scalatest.matchers.should.Matchers import org.ergoplatform.core.VersionTag import scorex.crypto.authds.avltree.batch.Remove @@ -24,17 +24,17 @@ import scala.collection.mutable import scala.util.{Failure, Random, Success} trait ValidBlocksGenerators - extends TestkitHelpers with TestFileUtils with Matchers with ChainGenerator with ErgoTransactionGenerators { + extends TestkitHelpers with FileUtils with Matchers with ChainGenerator with ErgoTransactionGenerators { def createUtxoState(settings: ErgoSettings): (UtxoState, BoxHolder) = { - ErgoState.generateGenesisUtxoState(createTempDir(), settings) + ErgoState.generateGenesisUtxoState(createTempDir, settings) } def createUtxoState(bh: BoxHolder, parameters: Parameters): UtxoState = - UtxoState.fromBoxHolder(bh, None, createTempDir(), settings, parameters) + UtxoState.fromBoxHolder(bh, None, createTempDir, settings, parameters) def createDigestState(version: VersionTag, digest: ADDigest): DigestState = - DigestState.create(Some(version), Some(digest), createTempDir(), settings) + DigestState.create(Some(version), Some(digest), createTempDir, settings) def validTransactionsFromBoxHolder(boxHolder: BoxHolder): (Seq[ErgoTransaction], BoxHolder) = validTransactionsFromBoxHolder(boxHolder, new RandomWrapper) From 7c5c98b70c2e47e741ba57651022afb0eb870fbe Mon Sep 17 00:00:00 2001 From: jellymlg Date: Wed, 28 Feb 2024 23:35:07 +0100 Subject: [PATCH 10/15] Renamed *LDB* to *RocksDB* --- .../benchmarks/AVLTreeBatchPerformance.scala | 8 +++--- .../crypto/authds/benchmarks/Helper.scala | 8 +++--- .../serialization/ManifestSerializer.scala | 4 +-- .../serialization/SubtreeSerializer.scala | 4 +-- .../avltree/batch/ProverNodeSerializer.scala | 6 ++-- .../batch/ProxyInternalProverNode.scala | 8 +++--- ...scala => VersionedRocksDBAVLStorage.scala} | 28 +++++++++---------- .../main/scala/scorex/db/KVStoreReader.scala | 2 +- ...{LDBFactory.scala => RocksDBFactory.scala} | 2 +- ...{LDBKVStore.scala => RocksDBKVStore.scala} | 4 +-- ...tore.scala => RocksDBVersionedStore.scala} | 10 +++---- .../AVLStorageWithPersistentProverSpec.scala | 8 +++--- ... RocksDBVersionedStoreSpecification.scala} | 8 +++--- ...ionedRocksDBAVLStorageSpecification.scala} | 12 ++++---- ...ksDBAVLStorageStatefulSpecification.scala} | 10 +++---- .../batch/benchmark/BatchingBenchmark.scala | 8 +++--- .../avltree/batch/benchmark/OOMTest.scala | 6 ++-- ...a => RocksDBVersionedStoreBenchmark.scala} | 6 ++-- .../avltree/batch/helpers/TestHelper.scala | 12 ++++---- ....scala => RocksDBVersionedStoreSpec.scala} | 6 ++-- .../network/ErgoNodeViewSynchronizer.scala | 2 +- .../network/peer/PeerDatabase.scala | 4 +-- .../history/storage/HistoryStorage.scala | 10 +++---- .../UtxoSetSnapshotProcessor.scala | 12 ++++---- .../nodeView/state/DigestState.scala | 8 +++--- .../nodeView/state/ErgoStateReader.scala | 6 ++-- .../nodeView/state/SnapshotsDb.scala | 10 +++---- .../state/UtxoSetSnapshotPersistence.scala | 4 +-- .../nodeView/state/UtxoState.scala | 12 ++++---- .../nodeView/state/UtxoStateReader.scala | 4 +-- .../wallet/persistence/WalletRegistry.scala | 12 ++++---- .../wallet/persistence/WalletStorage.scala | 6 ++-- .../scala/org/ergoplatform/db/DBSpec.scala | 14 +++++----- ...oreSpec.scala => RocksDBKVStoreSpec.scala} | 2 +- ...txoSetSnapshotProcessorSpecification.scala | 4 +-- .../state/wrapped/WrappedUtxoState.scala | 4 +-- .../wallet/ErgoWalletServiceSpec.scala | 4 +-- .../persistence/WalletStorageSpec.scala | 4 +-- 38 files changed, 141 insertions(+), 141 deletions(-) rename avldb/src/main/scala/scorex/crypto/authds/avltree/batch/{VersionedLDBAVLStorage.scala => VersionedRocksDBAVLStorage.scala} (90%) rename avldb/src/main/scala/scorex/db/{LDBFactory.scala => RocksDBFactory.scala} (98%) rename avldb/src/main/scala/scorex/db/{LDBKVStore.scala => RocksDBKVStore.scala} (94%) rename avldb/src/main/scala/scorex/db/{LDBVersionedStore.scala => RocksDBVersionedStore.scala} (98%) rename avldb/src/test/scala/scorex/crypto/authds/avltree/batch/{LDBVersionedStoreSpecification.scala => RocksDBVersionedStoreSpecification.scala} (83%) rename avldb/src/test/scala/scorex/crypto/authds/avltree/batch/{VersionedLDBAVLStorageSpecification.scala => VersionedRocksDBAVLStorageSpecification.scala} (97%) rename avldb/src/test/scala/scorex/crypto/authds/avltree/batch/{VersionedLDBAVLStorageStatefulSpecification.scala => VersionedRocksDBAVLStorageStatefulSpecification.scala} (97%) rename avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/{LDBVersionedStoreBenchmark.scala => RocksDBVersionedStoreBenchmark.scala} (86%) rename avldb/src/test/scala/scorex/db/{LDBVersionedStoreSpec.scala => RocksDBVersionedStoreSpec.scala} (94%) rename src/test/scala/org/ergoplatform/db/{LDBKVStoreSpec.scala => RocksDBKVStoreSpec.scala} (93%) diff --git a/avldb/benchmarks/src/main/scala/scorex/crypto/authds/benchmarks/AVLTreeBatchPerformance.scala b/avldb/benchmarks/src/main/scala/scorex/crypto/authds/benchmarks/AVLTreeBatchPerformance.scala index 9742eaa13b..39c7fa2434 100644 --- a/avldb/benchmarks/src/main/scala/scorex/crypto/authds/benchmarks/AVLTreeBatchPerformance.scala +++ b/avldb/benchmarks/src/main/scala/scorex/crypto/authds/benchmarks/AVLTreeBatchPerformance.scala @@ -4,9 +4,9 @@ import java.util.concurrent.TimeUnit import org.openjdk.jmh.annotations._ import org.slf4j.LoggerFactory -import scorex.crypto.authds.avltree.batch.{Operation, PersistentBatchAVLProver, VersionedLDBAVLStorage} +import scorex.crypto.authds.avltree.batch.{Operation, PersistentBatchAVLProver, VersionedRocksDBAVLStorage} import scorex.crypto.hash.{Blake2b256, Digest32} -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore object AVLTreeBatchPerformance extends { @@ -20,8 +20,8 @@ object AVLTreeBatchPerformance extends { val logger = LoggerFactory.getLogger("TEST") var prover: Prover = _ - var store: LDBVersionedStore = _ - var storage: VersionedLDBAVLStorage = _ + var store: RocksDBVersionedStore = _ + var storage: VersionedRocksDBAVLStorage = _ var operations: Array[Operation] = _ @Setup(Level.Iteration) diff --git a/avldb/benchmarks/src/main/scala/scorex/crypto/authds/benchmarks/Helper.scala b/avldb/benchmarks/src/main/scala/scorex/crypto/authds/benchmarks/Helper.scala index ab87dc73a9..577dd2b890 100644 --- a/avldb/benchmarks/src/main/scala/scorex/crypto/authds/benchmarks/Helper.scala +++ b/avldb/benchmarks/src/main/scala/scorex/crypto/authds/benchmarks/Helper.scala @@ -4,7 +4,7 @@ import com.google.common.primitives.Longs import scorex.crypto.authds.avltree.batch._ import scorex.crypto.authds.{ADKey, ADValue} import scorex.crypto.hash.{Blake2b256, Digest32} -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore import scorex.utils.Random object Helper { @@ -34,11 +34,11 @@ object Helper { } def persistentProverWithVersionedStore(initialKeepVersions: Int, - baseOperationsCount: Int = 0): (Prover, LDBVersionedStore, VersionedLDBAVLStorage) = { + baseOperationsCount: Int = 0): (Prover, RocksDBVersionedStore, VersionedRocksDBAVLStorage) = { val dir = java.nio.file.Files.createTempDirectory("bench_testing_" + scala.util.Random.alphanumeric.take(15)).toFile dir.deleteOnExit() - val store = new LDBVersionedStore(dir, initialKeepVersions = initialKeepVersions) - val storage = new VersionedLDBAVLStorage(store) + val store = new RocksDBVersionedStore(dir, initialKeepVersions = initialKeepVersions) + val storage = new VersionedRocksDBAVLStorage(store) require(storage.isEmpty) val prover = new BatchAVLProver[Digest32, HF](kl, Some(vl)) diff --git a/avldb/src/main/scala/org/ergoplatform/serialization/ManifestSerializer.scala b/avldb/src/main/scala/org/ergoplatform/serialization/ManifestSerializer.scala index 985eb421fe..4e3246b2e0 100644 --- a/avldb/src/main/scala/org/ergoplatform/serialization/ManifestSerializer.scala +++ b/avldb/src/main/scala/org/ergoplatform/serialization/ManifestSerializer.scala @@ -2,14 +2,14 @@ package org.ergoplatform.serialization import scorex.crypto.authds.avltree.batch.Constants.DigestType import scorex.crypto.authds.avltree.batch.serialization.{BatchAVLProverManifest, ProxyInternalNode} -import scorex.crypto.authds.avltree.batch.{InternalProverNode, ProverLeaf, ProverNodes, VersionedLDBAVLStorage} +import scorex.crypto.authds.avltree.batch.{InternalProverNode, ProverLeaf, ProverNodes, VersionedRocksDBAVLStorage} import scorex.util.serialization.{Reader, Writer} /** * Serializer of manifest, a tree which is cut at some `manifestDepth` from root */ class ManifestSerializer(manifestDepth: Byte) extends ErgoSerializer[BatchAVLProverManifest[DigestType]] { - private val nodeSerializer = VersionedLDBAVLStorage.noStoreSerializer + private val nodeSerializer = VersionedRocksDBAVLStorage.noStoreSerializer /** * Serialize manifest provided as top subtree and height separately. Used in tests. diff --git a/avldb/src/main/scala/org/ergoplatform/serialization/SubtreeSerializer.scala b/avldb/src/main/scala/org/ergoplatform/serialization/SubtreeSerializer.scala index f4822bbe3e..558a7b5eb6 100644 --- a/avldb/src/main/scala/org/ergoplatform/serialization/SubtreeSerializer.scala +++ b/avldb/src/main/scala/org/ergoplatform/serialization/SubtreeSerializer.scala @@ -1,7 +1,7 @@ package org.ergoplatform.serialization import scorex.crypto.authds.avltree.batch.Constants.DigestType -import scorex.crypto.authds.avltree.batch.{InternalProverNode, ProverLeaf, ProverNodes, VersionedLDBAVLStorage} +import scorex.crypto.authds.avltree.batch.{InternalProverNode, ProverLeaf, ProverNodes, VersionedRocksDBAVLStorage} import scorex.crypto.authds.avltree.batch.serialization.{BatchAVLProverSubtree, ProxyInternalNode} import scorex.util.ScorexLogging import scorex.util.serialization.{Reader, Writer} @@ -10,7 +10,7 @@ import scorex.util.serialization.{Reader, Writer} * Serializer for subtree */ object SubtreeSerializer extends ErgoSerializer[BatchAVLProverSubtree[DigestType]] with ScorexLogging { - private val nodeSerializer = VersionedLDBAVLStorage.noStoreSerializer + private val nodeSerializer = VersionedRocksDBAVLStorage.noStoreSerializer override def serialize(subtree: BatchAVLProverSubtree[DigestType], w: Writer): Unit = { def loop(node: ProverNodes[DigestType]): Unit = { diff --git a/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/ProverNodeSerializer.scala b/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/ProverNodeSerializer.scala index 9199acdd8f..9c5a12fe2a 100644 --- a/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/ProverNodeSerializer.scala +++ b/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/ProverNodeSerializer.scala @@ -6,7 +6,7 @@ import scorex.crypto.authds.{ADKey, ADValue, Balance} import scorex.crypto.authds.avltree.batch.Constants.{DigestType, hashFn} import scorex.crypto.authds.avltree.batch.serialization.ProxyInternalNode import scorex.crypto.hash.Digest32 -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore import scorex.util.serialization.{Reader, Writer} /** @@ -15,9 +15,9 @@ import scorex.util.serialization.{Reader, Writer} * * @param store - store which could be provided to fetch children of a node when a child is requested */ -class ProverNodeSerializer(store: LDBVersionedStore) extends ErgoSerializer[ProverNodes[DigestType]] { +class ProverNodeSerializer(store: RocksDBVersionedStore) extends ErgoSerializer[ProverNodes[DigestType]] { - import VersionedLDBAVLStorage.{InternalNodePrefix,LeafPrefix} + import VersionedRocksDBAVLStorage.{InternalNodePrefix,LeafPrefix} override def serialize(node: ProverNodes[DigestType], w: Writer): Unit = { node match { diff --git a/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/ProxyInternalProverNode.scala b/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/ProxyInternalProverNode.scala index c82b52cf3e..ebc1551bbf 100644 --- a/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/ProxyInternalProverNode.scala +++ b/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/ProxyInternalProverNode.scala @@ -1,7 +1,7 @@ package scorex.crypto.authds.avltree.batch import scorex.crypto.authds.{ADKey, Balance} -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore import InternalNode.InternalNodePrefix import scorex.crypto.authds.avltree.batch.Constants.{DigestType, hashFn} @@ -15,7 +15,7 @@ class ProxyInternalProverNode(protected var pk: ADKey, val leftLabel: ADKey, val rightLabel: ADKey, protected var pb: Balance = Balance @@ 0.toByte) - (store: LDBVersionedStore) + (store: RocksDBVersionedStore) extends InternalProverNode(k = pk, l = null, r = null, b = pb)(hashFn) { override protected def computeLabel: DigestType = { @@ -24,14 +24,14 @@ class ProxyInternalProverNode(protected var pk: ADKey, override def left: ProverNodes[DigestType] = { if (l == null) { - l = VersionedLDBAVLStorage.fetch(leftLabel)(store) + l = VersionedRocksDBAVLStorage.fetch(leftLabel)(store) } l } override def right: ProverNodes[DigestType] = { if (r == null) { - r = VersionedLDBAVLStorage.fetch(rightLabel)(store) + r = VersionedRocksDBAVLStorage.fetch(rightLabel)(store) } r } diff --git a/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorage.scala b/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/VersionedRocksDBAVLStorage.scala similarity index 90% rename from avldb/src/main/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorage.scala rename to avldb/src/main/scala/scorex/crypto/authds/avltree/batch/VersionedRocksDBAVLStorage.scala index 95d8e536dc..ec98a933a3 100644 --- a/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorage.scala +++ b/avldb/src/main/scala/scorex/crypto/authds/avltree/batch/VersionedRocksDBAVLStorage.scala @@ -2,13 +2,13 @@ package scorex.crypto.authds.avltree.batch import com.google.common.primitives.Ints import scorex.crypto.authds.avltree.batch.Constants.{DigestType, HashFnType, hashFn} -import scorex.crypto.authds.avltree.batch.VersionedLDBAVLStorage.{topNodeHashKey, topNodeHeightKey} +import scorex.crypto.authds.avltree.batch.VersionedRocksDBAVLStorage.{topNodeHashKey, topNodeHeightKey} import scorex.crypto.authds.avltree.batch.serialization.{BatchAVLProverManifest, BatchAVLProverSubtree, ProxyInternalNode} import scorex.crypto.authds.{ADDigest, ADKey} import scorex.util.encode.Base16 import scorex.crypto.hash import scorex.crypto.hash.Digest32 -import scorex.db.{LDBKVStore, LDBVersionedStore} +import scorex.db.{RocksDBKVStore, RocksDBVersionedStore} import scorex.util.ScorexLogging import scala.collection.mutable @@ -19,13 +19,13 @@ import scala.util.{Failure, Try} * * @param store - level db storage to save the tree in */ -class VersionedLDBAVLStorage(store: LDBVersionedStore) +class VersionedRocksDBAVLStorage(store: RocksDBVersionedStore) extends VersionedAVLStorage[DigestType] with ScorexLogging { - import VersionedLDBAVLStorage.nodeLabel + import VersionedRocksDBAVLStorage.nodeLabel private def restorePrunedRootNode(): Try[(ProverNodes[DigestType], Int)] = Try { - val rootNode = VersionedLDBAVLStorage.fetch(ADKey @@ store.get(topNodeHashKey).get)(store) + val rootNode = VersionedRocksDBAVLStorage.fetch(ADKey @@ store.get(topNodeHashKey).get)(store) val rootHeight = Ints.fromByteArray(store.get(topNodeHeightKey).get) rootNode -> rootHeight @@ -73,7 +73,7 @@ class VersionedLDBAVLStorage(store: LDBVersionedStore) isTop: Boolean): Array[(Array[Byte], Array[Byte])] = { // Should always serialize top node. It may not be new if it is the creation of the tree if (node.isNew || isTop) { - val pair: (Array[Byte], Array[Byte]) = (nodeLabel(node), VersionedLDBAVLStorage.noStoreSerializer.toBytes(node)) + val pair: (Array[Byte], Array[Byte]) = (nodeLabel(node), VersionedRocksDBAVLStorage.noStoreSerializer.toBytes(node)) node match { case n: InternalProverNode[DigestType] => val leftSubtree = serializedVisitedNodes(n.left, isTop = false) @@ -98,13 +98,13 @@ class VersionedLDBAVLStorage(store: LDBVersionedStore) * @param expectedRootHash - expected UTXO set authenticating tree root hash * @return - hash of root node of tree, or failure if an error (e.g. in database) happened */ - def dumpSnapshot(dumpStorage: LDBKVStore, manifestDepth: Byte, expectedRootHash: Array[Byte]): Try[Array[Byte]] = { + def dumpSnapshot(dumpStorage: RocksDBKVStore, manifestDepth: Byte, expectedRootHash: Array[Byte]): Try[Array[Byte]] = { store.processSnapshot { dbReader => def subtreeLoop(label: DigestType, builder: mutable.ArrayBuilder[Byte]): Unit = { val nodeBytes = dbReader.get(label) builder ++= nodeBytes - val node = VersionedLDBAVLStorage.noStoreSerializer.parseBytes(nodeBytes) + val node = VersionedRocksDBAVLStorage.noStoreSerializer.parseBytes(nodeBytes) node match { case in: ProxyInternalNode[DigestType] => subtreeLoop(Digest32 @@@ in.leftLabel, builder) @@ -123,7 +123,7 @@ class VersionedLDBAVLStorage(store: LDBVersionedStore) def manifestLoop(nodeDbKey: Array[Byte], level: Int, manifestBuilder: mutable.ArrayBuilder[Byte]): Unit = { val nodeBytes = dbReader.get(nodeDbKey) manifestBuilder ++= nodeBytes - val node = VersionedLDBAVLStorage.noStoreSerializer.parseBytes(nodeBytes) + val node = VersionedRocksDBAVLStorage.noStoreSerializer.parseBytes(nodeBytes) node match { case in: ProxyInternalNode[DigestType] if level == manifestDepth => dumpSubtree(Digest32 @@@ in.leftLabel) @@ -156,7 +156,7 @@ class VersionedLDBAVLStorage(store: LDBVersionedStore) } -object VersionedLDBAVLStorage { +object VersionedRocksDBAVLStorage { private[batch] val topNodeHashKey: Array[Byte] = Array.fill(StateTreeParameters.labelSize)(123: Byte) private[batch] val topNodeHeightKey: Array[Byte] = Array.fill(StateTreeParameters.labelSize)(124: Byte) @@ -179,7 +179,7 @@ object VersionedLDBAVLStorage { * @return node read from the database (or throws exception if there is no such node), in case of internal node it * returns pruned internal node, so a node where left and right children not stored, only their hashes */ - def fetch(dbKey: ADKey)(store: LDBVersionedStore): ProverNodes[DigestType] = { + def fetch(dbKey: ADKey)(store: RocksDBVersionedStore): ProverNodes[DigestType] = { val bytes = store(dbKey) val node = new ProverNodeSerializer(store).parseBytes(bytes) node.isNew = false @@ -202,7 +202,7 @@ object VersionedLDBAVLStorage { def recreate(manifest: BatchAVLProverManifest[DigestType], chunks: Iterator[BatchAVLProverSubtree[DigestType]], additionalData: Iterator[(Array[Byte], Array[Byte])], - store: LDBVersionedStore): Try[VersionedLDBAVLStorage] = { + store: RocksDBVersionedStore): Try[VersionedRocksDBAVLStorage] = { //todo: the function below copy-pasted from BatchAVLProver, eliminate boilerplate? def idCollector(node: ProverNodes[DigestType]): Iterator[(Array[Byte], Array[Byte])] = { @@ -219,12 +219,12 @@ object VersionedLDBAVLStorage { val rootNode = manifest.root val rootNodeHeight = manifest.rootHeight - val digestWrapper = VersionedLDBAVLStorage.digest(rootNode.label, rootNodeHeight) + val digestWrapper = VersionedRocksDBAVLStorage.digest(rootNode.label, rootNodeHeight) val indices = Iterator(topNodeHashKey -> nodeLabel(rootNode), topNodeHeightKey -> Ints.toByteArray(rootNodeHeight)) val nodesIterator = idCollector(manifest.root) ++ chunks.flatMap(subtree => idCollector(subtree.subtreeTop)) store.update(digestWrapper, toRemove = Nil, toUpdate = indices ++ nodesIterator ++ additionalData).map { _ => - new VersionedLDBAVLStorage(store) + new VersionedRocksDBAVLStorage(store) } } diff --git a/avldb/src/main/scala/scorex/db/KVStoreReader.scala b/avldb/src/main/scala/scorex/db/KVStoreReader.scala index cc37af3f8b..c46a6dc4cb 100644 --- a/avldb/src/main/scala/scorex/db/KVStoreReader.scala +++ b/avldb/src/main/scala/scorex/db/KVStoreReader.scala @@ -1,7 +1,7 @@ package scorex.db import org.rocksdb.ReadOptions -import scorex.db.LDBFactory.RegisteredDB +import scorex.db.RocksDBFactory.RegisteredDB import java.util.concurrent.locks.ReentrantReadWriteLock import scala.collection.mutable diff --git a/avldb/src/main/scala/scorex/db/LDBFactory.scala b/avldb/src/main/scala/scorex/db/RocksDBFactory.scala similarity index 98% rename from avldb/src/main/scala/scorex/db/LDBFactory.scala rename to avldb/src/main/scala/scorex/db/RocksDBFactory.scala index d6afa891bc..d52095be42 100644 --- a/avldb/src/main/scala/scorex/db/LDBFactory.scala +++ b/avldb/src/main/scala/scorex/db/RocksDBFactory.scala @@ -15,7 +15,7 @@ import scala.collection.mutable * And ergo application (mostly tests) quite frequently doesn't not explicitly close * database and tries to reopen it. */ -object LDBFactory extends ScorexLogging { +object RocksDBFactory extends ScorexLogging { RocksDB.loadLibrary() diff --git a/avldb/src/main/scala/scorex/db/LDBKVStore.scala b/avldb/src/main/scala/scorex/db/RocksDBKVStore.scala similarity index 94% rename from avldb/src/main/scala/scorex/db/LDBKVStore.scala rename to avldb/src/main/scala/scorex/db/RocksDBKVStore.scala index af50bb3150..b24519ceca 100644 --- a/avldb/src/main/scala/scorex/db/LDBKVStore.scala +++ b/avldb/src/main/scala/scorex/db/RocksDBKVStore.scala @@ -1,7 +1,7 @@ package scorex.db import org.rocksdb.{WriteBatch, WriteOptions} -import scorex.db.LDBFactory.RegisteredDB +import scorex.db.RocksDBFactory.RegisteredDB import scorex.util.ScorexLogging import spire.syntax.all.cfor @@ -13,7 +13,7 @@ import scala.util.{Failure, Success, Try} * * Both keys and values are var-sized byte arrays. */ -class LDBKVStore(protected val db: RegisteredDB) extends KVStoreReader with ScorexLogging { +class RocksDBKVStore(protected val db: RegisteredDB) extends KVStoreReader with ScorexLogging { /** Immutable empty array can be shared to avoid allocations. */ private val emptyArrayOfByteArray = Array.empty[Array[Byte]] diff --git a/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala b/avldb/src/main/scala/scorex/db/RocksDBVersionedStore.scala similarity index 98% rename from avldb/src/main/scala/scorex/db/LDBVersionedStore.scala rename to avldb/src/main/scala/scorex/db/RocksDBVersionedStore.scala index 36f0cdaff1..3c46cf327e 100644 --- a/avldb/src/main/scala/scorex/db/LDBVersionedStore.scala +++ b/avldb/src/main/scala/scorex/db/RocksDBVersionedStore.scala @@ -2,8 +2,8 @@ package scorex.db import org.rocksdb.{ReadOptions, WriteBatch, WriteOptions} import scorex.crypto.hash.Blake2b256 -import scorex.db.LDBFactory.RegisteredDB -import scorex.db.LDBVersionedStore.SnapshotReadInterface +import scorex.db.RocksDBFactory.RegisteredDB +import scorex.db.RocksDBVersionedStore.SnapshotReadInterface import scorex.util.ScorexLogging import java.io.File @@ -25,7 +25,7 @@ import scala.util.{Failure, Success, Try} * @param initialKeepVersions - number of versions to keep when the store is created. Can be changed after. * */ -class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) +class RocksDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) extends KVStoreReader with ScorexLogging { type VersionID = Array[Byte] @@ -52,7 +52,7 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) private val writeOptions = new WriteOptions() private def createDB(dir: File, storeName: String): RegisteredDB = - LDBFactory.open(new File(dir, storeName)) + RocksDBFactory.open(new File(dir, storeName)) /** Set new keep versions threshold, remove not needed versions and return old value of keep versions */ def setKeepVersions(newKeepVersions: Int): Int = { @@ -432,7 +432,7 @@ class LDBVersionedStore(protected val dir: File, val initialKeepVersions: Int) } -object LDBVersionedStore { +object RocksDBVersionedStore { /** * Interface to read from versioned database snapshot which can be provided to clients in order to serve them with diff --git a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/AVLStorageWithPersistentProverSpec.scala b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/AVLStorageWithPersistentProverSpec.scala index 47588f0af5..6c979225e3 100644 --- a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/AVLStorageWithPersistentProverSpec.scala +++ b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/AVLStorageWithPersistentProverSpec.scala @@ -2,11 +2,11 @@ package scorex.crypto.authds.avltree.batch import org.scalatest.matchers.should.Matchers import org.scalatest.propspec.AnyPropSpec -import scorex.crypto.authds.avltree.batch.benchmark.LDBVersionedStoreBenchmark.getRandomTempDir +import scorex.crypto.authds.avltree.batch.benchmark.RocksDBVersionedStoreBenchmark.getRandomTempDir import scorex.crypto.authds.{ADDigest, ADKey, SerializedAdProof, ADValue} import scorex.crypto.hash.{Digest32, Blake2b256} import scorex.utils.Random -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore import scala.util.{Success, Failure, Try} @@ -15,9 +15,9 @@ class AVLStorageWithPersistentProverSpec extends AnyPropSpec with Matchers { type HF = Blake2b256.type implicit val hf: HF = Blake2b256 - val stateStore = new LDBVersionedStore(getRandomTempDir, 10) + val stateStore = new RocksDBVersionedStore(getRandomTempDir, 10) - protected lazy val storage = new VersionedLDBAVLStorage(stateStore) + protected lazy val storage = new VersionedRocksDBAVLStorage(stateStore) protected lazy val persistentProver: PersistentBatchAVLProver[Digest32, HF] = PersistentBatchAVLProver.create( diff --git a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/LDBVersionedStoreSpecification.scala b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/RocksDBVersionedStoreSpecification.scala similarity index 83% rename from avldb/src/test/scala/scorex/crypto/authds/avltree/batch/LDBVersionedStoreSpecification.scala rename to avldb/src/test/scala/scorex/crypto/authds/avltree/batch/RocksDBVersionedStoreSpecification.scala index c1fe98d5df..001bf01cd7 100644 --- a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/LDBVersionedStoreSpecification.scala +++ b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/RocksDBVersionedStoreSpecification.scala @@ -6,11 +6,11 @@ import org.scalatest.propspec.AnyPropSpec import org.scalatestplus.scalacheck.ScalaCheckPropertyChecks import scorex.crypto.authds.avltree.batch.helpers.TestHelper import scorex.crypto.hash.Blake2b256 -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore import scala.collection.mutable.ArrayBuffer -class LDBVersionedStoreSpecification extends AnyPropSpec +class RocksDBVersionedStoreSpecification extends AnyPropSpec with ScalaCheckPropertyChecks with Matchers with TestHelper { @@ -18,7 +18,7 @@ class LDBVersionedStoreSpecification extends AnyPropSpec override protected val KL = 32 override protected val VL = 8 - val storeTest: LDBVersionedStore => Unit = { store => + val storeTest: RocksDBVersionedStore => Unit = { store => var version = store.lastVersionID val keys: ArrayBuffer[(Array[Byte], Array[Byte])] = ArrayBuffer() forAll { b: Array[Byte] => @@ -37,6 +37,6 @@ class LDBVersionedStoreSpecification extends AnyPropSpec } } - property("LDBVersionedStore") { storeTest(createVersionedStore()) } + property("RocksDBVersionedStore") { storeTest(createVersionedStore()) } } diff --git a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageSpecification.scala b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedRocksDBAVLStorageSpecification.scala similarity index 97% rename from avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageSpecification.scala rename to avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedRocksDBAVLStorageSpecification.scala index 605609f108..aea3535589 100644 --- a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageSpecification.scala +++ b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedRocksDBAVLStorageSpecification.scala @@ -11,7 +11,7 @@ import scorex.crypto.authds.avltree.batch.helpers.TestHelper import scorex.crypto.authds.{ADDigest, ADKey, ADValue, SerializedAdProof} import scorex.util.encode.Base16 import scorex.crypto.hash.{Blake2b256, Digest32} -import scorex.db.{LDBFactory, LDBKVStore, LDBVersionedStore} +import scorex.db.{RocksDBFactory, RocksDBKVStore, RocksDBVersionedStore} import scorex.util.ByteArrayBuilder import scorex.util.serialization.VLQByteBufferWriter import scorex.utils.{Random => RandomBytes} @@ -21,7 +21,7 @@ import scala.concurrent.duration._ import scala.concurrent.{Await, Future} import scala.util.{Success, Try} -class VersionedLDBAVLStorageSpecification +class VersionedRocksDBAVLStorageSpecification extends AnyPropSpec with ScalaCheckPropertyChecks with Matchers @@ -204,7 +204,7 @@ class VersionedLDBAVLStorageSpecification noException should be thrownBy storage.rollbackVersions.foreach(v => prover.rollback(v).get) } - def testAddInfoSaving(createStore: Int => LDBVersionedStore): Assertion = { + def testAddInfoSaving(createStore: Int => RocksDBVersionedStore): Assertion = { val store = createStore(1000) val storage = createVersionedStorage(store) val prover = createPersistentProver(storage) @@ -247,7 +247,7 @@ class VersionedLDBAVLStorageSpecification store.get(addInfo2._1) shouldBe None } - def removeFromLargerSetSingleRandomElementTest(createStore: Int => LDBVersionedStore): Unit = { + def removeFromLargerSetSingleRandomElementTest(createStore: Int => RocksDBVersionedStore): Unit = { val minSetSize = 10000 val maxSetSize = 200000 @@ -349,8 +349,8 @@ class VersionedLDBAVLStorageSpecification val prover = createPersistentProver() blockchainWorkflowTest(prover) - val storage = prover.storage.asInstanceOf[VersionedLDBAVLStorage] - val store = new LDBKVStore(LDBFactory.open(getRandomTempDir)) + val storage = prover.storage.asInstanceOf[VersionedRocksDBAVLStorage] + val store = new RocksDBKVStore(RocksDBFactory.open(getRandomTempDir)) val rootNodeLabel = storage.dumpSnapshot(store, manifestDepth, prover.digest.dropRight(1)).get rootNodeLabel.sameElements(prover.digest.dropRight(1)) shouldBe true diff --git a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageStatefulSpecification.scala b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedRocksDBAVLStorageStatefulSpecification.scala similarity index 97% rename from avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageStatefulSpecification.scala rename to avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedRocksDBAVLStorageStatefulSpecification.scala index 3d393582c6..7a0a3c62ca 100644 --- a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedLDBAVLStorageStatefulSpecification.scala +++ b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/VersionedRocksDBAVLStorageStatefulSpecification.scala @@ -12,18 +12,18 @@ import scorex.utils.{Random => RandomBytes} import scala.util.{Random, Success, Failure, Try} -class VersionedLDBAVLStorageStatefulSpecification extends AnyPropSpec { +class VersionedRocksDBAVLStorageStatefulSpecification extends AnyPropSpec { val params = Parameters.default .withMinSize(10) .withMaxSize(50) .withMinSuccessfulTests(15) - property("LDBAVLStorage: rollback in stateful environment") { - WithLDB.property().check(params) + property("RocksDBAVLStorage: rollback in stateful environment") { + WithRocksDB.property().check(params) } } -object WithLDB extends VersionedLDBAVLStorageStatefulCommands with TestHelper { +object WithRocksDB extends VersionedRocksDBAVLStorageStatefulCommands with TestHelper { override protected val KL = 32 override protected val VL = 8 @@ -33,7 +33,7 @@ object WithLDB extends VersionedLDBAVLStorageStatefulCommands with TestHelper { } } -trait VersionedLDBAVLStorageStatefulCommands extends Commands { this: TestHelper => +trait VersionedRocksDBAVLStorageStatefulCommands extends Commands { this: TestHelper => override type State = Operations override type Sut = PersistentBatchAVLProver[Digest32, HF] diff --git a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/BatchingBenchmark.scala b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/BatchingBenchmark.scala index 6179c933cc..5e9a81aad4 100644 --- a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/BatchingBenchmark.scala +++ b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/BatchingBenchmark.scala @@ -2,10 +2,10 @@ package scorex.crypto.authds.avltree.batch.benchmark import scorex.crypto.authds._ import scorex.crypto.authds.avltree.batch.helpers.FileHelper -import scorex.crypto.authds.avltree.batch.{VersionedLDBAVLStorage, _} +import scorex.crypto.authds.avltree.batch._ import scorex.crypto.hash.{Blake2b256, Digest32} import scorex.utils.Random -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore object BatchingBenchmark extends App with FileHelper { val KeyLength = 26 @@ -18,8 +18,8 @@ object BatchingBenchmark extends App with FileHelper { implicit val hf: HF = Blake2b256 type HF = Blake2b256.type - val store = new LDBVersionedStore(getRandomTempDir, initialKeepVersions = 10) - val storage = new VersionedLDBAVLStorage(store) + val store = new RocksDBVersionedStore(getRandomTempDir, initialKeepVersions = 10) + val storage = new VersionedRocksDBAVLStorage(store) require(storage.isEmpty) val mods = generateModifications() var digest: ADDigest = ADDigest @@ Array[Byte]() diff --git a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/OOMTest.scala b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/OOMTest.scala index 03b34284e2..e5c35b688e 100644 --- a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/OOMTest.scala +++ b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/OOMTest.scala @@ -8,7 +8,7 @@ import scorex.crypto.authds.{ADDigest, ADKey, ADValue} import scorex.crypto.authds.avltree.batch._ import scorex.util.encode.Base16 import scorex.crypto.hash.{Blake2b256, Digest32} -import scorex.db.{ByteArrayWrapper, LDBVersionedStore} +import scorex.db.{ByteArrayWrapper, RocksDBVersionedStore} import scala.collection.immutable.SortedMap @@ -20,10 +20,10 @@ object OOMTest extends App { protected implicit val hf = Blake2b256 val dir: File = Files.createTempDirectory("oom-test").toFile - val store = new LDBVersionedStore(dir, initialKeepVersions = 200) + val store = new RocksDBVersionedStore(dir, initialKeepVersions = 200) val bestVersionKey = Blake2b256("best state version") - protected lazy val storage = new VersionedLDBAVLStorage(store) + protected lazy val storage = new VersionedRocksDBAVLStorage(store) val afterGenesisStateDigestHex: String = "78b130095239561ecf5449a7794c0615326d1fd007cc79dcc286e46e4beb1d3f01" val afterGenesisStateDigest: ADDigest = ADDigest @@ Base16.decode(afterGenesisStateDigestHex).get diff --git a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/LDBVersionedStoreBenchmark.scala b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/RocksDBVersionedStoreBenchmark.scala similarity index 86% rename from avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/LDBVersionedStoreBenchmark.scala rename to avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/RocksDBVersionedStoreBenchmark.scala index 62c16fcc10..2c5b10e1a0 100644 --- a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/LDBVersionedStoreBenchmark.scala +++ b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/benchmark/RocksDBVersionedStoreBenchmark.scala @@ -3,16 +3,16 @@ package scorex.crypto.authds.avltree.batch.benchmark import com.google.common.primitives.Longs import scorex.crypto.authds.avltree.batch.helpers.FileHelper import scorex.utils.Random -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore -object LDBVersionedStoreBenchmark extends App with FileHelper { +object RocksDBVersionedStoreBenchmark extends App with FileHelper { val KL = 32 val VL = 8 val LL = 32 val NumMods = 2000000 val Step = 1000 - val store = new LDBVersionedStore(getRandomTempDir, 10) + val store = new RocksDBVersionedStore(getRandomTempDir, 10) val mods = generateModifications() var currentVersion: Option[Long] = None diff --git a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/helpers/TestHelper.scala b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/helpers/TestHelper.scala index 0b2c4d79a9..dde6f8f99c 100644 --- a/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/helpers/TestHelper.scala +++ b/avldb/src/test/scala/scorex/crypto/authds/avltree/batch/helpers/TestHelper.scala @@ -4,7 +4,7 @@ import scorex.crypto.authds.avltree.batch._ import scorex.crypto.authds.{ADDigest, SerializedAdProof} import scorex.util.encode.Base58 import scorex.crypto.hash.{Blake2b256, Digest32} -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore trait TestHelper extends FileHelper { @@ -15,20 +15,20 @@ trait TestHelper extends FileHelper { type PROVER = BatchAVLProver[D, HF] type VERIFIER = BatchAVLVerifier[D, HF] type PERSISTENT_PROVER = PersistentBatchAVLProver[D, HF] - type STORAGE = VersionedLDBAVLStorage + type STORAGE = VersionedRocksDBAVLStorage protected val KL: Int protected val VL: Int implicit val hf: HF = Blake2b256 - def createVersionedStore(initialKeepVersions: Int = 10): LDBVersionedStore = { + def createVersionedStore(initialKeepVersions: Int = 10): RocksDBVersionedStore = { val dir = getRandomTempDir - new LDBVersionedStore(dir, initialKeepVersions = initialKeepVersions) + new RocksDBVersionedStore(dir, initialKeepVersions = initialKeepVersions) } - def createVersionedStorage(store: LDBVersionedStore): STORAGE = - new VersionedLDBAVLStorage(store) + def createVersionedStorage(store: RocksDBVersionedStore): STORAGE = + new VersionedRocksDBAVLStorage(store) def createPersistentProver(storage: STORAGE): PERSISTENT_PROVER = { val prover = new BatchAVLProver[D, HF](KL, Some(VL)) diff --git a/avldb/src/test/scala/scorex/db/LDBVersionedStoreSpec.scala b/avldb/src/test/scala/scorex/db/RocksDBVersionedStoreSpec.scala similarity index 94% rename from avldb/src/test/scala/scorex/db/LDBVersionedStoreSpec.scala rename to avldb/src/test/scala/scorex/db/RocksDBVersionedStoreSpec.scala index 9a7bfa4f70..f52b54d900 100644 --- a/avldb/src/test/scala/scorex/db/LDBVersionedStoreSpec.scala +++ b/avldb/src/test/scala/scorex/db/RocksDBVersionedStoreSpec.scala @@ -3,16 +3,16 @@ package scorex.db import com.google.common.primitives.Longs import org.scalatest.matchers.should.Matchers import org.scalatest.propspec.AnyPropSpec -import scorex.crypto.authds.avltree.batch.benchmark.LDBVersionedStoreBenchmark.getRandomTempDir +import scorex.crypto.authds.avltree.batch.benchmark.RocksDBVersionedStoreBenchmark.getRandomTempDir import scala.collection.mutable import scala.util.Random //todo: rollbacks and pruning are checked in VersionedStoreSpec, merge both tests? -class LDBVersionedStoreSpec extends AnyPropSpec with Matchers { +class RocksDBVersionedStoreSpec extends AnyPropSpec with Matchers { private val dir = getRandomTempDir - private val store = new LDBVersionedStore(dir, 100) + private val store = new RocksDBVersionedStore(dir, 100) property("last version correct && versionIdExists && rollbackVersions") { val versionNum = Random.nextInt().toLong diff --git a/src/main/scala/org/ergoplatform/network/ErgoNodeViewSynchronizer.scala b/src/main/scala/org/ergoplatform/network/ErgoNodeViewSynchronizer.scala index e2a550282a..015f872939 100644 --- a/src/main/scala/org/ergoplatform/network/ErgoNodeViewSynchronizer.scala +++ b/src/main/scala/org/ergoplatform/network/ErgoNodeViewSynchronizer.scala @@ -35,7 +35,7 @@ import org.ergoplatform.modifiers.history.{ADProofs, ADProofsSerializer, BlockTr import org.ergoplatform.modifiers.history.extension.{Extension, ExtensionSerializer} import org.ergoplatform.modifiers.transaction.TooHighCostError import org.ergoplatform.serialization.{ErgoSerializer, ManifestSerializer, SubtreeSerializer} -import scorex.crypto.authds.avltree.batch.VersionedLDBAVLStorage.splitDigest +import scorex.crypto.authds.avltree.batch.VersionedRocksDBAVLStorage.splitDigest import scala.annotation.tailrec import scala.collection.mutable diff --git a/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala b/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala index ef99cdbd2a..e68f54d8b3 100644 --- a/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala +++ b/src/main/scala/org/ergoplatform/network/peer/PeerDatabase.scala @@ -5,7 +5,7 @@ import org.ergoplatform.nodeView.history.ErgoHistoryUtils._ import java.io.{ByteArrayInputStream, ByteArrayOutputStream, File, ObjectInputStream, ObjectOutputStream} import java.net.{InetAddress, InetSocketAddress} import org.ergoplatform.settings.ErgoSettings -import scorex.db.{LDBFactory, LDBKVStore} +import scorex.db.{RocksDBFactory, RocksDBKVStore} import scorex.util.ScorexLogging import scala.concurrent.duration._ @@ -16,7 +16,7 @@ import scala.util.{Failure, Success, Try} */ final class PeerDatabase(settings: ErgoSettings) extends ScorexLogging { - private val persistentStore = new LDBKVStore(LDBFactory.open(new File(s"${settings.directory}/peers"))) + private val persistentStore = new RocksDBKVStore(RocksDBFactory.open(new File(s"${settings.directory}/peers"))) private var peers = loadPeers match { diff --git a/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala b/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala index bf11d3d23a..4fa74b963c 100644 --- a/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala +++ b/src/main/scala/org/ergoplatform/nodeView/history/storage/HistoryStorage.scala @@ -7,7 +7,7 @@ import org.ergoplatform.modifiers.history.header.Header import org.ergoplatform.nodeView.history.extra.{ExtraIndex, ExtraIndexSerializer, Segment} import org.ergoplatform.settings.{Algos, CacheSettings, ErgoSettings} import org.ergoplatform.utils.ScorexEncoding -import scorex.db.{ByteArrayWrapper, LDBFactory, LDBKVStore} +import scorex.db.{ByteArrayWrapper, RocksDBFactory, RocksDBKVStore} import scorex.util.{ModifierId, ScorexLogging, idToBytes} import scala.util.{Failure, Success, Try} @@ -26,7 +26,7 @@ import scala.jdk.CollectionConverters.asScalaIteratorConverter * @param extraStore - key-value store, where key is id of Index and value is it's bytes * @param config - cache configs */ -class HistoryStorage private(indexStore: LDBKVStore, objectsStore: LDBKVStore, extraStore: LDBKVStore, config: CacheSettings) +class HistoryStorage private(indexStore: RocksDBKVStore, objectsStore: RocksDBKVStore, extraStore: RocksDBKVStore, config: CacheSettings) extends ScorexLogging with AutoCloseable with ScorexEncoding { @@ -229,9 +229,9 @@ class HistoryStorage private(indexStore: LDBKVStore, objectsStore: LDBKVStore, e object HistoryStorage { def apply(ergoSettings: ErgoSettings): HistoryStorage = { - val indexStore = new LDBKVStore(LDBFactory.open(new File(s"${ergoSettings.directory}/history/index"))) - val objectsStore = new LDBKVStore(LDBFactory.open(new File(s"${ergoSettings.directory}/history/objects"))) - val extraStore = new LDBKVStore(LDBFactory.open(new File(s"${ergoSettings.directory}/history/extra"))) + val indexStore = new RocksDBKVStore(RocksDBFactory.open(new File(s"${ergoSettings.directory}/history/index"))) + val objectsStore = new RocksDBKVStore(RocksDBFactory.open(new File(s"${ergoSettings.directory}/history/objects"))) + val extraStore = new RocksDBKVStore(RocksDBFactory.open(new File(s"${ergoSettings.directory}/history/extra"))) new HistoryStorage(indexStore, objectsStore, extraStore, ergoSettings.cacheSettings) } } diff --git a/src/main/scala/org/ergoplatform/nodeView/history/storage/modifierprocessors/UtxoSetSnapshotProcessor.scala b/src/main/scala/org/ergoplatform/nodeView/history/storage/modifierprocessors/UtxoSetSnapshotProcessor.scala index 1d2972cfe9..c886cdf523 100644 --- a/src/main/scala/org/ergoplatform/nodeView/history/storage/modifierprocessors/UtxoSetSnapshotProcessor.scala +++ b/src/main/scala/org/ergoplatform/nodeView/history/storage/modifierprocessors/UtxoSetSnapshotProcessor.scala @@ -13,12 +13,12 @@ import scorex.core.network.ConnectedPeer import org.ergoplatform.serialization.SubtreeSerializer import scorex.crypto.authds.avltree.batch.serialization.{BatchAVLProverManifest, BatchAVLProverSubtree} import scorex.crypto.hash.{Blake2b256, Digest32} -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore import scorex.util.{ModifierId, ScorexLogging} import spire.syntax.all.cfor import scala.util.{Failure, Random, Success, Try} -import scorex.crypto.authds.avltree.batch.{BatchAVLProver, PersistentBatchAVLProver, VersionedLDBAVLStorage} +import scorex.crypto.authds.avltree.batch.{BatchAVLProver, PersistentBatchAVLProver, VersionedRocksDBAVLStorage} /** * Parts of history processing and storage corresponding to UTXO set snapshot processing and storage @@ -204,7 +204,7 @@ trait UtxoSetSnapshotProcessor extends MinimalFullBlockHeightFunctions with Scor * @param blockId - id of a block corresponding to the tree (tree is on top of a state after the block) * @return prover with initialized tree database */ - def createPersistentProver(stateStore: LDBVersionedStore, + def createPersistentProver(stateStore: RocksDBVersionedStore, historyReader: ErgoHistoryReader, height: Height, blockId: ModifierId): Try[PersistentBatchAVLProver[Digest32, HF]] = { @@ -213,15 +213,15 @@ trait UtxoSetSnapshotProcessor extends MinimalFullBlockHeightFunctions with Scor log.info("Starting UTXO set snapshot transfer into state database") ErgoStateReader.reconstructStateContextBeforeEpoch(historyReader, height, settings) match { case Success(esc) => - val metadata = UtxoState.metadata(VersionTag @@@ blockId, VersionedLDBAVLStorage.digest(manifest.id, manifest.rootHeight), None, esc) - VersionedLDBAVLStorage.recreate(manifest, downloadedChunksIterator(), additionalData = metadata.toIterator, stateStore).flatMap { + val metadata = UtxoState.metadata(VersionTag @@@ blockId, VersionedRocksDBAVLStorage.digest(manifest.id, manifest.rootHeight), None, esc) + VersionedRocksDBAVLStorage.recreate(manifest, downloadedChunksIterator(), additionalData = metadata.toIterator, stateStore).flatMap { ldbStorage => log.info("Finished UTXO set snapshot transfer into state database") ldbStorage.restorePrunedProver().map { prunedAvlProver => new PersistentBatchAVLProver[Digest32, HF] { override var avlProver: BatchAVLProver[Digest32, ErgoAlgos.HF] = prunedAvlProver - override val storage: VersionedLDBAVLStorage = ldbStorage + override val storage: VersionedRocksDBAVLStorage = ldbStorage } } } diff --git a/src/main/scala/org/ergoplatform/nodeView/state/DigestState.scala b/src/main/scala/org/ergoplatform/nodeView/state/DigestState.scala index 3c5a3a4721..b9bb6bd5c0 100644 --- a/src/main/scala/org/ergoplatform/nodeView/state/DigestState.scala +++ b/src/main/scala/org/ergoplatform/nodeView/state/DigestState.scala @@ -11,7 +11,7 @@ import org.ergoplatform.nodeView.state.ErgoState.ModifierProcessing import org.ergoplatform.settings._ import org.ergoplatform.utils.LoggingUtil import org.ergoplatform.wallet.boxes.ErgoBoxSerializer -import scorex.db.{ByteArrayWrapper, LDBVersionedStore} +import scorex.db.{ByteArrayWrapper, RocksDBVersionedStore} import org.ergoplatform.core._ import org.ergoplatform.nodeView.LocallyGeneratedModifier import org.ergoplatform.utils.ScorexEncoding @@ -26,7 +26,7 @@ import scala.util.{Failure, Success, Try} */ class DigestState protected(override val version: VersionTag, override val rootDigest: ADDigest, - override val store: LDBVersionedStore, + override val store: RocksDBVersionedStore, override val ergoSettings: ErgoSettings) extends ErgoState[DigestState] with ScorexLogging @@ -161,7 +161,7 @@ object DigestState extends ScorexLogging with ScorexEncoding { stateContext: ErgoStateContext, dir: File, settings: ErgoSettings): Try[DigestState] = { - val store = new LDBVersionedStore(dir, initialKeepVersions = settings.nodeSettings.keepVersions) + val store = new RocksDBVersionedStore(dir, initialKeepVersions = settings.nodeSettings.keepVersions) val toUpdate = DigestState.metadata(version, rootHash, stateContext) store.update(org.ergoplatform.core.versionToBytes(version), Seq.empty, toUpdate).map { _ => @@ -176,7 +176,7 @@ object DigestState extends ScorexLogging with ScorexEncoding { rootHashOpt: Option[ADDigest], dir: File, settings: ErgoSettings): DigestState = { - val store = new LDBVersionedStore(dir, initialKeepVersions = settings.nodeSettings.keepVersions) + val store = new RocksDBVersionedStore(dir, initialKeepVersions = settings.nodeSettings.keepVersions) Try { val context = ErgoStateReader.storageStateContext(store, settings) (versionOpt, rootHashOpt) match { diff --git a/src/main/scala/org/ergoplatform/nodeView/state/ErgoStateReader.scala b/src/main/scala/org/ergoplatform/nodeView/state/ErgoStateReader.scala index 9b57c853cf..157561e296 100644 --- a/src/main/scala/org/ergoplatform/nodeView/state/ErgoStateReader.scala +++ b/src/main/scala/org/ergoplatform/nodeView/state/ErgoStateReader.scala @@ -7,7 +7,7 @@ import org.ergoplatform.settings.{Algos, Constants, ErgoSettings, LaunchParamete import org.ergoplatform.core.VersionTag import scorex.crypto.authds.ADDigest import scorex.crypto.hash.Digest32 -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore import scorex.util.ScorexLogging import scala.util.{Failure, Success, Try} @@ -29,7 +29,7 @@ trait ErgoStateReader extends NodeViewComponent with ScorexLogging { */ def version: VersionTag - val store: LDBVersionedStore + val store: RocksDBVersionedStore protected def ergoSettings: ErgoSettings @@ -64,7 +64,7 @@ object ErgoStateReader extends ScorexLogging { /** * Read blockchain-related state context from `store` database */ - def storageStateContext(store: LDBVersionedStore, settings: ErgoSettings): ErgoStateContext = { + def storageStateContext(store: RocksDBVersionedStore, settings: ErgoSettings): ErgoStateContext = { store.get(ErgoStateReader.ContextKey) .flatMap(b => ErgoStateContextSerializer(settings.chainSettings).parseBytesTry(b).toOption) .getOrElse { diff --git a/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala b/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala index bf28fe47db..b7623cbea9 100644 --- a/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala +++ b/src/main/scala/org/ergoplatform/nodeView/state/SnapshotsDb.scala @@ -4,9 +4,9 @@ import org.ergoplatform.ErgoLikeContext.Height import org.ergoplatform.nodeView.state.UtxoState.{ManifestId, SubtreeId} import org.ergoplatform.settings.{Algos, ErgoSettings} import org.ergoplatform.serialization.ManifestSerializer -import scorex.crypto.authds.avltree.batch.VersionedLDBAVLStorage +import scorex.crypto.authds.avltree.batch.VersionedRocksDBAVLStorage import scorex.crypto.hash.Digest32 -import scorex.db.{LDBFactory, LDBKVStore} +import scorex.db.{RocksDBFactory, RocksDBKVStore} import scorex.util.ScorexLogging import scorex.util.encode.Base16 @@ -18,7 +18,7 @@ import scala.util.{Failure, Success, Try} /** * Interface for a (non-versioned) database storing UTXO set snapshots and metadata about them */ -class SnapshotsDb(store: LDBKVStore) extends ScorexLogging { +class SnapshotsDb(store: RocksDBKVStore) extends ScorexLogging { private val snapshotInfoKey: Array[Byte] = Array.fill(32)(0: Byte) @@ -106,7 +106,7 @@ class SnapshotsDb(store: LDBKVStore) extends ScorexLogging { * @return - id of the snapshot (root hash of its authenticating AVL+ tree), * or error happened during read-write process */ - def writeSnapshot(pullFrom: VersionedLDBAVLStorage, + def writeSnapshot(pullFrom: VersionedRocksDBAVLStorage, height: Height, expectedRootHash: Array[Byte], manifestDepth: Byte = ManifestSerializer.MainnetManifestDepth): Try[Array[Byte]] = { @@ -142,7 +142,7 @@ object SnapshotsDb { // internal method to open or init snapshots database in given folder // private[nodeView] to use it in tests also private[nodeView] def create(dir: String): SnapshotsDb = { - val store = new LDBKVStore(LDBFactory.open(new File(dir))) + val store = new RocksDBKVStore(RocksDBFactory.open(new File(dir))) new SnapshotsDb(store) } diff --git a/src/main/scala/org/ergoplatform/nodeView/state/UtxoSetSnapshotPersistence.scala b/src/main/scala/org/ergoplatform/nodeView/state/UtxoSetSnapshotPersistence.scala index 1416bc11d7..80e999635c 100644 --- a/src/main/scala/org/ergoplatform/nodeView/state/UtxoSetSnapshotPersistence.scala +++ b/src/main/scala/org/ergoplatform/nodeView/state/UtxoSetSnapshotPersistence.scala @@ -3,7 +3,7 @@ package org.ergoplatform.nodeView.state import org.ergoplatform.ErgoLikeContext.Height import org.ergoplatform.nodeView.state.UtxoState.{ManifestId, SubtreeId} import org.ergoplatform.settings.Algos.HF -import scorex.crypto.authds.avltree.batch.{PersistentBatchAVLProver, VersionedLDBAVLStorage} +import scorex.crypto.authds.avltree.batch.{PersistentBatchAVLProver, VersionedRocksDBAVLStorage} import scorex.crypto.hash.Digest32 import scorex.util.ScorexLogging import org.ergoplatform.settings.ErgoSettings @@ -27,7 +27,7 @@ trait UtxoSetSnapshotPersistence extends ScorexLogging { private[nodeView] def dumpSnapshot(height: Height, expectedRootHash: Array[Byte], manifestDepth: Byte = ManifestSerializer.MainnetManifestDepth): Try[Array[Byte]] = { - val storage = persistentProver.storage.asInstanceOf[VersionedLDBAVLStorage] + val storage = persistentProver.storage.asInstanceOf[VersionedRocksDBAVLStorage] snapshotsDb.writeSnapshot(storage, height, expectedRootHash, manifestDepth) } diff --git a/src/main/scala/org/ergoplatform/nodeView/state/UtxoState.scala b/src/main/scala/org/ergoplatform/nodeView/state/UtxoState.scala index 01b98a59ef..3de8befdb1 100644 --- a/src/main/scala/org/ergoplatform/nodeView/state/UtxoState.scala +++ b/src/main/scala/org/ergoplatform/nodeView/state/UtxoState.scala @@ -20,7 +20,7 @@ import scorex.crypto.authds.avltree.batch._ import scorex.crypto.authds.avltree.batch.serialization.{BatchAVLProverManifest, BatchAVLProverSubtree} import scorex.crypto.authds.{ADDigest, ADValue} import scorex.crypto.hash.Digest32 -import scorex.db.{ByteArrayWrapper, LDBVersionedStore} +import scorex.db.{ByteArrayWrapper, RocksDBVersionedStore} import scorex.util.ModifierId import scala.util.{Failure, Success, Try} @@ -35,7 +35,7 @@ import scala.util.{Failure, Success, Try} */ class UtxoState(override val persistentProver: PersistentBatchAVLProver[Digest32, HF], override val version: VersionTag, - override val store: LDBVersionedStore, + override val store: RocksDBVersionedStore, override protected val ergoSettings: ErgoSettings) extends ErgoState[UtxoState] with UtxoStateReader @@ -284,12 +284,12 @@ object UtxoState { * @return UTXO set based state on top of existing database, or genesis state if the database is empty */ def create(dir: File, settings: ErgoSettings): UtxoState = { - val store = new LDBVersionedStore(dir, initialKeepVersions = settings.nodeSettings.keepVersions) + val store = new RocksDBVersionedStore(dir, initialKeepVersions = settings.nodeSettings.keepVersions) val version = store.get(bestVersionKey).map(w => bytesToVersion(w)) .getOrElse(ErgoState.genesisStateVersion) val persistentProver: PersistentBatchAVLProver[Digest32, HF] = { val bp = new BatchAVLProver[Digest32, HF](keyLength = 32, valueLengthOpt = None) - val storage = new VersionedLDBAVLStorage(store) + val storage = new VersionedRocksDBAVLStorage(store) PersistentBatchAVLProver.create(bp, storage).get } new UtxoState(persistentProver, version, store, settings) @@ -309,10 +309,10 @@ object UtxoState { p.performOneOperation(Insert(b.id, ADValue @@ b.bytes)).ensuring(_.isSuccess) } - val store = new LDBVersionedStore(dir, initialKeepVersions = settings.nodeSettings.keepVersions) + val store = new RocksDBVersionedStore(dir, initialKeepVersions = settings.nodeSettings.keepVersions) val defaultStateContext = ErgoStateContext.empty(settings.chainSettings, parameters) - val storage = new VersionedLDBAVLStorage(store) + val storage = new VersionedRocksDBAVLStorage(store) val persistentProver = PersistentBatchAVLProver.create( p, storage, diff --git a/src/main/scala/org/ergoplatform/nodeView/state/UtxoStateReader.scala b/src/main/scala/org/ergoplatform/nodeView/state/UtxoStateReader.scala index d891a6e30e..97ab31261d 100644 --- a/src/main/scala/org/ergoplatform/nodeView/state/UtxoStateReader.scala +++ b/src/main/scala/org/ergoplatform/nodeView/state/UtxoStateReader.scala @@ -11,7 +11,7 @@ import org.ergoplatform.settings.Algos.HF import org.ergoplatform.wallet.boxes.ErgoBoxSerializer import org.ergoplatform.wallet.interpreter.ErgoInterpreter import org.ergoplatform.validation.MalformedModifierError -import scorex.crypto.authds.avltree.batch.{Lookup, PersistentBatchAVLProver, VersionedLDBAVLStorage} +import scorex.crypto.authds.avltree.batch.{Lookup, PersistentBatchAVLProver, VersionedRocksDBAVLStorage} import scorex.crypto.authds.{ADDigest, ADKey, SerializedAdProof} import scorex.crypto.hash.Digest32 @@ -31,7 +31,7 @@ trait UtxoStateReader extends ErgoStateReader with UtxoSetSnapshotPersistence { /** * Versioned database where UTXO set and its authenticating AVL+ tree are stored */ - protected lazy val storage = new VersionedLDBAVLStorage(store) + protected lazy val storage = new VersionedRocksDBAVLStorage(store) protected val persistentProver: PersistentBatchAVLProver[Digest32, HF] diff --git a/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletRegistry.scala b/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletRegistry.scala index f3e7deba48..8389b9dbce 100644 --- a/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletRegistry.scala +++ b/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletRegistry.scala @@ -15,7 +15,7 @@ import org.ergoplatform.wallet.boxes.{TrackedBox, TrackedBoxSerializer} import org.ergoplatform.wallet.transactions.TransactionBuilder import org.ergoplatform.core.VersionTag import scorex.crypto.authds.ADKey -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore import scorex.util.encode.Base16 import scorex.util.{ModifierId, ScorexLogging, bytesToId, idToBytes} @@ -31,7 +31,7 @@ import scala.util.{Failure, Success, Try} * * boxes, spent or not * */ -class WalletRegistry(private val store: LDBVersionedStore)(ws: WalletSettings) extends ScorexLogging { +class WalletRegistry(private val store: RocksDBVersionedStore)(ws: WalletSettings) extends ScorexLogging { import WalletRegistry._ @@ -454,7 +454,7 @@ object WalletRegistry { def apply(settings: ErgoSettings): Try[WalletRegistry] = Try { val dir = registryFolder(settings) dir.mkdirs() - new LDBVersionedStore(dir, settings.nodeSettings.keepVersions) + new RocksDBVersionedStore(dir, settings.nodeSettings.keepVersions) }.flatMap { case store if !store.versionIdExists(PreGenesisStateVersion) => // Create pre-genesis state checkpoint @@ -679,14 +679,14 @@ case class KeyValuePairsBag(toInsert: Seq[(Array[Byte], Array[Byte])], * Applies non-versioned transaction to a given `store`. * */ - def transact(store: LDBVersionedStore): Try[Unit] = transact(store, None) + def transact(store: RocksDBVersionedStore): Try[Unit] = transact(store, None) /** * Applies versioned transaction to a given `store`. */ - def transact(store: LDBVersionedStore, version: Array[Byte]): Try[Unit] = transact(store, Some(version)) + def transact(store: RocksDBVersionedStore, version: Array[Byte]): Try[Unit] = transact(store, Some(version)) - private def transact(store: LDBVersionedStore, versionOpt: Option[Array[Byte]]): Try[Unit] = + private def transact(store: RocksDBVersionedStore, versionOpt: Option[Array[Byte]]): Try[Unit] = if (toInsert.nonEmpty || toRemove.nonEmpty) { store.update(versionOpt.getOrElse(scorex.utils.Random.randomBytes()), toRemove, toInsert) } else { diff --git a/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala b/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala index 21e46ff2c6..c0140619b3 100644 --- a/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala +++ b/src/main/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorage.scala @@ -8,7 +8,7 @@ import org.ergoplatform.sdk.wallet.secrets.{DerivationPath, DerivationPathSerial import org.ergoplatform.settings.{Constants, ErgoSettings, Parameters} import org.ergoplatform.wallet.Constants.{PaymentsScanId, ScanId} import scorex.crypto.hash.Blake2b256 -import scorex.db.{LDBFactory, LDBKVStore} +import scorex.db.{RocksDBFactory, RocksDBKVStore} import scorex.util.ScorexLogging import sigmastate.serialization.SigmaSerializer @@ -25,7 +25,7 @@ import scala.util.{Failure, Success, Try} * * ErgoStateContext (not version-agnostic, but state changes including rollbacks it is updated externally) * * external scans */ -final class WalletStorage(store: LDBKVStore, settings: ErgoSettings) extends ScorexLogging { +final class WalletStorage(store: RocksDBKVStore, settings: ErgoSettings) extends ScorexLogging { import WalletStorage._ @@ -245,7 +245,7 @@ object WalletStorage { def storageFolder(settings: ErgoSettings): File = new File(s"${settings.directory}/wallet/storage") def readOrCreate(settings: ErgoSettings): WalletStorage = { - val db = new LDBKVStore(LDBFactory.open(storageFolder(settings))) + val db = new RocksDBKVStore(RocksDBFactory.open(storageFolder(settings))) new WalletStorage(db, settings) } diff --git a/src/test/scala/org/ergoplatform/db/DBSpec.scala b/src/test/scala/org/ergoplatform/db/DBSpec.scala index c899372833..663057b8d5 100644 --- a/src/test/scala/org/ergoplatform/db/DBSpec.scala +++ b/src/test/scala/org/ergoplatform/db/DBSpec.scala @@ -3,8 +3,8 @@ package org.ergoplatform.db import akka.util.ByteString import org.ergoplatform.settings.Algos import org.ergoplatform.wallet.utils.FileUtils -import scorex.db.LDBFactory.RegisteredDB -import scorex.db.{LDBFactory, LDBKVStore, LDBVersionedStore} +import scorex.db.RocksDBFactory.RegisteredDB +import scorex.db.{RocksDBFactory, RocksDBKVStore, RocksDBVersionedStore} trait DBSpec extends FileUtils { @@ -21,17 +21,17 @@ trait DBSpec extends FileUtils { protected def byteString32(s: String): Array[Byte] = Algos.hash(byteString(s)) protected def withDb[T](body: RegisteredDB => T): T = { - val db = LDBFactory.open(createTempDir) + val db = RocksDBFactory.open(createTempDir) try body(db) finally db.close() } protected def versionId(s: String): Array[Byte] = byteString32(s) - protected def withStore[T](body: LDBKVStore => T): T = - withDb { db: RegisteredDB => body(new LDBKVStore(db)) } + protected def withStore[T](body: RocksDBKVStore => T): T = + withDb { db: RegisteredDB => body(new RocksDBKVStore(db)) } - protected def withVersionedStore[T](keepVersions: Int)(body: LDBVersionedStore => T): T = { - val db = new LDBVersionedStore(createTempDir, keepVersions) + protected def withVersionedStore[T](keepVersions: Int)(body: RocksDBVersionedStore => T): T = { + val db = new RocksDBVersionedStore(createTempDir, keepVersions) try body(db) finally db.close() } diff --git a/src/test/scala/org/ergoplatform/db/LDBKVStoreSpec.scala b/src/test/scala/org/ergoplatform/db/RocksDBKVStoreSpec.scala similarity index 93% rename from src/test/scala/org/ergoplatform/db/LDBKVStoreSpec.scala rename to src/test/scala/org/ergoplatform/db/RocksDBKVStoreSpec.scala index f43452160a..eb2527f4aa 100644 --- a/src/test/scala/org/ergoplatform/db/LDBKVStoreSpec.scala +++ b/src/test/scala/org/ergoplatform/db/RocksDBKVStoreSpec.scala @@ -3,7 +3,7 @@ package org.ergoplatform.db import org.scalatest.matchers.should.Matchers import org.scalatest.propspec.AnyPropSpec -class LDBKVStoreSpec extends AnyPropSpec with Matchers with DBSpec { +class RocksDBKVStoreSpec extends AnyPropSpec with Matchers with DBSpec { property("put/get/getAll/delete") { withStore { store => diff --git a/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala b/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala index def8600a21..74f1b66c62 100644 --- a/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala +++ b/src/test/scala/org/ergoplatform/nodeView/history/UtxoSetSnapshotProcessorSpecification.scala @@ -8,7 +8,7 @@ import org.ergoplatform.settings.{Algos, ErgoSettings} import org.ergoplatform.utils.HistoryTestHelpers import org.ergoplatform.core.VersionTag import org.ergoplatform.serialization.{ManifestSerializer, SubtreeSerializer} -import scorex.db.LDBVersionedStore +import scorex.db.RocksDBVersionedStore import scorex.util.ModifierId import scala.util.Random @@ -79,7 +79,7 @@ class UtxoSetSnapshotProcessorSpecification extends HistoryTestHelpers { s shouldBe subtreeIdsEncoded val dir = createTempDir - val store = new LDBVersionedStore(dir, initialKeepVersions = 100) + val store = new RocksDBVersionedStore(dir, initialKeepVersions = 100) val restoredProver = utxoSetSnapshotProcessor.createPersistentProver(store, history, snapshotHeight, blockId).get bh.sortedBoxes.foreach { box => restoredProver.unauthenticatedLookup(box.id).isDefined shouldBe true diff --git a/src/test/scala/org/ergoplatform/nodeView/state/wrapped/WrappedUtxoState.scala b/src/test/scala/org/ergoplatform/nodeView/state/wrapped/WrappedUtxoState.scala index 563387eea6..dbb1b08d57 100644 --- a/src/test/scala/org/ergoplatform/nodeView/state/wrapped/WrappedUtxoState.scala +++ b/src/test/scala/org/ergoplatform/nodeView/state/wrapped/WrappedUtxoState.scala @@ -13,13 +13,13 @@ import org.ergoplatform.core.{VersionTag, idToVersion} import org.ergoplatform.nodeView.LocallyGeneratedModifier import scorex.crypto.authds.avltree.batch._ import scorex.crypto.hash.Digest32 -import scorex.db.{ByteArrayWrapper, LDBVersionedStore} +import scorex.db.{ByteArrayWrapper, RocksDBVersionedStore} import scala.util.{Failure, Success, Try} class WrappedUtxoState(prover: PersistentBatchAVLProver[Digest32, HF], override val version: VersionTag, - store: LDBVersionedStore, + store: RocksDBVersionedStore, val versionedBoxHolder: VersionedInMemoryBoxHolder, settings: ErgoSettings) extends UtxoState(prover, version, store, settings) { diff --git a/src/test/scala/org/ergoplatform/nodeView/wallet/ErgoWalletServiceSpec.scala b/src/test/scala/org/ergoplatform/nodeView/wallet/ErgoWalletServiceSpec.scala index c492591ac0..8be9c7e759 100644 --- a/src/test/scala/org/ergoplatform/nodeView/wallet/ErgoWalletServiceSpec.scala +++ b/src/test/scala/org/ergoplatform/nodeView/wallet/ErgoWalletServiceSpec.scala @@ -22,7 +22,7 @@ import org.ergoplatform.wallet.interface4j.SecretString import org.ergoplatform.wallet.mnemonic.Mnemonic import org.scalacheck.Gen import org.scalatest.BeforeAndAfterAll -import scorex.db.{LDBKVStore, LDBVersionedStore} +import scorex.db.{RocksDBKVStore, RocksDBVersionedStore} import scorex.util.encode.Base16 import sigmastate.Values.{ByteArrayConstant, EvaluatedValue} import sigmastate.eval.Extensions.ArrayOps @@ -50,7 +50,7 @@ class ErgoWalletServiceSpec override def afterAll(): Unit = try super.afterAll() finally x.stop() - private def initialState(store: LDBKVStore, versionedStore: LDBVersionedStore, mempool: Option[ErgoMemPoolReader] = None) = { + private def initialState(store: RocksDBKVStore, versionedStore: RocksDBVersionedStore, mempool: Option[ErgoMemPoolReader] = None) = { ErgoWalletState( new WalletStorage(store, settings), secretStorageOpt = Option.empty, diff --git a/src/test/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorageSpec.scala b/src/test/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorageSpec.scala index c0a1826bab..aa2218afc7 100644 --- a/src/test/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorageSpec.scala +++ b/src/test/scala/org/ergoplatform/nodeView/wallet/persistence/WalletStorageSpec.scala @@ -10,7 +10,7 @@ import org.scalacheck.Gen import org.scalatest.flatspec.AnyFlatSpec import org.scalatest.matchers.should.Matchers import org.scalatestplus.scalacheck.ScalaCheckPropertyChecks -import scorex.db.LDBKVStore +import scorex.db.RocksDBKVStore class WalletStorageSpec extends AnyFlatSpec @@ -20,7 +20,7 @@ class WalletStorageSpec with DBSpec { it should "add and read derivation paths" in { - def addPath(store: LDBKVStore, storedPaths: Seq[DerivationPath], derivationPath: DerivationPath): Unit = { + def addPath(store: RocksDBKVStore, storedPaths: Seq[DerivationPath], derivationPath: DerivationPath): Unit = { val updatedPaths = (storedPaths :+ derivationPath).toSet val toInsert = Ints.toByteArray(updatedPaths.size) ++ updatedPaths .foldLeft(Array.empty[Byte]) { case (acc, path) => From 3974f64a0c8e1ddf2026ab5a27a262b805348fba Mon Sep 17 00:00:00 2001 From: jellymlg Date: Thu, 29 Feb 2024 00:16:13 +0100 Subject: [PATCH 11/15] Undo changes to ErgoSettingsReader --- .../org/ergoplatform/settings/ErgoSettingsReader.scala | 10 ++-------- 1 file changed, 2 insertions(+), 8 deletions(-) diff --git a/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala b/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala index bd83826f90..4cacd4be02 100644 --- a/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala +++ b/src/main/scala/org/ergoplatform/settings/ErgoSettingsReader.scala @@ -8,7 +8,6 @@ import com.typesafe.config.{Config, ConfigFactory, ConfigValueFactory} import net.ceedubs.ficus.Ficus._ import net.ceedubs.ficus.readers.ArbitraryTypeReader._ import org.ergoplatform.nodeView.state.StateType.Digest -import org.ergoplatform.wallet.utils.FileUtils import org.ergoplatform.ErgoApp import scorex.util.ScorexLogging import org.ergoplatform.settings.ErgoSettings.{configPath, scorexConfigPath} @@ -22,19 +21,14 @@ import scala.util.Try object ErgoSettingsReader extends ScorexLogging with PowSchemeReaders with NodeConfigurationReaders - with SettingsReaders - with FileUtils { + with SettingsReaders { def read(args: Args = Args.empty): ErgoSettings = { fromConfig(readConfig(args), args.networkTypeOpt) } def fromConfig(config: Config, desiredNetworkTypeOpt: Option[NetworkType] = None): ErgoSettings = { - val directory = - if(System.getProperty("env") == "test") - createTempDir.getAbsolutePath - else - config.as[String](s"$configPath.directory") + val directory = config.as[String](s"$configPath.directory") val networkTypeName = config.as[String](s"$configPath.networkType") val networkType = NetworkType.fromString(networkTypeName) .getOrElse(throw new Error(s"Unknown `networkType = $networkTypeName`")) From 9ef6f9dfa0680a359c8f1408a3c50c80e3a205dd Mon Sep 17 00:00:00 2001 From: Alexander Chepurnoy Date: Tue, 5 Mar 2024 20:19:38 +0300 Subject: [PATCH 12/15] updating rocksdb to 8.11.3 --- build.sbt | 3 +-- 1 file changed, 1 insertion(+), 2 deletions(-) diff --git a/build.sbt b/build.sbt index afa3255ebf..90ec83877b 100644 --- a/build.sbt +++ b/build.sbt @@ -18,7 +18,6 @@ lazy val commonSettings = Seq( // without the tag version resolves to [branch name]-[git commit hash]-SNAPSHOT // don't set the version manually resolvers ++= Seq("Sonatype Releases" at "https://oss.sonatype.org/content/repositories/releases/", - "Bintray" at "https://jcenter.bintray.com/", //for org.ethereum % leveldbjni-all "SonaType" at "https://oss.sonatype.org/content/groups/public", "Typesafe maven releases" at "https://dl.bintray.com/typesafe/maven-releases/", "Sonatype Snapshots" at "https://oss.sonatype.org/content/repositories/snapshots/"), @@ -217,7 +216,7 @@ lazy val avldb = (project in file("avldb")) javacOptions in(Compile, compile) ++= javacReleaseOption, libraryDependencies ++= Seq( // database dependencies - "org.rocksdb" % "rocksdbjni" % "8.9.1" + "org.rocksdb" % "rocksdbjni" % "8.11.3" ) ) From 09fb1f70c5a535bfb9ffa95786e5866fbee01b95 Mon Sep 17 00:00:00 2001 From: Alexander Chepurnoy Date: Thu, 17 Oct 2024 13:49:03 +0300 Subject: [PATCH 13/15] syncwal before close --- avldb/src/main/scala/scorex/db/RocksDBFactory.scala | 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-) diff --git a/avldb/src/main/scala/scorex/db/RocksDBFactory.scala b/avldb/src/main/scala/scorex/db/RocksDBFactory.scala index d52095be42..198369ae33 100644 --- a/avldb/src/main/scala/scorex/db/RocksDBFactory.scala +++ b/avldb/src/main/scala/scorex/db/RocksDBFactory.scala @@ -47,6 +47,7 @@ object RocksDBFactory extends ScorexLogging { lock.writeLock().lock() try { map.remove(path) + impl.syncWal() impl.close() open.set(false) } finally { @@ -57,8 +58,8 @@ object RocksDBFactory extends ScorexLogging { private val normalOptions: Options = new Options() .setCreateIfMissing(true) - .setWriteBufferSize(32 * SizeUnit.MB) .setAllowMmapReads(true) + .setAtomicFlush(true) .setIncreaseParallelism(4) .setCompressionType(CompressionType.LZ4_COMPRESSION) .setCompactionStyle(CompactionStyle.LEVEL) From aac8d388250523688f596747dedb553bb588697e Mon Sep 17 00:00:00 2001 From: jellymlg Date: Fri, 13 Dec 2024 19:26:02 +0100 Subject: [PATCH 14/15] Fix tests --- .../main/scala/scorex/db/RocksDBFactory.scala | 2 +- .../nodeView/history/extra/SegmentSpec.scala | 40 ++++++------------- .../scala/org/ergoplatform/utils/Stubs.scala | 2 +- .../generators/ValidBlocksGenerators.scala | 4 +- 4 files changed, 16 insertions(+), 32 deletions(-) diff --git a/avldb/src/main/scala/scorex/db/RocksDBFactory.scala b/avldb/src/main/scala/scorex/db/RocksDBFactory.scala index 198369ae33..571ab14ce8 100644 --- a/avldb/src/main/scala/scorex/db/RocksDBFactory.scala +++ b/avldb/src/main/scala/scorex/db/RocksDBFactory.scala @@ -47,7 +47,7 @@ object RocksDBFactory extends ScorexLogging { lock.writeLock().lock() try { map.remove(path) - impl.syncWal() + if(System.getProperty("env") != "test") impl.syncWal() impl.close() open.set(false) } finally { diff --git a/src/test/scala/org/ergoplatform/nodeView/history/extra/SegmentSpec.scala b/src/test/scala/org/ergoplatform/nodeView/history/extra/SegmentSpec.scala index 4ab6dd4e82..069b8a5a6d 100644 --- a/src/test/scala/org/ergoplatform/nodeView/history/extra/SegmentSpec.scala +++ b/src/test/scala/org/ergoplatform/nodeView/history/extra/SegmentSpec.scala @@ -12,6 +12,7 @@ import org.ergoplatform.nodeView.history.ErgoHistoryReader import org.ergoplatform.nodeView.history.storage.HistoryStorage import org.ergoplatform.settings.ErgoSettings +import scala.reflect.ClassTag import scala.util.Try class SegmentSpec extends ErgoCorePropertyTest { @@ -31,40 +32,23 @@ class SegmentSpec extends ErgoCorePropertyTest { val ia = IndexedErgoAddress(hash, new ArrayBuffer[Long](), boxes.slice(segmentTreshold * 3, boxes.size).reverse) val hr = new ErgoHistoryReader { - override protected[history] val historyStorage: HistoryStorage = new HistoryStorage(null, null, null ,null) { - override def getExtraIndex(id: ModifierId): Option[ExtraIndex] = { - val b = Base16.decode(id).get.head - if(b == 0) { - Some(segment0) - } else if(b == 1) { - Some(segment1) - } else if(b == 2) { - Some(segment2) - } else { - None - } + override def typedExtraIndexById[T <: ExtraIndex : ClassTag](id: ModifierId): Option[T] = { + val b = Base16.decode(id).get.head + if(b == 0) { + Some(segment0.asInstanceOf[T]) + } else if(b == 1) { + Some(segment1.asInstanceOf[T]) + } else if(b == 2) { + Some(segment2.asInstanceOf[T]) + } else { + None } } - + override protected[history] val historyStorage: HistoryStorage = null override protected val settings: ErgoSettings = null - - /** - * Whether state requires to download adProofs before full block application - */ override protected def requireProofs: Boolean = ??? - - /** - * @param m - modifier to process - * @return ProgressInfo - info required for State to be consistent with History - */ override protected def process(m: NonHeaderBlockSection): Try[ProgressInfo[BlockSection]] = ??? - - /** - * @param m - modifier to validate - * @return Success() if modifier is valid from History point of view, Failure(error) otherwise - */ override protected def validate(m: NonHeaderBlockSection): Try[Unit] = ??? - override val powScheme: AutolykosPowScheme = null } diff --git a/src/test/scala/org/ergoplatform/utils/Stubs.scala b/src/test/scala/org/ergoplatform/utils/Stubs.scala index 4eca69eb01..fdef6d4eca 100644 --- a/src/test/scala/org/ergoplatform/utils/Stubs.scala +++ b/src/test/scala/org/ergoplatform/utils/Stubs.scala @@ -48,7 +48,7 @@ import scala.collection.mutable import scala.concurrent.duration._ import scala.util.{Failure, Success, Try} -trait Stubs extends ErgoTestHelpers with TestFileUtils { +trait Stubs extends ErgoTestHelpers with FileUtils { import org.ergoplatform.utils.ErgoNodeTestConstants._ import org.ergoplatform.utils.ErgoCoreTestConstants._ import org.ergoplatform.utils.generators.ChainGenerator._ diff --git a/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala b/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala index e09ef9c9cc..45f9cb1b5c 100644 --- a/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala +++ b/src/test/scala/org/ergoplatform/utils/generators/ValidBlocksGenerators.scala @@ -11,7 +11,7 @@ import org.ergoplatform.nodeView.state._ import org.ergoplatform.nodeView.state.wrapped.WrappedUtxoState import org.ergoplatform.settings.{Algos, Constants, ErgoSettings, Parameters} import org.ergoplatform.utils.{LoggingUtil, RandomLike, RandomWrapper} -import org.ergoplatform.wallet.utils.TestFileUtils +import org.ergoplatform.wallet.utils.FileUtils import org.scalatest.matchers.should.Matchers import org.ergoplatform.core.VersionTag import scorex.crypto.authds.avltree.batch.Remove @@ -25,7 +25,7 @@ import scala.collection.mutable import scala.util.{Failure, Random, Success} object ValidBlocksGenerators - extends TestkitHelpers with TestFileUtils with Matchers with ScorexLogging { + extends TestkitHelpers with FileUtils with Matchers with ScorexLogging { import org.ergoplatform.utils.ErgoNodeTestConstants._ import org.ergoplatform.utils.ErgoCoreTestConstants._ import org.ergoplatform.utils.generators.ErgoNodeTransactionGenerators._ From d0a4ec46b401197455b8b53b58da49a6ae73d092 Mon Sep 17 00:00:00 2001 From: jellymlg Date: Wed, 18 Dec 2024 00:22:03 +0100 Subject: [PATCH 15/15] Update RocksDB version --- avldb/build.sbt | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/avldb/build.sbt b/avldb/build.sbt index fecd59ab0d..bb48e0bba6 100644 --- a/avldb/build.sbt +++ b/avldb/build.sbt @@ -26,7 +26,7 @@ libraryDependencies ++= Seq( "org.scalacheck" %% "scalacheck" % "1.14.3" % "test", "org.scalatestplus" %% "scalatestplus-scalacheck" % "3.1.0.0-RC2" % Test, "com.storm-enroute" %% "scalameter" % Versions.scalameter(scalaVersion.value) % "test", - "org.rocksdb" % "rocksdbjni" % "8.9.1", + "org.rocksdb" % "rocksdbjni" % "9.7.3", "org.typelevel" %% "spire" % Versions.spire(scalaVersion.value) )