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:
parent
456e95d12c
commit
c4327b26f4
@ -45,6 +45,7 @@ set(memgraph_src_files
|
|||||||
storage/vertex_accessor.cpp
|
storage/vertex_accessor.cpp
|
||||||
threading/thread.cpp
|
threading/thread.cpp
|
||||||
transactions/engine_master.cpp
|
transactions/engine_master.cpp
|
||||||
|
transactions/engine_single_node.cpp
|
||||||
transactions/engine_worker.cpp
|
transactions/engine_worker.cpp
|
||||||
utils/watchdog.cpp
|
utils/watchdog.cpp
|
||||||
)
|
)
|
||||||
|
@ -11,6 +11,7 @@
|
|||||||
#include "storage/concurrent_id_mapper_master.hpp"
|
#include "storage/concurrent_id_mapper_master.hpp"
|
||||||
#include "storage/concurrent_id_mapper_worker.hpp"
|
#include "storage/concurrent_id_mapper_worker.hpp"
|
||||||
#include "transactions/engine_master.hpp"
|
#include "transactions/engine_master.hpp"
|
||||||
|
#include "transactions/engine_single_node.hpp"
|
||||||
#include "transactions/engine_worker.hpp"
|
#include "transactions/engine_worker.hpp"
|
||||||
#include "utils/timer.hpp"
|
#include "utils/timer.hpp"
|
||||||
|
|
||||||
@ -22,7 +23,7 @@ namespace fs = std::experimental::filesystem;
|
|||||||
properties_ = std::make_unique<type<GraphDbTypes::Property>>(__VA_ARGS__);
|
properties_ = std::make_unique<type<GraphDbTypes::Property>>(__VA_ARGS__);
|
||||||
|
|
||||||
GraphDb::GraphDb(Config config) : GraphDb(config, 0) {
|
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>();
|
counters_ = std::make_unique<database::SingleNodeCounters>();
|
||||||
INIT_MAPPERS(storage::SingleNodeConcurrentIdMapper);
|
INIT_MAPPERS(storage::SingleNodeConcurrentIdMapper);
|
||||||
Start();
|
Start();
|
||||||
@ -31,9 +32,7 @@ GraphDb::GraphDb(Config config) : GraphDb(config, 0) {
|
|||||||
GraphDb::GraphDb(communication::messaging::System &system,
|
GraphDb::GraphDb(communication::messaging::System &system,
|
||||||
distributed::MasterCoordination &master, Config config)
|
distributed::MasterCoordination &master, Config config)
|
||||||
: GraphDb(config, 0) {
|
: GraphDb(config, 0) {
|
||||||
auto tx_engine = std::make_unique<tx::MasterEngine>(&wal_);
|
tx_engine_ = std::make_unique<tx::MasterEngine>(system, &wal_);
|
||||||
tx_engine->StartServer(system);
|
|
||||||
tx_engine_ = std::move(tx_engine);
|
|
||||||
auto counters = std::make_unique<database::MasterCounters>(system);
|
auto counters = std::make_unique<database::MasterCounters>(system);
|
||||||
counters_ = std::move(counters);
|
counters_ = std::move(counters);
|
||||||
INIT_MAPPERS(storage::MasterConcurrentIdMapper, system);
|
INIT_MAPPERS(storage::MasterConcurrentIdMapper, system);
|
||||||
|
@ -75,7 +75,7 @@ class GraphDb {
|
|||||||
};
|
};
|
||||||
|
|
||||||
/** Single-node GraphDb ctor. */
|
/** Single-node GraphDb ctor. */
|
||||||
GraphDb(Config config = Config{});
|
explicit GraphDb(Config config = Config{});
|
||||||
|
|
||||||
/** Distributed master GraphDb ctor. */
|
/** Distributed master GraphDb ctor. */
|
||||||
GraphDb(communication::messaging::System &system,
|
GraphDb(communication::messaging::System &system,
|
||||||
|
@ -10,7 +10,7 @@
|
|||||||
#include "utils/on_scope_exit.hpp"
|
#include "utils/on_scope_exit.hpp"
|
||||||
|
|
||||||
GraphDbAccessor::GraphDbAccessor(GraphDb &db)
|
GraphDbAccessor::GraphDbAccessor(GraphDb &db)
|
||||||
: db_(db), transaction_(MasterEngine().Begin()) {}
|
: db_(db), transaction_(SingleNodeEngine().Begin()) {}
|
||||||
|
|
||||||
GraphDbAccessor::~GraphDbAccessor() {
|
GraphDbAccessor::~GraphDbAccessor() {
|
||||||
if (!commited_ && !aborted_) {
|
if (!commited_ && !aborted_) {
|
||||||
@ -24,18 +24,18 @@ tx::transaction_id_t GraphDbAccessor::transaction_id() const {
|
|||||||
|
|
||||||
void GraphDbAccessor::AdvanceCommand() {
|
void GraphDbAccessor::AdvanceCommand() {
|
||||||
DCHECK(!commited_ && !aborted_) << "Accessor committed or aborted";
|
DCHECK(!commited_ && !aborted_) << "Accessor committed or aborted";
|
||||||
MasterEngine().Advance(transaction_->id_);
|
SingleNodeEngine().Advance(transaction_->id_);
|
||||||
}
|
}
|
||||||
|
|
||||||
void GraphDbAccessor::Commit() {
|
void GraphDbAccessor::Commit() {
|
||||||
DCHECK(!commited_ && !aborted_) << "Already aborted or commited transaction.";
|
DCHECK(!commited_ && !aborted_) << "Already aborted or commited transaction.";
|
||||||
MasterEngine().Commit(*transaction_);
|
SingleNodeEngine().Commit(*transaction_);
|
||||||
commited_ = true;
|
commited_ = true;
|
||||||
}
|
}
|
||||||
|
|
||||||
void GraphDbAccessor::Abort() {
|
void GraphDbAccessor::Abort() {
|
||||||
DCHECK(!commited_ && !aborted_) << "Already aborted or commited transaction.";
|
DCHECK(!commited_ && !aborted_) << "Already aborted or commited transaction.";
|
||||||
MasterEngine().Abort(*transaction_);
|
SingleNodeEngine().Abort(*transaction_);
|
||||||
aborted_ = true;
|
aborted_ = true;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -610,13 +610,15 @@ class GraphDbAccessor {
|
|||||||
const RecordAccessor<Vertex> &vertex_accessor,
|
const RecordAccessor<Vertex> &vertex_accessor,
|
||||||
const Vertex *const vertex);
|
const Vertex *const vertex);
|
||||||
|
|
||||||
/** Casts the DB's engine to MasterEngine and returns it. If the DB's engine
|
/** Casts the DB's engine to SingleNodeEngine and returns it. If the DB's
|
||||||
* is RemoteEngine, this function will crash MG. */
|
* engine is RemoteEngine, this function will crash MG. It must be either
|
||||||
tx::MasterEngine &MasterEngine() {
|
* SingleNodeEngine, or MasterEngine (which inherits it). */
|
||||||
auto *master_engine =
|
tx::SingleNodeEngine &SingleNodeEngine() {
|
||||||
dynamic_cast<tx::MasterEngine *>(db_.tx_engine_.get());
|
auto *single_node_engine =
|
||||||
DCHECK(master_engine) << "Asked for MasterEngine on distributed worker";
|
dynamic_cast<tx::SingleNodeEngine *>(db_.tx_engine_.get());
|
||||||
return *master_engine;
|
DCHECK(single_node_engine)
|
||||||
|
<< "Asked for SingleNodeEngine on distributed worker";
|
||||||
|
return *single_node_engine;
|
||||||
}
|
}
|
||||||
|
|
||||||
GraphDb &db_;
|
GraphDb &db_;
|
||||||
|
@ -9,127 +9,31 @@
|
|||||||
|
|
||||||
namespace tx {
|
namespace tx {
|
||||||
|
|
||||||
MasterEngine::MasterEngine(durability::WriteAheadLog *wal) : wal_(wal) {}
|
MasterEngine::MasterEngine(communication::messaging::System &system,
|
||||||
|
durability::WriteAheadLog *wal)
|
||||||
Transaction *MasterEngine::Begin() {
|
: SingleNodeEngine(wal), rpc_server_(system, kTransactionEngineRpc) {
|
||||||
std::lock_guard<SpinLock> guard(lock_);
|
rpc_server_.Register<SnapshotRpc>([this](const SnapshotReq &req) {
|
||||||
|
|
||||||
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) {
|
|
||||||
// It is guaranteed that the Worker will not be requesting this for a
|
// 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.
|
// transaction that's done, and that there are no race conditions here.
|
||||||
auto found = store_.find(req.member);
|
return std::make_unique<SnapshotRes>(GetSnapshot(req.member));
|
||||||
DCHECK(found != store_.end())
|
|
||||||
<< "Can't return snapshot for an inactive transaction";
|
|
||||||
return std::make_unique<SnapshotRes>(found->second->snapshot());
|
|
||||||
});
|
});
|
||||||
|
|
||||||
rpc_server_->Register<GcSnapshotRpc>(
|
rpc_server_.Register<GcSnapshotRpc>(
|
||||||
[this](const communication::messaging::Message &) {
|
[this](const communication::messaging::Message &) {
|
||||||
return std::make_unique<SnapshotRes>(GlobalGcSnapshot());
|
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));
|
return std::make_unique<ClogInfoRes>(Info(req.member));
|
||||||
});
|
});
|
||||||
|
|
||||||
rpc_server_->Register<ActiveTransactionsRpc>(
|
rpc_server_.Register<ActiveTransactionsRpc>(
|
||||||
[this](const communication::messaging::Message &) {
|
[this](const communication::messaging::Message &) {
|
||||||
return std::make_unique<SnapshotRes>(GlobalActiveTransactions());
|
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));
|
return std::make_unique<IsActiveRes>(GlobalIsActive(req.member));
|
||||||
});
|
});
|
||||||
}
|
}
|
||||||
|
|
||||||
} // namespace tx
|
} // namespace tx
|
||||||
|
@ -1,89 +1,23 @@
|
|||||||
#pragma once
|
#pragma once
|
||||||
|
|
||||||
#include <atomic>
|
|
||||||
#include <experimental/optional>
|
|
||||||
#include <unordered_map>
|
|
||||||
|
|
||||||
#include "communication/messaging/distributed.hpp"
|
#include "communication/messaging/distributed.hpp"
|
||||||
#include "communication/rpc/rpc.hpp"
|
#include "communication/rpc/rpc.hpp"
|
||||||
#include "durability/wal.hpp"
|
#include "transactions/engine_single_node.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 {
|
namespace tx {
|
||||||
|
|
||||||
/** Indicates an error in transaction handling (currently
|
/** Distributed master transaction engine. Has complete engine functionality and
|
||||||
* only command id overflow). */
|
* exposes an RPC server to be used by distributed Workers. */
|
||||||
class TransactionError : public utils::BasicException {
|
class MasterEngine : public SingleNodeEngine {
|
||||||
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 {
|
|
||||||
public:
|
public:
|
||||||
/**
|
/**
|
||||||
* @param wal - Optional. If present, the Engine will write tx
|
* @param wal - Optional. If present, the Engine will write tx
|
||||||
* Begin/Commit/Abort atomically (while under lock).
|
* Begin/Commit/Abort atomically (while under lock).
|
||||||
*/
|
*/
|
||||||
MasterEngine(durability::WriteAheadLog *wal = nullptr);
|
MasterEngine(communication::messaging::System &system,
|
||||||
|
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);
|
|
||||||
|
|
||||||
private:
|
private:
|
||||||
std::atomic<transaction_id_t> counter_{0};
|
communication::rpc::Server rpc_server_;
|
||||||
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_;
|
|
||||||
};
|
};
|
||||||
} // namespace tx
|
} // namespace tx
|
||||||
|
@ -8,6 +8,8 @@
|
|||||||
|
|
||||||
namespace tx {
|
namespace tx {
|
||||||
|
|
||||||
|
const std::string kTransactionEngineRpc = "transaction_engine_rpc";
|
||||||
|
|
||||||
RPC_SINGLE_MEMBER_MESSAGE(SnapshotReq, transaction_id_t)
|
RPC_SINGLE_MEMBER_MESSAGE(SnapshotReq, transaction_id_t)
|
||||||
RPC_SINGLE_MEMBER_MESSAGE(SnapshotRes, Snapshot)
|
RPC_SINGLE_MEMBER_MESSAGE(SnapshotRes, Snapshot)
|
||||||
RPC_NO_MEMBER_MESSAGE(GcSnapshotReq)
|
RPC_NO_MEMBER_MESSAGE(GcSnapshotReq)
|
||||||
|
113
src/transactions/engine_single_node.cpp
Normal file
113
src/transactions/engine_single_node.cpp
Normal 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
|
83
src/transactions/engine_single_node.hpp
Normal file
83
src/transactions/engine_single_node.hpp
Normal 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
|
@ -11,7 +11,7 @@ static const auto kRpcTimeout = 100ms;
|
|||||||
|
|
||||||
WorkerEngine::WorkerEngine(communication::messaging::System &system,
|
WorkerEngine::WorkerEngine(communication::messaging::System &system,
|
||||||
const io::network::NetworkEndpoint &endpoint)
|
const io::network::NetworkEndpoint &endpoint)
|
||||||
: rpc_client_(system, endpoint, "tx_engine") {}
|
: rpc_client_(system, endpoint, kTransactionEngineRpc) {}
|
||||||
|
|
||||||
Transaction *WorkerEngine::LocalBegin(transaction_id_t tx_id) {
|
Transaction *WorkerEngine::LocalBegin(transaction_id_t tx_id) {
|
||||||
auto accessor = active_.access();
|
auto accessor = active_.access();
|
||||||
|
@ -13,7 +13,8 @@
|
|||||||
|
|
||||||
namespace tx {
|
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 {
|
class WorkerEngine : public Engine {
|
||||||
public:
|
public:
|
||||||
WorkerEngine(communication::messaging::System &system,
|
WorkerEngine(communication::messaging::System &system,
|
||||||
|
@ -26,6 +26,7 @@ class Transaction {
|
|||||||
}
|
}
|
||||||
|
|
||||||
private:
|
private:
|
||||||
|
friend class SingleNodeEngine;
|
||||||
friend class MasterEngine;
|
friend class MasterEngine;
|
||||||
friend class WorkerEngine;
|
friend class WorkerEngine;
|
||||||
|
|
||||||
|
@ -4,7 +4,7 @@
|
|||||||
|
|
||||||
#include "mvcc/record.hpp"
|
#include "mvcc/record.hpp"
|
||||||
#include "mvcc/version_list.hpp"
|
#include "mvcc/version_list.hpp"
|
||||||
#include "transactions/engine_master.hpp"
|
#include "transactions/engine_single_node.hpp"
|
||||||
|
|
||||||
class Prop : public mvcc::Record<Prop> {
|
class Prop : public mvcc::Record<Prop> {
|
||||||
public:
|
public:
|
||||||
@ -19,7 +19,7 @@ class Prop : public mvcc::Record<Prop> {
|
|||||||
void MvccMix(benchmark::State &state) {
|
void MvccMix(benchmark::State &state) {
|
||||||
while (state.KeepRunning()) {
|
while (state.KeepRunning()) {
|
||||||
state.PauseTiming();
|
state.PauseTiming();
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
auto t1 = engine.Begin();
|
auto t1 = engine.Begin();
|
||||||
mvcc::VersionList<Prop> version_list(*t1, 0);
|
mvcc::VersionList<Prop> version_list(*t1, 0);
|
||||||
|
|
||||||
|
@ -4,7 +4,7 @@
|
|||||||
#include "database/graph_db_accessor.hpp"
|
#include "database/graph_db_accessor.hpp"
|
||||||
#include "database/graph_db_datatypes.hpp"
|
#include "database/graph_db_datatypes.hpp"
|
||||||
#include "storage/vertex.hpp"
|
#include "storage/vertex.hpp"
|
||||||
#include "transactions/engine_master.hpp"
|
#include "transactions/engine_single_node.hpp"
|
||||||
|
|
||||||
#include "mvcc_gc_common.hpp"
|
#include "mvcc_gc_common.hpp"
|
||||||
|
|
||||||
@ -15,7 +15,7 @@ TEST(LabelsIndex, UniqueInsert) {
|
|||||||
KeyIndex<GraphDbTypes::Label, Vertex> index;
|
KeyIndex<GraphDbTypes::Label, Vertex> index;
|
||||||
GraphDb db;
|
GraphDb db;
|
||||||
GraphDbAccessor dba(db);
|
GraphDbAccessor dba(db);
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
auto t1 = engine.Begin();
|
auto t1 = engine.Begin();
|
||||||
mvcc::VersionList<Vertex> vlist(*t1, 0);
|
mvcc::VersionList<Vertex> vlist(*t1, 0);
|
||||||
engine.Commit(*t1);
|
engine.Commit(*t1);
|
||||||
@ -43,7 +43,7 @@ TEST(LabelsIndex, UniqueFilter) {
|
|||||||
GraphDb db;
|
GraphDb db;
|
||||||
KeyIndex<GraphDbTypes::Label, Vertex> index;
|
KeyIndex<GraphDbTypes::Label, Vertex> index;
|
||||||
GraphDbAccessor dba(db);
|
GraphDbAccessor dba(db);
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
|
|
||||||
auto t1 = engine.Begin();
|
auto t1 = engine.Begin();
|
||||||
mvcc::VersionList<Vertex> vlist1(*t1, 0);
|
mvcc::VersionList<Vertex> vlist1(*t1, 0);
|
||||||
@ -83,7 +83,7 @@ TEST(LabelsIndex, Refresh) {
|
|||||||
KeyIndex<GraphDbTypes::Label, Vertex> index;
|
KeyIndex<GraphDbTypes::Label, Vertex> index;
|
||||||
GraphDb db;
|
GraphDb db;
|
||||||
GraphDbAccessor access(db);
|
GraphDbAccessor access(db);
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
|
|
||||||
// add two vertices to database
|
// add two vertices to database
|
||||||
auto t1 = engine.Begin();
|
auto t1 = engine.Begin();
|
||||||
|
@ -4,7 +4,7 @@
|
|||||||
#include "database/graph_db_accessor.hpp"
|
#include "database/graph_db_accessor.hpp"
|
||||||
#include "database/graph_db_datatypes.hpp"
|
#include "database/graph_db_datatypes.hpp"
|
||||||
#include "database/indexes/label_property_index.hpp"
|
#include "database/indexes/label_property_index.hpp"
|
||||||
#include "transactions/engine_master.hpp"
|
#include "transactions/engine_single_node.hpp"
|
||||||
|
|
||||||
#include "mvcc_gc_common.hpp"
|
#include "mvcc_gc_common.hpp"
|
||||||
|
|
||||||
@ -44,7 +44,7 @@ class LabelPropertyIndexComplexTest : public ::testing::Test {
|
|||||||
LabelPropertyIndex index;
|
LabelPropertyIndex index;
|
||||||
LabelPropertyIndex::Key *key;
|
LabelPropertyIndex::Key *key;
|
||||||
|
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
tx::Transaction *t{nullptr};
|
tx::Transaction *t{nullptr};
|
||||||
|
|
||||||
mvcc::VersionList<Vertex> *vlist;
|
mvcc::VersionList<Vertex> *vlist;
|
||||||
|
@ -10,7 +10,7 @@
|
|||||||
#include "storage/edge.hpp"
|
#include "storage/edge.hpp"
|
||||||
#include "storage/property_value_store.hpp"
|
#include "storage/property_value_store.hpp"
|
||||||
#include "storage/vertex.hpp"
|
#include "storage/vertex.hpp"
|
||||||
#include "transactions/engine_master.hpp"
|
#include "transactions/engine_single_node.hpp"
|
||||||
|
|
||||||
using namespace GraphDbTypes;
|
using namespace GraphDbTypes;
|
||||||
|
|
||||||
@ -118,7 +118,7 @@ TEST(DistributedSerialization, VertexProperties) {
|
|||||||
|
|
||||||
class DistributedSerializationMvcc : public ::testing::Test {
|
class DistributedSerializationMvcc : public ::testing::Test {
|
||||||
protected:
|
protected:
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
tx::Transaction *tx = engine.Begin();
|
tx::Transaction *tx = engine.Begin();
|
||||||
mvcc::VersionList<Vertex> v1_vlist{*tx, 0};
|
mvcc::VersionList<Vertex> v1_vlist{*tx, 0};
|
||||||
Vertex &v1 = *v1_vlist.Oldest();
|
Vertex &v1 = *v1_vlist.Oldest();
|
||||||
|
@ -5,13 +5,13 @@
|
|||||||
#include "mvcc/version.hpp"
|
#include "mvcc/version.hpp"
|
||||||
#include "mvcc/version_list.hpp"
|
#include "mvcc/version_list.hpp"
|
||||||
#include "threading/sync/lock_timeout_exception.hpp"
|
#include "threading/sync/lock_timeout_exception.hpp"
|
||||||
#include "transactions/engine_master.hpp"
|
#include "transactions/engine_single_node.hpp"
|
||||||
#include "transactions/transaction.hpp"
|
#include "transactions/transaction.hpp"
|
||||||
|
|
||||||
#include "mvcc_gc_common.hpp"
|
#include "mvcc_gc_common.hpp"
|
||||||
|
|
||||||
TEST(MVCC, Deadlock) {
|
TEST(MVCC, Deadlock) {
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
|
|
||||||
auto t0 = engine.Begin();
|
auto t0 = engine.Begin();
|
||||||
mvcc::VersionList<Prop> version_list1(*t0, 0);
|
mvcc::VersionList<Prop> version_list1(*t0, 0);
|
||||||
@ -31,7 +31,7 @@ TEST(MVCC, Deadlock) {
|
|||||||
TEST(MVCC, UpdateDontDelete) {
|
TEST(MVCC, UpdateDontDelete) {
|
||||||
std::atomic<int> count{0};
|
std::atomic<int> count{0};
|
||||||
{
|
{
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
auto t1 = engine.Begin();
|
auto t1 = engine.Begin();
|
||||||
mvcc::VersionList<DestrCountRec> version_list(*t1, 0, count);
|
mvcc::VersionList<DestrCountRec> version_list(*t1, 0, count);
|
||||||
engine.Commit(*t1);
|
engine.Commit(*t1);
|
||||||
@ -55,7 +55,7 @@ TEST(MVCC, UpdateDontDelete) {
|
|||||||
|
|
||||||
// Check that we get the oldest record.
|
// Check that we get the oldest record.
|
||||||
TEST(MVCC, Oldest) {
|
TEST(MVCC, Oldest) {
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
auto t1 = engine.Begin();
|
auto t1 = engine.Begin();
|
||||||
mvcc::VersionList<Prop> version_list(*t1, 0);
|
mvcc::VersionList<Prop> version_list(*t1, 0);
|
||||||
auto first = version_list.Oldest();
|
auto first = version_list.Oldest();
|
||||||
|
@ -5,13 +5,13 @@
|
|||||||
#include "mvcc/version.hpp"
|
#include "mvcc/version.hpp"
|
||||||
#include "mvcc/version_list.hpp"
|
#include "mvcc/version_list.hpp"
|
||||||
#include "threading/sync/lock_timeout_exception.hpp"
|
#include "threading/sync/lock_timeout_exception.hpp"
|
||||||
#include "transactions/engine_master.hpp"
|
#include "transactions/engine_single_node.hpp"
|
||||||
#include "transactions/transaction.hpp"
|
#include "transactions/transaction.hpp"
|
||||||
|
|
||||||
class TestClass : public mvcc::Record<TestClass> {
|
class TestClass : public mvcc::Record<TestClass> {
|
||||||
public:
|
public:
|
||||||
// constructs first version, size should be 0
|
// 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_;
|
++version_list_size_;
|
||||||
}
|
}
|
||||||
TestClass *CloneData() { return new TestClass(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
|
// variable where number of versions is stored
|
||||||
int version_list_size = 0;
|
int version_list_size = 0;
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
tx::Transaction *t1 = engine.Begin();
|
tx::Transaction *t1 = engine.Begin();
|
||||||
mvcc::VersionList<TestClass> version_list{*t1, 0, version_list_size};
|
mvcc::VersionList<TestClass> version_list{*t1, 0, version_list_size};
|
||||||
TestClass *v1 = nullptr;
|
TestClass *v1 = nullptr;
|
||||||
|
@ -12,13 +12,13 @@
|
|||||||
#include "mvcc/version_list.hpp"
|
#include "mvcc/version_list.hpp"
|
||||||
#include "storage/garbage_collector.hpp"
|
#include "storage/garbage_collector.hpp"
|
||||||
#include "storage/vertex.hpp"
|
#include "storage/vertex.hpp"
|
||||||
#include "transactions/engine_master.hpp"
|
#include "transactions/engine_single_node.hpp"
|
||||||
|
|
||||||
#include "mvcc_gc_common.hpp"
|
#include "mvcc_gc_common.hpp"
|
||||||
|
|
||||||
class MvccGcTest : public ::testing::Test {
|
class MvccGcTest : public ::testing::Test {
|
||||||
protected:
|
protected:
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
|
|
||||||
private:
|
private:
|
||||||
tx::Transaction *t0 = engine.Begin();
|
tx::Transaction *t0 = engine.Begin();
|
||||||
@ -116,7 +116,7 @@ TEST_F(MvccGcTest, OldestTransactionSnapshot) {
|
|||||||
*/
|
*/
|
||||||
TEST(GarbageCollector, GcClean) {
|
TEST(GarbageCollector, GcClean) {
|
||||||
ConcurrentMap<int64_t, mvcc::VersionList<DestrCountRec> *> collection;
|
ConcurrentMap<int64_t, mvcc::VersionList<DestrCountRec> *> collection;
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
DeferredDeleter<DestrCountRec> deleter;
|
DeferredDeleter<DestrCountRec> deleter;
|
||||||
DeferredDeleter<mvcc::VersionList<DestrCountRec>> vlist_deleter;
|
DeferredDeleter<mvcc::VersionList<DestrCountRec>> vlist_deleter;
|
||||||
GarbageCollector<decltype(collection), DestrCountRec> gc(collection, deleter,
|
GarbageCollector<decltype(collection), DestrCountRec> gc(collection, deleter,
|
||||||
|
@ -1,7 +1,7 @@
|
|||||||
#pragma once
|
#pragma once
|
||||||
|
|
||||||
#include "mvcc/record.hpp"
|
#include "mvcc/record.hpp"
|
||||||
#include "transactions/engine_master.hpp"
|
#include "transactions/engine_single_node.hpp"
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* @brief - Empty class which inherits from mvcc:Record.
|
* @brief - Empty class which inherits from mvcc:Record.
|
||||||
@ -18,7 +18,7 @@ class Prop : public mvcc::Record<Prop> {
|
|||||||
*/
|
*/
|
||||||
class DestrCountRec : public mvcc::Record<DestrCountRec> {
|
class DestrCountRec : public mvcc::Record<DestrCountRec> {
|
||||||
public:
|
public:
|
||||||
DestrCountRec(std::atomic<int> &count) : count_(count) {}
|
explicit DestrCountRec(std::atomic<int> &count) : count_(count) {}
|
||||||
DestrCountRec *CloneData() { return new DestrCountRec(count_); }
|
DestrCountRec *CloneData() { return new DestrCountRec(count_); }
|
||||||
~DestrCountRec() { ++count_; }
|
~DestrCountRec() { ++count_; }
|
||||||
|
|
||||||
@ -29,7 +29,7 @@ class DestrCountRec : public mvcc::Record<DestrCountRec> {
|
|||||||
// helper function for creating a GC snapshot
|
// helper function for creating a GC snapshot
|
||||||
// if given a nullptr it makes a GC snapshot like there
|
// if given a nullptr it makes a GC snapshot like there
|
||||||
// are no active transactions
|
// are no active transactions
|
||||||
auto GcSnapshot(tx::MasterEngine &engine, tx::Transaction *t) {
|
auto GcSnapshot(tx::SingleNodeEngine &engine, tx::Transaction *t) {
|
||||||
if (t != nullptr) {
|
if (t != nullptr) {
|
||||||
tx::Snapshot gc_snap = t->snapshot();
|
tx::Snapshot gc_snap = t->snapshot();
|
||||||
gc_snap.insert(t->id_);
|
gc_snap.insert(t->id_);
|
||||||
|
@ -16,12 +16,10 @@ class WorkerEngineTest : public testing::Test {
|
|||||||
const std::string local{"127.0.0.1"};
|
const std::string local{"127.0.0.1"};
|
||||||
|
|
||||||
System master_system_{local, 0};
|
System master_system_{local, 0};
|
||||||
MasterEngine master_;
|
MasterEngine master_{master_system_};
|
||||||
|
|
||||||
System worker_system_{local, 0};
|
System worker_system_{local, 0};
|
||||||
WorkerEngine worker_{worker_system_, master_system_.endpoint()};
|
WorkerEngine worker_{worker_system_, master_system_.endpoint()};
|
||||||
|
|
||||||
void SetUp() override { master_.StartServer(master_system_); }
|
|
||||||
};
|
};
|
||||||
|
|
||||||
TEST_F(WorkerEngineTest, LocalBegin) {
|
TEST_F(WorkerEngineTest, LocalBegin) {
|
@ -4,11 +4,11 @@
|
|||||||
#include <vector>
|
#include <vector>
|
||||||
|
|
||||||
#include "data_structures/concurrent/concurrent_set.hpp"
|
#include "data_structures/concurrent/concurrent_set.hpp"
|
||||||
#include "transactions/engine_master.hpp"
|
#include "transactions/engine_single_node.hpp"
|
||||||
#include "transactions/transaction.hpp"
|
#include "transactions/transaction.hpp"
|
||||||
|
|
||||||
TEST(Engine, GcSnapshot) {
|
TEST(Engine, GcSnapshot) {
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
ASSERT_EQ(engine.GlobalGcSnapshot(), tx::Snapshot({1}));
|
ASSERT_EQ(engine.GlobalGcSnapshot(), tx::Snapshot({1}));
|
||||||
|
|
||||||
std::vector<tx::Transaction *> transactions;
|
std::vector<tx::Transaction *> transactions;
|
||||||
@ -36,7 +36,7 @@ TEST(Engine, GcSnapshot) {
|
|||||||
}
|
}
|
||||||
|
|
||||||
TEST(Engine, Advance) {
|
TEST(Engine, Advance) {
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
|
|
||||||
auto t0 = engine.Begin();
|
auto t0 = engine.Begin();
|
||||||
auto t1 = engine.Begin();
|
auto t1 = engine.Begin();
|
||||||
@ -49,7 +49,7 @@ TEST(Engine, Advance) {
|
|||||||
}
|
}
|
||||||
|
|
||||||
TEST(Engine, ConcurrentBegin) {
|
TEST(Engine, ConcurrentBegin) {
|
||||||
tx::MasterEngine engine;
|
tx::SingleNodeEngine engine;
|
||||||
std::vector<std::thread> threads;
|
std::vector<std::thread> threads;
|
||||||
ConcurrentSet<tx::transaction_id_t> tx_ids;
|
ConcurrentSet<tx::transaction_id_t> tx_ids;
|
||||||
for (int i = 0; i < 10; ++i) {
|
for (int i = 0; i < 10; ++i) {
|
Loading…
Reference in New Issue
Block a user