Extract tx::SingleNodeEngine from tx::MasterEngine

Summary:
No logic changes, just split `tx::MasterEngine` into
`tx::SingleNodeEngine` and `tx::MasterEngine`. This gives better
responsibility separation and is more appropriate now there is no
Start/Shutdown.

Reviewers: dgleich, teon.banek, buda

Reviewed By: dgleich, teon.banek

Subscribers: pullbot

Differential Revision: https://phabricator.memgraph.io/D1099
This commit is contained in:
florijan 2018-01-10 15:10:22 +01:00
parent 456e95d12c
commit c4327b26f4
23 changed files with 264 additions and 226 deletions

View File

@ -45,6 +45,7 @@ set(memgraph_src_files
storage/vertex_accessor.cpp
threading/thread.cpp
transactions/engine_master.cpp
transactions/engine_single_node.cpp
transactions/engine_worker.cpp
utils/watchdog.cpp
)

View File

@ -11,6 +11,7 @@
#include "storage/concurrent_id_mapper_master.hpp"
#include "storage/concurrent_id_mapper_worker.hpp"
#include "transactions/engine_master.hpp"
#include "transactions/engine_single_node.hpp"
#include "transactions/engine_worker.hpp"
#include "utils/timer.hpp"
@ -22,7 +23,7 @@ namespace fs = std::experimental::filesystem;
properties_ = std::make_unique<type<GraphDbTypes::Property>>(__VA_ARGS__);
GraphDb::GraphDb(Config config) : GraphDb(config, 0) {
tx_engine_ = std::make_unique<tx::MasterEngine>(&wal_);
tx_engine_ = std::make_unique<tx::SingleNodeEngine>(&wal_);
counters_ = std::make_unique<database::SingleNodeCounters>();
INIT_MAPPERS(storage::SingleNodeConcurrentIdMapper);
Start();
@ -31,9 +32,7 @@ GraphDb::GraphDb(Config config) : GraphDb(config, 0) {
GraphDb::GraphDb(communication::messaging::System &system,
distributed::MasterCoordination &master, Config config)
: GraphDb(config, 0) {
auto tx_engine = std::make_unique<tx::MasterEngine>(&wal_);
tx_engine->StartServer(system);
tx_engine_ = std::move(tx_engine);
tx_engine_ = std::make_unique<tx::MasterEngine>(system, &wal_);
auto counters = std::make_unique<database::MasterCounters>(system);
counters_ = std::move(counters);
INIT_MAPPERS(storage::MasterConcurrentIdMapper, system);

View File

@ -75,7 +75,7 @@ class GraphDb {
};
/** Single-node GraphDb ctor. */
GraphDb(Config config = Config{});
explicit GraphDb(Config config = Config{});
/** Distributed master GraphDb ctor. */
GraphDb(communication::messaging::System &system,

View File

@ -10,7 +10,7 @@
#include "utils/on_scope_exit.hpp"
GraphDbAccessor::GraphDbAccessor(GraphDb &db)
: db_(db), transaction_(MasterEngine().Begin()) {}
: db_(db), transaction_(SingleNodeEngine().Begin()) {}
GraphDbAccessor::~GraphDbAccessor() {
if (!commited_ && !aborted_) {
@ -24,18 +24,18 @@ tx::transaction_id_t GraphDbAccessor::transaction_id() const {
void GraphDbAccessor::AdvanceCommand() {
DCHECK(!commited_ && !aborted_) << "Accessor committed or aborted";
MasterEngine().Advance(transaction_->id_);
SingleNodeEngine().Advance(transaction_->id_);
}
void GraphDbAccessor::Commit() {
DCHECK(!commited_ && !aborted_) << "Already aborted or commited transaction.";
MasterEngine().Commit(*transaction_);
SingleNodeEngine().Commit(*transaction_);
commited_ = true;
}
void GraphDbAccessor::Abort() {
DCHECK(!commited_ && !aborted_) << "Already aborted or commited transaction.";
MasterEngine().Abort(*transaction_);
SingleNodeEngine().Abort(*transaction_);
aborted_ = true;
}

View File

@ -610,13 +610,15 @@ class GraphDbAccessor {
const RecordAccessor<Vertex> &vertex_accessor,
const Vertex *const vertex);
/** Casts the DB's engine to MasterEngine and returns it. If the DB's engine
* is RemoteEngine, this function will crash MG. */
tx::MasterEngine &MasterEngine() {
auto *master_engine =
dynamic_cast<tx::MasterEngine *>(db_.tx_engine_.get());
DCHECK(master_engine) << "Asked for MasterEngine on distributed worker";
return *master_engine;
/** Casts the DB's engine to SingleNodeEngine and returns it. If the DB's
* engine is RemoteEngine, this function will crash MG. It must be either
* SingleNodeEngine, or MasterEngine (which inherits it). */
tx::SingleNodeEngine &SingleNodeEngine() {
auto *single_node_engine =
dynamic_cast<tx::SingleNodeEngine *>(db_.tx_engine_.get());
DCHECK(single_node_engine)
<< "Asked for SingleNodeEngine on distributed worker";
return *single_node_engine;
}
GraphDb &db_;

View File

@ -9,127 +9,31 @@
namespace tx {
MasterEngine::MasterEngine(durability::WriteAheadLog *wal) : wal_(wal) {}
Transaction *MasterEngine::Begin() {
std::lock_guard<SpinLock> guard(lock_);
transaction_id_t id{++counter_};
auto t = new Transaction(id, active_, *this);
active_.insert(id);
store_.emplace(id, t);
if (wal_) {
wal_->Emplace(database::StateDelta::TxBegin(id));
}
return t;
}
void MasterEngine::Advance(transaction_id_t id) {
std::lock_guard<SpinLock> guard(lock_);
auto it = store_.find(id);
DCHECK(it != store_.end())
<< "Transaction::advance on non-existing transaction";
Transaction *t = it->second.get();
if (t->cid_ == std::numeric_limits<command_id_t>::max())
throw TransactionError(
"Reached maximum number of commands in this "
"transaction.");
t->cid_++;
}
void MasterEngine::Commit(const Transaction &t) {
std::lock_guard<SpinLock> guard(lock_);
clog_.set_committed(t.id_);
active_.remove(t.id_);
if (wal_) {
wal_->Emplace(database::StateDelta::TxCommit(t.id_));
}
store_.erase(store_.find(t.id_));
}
void MasterEngine::Abort(const Transaction &t) {
std::lock_guard<SpinLock> guard(lock_);
clog_.set_aborted(t.id_);
active_.remove(t.id_);
if (wal_) {
wal_->Emplace(database::StateDelta::TxAbort(t.id_));
}
store_.erase(store_.find(t.id_));
}
CommitLog::Info MasterEngine::Info(transaction_id_t tx) const {
return clog_.fetch_info(tx);
}
Snapshot MasterEngine::GlobalGcSnapshot() {
std::lock_guard<SpinLock> guard(lock_);
// No active transactions.
if (active_.size() == 0) {
auto snapshot_copy = active_;
snapshot_copy.insert(counter_ + 1);
return snapshot_copy;
}
// There are active transactions.
auto snapshot_copy = store_.find(active_.front())->second->snapshot();
snapshot_copy.insert(active_.front());
return snapshot_copy;
}
Snapshot MasterEngine::GlobalActiveTransactions() {
std::lock_guard<SpinLock> guard(lock_);
Snapshot active_transactions = active_;
return active_transactions;
}
bool MasterEngine::GlobalIsActive(transaction_id_t tx) const {
return clog_.is_active(tx);
}
tx::transaction_id_t MasterEngine::LocalLast() const { return counter_.load(); }
void MasterEngine::LocalForEachActiveTransaction(
std::function<void(Transaction &)> f) {
std::lock_guard<SpinLock> guard(lock_);
for (auto transaction : active_) {
f(*store_.find(transaction)->second);
}
}
void MasterEngine::StartServer(communication::messaging::System &system) {
CHECK(!rpc_server_) << "Can't start a running server";
rpc_server_.emplace(system, "tx_engine");
rpc_server_->Register<SnapshotRpc>([this](const SnapshotReq &req) {
MasterEngine::MasterEngine(communication::messaging::System &system,
durability::WriteAheadLog *wal)
: SingleNodeEngine(wal), rpc_server_(system, kTransactionEngineRpc) {
rpc_server_.Register<SnapshotRpc>([this](const SnapshotReq &req) {
// It is guaranteed that the Worker will not be requesting this for a
// transaction that's done, and that there are no race conditions here.
auto found = store_.find(req.member);
DCHECK(found != store_.end())
<< "Can't return snapshot for an inactive transaction";
return std::make_unique<SnapshotRes>(found->second->snapshot());
return std::make_unique<SnapshotRes>(GetSnapshot(req.member));
});
rpc_server_->Register<GcSnapshotRpc>(
rpc_server_.Register<GcSnapshotRpc>(
[this](const communication::messaging::Message &) {
return std::make_unique<SnapshotRes>(GlobalGcSnapshot());
});
rpc_server_->Register<ClogInfoRpc>([this](const ClogInfoReq &req) {
rpc_server_.Register<ClogInfoRpc>([this](const ClogInfoReq &req) {
return std::make_unique<ClogInfoRes>(Info(req.member));
});
rpc_server_->Register<ActiveTransactionsRpc>(
rpc_server_.Register<ActiveTransactionsRpc>(
[this](const communication::messaging::Message &) {
return std::make_unique<SnapshotRes>(GlobalActiveTransactions());
});
rpc_server_->Register<IsActiveRpc>([this](const IsActiveReq &req) {
rpc_server_.Register<IsActiveRpc>([this](const IsActiveReq &req) {
return std::make_unique<IsActiveRes>(GlobalIsActive(req.member));
});
}
} // namespace tx

View File

@ -1,89 +1,23 @@
#pragma once
#include <atomic>
#include <experimental/optional>
#include <unordered_map>
#include "communication/messaging/distributed.hpp"
#include "communication/rpc/rpc.hpp"
#include "durability/wal.hpp"
#include "threading/sync/spinlock.hpp"
#include "transactions/commit_log.hpp"
#include "transactions/engine.hpp"
#include "transactions/transaction.hpp"
#include "utils/exceptions.hpp"
#include "transactions/engine_single_node.hpp"
namespace tx {
/** Indicates an error in transaction handling (currently
* only command id overflow). */
class TransactionError : public utils::BasicException {
public:
using utils::BasicException::BasicException;
};
/**
* A transaction engine that contains everything necessary for transactional
* handling. Used for single-node Memgraph deployments and for the master in a
* distributed system.
*/
class MasterEngine : public Engine {
/** Distributed master transaction engine. Has complete engine functionality and
* exposes an RPC server to be used by distributed Workers. */
class MasterEngine : public SingleNodeEngine {
public:
/**
* @param wal - Optional. If present, the Engine will write tx
* Begin/Commit/Abort atomically (while under lock).
*/
MasterEngine(durability::WriteAheadLog *wal = nullptr);
/**
* Begins a transaction and returns a pointer to
* it's object.
*
* The transaction object is owned by this engine.
* It will be released when the transaction gets
* committted or aborted.
*/
Transaction *Begin();
/**
* Advances the command on the transaction with the
* given id.
*
* @param id - Transation id. That transaction must
* be currently active.
*/
void Advance(transaction_id_t id);
/** Comits the given transaction. Deletes the transaction object, it's not
* valid after this function executes. */
void Commit(const Transaction &t);
/** Aborts the given transaction. Deletes the transaction object, it's not
* valid after this function executes. */
void Abort(const Transaction &t);
CommitLog::Info Info(transaction_id_t tx) const override;
Snapshot GlobalGcSnapshot() override;
Snapshot GlobalActiveTransactions() override;
bool GlobalIsActive(transaction_id_t tx) const override;
tx::transaction_id_t LocalLast() const override;
void LocalForEachActiveTransaction(
std::function<void(Transaction &)> f) override;
/** Starts the RPC server of the master transactional engine. */
void StartServer(communication::messaging::System &system);
MasterEngine(communication::messaging::System &system,
durability::WriteAheadLog *wal = nullptr);
private:
std::atomic<transaction_id_t> counter_{0};
CommitLog clog_;
std::unordered_map<transaction_id_t, std::unique_ptr<Transaction>> store_;
Snapshot active_;
SpinLock lock_;
// Optional. If present, the Engine will write tx Begin/Commit/Abort
// atomically (while under lock).
durability::WriteAheadLog *wal_{nullptr};
// Optional RPC server, only used in distributed, not in single_node.
std::experimental::optional<communication::rpc::Server> rpc_server_;
communication::rpc::Server rpc_server_;
};
} // namespace tx

View File

@ -8,6 +8,8 @@
namespace tx {
const std::string kTransactionEngineRpc = "transaction_engine_rpc";
RPC_SINGLE_MEMBER_MESSAGE(SnapshotReq, transaction_id_t)
RPC_SINGLE_MEMBER_MESSAGE(SnapshotRes, Snapshot)
RPC_NO_MEMBER_MESSAGE(GcSnapshotReq)

View File

@ -0,0 +1,113 @@
#include <limits>
#include <mutex>
#include "glog/logging.h"
#include "database/state_delta.hpp"
#include "transactions/engine_rpc_messages.hpp"
#include "transactions/engine_single_node.hpp"
namespace tx {
SingleNodeEngine::SingleNodeEngine(durability::WriteAheadLog *wal)
: wal_(wal) {}
Transaction *SingleNodeEngine::Begin() {
std::lock_guard<SpinLock> guard(lock_);
transaction_id_t id{++counter_};
auto t = new Transaction(id, active_, *this);
active_.insert(id);
store_.emplace(id, t);
if (wal_) {
wal_->Emplace(database::StateDelta::TxBegin(id));
}
return t;
}
void SingleNodeEngine::Advance(transaction_id_t id) {
std::lock_guard<SpinLock> guard(lock_);
auto it = store_.find(id);
DCHECK(it != store_.end())
<< "Transaction::advance on non-existing transaction";
Transaction *t = it->second.get();
if (t->cid_ == std::numeric_limits<command_id_t>::max())
throw TransactionError(
"Reached maximum number of commands in this "
"transaction.");
t->cid_++;
}
void SingleNodeEngine::Commit(const Transaction &t) {
std::lock_guard<SpinLock> guard(lock_);
clog_.set_committed(t.id_);
active_.remove(t.id_);
if (wal_) {
wal_->Emplace(database::StateDelta::TxCommit(t.id_));
}
store_.erase(store_.find(t.id_));
}
void SingleNodeEngine::Abort(const Transaction &t) {
std::lock_guard<SpinLock> guard(lock_);
clog_.set_aborted(t.id_);
active_.remove(t.id_);
if (wal_) {
wal_->Emplace(database::StateDelta::TxAbort(t.id_));
}
store_.erase(store_.find(t.id_));
}
CommitLog::Info SingleNodeEngine::Info(transaction_id_t tx) const {
return clog_.fetch_info(tx);
}
Snapshot SingleNodeEngine::GlobalGcSnapshot() {
std::lock_guard<SpinLock> guard(lock_);
// No active transactions.
if (active_.size() == 0) {
auto snapshot_copy = active_;
snapshot_copy.insert(counter_ + 1);
return snapshot_copy;
}
// There are active transactions.
auto snapshot_copy = store_.find(active_.front())->second->snapshot();
snapshot_copy.insert(active_.front());
return snapshot_copy;
}
Snapshot SingleNodeEngine::GlobalActiveTransactions() {
std::lock_guard<SpinLock> guard(lock_);
Snapshot active_transactions = active_;
return active_transactions;
}
bool SingleNodeEngine::GlobalIsActive(transaction_id_t tx) const {
return clog_.is_active(tx);
}
tx::transaction_id_t SingleNodeEngine::LocalLast() const {
return counter_.load();
}
void SingleNodeEngine::LocalForEachActiveTransaction(
std::function<void(Transaction &)> f) {
std::lock_guard<SpinLock> guard(lock_);
for (auto transaction : active_) {
f(*store_.find(transaction)->second);
}
}
Snapshot SingleNodeEngine::GetSnapshot(tx::transaction_id_t tx_id) {
std::lock_guard<SpinLock> guard(lock_);
auto found = store_.find(tx_id);
DCHECK(found != store_.end())
<< "Can't return snapshot for an inactive transaction";
return found->second->snapshot();
}
} // namespace tx

View File

@ -0,0 +1,83 @@
#pragma once
#include <atomic>
#include <experimental/optional>
#include <unordered_map>
#include "communication/messaging/distributed.hpp"
#include "communication/rpc/rpc.hpp"
#include "durability/wal.hpp"
#include "threading/sync/spinlock.hpp"
#include "transactions/commit_log.hpp"
#include "transactions/engine.hpp"
#include "transactions/transaction.hpp"
#include "utils/exceptions.hpp"
namespace tx {
/** Indicates an error in transaction handling (currently
* only command id overflow). */
class TransactionError : public utils::BasicException {
public:
using utils::BasicException::BasicException;
};
/** Single-node deployment transaction engine. Has complete functionality. */
class SingleNodeEngine : public Engine {
public:
/**
* @param wal - Optional. If present, the Engine will write tx
* Begin/Commit/Abort atomically (while under lock).
*/
explicit SingleNodeEngine(durability::WriteAheadLog *wal = nullptr);
/**
* Begins a transaction and returns a pointer to
* it's object.
*
* The transaction object is owned by this engine.
* It will be released when the transaction gets
* committted or aborted.
*/
Transaction *Begin();
/**
* Advances the command on the transaction with the
* given id.
*
* @param id - Transation id. That transaction must
* be currently active.
*/
void Advance(transaction_id_t id);
/** Comits the given transaction. Deletes the transaction object, it's not
* valid after this function executes. */
void Commit(const Transaction &t);
/** Aborts the given transaction. Deletes the transaction object, it's not
* valid after this function executes. */
void Abort(const Transaction &t);
CommitLog::Info Info(transaction_id_t tx) const override;
Snapshot GlobalGcSnapshot() override;
Snapshot GlobalActiveTransactions() override;
bool GlobalIsActive(transaction_id_t tx) const override;
tx::transaction_id_t LocalLast() const override;
void LocalForEachActiveTransaction(
std::function<void(Transaction &)> f) override;
protected:
// Exposed for MasterEngine. Transaction for tx_id must be alive.
Snapshot GetSnapshot(tx::transaction_id_t tx_id);
private:
std::atomic<transaction_id_t> counter_{0};
CommitLog clog_;
std::unordered_map<transaction_id_t, std::unique_ptr<Transaction>> store_;
Snapshot active_;
SpinLock lock_;
// Optional. If present, the Engine will write tx Begin/Commit/Abort
// atomically (while under lock).
durability::WriteAheadLog *wal_{nullptr};
};
} // namespace tx

View File

@ -11,7 +11,7 @@ static const auto kRpcTimeout = 100ms;
WorkerEngine::WorkerEngine(communication::messaging::System &system,
const io::network::NetworkEndpoint &endpoint)
: rpc_client_(system, endpoint, "tx_engine") {}
: rpc_client_(system, endpoint, kTransactionEngineRpc) {}
Transaction *WorkerEngine::LocalBegin(transaction_id_t tx_id) {
auto accessor = active_.access();

View File

@ -13,7 +13,8 @@
namespace tx {
/** A transactional engine for the worker in a distributed system. */
/** Distributed worker transaction engine. Connects to a MasterEngine (single
* source of truth) to obtain transactional info. Caches most info locally. */
class WorkerEngine : public Engine {
public:
WorkerEngine(communication::messaging::System &system,

View File

@ -26,6 +26,7 @@ class Transaction {
}
private:
friend class SingleNodeEngine;
friend class MasterEngine;
friend class WorkerEngine;

View File

@ -4,7 +4,7 @@
#include "mvcc/record.hpp"
#include "mvcc/version_list.hpp"
#include "transactions/engine_master.hpp"
#include "transactions/engine_single_node.hpp"
class Prop : public mvcc::Record<Prop> {
public:
@ -19,7 +19,7 @@ class Prop : public mvcc::Record<Prop> {
void MvccMix(benchmark::State &state) {
while (state.KeepRunning()) {
state.PauseTiming();
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
auto t1 = engine.Begin();
mvcc::VersionList<Prop> version_list(*t1, 0);

View File

@ -4,7 +4,7 @@
#include "database/graph_db_accessor.hpp"
#include "database/graph_db_datatypes.hpp"
#include "storage/vertex.hpp"
#include "transactions/engine_master.hpp"
#include "transactions/engine_single_node.hpp"
#include "mvcc_gc_common.hpp"
@ -15,7 +15,7 @@ TEST(LabelsIndex, UniqueInsert) {
KeyIndex<GraphDbTypes::Label, Vertex> index;
GraphDb db;
GraphDbAccessor dba(db);
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
auto t1 = engine.Begin();
mvcc::VersionList<Vertex> vlist(*t1, 0);
engine.Commit(*t1);
@ -43,7 +43,7 @@ TEST(LabelsIndex, UniqueFilter) {
GraphDb db;
KeyIndex<GraphDbTypes::Label, Vertex> index;
GraphDbAccessor dba(db);
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
auto t1 = engine.Begin();
mvcc::VersionList<Vertex> vlist1(*t1, 0);
@ -83,7 +83,7 @@ TEST(LabelsIndex, Refresh) {
KeyIndex<GraphDbTypes::Label, Vertex> index;
GraphDb db;
GraphDbAccessor access(db);
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
// add two vertices to database
auto t1 = engine.Begin();

View File

@ -4,7 +4,7 @@
#include "database/graph_db_accessor.hpp"
#include "database/graph_db_datatypes.hpp"
#include "database/indexes/label_property_index.hpp"
#include "transactions/engine_master.hpp"
#include "transactions/engine_single_node.hpp"
#include "mvcc_gc_common.hpp"
@ -44,7 +44,7 @@ class LabelPropertyIndexComplexTest : public ::testing::Test {
LabelPropertyIndex index;
LabelPropertyIndex::Key *key;
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
tx::Transaction *t{nullptr};
mvcc::VersionList<Vertex> *vlist;

View File

@ -10,7 +10,7 @@
#include "storage/edge.hpp"
#include "storage/property_value_store.hpp"
#include "storage/vertex.hpp"
#include "transactions/engine_master.hpp"
#include "transactions/engine_single_node.hpp"
using namespace GraphDbTypes;
@ -118,7 +118,7 @@ TEST(DistributedSerialization, VertexProperties) {
class DistributedSerializationMvcc : public ::testing::Test {
protected:
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
tx::Transaction *tx = engine.Begin();
mvcc::VersionList<Vertex> v1_vlist{*tx, 0};
Vertex &v1 = *v1_vlist.Oldest();

View File

@ -5,13 +5,13 @@
#include "mvcc/version.hpp"
#include "mvcc/version_list.hpp"
#include "threading/sync/lock_timeout_exception.hpp"
#include "transactions/engine_master.hpp"
#include "transactions/engine_single_node.hpp"
#include "transactions/transaction.hpp"
#include "mvcc_gc_common.hpp"
TEST(MVCC, Deadlock) {
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
auto t0 = engine.Begin();
mvcc::VersionList<Prop> version_list1(*t0, 0);
@ -31,7 +31,7 @@ TEST(MVCC, Deadlock) {
TEST(MVCC, UpdateDontDelete) {
std::atomic<int> count{0};
{
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
auto t1 = engine.Begin();
mvcc::VersionList<DestrCountRec> version_list(*t1, 0, count);
engine.Commit(*t1);
@ -55,7 +55,7 @@ TEST(MVCC, UpdateDontDelete) {
// Check that we get the oldest record.
TEST(MVCC, Oldest) {
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
auto t1 = engine.Begin();
mvcc::VersionList<Prop> version_list(*t1, 0);
auto first = version_list.Oldest();

View File

@ -5,13 +5,13 @@
#include "mvcc/version.hpp"
#include "mvcc/version_list.hpp"
#include "threading/sync/lock_timeout_exception.hpp"
#include "transactions/engine_master.hpp"
#include "transactions/engine_single_node.hpp"
#include "transactions/transaction.hpp"
class TestClass : public mvcc::Record<TestClass> {
public:
// constructs first version, size should be 0
TestClass(int &version_list_size) : version_list_size_(version_list_size) {
explicit TestClass(int &version_list_size) : version_list_size_(version_list_size) {
++version_list_size_;
}
TestClass *CloneData() { return new TestClass(version_list_size_); }
@ -58,7 +58,7 @@ class Mvcc : public ::testing::Test {
}
// variable where number of versions is stored
int version_list_size = 0;
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
tx::Transaction *t1 = engine.Begin();
mvcc::VersionList<TestClass> version_list{*t1, 0, version_list_size};
TestClass *v1 = nullptr;

View File

@ -12,13 +12,13 @@
#include "mvcc/version_list.hpp"
#include "storage/garbage_collector.hpp"
#include "storage/vertex.hpp"
#include "transactions/engine_master.hpp"
#include "transactions/engine_single_node.hpp"
#include "mvcc_gc_common.hpp"
class MvccGcTest : public ::testing::Test {
protected:
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
private:
tx::Transaction *t0 = engine.Begin();
@ -116,7 +116,7 @@ TEST_F(MvccGcTest, OldestTransactionSnapshot) {
*/
TEST(GarbageCollector, GcClean) {
ConcurrentMap<int64_t, mvcc::VersionList<DestrCountRec> *> collection;
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
DeferredDeleter<DestrCountRec> deleter;
DeferredDeleter<mvcc::VersionList<DestrCountRec>> vlist_deleter;
GarbageCollector<decltype(collection), DestrCountRec> gc(collection, deleter,

View File

@ -1,7 +1,7 @@
#pragma once
#include "mvcc/record.hpp"
#include "transactions/engine_master.hpp"
#include "transactions/engine_single_node.hpp"
/**
* @brief - Empty class which inherits from mvcc:Record.
@ -18,7 +18,7 @@ class Prop : public mvcc::Record<Prop> {
*/
class DestrCountRec : public mvcc::Record<DestrCountRec> {
public:
DestrCountRec(std::atomic<int> &count) : count_(count) {}
explicit DestrCountRec(std::atomic<int> &count) : count_(count) {}
DestrCountRec *CloneData() { return new DestrCountRec(count_); }
~DestrCountRec() { ++count_; }
@ -29,7 +29,7 @@ class DestrCountRec : public mvcc::Record<DestrCountRec> {
// helper function for creating a GC snapshot
// if given a nullptr it makes a GC snapshot like there
// are no active transactions
auto GcSnapshot(tx::MasterEngine &engine, tx::Transaction *t) {
auto GcSnapshot(tx::SingleNodeEngine &engine, tx::Transaction *t) {
if (t != nullptr) {
tx::Snapshot gc_snap = t->snapshot();
gc_snap.insert(t->id_);

View File

@ -16,12 +16,10 @@ class WorkerEngineTest : public testing::Test {
const std::string local{"127.0.0.1"};
System master_system_{local, 0};
MasterEngine master_;
MasterEngine master_{master_system_};
System worker_system_{local, 0};
WorkerEngine worker_{worker_system_, master_system_.endpoint()};
void SetUp() override { master_.StartServer(master_system_); }
};
TEST_F(WorkerEngineTest, LocalBegin) {

View File

@ -4,11 +4,11 @@
#include <vector>
#include "data_structures/concurrent/concurrent_set.hpp"
#include "transactions/engine_master.hpp"
#include "transactions/engine_single_node.hpp"
#include "transactions/transaction.hpp"
TEST(Engine, GcSnapshot) {
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
ASSERT_EQ(engine.GlobalGcSnapshot(), tx::Snapshot({1}));
std::vector<tx::Transaction *> transactions;
@ -36,7 +36,7 @@ TEST(Engine, GcSnapshot) {
}
TEST(Engine, Advance) {
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
auto t0 = engine.Begin();
auto t1 = engine.Begin();
@ -49,7 +49,7 @@ TEST(Engine, Advance) {
}
TEST(Engine, ConcurrentBegin) {
tx::MasterEngine engine;
tx::SingleNodeEngine engine;
std::vector<std::thread> threads;
ConcurrentSet<tx::transaction_id_t> tx_ids;
for (int i = 0; i < 10; ++i) {