Migrate cereal to boost_serialization

Reviewers: teon.banek

Reviewed By: teon.banek

Subscribers: pullbot

Differential Revision: https://phabricator.memgraph.io/D1077
This commit is contained in:
Mislav Bradac 2017-12-22 14:36:25 +01:00
parent 7b3d298741
commit d3623585e7
38 changed files with 476 additions and 504 deletions

View File

@ -164,9 +164,3 @@ import_header_library(cppitertools ${CMAKE_CURRENT_SOURCE_DIR})
# Setup json
import_header_library(json ${CMAKE_CURRENT_SOURCE_DIR})
# Setup cereal
import_header_library(cereal "${CMAKE_CURRENT_SOURCE_DIR}/cereal/include")
# Make cereal multithreaded by passing -DCEREAL_THREAD_SAFE=1 (note that -D is omitted below).
set_property(TARGET cereal PROPERTY
INTERFACE_COMPILE_DEFINITIONS CEREAL_THREAD_SAFE=1)

View File

@ -123,7 +123,3 @@ cd ..
# git clone https://github.com/r-lyeh/ltalloc.git
ltalloc_tag="43b51c14857111f993f277c46151fdfac91525a2" # Nov 16, 2017
clone git://deps.memgraph.io/ltalloc.git ltalloc $ltalloc_tag
# cereal
#git clone https://github.com/USCiLab/cereal.git
clone git://deps.memgraph.io/cereal.git cereal v1.2.2

View File

@ -10,8 +10,8 @@ set(memgraph_src_files
communication/rpc/rpc.cpp
data_structures/concurrent/skiplist_gc.cpp
database/graph_db.cpp
database/graph_db_config.cpp
database/graph_db_accessor.cpp
database/graph_db_config.cpp
database/state_delta.cpp
distributed/coordination_master.cpp
distributed/coordination_worker.cpp
@ -50,7 +50,7 @@ set(memgraph_src_files
# -----------------------------------------------------------------------------
# memgraph_lib depend on these libraries
set(MEMGRAPH_ALL_LIBS stdc++fs Threads::Threads fmt cppitertools cereal
set(MEMGRAPH_ALL_LIBS stdc++fs Threads::Threads fmt cppitertools
antlr_opencypher_parser_lib dl glog gflags ${Boost_SERIALIZATION_LIBRARY_RELEASE})
if (USE_LTALLOC)
@ -124,4 +124,3 @@ set(examples ${CMAKE_SOURCE_DIR}/release/examples)
install(CODE "execute_process(COMMAND ${examples}/build_examples
WORKING_DIRECTORY ${examples})")
install(DIRECTORY ${examples}/build/ DESTINATION share/memgraph/examples)

View File

@ -18,14 +18,6 @@
#include "data_structures/queue.hpp"
#include "protocol.hpp"
#include "cereal/archives/binary.hpp"
#include "cereal/types/base_class.hpp"
#include "cereal/types/memory.hpp"
#include "cereal/types/polymorphic.hpp"
#include "cereal/types/string.hpp"
#include "cereal/types/utility.hpp"
#include "cereal/types/vector.hpp"
#include "communication/server.hpp"
#include "io/network/network_endpoint.hpp"
#include "threading/sync/spinlock.hpp"

View File

@ -58,4 +58,5 @@ std::unique_ptr<Message> EventStream::Await(
};
void EventStream::Shutdown() { queue_.Shutdown(); }
}
} // namespace communication::messaging

View File

@ -4,8 +4,9 @@
#include <string>
#include <type_traits>
#include <typeindex>
#include <unordered_map>
#include "cereal/types/memory.hpp"
#include "boost/serialization/access.hpp"
#include "data_structures/queue.hpp"
@ -18,9 +19,6 @@ class Message {
public:
virtual ~Message() {}
template <class Archive>
void serialize(Archive &) {}
/**
* Run-time type identification that is used for callbacks.
*
@ -28,6 +26,12 @@ class Message {
* this class
*/
std::type_index type_index() const { return typeid(*this); }
private:
friend boost::serialization::access;
template <class TArchive>
void serialize(TArchive &, unsigned int) {}
};
class EventStream;
@ -106,4 +110,5 @@ class EventStream {
std::string name_;
Queue<std::unique_ptr<Message>> queue_;
};
} // namespace communication::messaging

View File

@ -1,11 +1,15 @@
#include <sstream>
#include "boost/archive/binary_iarchive.hpp"
#include "boost/archive/binary_oarchive.hpp"
#include "boost/serialization/unique_ptr.hpp"
#include "fmt/format.h"
#include "glog/logging.h"
#include "communication/messaging/distributed.hpp"
#include "communication/messaging/local.hpp"
#include "communication/messaging/protocol.hpp"
#include "fmt/format.h"
#include "glog/logging.h"
#include "communication/rpc/messages-inl.hpp"
namespace communication::messaging {
@ -33,11 +37,11 @@ void Session::Execute() {
buffer_.Shift(sizeof(SizeT));
// TODO: check for exceptions
std::istringstream stream;
std::stringstream stream;
stream.str(std::string(reinterpret_cast<char *>(buffer_.data()), len_data));
::cereal::BinaryInputArchive iarchive{stream};
boost::archive::binary_iarchive archive(stream);
std::unique_ptr<Message> message{nullptr};
iarchive(message);
archive >> message;
buffer_.Shift(len_data);
LocalWriter writer(system_, channel);
@ -86,9 +90,9 @@ void SendMessage(const std::string &address, uint16_t port,
}
// Serialize and send message
std::ostringstream stream;
::cereal::BinaryOutputArchive oarchive(stream);
oarchive(message);
std::stringstream stream;
boost::archive::binary_oarchive archive(stream);
archive << message;
const std::string &buffer = stream.str();
int64_t message_size = 2 * sizeof(SizeT) + buffer.size() + channel.size();

View File

@ -1,6 +1,7 @@
#pragma once
#include "cereal/cereal.hpp"
#include "boost/serialization/access.hpp"
#include "boost/serialization/base_object.hpp"
#include "communication/messaging/distributed.hpp"
#include "communication/raft/raft.hpp"
@ -15,10 +16,15 @@ struct PeerRpcRequest : public messaging::Message {
RequestVoteRequest request_vote;
AppendEntriesRequest<State> append_entries;
template <class Archive>
void serialize(Archive &ar) {
ar(cereal::virtual_base_class<messaging::Message>(this), type, request_vote,
append_entries);
private:
friend class boost::serialization::access;
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &boost::serialization::base_object<messaging::Message>(*this);
ar &type;
ar &request_vote;
ar &append_entries;
}
};
@ -27,10 +33,15 @@ struct PeerRpcReply : public messaging::Message {
RequestVoteReply request_vote;
AppendEntriesReply append_entries;
template <class Archive>
void serialize(Archive &ar) {
ar(cereal::virtual_base_class<messaging::Message>(this), type, request_vote,
append_entries);
private:
friend class boost::serialization::access;
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &boost::serialization::base_object<messaging::Message>(*this);
ar &type;
ar &request_vote;
ar &append_entries;
}
};

View File

@ -10,9 +10,10 @@
#include <thread>
#include <vector>
#include "boost/serialization/vector.hpp"
#include "glog/logging.h"
#include "utils/cereal_optional.hpp"
#include "utils/serialization_optional.hpp"
namespace communication::raft {
@ -43,9 +44,10 @@ struct LogEntry {
}
bool operator!=(const LogEntry &rhs) const { return !(*this == rhs); }
template <class Archive>
void serialize(Archive &ar) {
ar(term, command);
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &term;
ar &command;
}
};
@ -56,9 +58,12 @@ struct RequestVoteRequest {
LogIndex last_log_index;
TermId last_log_term;
template <class Archive>
void serialize(Archive &ar) {
ar(candidate_term, candidate_id, last_log_index, last_log_term);
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &candidate_term;
ar &candidate_id;
ar &last_log_index;
ar &last_log_term;
}
};
@ -66,9 +71,10 @@ struct RequestVoteReply {
TermId term;
bool vote_granted;
template <class Archive>
void serialize(Archive &ar) {
ar(term, vote_granted);
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &term;
ar &vote_granted;
}
};
@ -81,10 +87,14 @@ struct AppendEntriesRequest {
std::vector<LogEntry<State>> entries;
LogIndex leader_commit;
template <class Archive>
void serialize(Archive &ar) {
ar(leader_term, leader_id, prev_log_index, prev_log_term, entries,
leader_commit);
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &leader_term;
ar &leader_id;
ar &prev_log_index;
ar &prev_log_term;
ar &entries;
ar &leader_commit;
}
};
@ -92,9 +102,10 @@ struct AppendEntriesReply {
TermId term;
bool success;
template <class Archive>
void serialize(Archive &ar) {
ar(term, success);
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &term;
ar &success;
}
};

View File

@ -10,12 +10,12 @@ struct DummyState {
bool operator==(const Change &) const { return true; }
bool operator!=(const Change &) const { return false; }
template <class Archive>
void serialize(Archive &ar) {}
template <class TArchive>
void serialize(TArchive &, unsigned int) {}
};
template <class Archive>
void serialize(Archive &ar) {}
template <class TArchive>
void serialize(TArchive &, unsigned int) {}
};
struct IntState {
@ -31,15 +31,16 @@ struct IntState {
}
bool operator!=(const Change &rhs) const { return !(*this == rhs); };
template <class Archive>
void serialize(Archive &ar) {
ar(t, d);
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &t;
ar &d;
}
};
template <class Archive>
void serialize(Archive &ar) {
ar(x);
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &x;
}
};
@ -52,21 +53,20 @@ class NoOpNetworkInterface : public RaftNetworkInterface<State> {
public:
~NoOpNetworkInterface() {}
virtual bool SendRequestVote(const MemberId &recipient,
const RequestVoteRequest &request,
RequestVoteReply &reply,
std::chrono::milliseconds timeout) override {
virtual bool SendRequestVote(const MemberId &, const RequestVoteRequest &,
RequestVoteReply &,
std::chrono::milliseconds) override {
return false;
}
virtual bool SendAppendEntries(const MemberId &recipient,
const AppendEntriesRequest<State> &request,
AppendEntriesReply &reply,
std::chrono::milliseconds timeout) override {
virtual bool SendAppendEntries(const MemberId &,
const AppendEntriesRequest<State> &,
AppendEntriesReply &,
std::chrono::milliseconds) override {
return false;
}
virtual void Start(RaftMember<State> &member) override {}
virtual void Start(RaftMember<State> &) override {}
virtual void Shutdown() override {}
};
@ -80,10 +80,10 @@ class NextReplyNetworkInterface : public RaftNetworkInterface<State> {
public:
~NextReplyNetworkInterface() {}
virtual bool SendRequestVote(const MemberId &recipient,
virtual bool SendRequestVote(const MemberId &,
const RequestVoteRequest &request,
RequestVoteReply &reply,
std::chrono::milliseconds timeout) override {
std::chrono::milliseconds) override {
PeerRpcRequest<State> req;
req.type = RpcType::REQUEST_VOTE;
req.request_vote = request;
@ -97,10 +97,10 @@ class NextReplyNetworkInterface : public RaftNetworkInterface<State> {
return true;
}
virtual bool SendAppendEntries(const MemberId &recipient,
virtual bool SendAppendEntries(const MemberId &,
const AppendEntriesRequest<State> &request,
AppendEntriesReply &reply,
std::chrono::milliseconds timeout) override {
std::chrono::milliseconds) override {
PeerRpcRequest<State> req;
req.type = RpcType::APPEND_ENTRIES;
req.append_entries = request;
@ -114,7 +114,7 @@ class NextReplyNetworkInterface : public RaftNetworkInterface<State> {
return true;
}
virtual void Start(RaftMember<State> &member) override {}
virtual void Start(RaftMember<State> &) override {}
virtual void Shutdown() override {}

View File

@ -0,0 +1,35 @@
#include "boost/archive/binary_iarchive.hpp"
#include "boost/archive/binary_oarchive.hpp"
#include "boost/serialization/export.hpp"
#include "distributed/coordination_rpc_messages.hpp"
#include "storage/concurrent_id_mapper_rpc_messages.hpp"
#include "transactions/engine_rpc_messages.hpp"
BOOST_CLASS_EXPORT(tx::SnapshotReq);
BOOST_CLASS_EXPORT(tx::SnapshotRes);
BOOST_CLASS_EXPORT(tx::GcSnapshotReq);
BOOST_CLASS_EXPORT(tx::ClogInfoReq);
BOOST_CLASS_EXPORT(tx::ClogInfoRes);
BOOST_CLASS_EXPORT(tx::ActiveTransactionsReq);
BOOST_CLASS_EXPORT(tx::IsActiveReq);
BOOST_CLASS_EXPORT(tx::IsActiveRes);
#define ID_VALUE_EXPORT_BOOST_TYPE(type) \
BOOST_CLASS_EXPORT(storage::type##IdReq); \
BOOST_CLASS_EXPORT(storage::type##IdRes); \
BOOST_CLASS_EXPORT(storage::Id##type##Req); \
BOOST_CLASS_EXPORT(storage::Id##type##Res);
ID_VALUE_EXPORT_BOOST_TYPE(Label)
ID_VALUE_EXPORT_BOOST_TYPE(EdgeType)
ID_VALUE_EXPORT_BOOST_TYPE(Property)
#undef ID_VALUE_EXPORT_BOOST_TYPE
BOOST_CLASS_EXPORT(distributed::RegisterWorkerReq);
BOOST_CLASS_EXPORT(distributed::RegisterWorkerRes);
BOOST_CLASS_EXPORT(distributed::GetEndpointReq);
BOOST_CLASS_EXPORT(distributed::GetEndpointRes);
BOOST_CLASS_EXPORT(distributed::StopWorkerReq);
BOOST_CLASS_EXPORT(distributed::StopWorkerRes);

View File

@ -1,6 +1,13 @@
#include <iterator>
#include <random>
#include "boost/archive/binary_iarchive.hpp"
#include "boost/archive/binary_oarchive.hpp"
#include "boost/serialization/access.hpp"
#include "boost/serialization/base_object.hpp"
#include "boost/serialization/export.hpp"
#include "boost/serialization/unique_ptr.hpp"
#include "communication/rpc/rpc.hpp"
#include "utils/string.hpp"
@ -24,15 +31,19 @@ class Request : public messaging::Message {
const std::string &message_id() const { return message_id_; }
const messaging::Message &message() const { return *message_; }
template <class Archive>
void serialize(Archive &ar) {
ar(cereal::virtual_base_class<messaging::Message>(this), address_, port_,
stream_, message_id_, message_);
}
private:
friend class boost::serialization::access;
Request() {} // Needed for serialization.
protected:
friend class cereal::access;
Request() {} // Cereal needs access to a default constructor.
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &boost::serialization::base_object<messaging::Message>(*this);
ar &address_;
ar &port_;
ar &stream_;
ar &message_id_;
ar &message_;
}
std::string address_;
uint16_t port_;
@ -47,18 +58,20 @@ class Response : public messaging::Message {
std::unique_ptr<messaging::Message> message)
: message_id_(message_id), message_(std::move(message)) {}
template <class Archive>
void serialize(Archive &ar) {
ar(cereal::virtual_base_class<messaging::Message>(this), message_id_,
message_);
}
const auto &message_id() const { return message_id_; }
auto &message() { return message_; }
protected:
Response() {} // Cereal needs access to a default constructor.
friend class cereal::access;
private:
friend class boost::serialization::access;
Response() {} // Needed for serialization.
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &boost::serialization::base_object<Message>(*this);
ar &message_id_;
ar &message_;
}
std::string message_id_;
std::unique_ptr<messaging::Message> message_;
};
@ -138,6 +151,8 @@ void Server::Shutdown() {
stream_->Shutdown();
if (running_thread_.joinable()) running_thread_.join();
}
} // namespace communication::rpc
CEREAL_REGISTER_TYPE(communication::rpc::Request);
CEREAL_REGISTER_TYPE(communication::rpc::Response);
BOOST_CLASS_EXPORT(communication::rpc::Request);
BOOST_CLASS_EXPORT(communication::rpc::Response);

View File

@ -73,9 +73,8 @@ class Server {
typename TRequestResponse::Response>::value,
"TRequestResponse::Response must be derived from Message");
auto got = callbacks_.emplace(
typeid(typename TRequestResponse::Request), [callback = callback](
const messaging::Message
&base_message) {
typeid(typename TRequestResponse::Request),
[callback = callback](const messaging::Message &base_message) {
const auto &message =
dynamic_cast<const typename TRequestResponse::Request &>(
base_message);
@ -99,4 +98,5 @@ class Server {
std::thread running_thread_;
bool started_{false};
};
} // namespace communication::rpc

View File

@ -3,7 +3,6 @@
#include <string>
#include "boost/serialization/base_object.hpp"
#include "cereal/types/base_class.hpp"
#include "utils/total_ordering.hpp"
@ -28,21 +27,15 @@ class Common : TotalOrdering<TSpecificType> {
size_t operator()(const TSpecificType &t) const { return hash(t.storage_); }
};
/** Required for cereal serialization. */
template <class Archive>
void serialize(Archive &archive) {
archive(storage_);
}
private:
StorageT storage_{0};
friend class boost::serialization::access;
template <class TArchive>
void serialize(TArchive &ar, const unsigned int) {
ar & storage_;
ar &storage_;
}
StorageT storage_{0};
};
class Label : public Common<Label> {
@ -52,14 +45,7 @@ class Label : public Common<Label> {
template <class TArchive>
void serialize(TArchive &ar, const unsigned int) {
ar & boost::serialization::base_object<Common<Label>>(*this);
}
public:
/** Required for cereal serialization. */
template <class Archive>
void serialize(Archive &archive) {
archive(cereal::base_class<Common<Label>>(this));
ar &boost::serialization::base_object<Common<Label>>(*this);
}
};
@ -70,14 +56,7 @@ class EdgeType : public Common<EdgeType> {
template <class TArchive>
void serialize(TArchive &ar, const unsigned int) {
ar & boost::serialization::base_object<Common<EdgeType>>(*this);
}
public:
/** Required for cereal serialization. */
template <class Archive>
void serialize(Archive &archive) {
archive(cereal::base_class<Common<EdgeType>>(this));
ar &boost::serialization::base_object<Common<EdgeType>>(*this);
}
};
@ -88,20 +67,14 @@ class Property : public Common<Property> {
template <class TArchive>
void serialize(TArchive &ar, const unsigned int) {
ar & boost::serialization::base_object<Common<Property>>(*this);
}
public:
/** Required for cereal serialization. */
template <class Archive>
void serialize(Archive &archive) {
archive(cereal::base_class<Common<Property>>(this));
ar &boost::serialization::base_object<Common<Property>>(*this);
}
};
}; // namespace GraphDbTypes
namespace std {
template <>
struct hash<GraphDbTypes::Label>
: public GraphDbTypes::Common<GraphDbTypes::Label>::Hash {};
@ -111,4 +84,5 @@ struct hash<GraphDbTypes::EdgeType>
template <>
struct hash<GraphDbTypes::Property>
: public GraphDbTypes::Common<GraphDbTypes::Property>::Hash {};
}; // namespace std
} // namespace std

View File

@ -1,5 +1,8 @@
#pragma once
#include "boost/serialization/access.hpp"
#include "boost/serialization/base_object.hpp"
#include "communication/messaging/local.hpp"
#include "communication/rpc/rpc.hpp"
#include "io/network/network_endpoint.hpp"
@ -20,9 +23,14 @@ struct RegisterWorkerReq : public Message {
int desired_worker_id;
Endpoint endpoint;
template <class Archive>
void serialize(Archive &ar) {
ar(cereal::virtual_base_class<Message>(this), desired_worker_id, endpoint);
private:
friend class boost::serialization::access;
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &boost::serialization::base_object<Message>(*this);
ar &desired_worker_id;
ar &endpoint;
}
};
@ -40,10 +48,3 @@ using StopWorkerRpc =
communication::rpc::RequestResponse<StopWorkerReq, StopWorkerRes>;
} // namespace distributed
CEREAL_REGISTER_TYPE(distributed::RegisterWorkerReq);
CEREAL_REGISTER_TYPE(distributed::RegisterWorkerRes);
CEREAL_REGISTER_TYPE(distributed::GetEndpointReq);
CEREAL_REGISTER_TYPE(distributed::GetEndpointRes);
CEREAL_REGISTER_TYPE(distributed::StopWorkerReq);
CEREAL_REGISTER_TYPE(distributed::StopWorkerRes);

View File

@ -49,4 +49,5 @@ bool NetworkEndpoint::operator==(const NetworkEndpoint &other) const {
std::begin(other.address_)) &&
port_ == other.port_ && family_ == other.family_;
}
} // namespace io::network

View File

@ -4,6 +4,8 @@
#include <cstdint>
#include <string>
#include "boost/serialization/access.hpp"
#include "utils/exceptions.hpp"
namespace io::network {
@ -25,18 +27,23 @@ class NetworkEndpoint {
uint16_t port() const { return port_; }
unsigned char family() const { return family_; }
/** Required for cereal serialization. */
template <class Archive>
void serialize(Archive &archive) {
archive(address_, port_str_, port_, family_);
}
bool operator==(const NetworkEndpoint &other) const;
private:
friend class boost::serialization::access;
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &address_;
ar &port_str_;
ar &port_;
ar &family_;
}
char address_[INET6_ADDRSTRLEN];
char port_str_[6];
uint16_t port_;
unsigned char family_;
};
}
} // namespace io::network

View File

@ -1,5 +1,10 @@
#include "query/frontend/ast/ast.hpp"
// Include archives before registering most derived types.
#include "boost/archive/binary_iarchive.hpp"
#include "boost/archive/binary_oarchive.hpp"
#include "boost/serialization/export.hpp"
namespace query {
AstTreeStorage::AstTreeStorage() {
@ -35,62 +40,56 @@ ReturnBody CloneReturnBody(AstTreeStorage &storage, const ReturnBody &body) {
} // namespace query
// Include archives before registering most derived types.
#include "boost/archive/binary_iarchive.hpp"
#include "boost/archive/binary_oarchive.hpp"
#include "boost/archive/text_iarchive.hpp"
#include "boost/archive/text_oarchive.hpp"
BOOST_CLASS_EXPORT_IMPLEMENT(query::Query);
BOOST_CLASS_EXPORT_IMPLEMENT(query::SingleQuery);
BOOST_CLASS_EXPORT_IMPLEMENT(query::CypherUnion);
BOOST_CLASS_EXPORT_IMPLEMENT(query::NamedExpression);
BOOST_CLASS_EXPORT_IMPLEMENT(query::OrOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::XorOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::AndOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::NotOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::AdditionOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::SubtractionOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::MultiplicationOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::DivisionOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::ModOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::NotEqualOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::EqualOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::LessOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::GreaterOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::LessEqualOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::GreaterEqualOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::InListOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::ListMapIndexingOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::ListSlicingOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::IfOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::UnaryPlusOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::UnaryMinusOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::IsNullOperator);
BOOST_CLASS_EXPORT_IMPLEMENT(query::ListLiteral);
BOOST_CLASS_EXPORT_IMPLEMENT(query::MapLiteral);
BOOST_CLASS_EXPORT_IMPLEMENT(query::PropertyLookup);
BOOST_CLASS_EXPORT_IMPLEMENT(query::LabelsTest);
BOOST_CLASS_EXPORT_IMPLEMENT(query::Aggregation);
BOOST_CLASS_EXPORT_IMPLEMENT(query::Function);
BOOST_CLASS_EXPORT_IMPLEMENT(query::All);
BOOST_CLASS_EXPORT_IMPLEMENT(query::ParameterLookup);
BOOST_CLASS_EXPORT_IMPLEMENT(query::Create);
BOOST_CLASS_EXPORT_IMPLEMENT(query::Match);
BOOST_CLASS_EXPORT_IMPLEMENT(query::Return);
BOOST_CLASS_EXPORT_IMPLEMENT(query::With);
BOOST_CLASS_EXPORT_IMPLEMENT(query::Pattern);
BOOST_CLASS_EXPORT_IMPLEMENT(query::NodeAtom);
BOOST_CLASS_EXPORT_IMPLEMENT(query::EdgeAtom);
BOOST_CLASS_EXPORT_IMPLEMENT(query::Delete);
BOOST_CLASS_EXPORT_IMPLEMENT(query::Where);
BOOST_CLASS_EXPORT_IMPLEMENT(query::SetProperty);
BOOST_CLASS_EXPORT_IMPLEMENT(query::SetProperties);
BOOST_CLASS_EXPORT_IMPLEMENT(query::SetLabels);
BOOST_CLASS_EXPORT_IMPLEMENT(query::RemoveProperty);
BOOST_CLASS_EXPORT_IMPLEMENT(query::RemoveLabels);
BOOST_CLASS_EXPORT_IMPLEMENT(query::Merge);
BOOST_CLASS_EXPORT_IMPLEMENT(query::Unwind);
BOOST_CLASS_EXPORT_IMPLEMENT(query::Identifier);
BOOST_CLASS_EXPORT_IMPLEMENT(query::PrimitiveLiteral);
BOOST_CLASS_EXPORT_IMPLEMENT(query::CreateIndex);
BOOST_CLASS_EXPORT(query::Query);
BOOST_CLASS_EXPORT(query::SingleQuery);
BOOST_CLASS_EXPORT(query::CypherUnion);
BOOST_CLASS_EXPORT(query::NamedExpression);
BOOST_CLASS_EXPORT(query::OrOperator);
BOOST_CLASS_EXPORT(query::XorOperator);
BOOST_CLASS_EXPORT(query::AndOperator);
BOOST_CLASS_EXPORT(query::NotOperator);
BOOST_CLASS_EXPORT(query::AdditionOperator);
BOOST_CLASS_EXPORT(query::SubtractionOperator);
BOOST_CLASS_EXPORT(query::MultiplicationOperator);
BOOST_CLASS_EXPORT(query::DivisionOperator);
BOOST_CLASS_EXPORT(query::ModOperator);
BOOST_CLASS_EXPORT(query::NotEqualOperator);
BOOST_CLASS_EXPORT(query::EqualOperator);
BOOST_CLASS_EXPORT(query::LessOperator);
BOOST_CLASS_EXPORT(query::GreaterOperator);
BOOST_CLASS_EXPORT(query::LessEqualOperator);
BOOST_CLASS_EXPORT(query::GreaterEqualOperator);
BOOST_CLASS_EXPORT(query::InListOperator);
BOOST_CLASS_EXPORT(query::ListMapIndexingOperator);
BOOST_CLASS_EXPORT(query::ListSlicingOperator);
BOOST_CLASS_EXPORT(query::IfOperator);
BOOST_CLASS_EXPORT(query::UnaryPlusOperator);
BOOST_CLASS_EXPORT(query::UnaryMinusOperator);
BOOST_CLASS_EXPORT(query::IsNullOperator);
BOOST_CLASS_EXPORT(query::ListLiteral);
BOOST_CLASS_EXPORT(query::MapLiteral);
BOOST_CLASS_EXPORT(query::PropertyLookup);
BOOST_CLASS_EXPORT(query::LabelsTest);
BOOST_CLASS_EXPORT(query::Aggregation);
BOOST_CLASS_EXPORT(query::Function);
BOOST_CLASS_EXPORT(query::All);
BOOST_CLASS_EXPORT(query::ParameterLookup);
BOOST_CLASS_EXPORT(query::Create);
BOOST_CLASS_EXPORT(query::Match);
BOOST_CLASS_EXPORT(query::Return);
BOOST_CLASS_EXPORT(query::With);
BOOST_CLASS_EXPORT(query::Pattern);
BOOST_CLASS_EXPORT(query::NodeAtom);
BOOST_CLASS_EXPORT(query::EdgeAtom);
BOOST_CLASS_EXPORT(query::Delete);
BOOST_CLASS_EXPORT(query::Where);
BOOST_CLASS_EXPORT(query::SetProperty);
BOOST_CLASS_EXPORT(query::SetProperties);
BOOST_CLASS_EXPORT(query::SetLabels);
BOOST_CLASS_EXPORT(query::RemoveProperty);
BOOST_CLASS_EXPORT(query::RemoveLabels);
BOOST_CLASS_EXPORT(query::Merge);
BOOST_CLASS_EXPORT(query::Unwind);
BOOST_CLASS_EXPORT(query::Identifier);
BOOST_CLASS_EXPORT(query::PrimitiveLiteral);
BOOST_CLASS_EXPORT(query::CreateIndex);

View File

@ -6,7 +6,6 @@
#include <vector>
#include "boost/serialization/base_object.hpp"
#include "boost/serialization/export.hpp"
#include "boost/serialization/split_member.hpp"
#include "boost/serialization/string.hpp"
#include "boost/serialization/vector.hpp"
@ -39,23 +38,23 @@ namespace query {
#define CLONE_BINARY_EXPRESSION \
auto Clone(AstTreeStorage &storage) const->std::remove_const< \
std::remove_pointer<decltype(this)>::type>::type *override { \
std::remove_pointer<decltype(this)>::type>::type * override { \
return storage.Create< \
std::remove_cv<std::remove_reference<decltype(*this)>::type>::type>( \
expression1_->Clone(storage), expression2_->Clone(storage)); \
}
#define CLONE_UNARY_EXPRESSION \
auto Clone(AstTreeStorage &storage) const->std::remove_const< \
std::remove_pointer<decltype(this)>::type>::type *override { \
std::remove_pointer<decltype(this)>::type>::type * override { \
return storage.Create< \
std::remove_cv<std::remove_reference<decltype(*this)>::type>::type>( \
expression_->Clone(storage)); \
}
#define SERIALIZE_USING_BASE(BaseClass) \
template <class TArchive> \
void serialize(TArchive &ar, const unsigned int) { \
ar & boost::serialization::base_object<BaseClass>(*this); \
#define SERIALIZE_USING_BASE(BaseClass) \
template <class TArchive> \
void serialize(TArchive &ar, const unsigned int) { \
ar &boost::serialization::base_object<BaseClass>(*this); \
}
class Tree;
@ -173,7 +172,7 @@ class Tree : public ::utils::Visitable<HierarchicalTreeVisitor>,
template <class TArchive>
void serialize(TArchive &ar, const unsigned int) {
ar & uid_;
ar &uid_;
}
};
@ -1244,9 +1243,9 @@ class Identifier : public Expression {
template <class TArchive>
void serialize(TArchive &ar, const unsigned int) {
ar & boost::serialization::base_object<Expression>(*this);
ar & name_;
ar & user_declared_;
ar &boost::serialization::base_object<Expression>(*this);
ar &name_;
ar &user_declared_;
}
template <class TArchive>
@ -1482,8 +1481,8 @@ class Aggregation : public BinaryOperator {
template <class TArchive>
void serialize(TArchive &ar, const unsigned int) {
ar & boost::serialization::base_object<BinaryOperator>(*this);
ar & op_;
ar &boost::serialization::base_object<BinaryOperator>(*this);
ar &op_;
}
template <class TArchive>
@ -1575,8 +1574,8 @@ class ParameterLookup : public Expression {
template <class TArchive>
void serialize(TArchive &ar, const unsigned int) {
ar & boost::serialization::base_object<Expression>(*this);
ar & token_position_;
ar &boost::serialization::base_object<Expression>(*this);
ar &token_position_;
}
template <class TArchive>
@ -2361,8 +2360,8 @@ class Return : public Clause {
template <class TArchive>
void serialize(TArchive &ar, const unsigned int) {
ar & boost::serialization::base_object<Clause>(*this);
ar & body_;
ar &boost::serialization::base_object<Clause>(*this);
ar &body_;
}
template <class TArchive>
@ -2890,9 +2889,9 @@ class CreateIndex : public Clause {
template <class TArchive>
void serialize(TArchive &ar, const unsigned int) {
ar & boost::serialization::base_object<Clause>(*this);
ar & label_;
ar & property_;
ar &boost::serialization::base_object<Clause>(*this);
ar &label_;
ar &property_;
}
template <class TArchive>
@ -2904,72 +2903,20 @@ class CreateIndex : public Clause {
#undef CLONE_BINARY_EXPRESSION
#undef CLONE_UNARY_EXPRESSION
#undef SERIALIZE_USING_BASE
} // namespace query
// All of the serialization cruft follows
BOOST_CLASS_EXPORT_KEY(query::Query);
BOOST_CLASS_EXPORT_KEY(query::SingleQuery);
BOOST_CLASS_EXPORT_KEY(query::CypherUnion);
BOOST_CLASS_EXPORT_KEY(query::NamedExpression);
BOOST_CLASS_EXPORT_KEY(query::OrOperator);
BOOST_CLASS_EXPORT_KEY(query::XorOperator);
BOOST_CLASS_EXPORT_KEY(query::AndOperator);
BOOST_CLASS_EXPORT_KEY(query::NotOperator);
BOOST_CLASS_EXPORT_KEY(query::AdditionOperator);
BOOST_CLASS_EXPORT_KEY(query::SubtractionOperator);
BOOST_CLASS_EXPORT_KEY(query::MultiplicationOperator);
BOOST_CLASS_EXPORT_KEY(query::DivisionOperator);
BOOST_CLASS_EXPORT_KEY(query::ModOperator);
BOOST_CLASS_EXPORT_KEY(query::NotEqualOperator);
BOOST_CLASS_EXPORT_KEY(query::EqualOperator);
BOOST_CLASS_EXPORT_KEY(query::LessOperator);
BOOST_CLASS_EXPORT_KEY(query::GreaterOperator);
BOOST_CLASS_EXPORT_KEY(query::LessEqualOperator);
BOOST_CLASS_EXPORT_KEY(query::GreaterEqualOperator);
BOOST_CLASS_EXPORT_KEY(query::InListOperator);
BOOST_CLASS_EXPORT_KEY(query::ListMapIndexingOperator);
BOOST_CLASS_EXPORT_KEY(query::ListSlicingOperator);
BOOST_CLASS_EXPORT_KEY(query::IfOperator);
BOOST_CLASS_EXPORT_KEY(query::UnaryPlusOperator);
BOOST_CLASS_EXPORT_KEY(query::UnaryMinusOperator);
BOOST_CLASS_EXPORT_KEY(query::IsNullOperator);
BOOST_CLASS_EXPORT_KEY(query::ListLiteral);
BOOST_CLASS_EXPORT_KEY(query::MapLiteral);
BOOST_CLASS_EXPORT_KEY(query::PropertyLookup);
BOOST_CLASS_EXPORT_KEY(query::LabelsTest);
BOOST_CLASS_EXPORT_KEY(query::Aggregation);
BOOST_CLASS_EXPORT_KEY(query::Function);
BOOST_CLASS_EXPORT_KEY(query::All);
BOOST_CLASS_EXPORT_KEY(query::ParameterLookup);
BOOST_CLASS_EXPORT_KEY(query::Create);
BOOST_CLASS_EXPORT_KEY(query::Match);
BOOST_CLASS_EXPORT_KEY(query::Return);
BOOST_CLASS_EXPORT_KEY(query::With);
BOOST_CLASS_EXPORT_KEY(query::Pattern);
BOOST_CLASS_EXPORT_KEY(query::NodeAtom);
BOOST_CLASS_EXPORT_KEY(query::EdgeAtom);
BOOST_CLASS_EXPORT_KEY(query::Delete);
BOOST_CLASS_EXPORT_KEY(query::Where);
BOOST_CLASS_EXPORT_KEY(query::SetProperty);
BOOST_CLASS_EXPORT_KEY(query::SetProperties);
BOOST_CLASS_EXPORT_KEY(query::SetLabels);
BOOST_CLASS_EXPORT_KEY(query::RemoveProperty);
BOOST_CLASS_EXPORT_KEY(query::RemoveLabels);
BOOST_CLASS_EXPORT_KEY(query::Merge);
BOOST_CLASS_EXPORT_KEY(query::Unwind);
BOOST_CLASS_EXPORT_KEY(query::Identifier);
BOOST_CLASS_EXPORT_KEY(query::PrimitiveLiteral);
BOOST_CLASS_EXPORT_KEY(query::CreateIndex);
#define LOAD_AND_CONSTRUCT(DerivedClass, ...) \
template <class TArchive> \
void load_construct_data(TArchive &ar, DerivedClass *cls, \
const unsigned int) { \
::new (cls) DerivedClass(__VA_ARGS__); \
#define LOAD_AND_CONSTRUCT(DerivedClass, ...) \
template <class TArchive> \
void load_construct_data(TArchive &, DerivedClass *cls, \
const unsigned int) { \
::new (cls) DerivedClass(__VA_ARGS__); \
}
namespace boost::serialization {
LOAD_AND_CONSTRUCT(query::Where, 0);
LOAD_AND_CONSTRUCT(query::OrOperator, 0);
LOAD_AND_CONSTRUCT(query::XorOperator, 0);
@ -3026,7 +2973,7 @@ LOAD_AND_CONSTRUCT(query::RemoveLabels, 0);
LOAD_AND_CONSTRUCT(query::Merge, 0);
LOAD_AND_CONSTRUCT(query::Unwind, 0);
LOAD_AND_CONSTRUCT(query::CreateIndex, 0);
} // namespace boost::serialization
#undef LOAD_AND_CONSTRUCT

View File

@ -19,4 +19,5 @@ class ConcurrentIdMapper {
virtual const std::string &id_to_value(const TId &id) = 0;
virtual ~ConcurrentIdMapper() {}
};
} // namespace storage

View File

@ -31,15 +31,3 @@ ID_VALUE_RPC(Property)
#undef ID_VALUE_RPC
} // namespace storage
#define ID_VALUE_REGISTER_CEREAL_TYPE(type) \
CEREAL_REGISTER_TYPE(storage::type##IdReq); \
CEREAL_REGISTER_TYPE(storage::type##IdRes); \
CEREAL_REGISTER_TYPE(storage::Id##type##Req); \
CEREAL_REGISTER_TYPE(storage::Id##type##Res);
ID_VALUE_REGISTER_CEREAL_TYPE(Label)
ID_VALUE_REGISTER_CEREAL_TYPE(EdgeType)
ID_VALUE_REGISTER_CEREAL_TYPE(Property)
#undef ID_VALUE_REGISTER_CEREAL_TYPE

View File

@ -1,5 +1,7 @@
#pragma once
#include "boost/serialization/access.hpp"
#include "data_structures/bitset/dynamic_bitset.hpp"
#include "type.hpp"
@ -33,13 +35,13 @@ class CommitLog {
class Info {
public:
Info() {} // Needed for serialization.
enum Status {
ACTIVE = 0, // 00
COMMITTED = 1, // 01
ABORTED = 2, // 10
};
Info() = default; // Required for cereal serialization
explicit Info(uint8_t flags) : flags_(flags) {}
bool is_active() const { return flags_ == ACTIVE; }
@ -50,13 +52,14 @@ class CommitLog {
operator uint8_t() const { return flags_; }
/** Required for cereal serialization. */
template <class Archive>
void serialize(Archive &archive) {
archive(flags_);
private:
friend class boost::serialization::access;
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &flags_;
}
private:
uint8_t flags_{0};
};
@ -65,4 +68,5 @@ class CommitLog {
private:
DynamicBitset<uint8_t, 32768> log;
};
} // namespace tx

View File

@ -1,11 +1,10 @@
#pragma once
#include "utils/rpc_pimp.hpp"
#include "communication/rpc/rpc.hpp"
#include "transactions/commit_log.hpp"
#include "transactions/snapshot.hpp"
#include "transactions/type.hpp"
#include "utils/rpc_pimp.hpp"
namespace tx {
@ -30,13 +29,5 @@ using ActiveTransactionsRpc =
communication::rpc::RequestResponse<ActiveTransactionsReq, SnapshotRes>;
using IsActiveRpc =
communication::rpc::RequestResponse<IsActiveReq, IsActiveRes>;
} // namespace tx
CEREAL_REGISTER_TYPE(tx::SnapshotReq);
CEREAL_REGISTER_TYPE(tx::SnapshotRes);
CEREAL_REGISTER_TYPE(tx::GcSnapshotReq);
CEREAL_REGISTER_TYPE(tx::ClogInfoReq);
CEREAL_REGISTER_TYPE(tx::ClogInfoRes);
CEREAL_REGISTER_TYPE(tx::ActiveTransactionsReq);
CEREAL_REGISTER_TYPE(tx::IsActiveReq);
CEREAL_REGISTER_TYPE(tx::IsActiveRes);
} // namespace tx

View File

@ -7,7 +7,7 @@
namespace tx {
namespace {
static const auto kRpcTimeout = 100ms;
}
} // namespace
WorkerEngine::WorkerEngine(communication::messaging::System &system,
const io::network::NetworkEndpoint &endpoint)

View File

@ -12,6 +12,7 @@
#include "transactions/transaction.hpp"
namespace tx {
/** A transactional engine for the worker in a distributed system. */
class WorkerEngine : public Engine {
public:
@ -38,4 +39,5 @@ class WorkerEngine : public Engine {
// Communication to the transactional master.
mutable communication::rpc::Client rpc_client_;
};
} // namespace tx

View File

@ -4,6 +4,9 @@
#include <iostream>
#include <vector>
#include "boost/serialization/access.hpp"
#include "boost/serialization/vector.hpp"
#include "glog/logging.h"
#include "transactions/type.hpp"
#include "utils/algorithm.hpp"
@ -84,13 +87,14 @@ class Snapshot {
return stream;
}
/** Required for cereal serialization. */
template <class Archive>
void serialize(Archive &archive) {
archive(transaction_ids_);
private:
friend class boost::serialization::access;
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &transaction_ids_;
}
private:
std::vector<transaction_id_t> transaction_ids_;
};
} // namespace tx

View File

@ -1,28 +0,0 @@
#pragma once
#include <experimental/optional>
namespace cereal {
template <class Archive, class T>
void save(Archive &ar, const std::experimental::optional<T> &opt) {
ar(static_cast<bool>(opt));
if (opt) {
ar(*opt);
}
}
template <class Archive, class T>
void load(Archive &ar, std::experimental::optional<T> &opt) {
bool has_value;
ar(has_value);
if (has_value) {
T tmp;
ar(tmp);
opt = std::move(tmp);
} else {
opt = std::experimental::nullopt;
}
}
} // namespace cereal

View File

@ -1,34 +1,36 @@
#pragma once
#include "cereal/archives/binary.hpp"
#include "cereal/types/base_class.hpp"
#include "cereal/types/memory.hpp"
#include "cereal/types/polymorphic.hpp"
#include "cereal/types/string.hpp"
#include "cereal/types/utility.hpp"
#include "cereal/types/vector.hpp"
#include "boost/serialization/base_object.hpp"
#include "communication/messaging/local.hpp"
#define RPC_NO_MEMBER_MESSAGE(name) \
using communication::messaging::Message; \
struct name : public Message { \
name() {} \
template <class Archive> \
void serialize(Archive &ar) { \
ar(::cereal::virtual_base_class<Message>(this)); \
} \
#define RPC_NO_MEMBER_MESSAGE(name) \
struct name : public communication::messaging::Message { \
name() {} \
\
private: \
friend class boost::serialization::access; \
\
template <class TArchive> \
void serialize(TArchive &ar, unsigned int) { \
ar &boost::serialization::base_object< \
communication::messaging::Message>(*this); \
} \
};
#define RPC_SINGLE_MEMBER_MESSAGE(name, type) \
using communication::messaging::Message; \
struct name : public Message { \
name() {} \
name(const type &member) : member(member) {} \
type member; \
template <class Archive> \
void serialize(Archive &ar) { \
ar(::cereal::virtual_base_class<Message>(this), member); \
} \
#define RPC_SINGLE_MEMBER_MESSAGE(name, type) \
struct name : public communication::messaging::Message { \
name() {} \
name(const type &member) : member(member) {} \
type member; \
\
private: \
friend class boost::serialization::access; \
\
template <class TArchive> \
void serialize(TArchive &ar, unsigned int) { \
ar &boost::serialization::base_object< \
communication::messaging::Message>(*this); \
ar &member; \
} \
};

View File

@ -0,0 +1,35 @@
#include <experimental/optional>
#include "boost/serialization/split_free.hpp"
namespace boost::serialization {
template <class TArchive, class T>
inline void serialize(TArchive &ar, std::experimental::optional<T> &opt,
unsigned int version) {
split_free(ar, opt, version);
}
template <class TArchive, class T>
void save(TArchive &ar, const std::experimental::optional<T> &opt,
unsigned int) {
ar << static_cast<bool>(opt);
if (opt) {
ar << *opt;
}
}
template <class TArchive, class T>
void load(TArchive &ar, std::experimental::optional<T> &opt, unsigned int) {
bool has_value;
ar >> has_value;
if (has_value) {
T tmp;
ar >> tmp;
opt = std::move(tmp);
} else {
opt = std::experimental::nullopt;
}
}
} // boost::serialization

View File

@ -1,3 +1,11 @@
#include "boost/serialization/export.hpp"
#include "boost/archive/binary_iarchive.hpp"
#include "boost/archive/binary_oarchive.hpp"
#include "boost/archive/text_iarchive.hpp"
#include "boost/archive/text_oarchive.hpp"
#include "boost/serialization/export.hpp"
#include "communication/messaging/distributed.hpp"
#include "communication/raft/rpc.hpp"
#include "communication/raft/test_utils.hpp"
@ -12,8 +20,8 @@ using raft::test_utils::InMemoryStorageInterface;
DEFINE_string(member_id, "", "id of RaftMember");
CEREAL_REGISTER_TYPE(raft::PeerRpcReply);
CEREAL_REGISTER_TYPE(raft::PeerRpcRequest<DummyState>);
BOOST_CLASS_EXPORT(raft::PeerRpcReply);
BOOST_CLASS_EXPORT(raft::PeerRpcRequest<DummyState>);
/* Start cluster members with:
* ./raft_rpc --member-id a
@ -42,8 +50,9 @@ int main(int argc, char *argv[]) {
{
raft::RaftMember<DummyState> raft_member(network, storage, FLAGS_member_id,
config);
while (true)
;
while (true) {
continue;
}
}
my_system.Shutdown();

View File

@ -1,29 +0,0 @@
#include <sstream>
#include "cereal/archives/json.hpp"
#include "gtest/gtest.h"
#include "utils/cereal_optional.hpp"
using std::experimental::optional;
TEST(CerealOptionalTest, SerializeAndDeserialize) {
std::stringstream ss;
optional<int> x1 = {};
optional<int> x2 = 42;
optional<int> y1, y2;
{
cereal::JSONOutputArchive oarchive(ss);
oarchive(x1, x2);
}
{
cereal::JSONInputArchive iarchive(ss);
iarchive(y1, y2);
}
EXPECT_EQ(x1, y1);
EXPECT_EQ(x2, y2);
}

View File

@ -6,8 +6,8 @@
#include <vector>
#include "antlr4-runtime.h"
#include "boost/archive/text_iarchive.hpp"
#include "boost/archive/text_oarchive.hpp"
#include "boost/archive/binary_iarchive.hpp"
#include "boost/archive/binary_oarchive.hpp"
#include "gmock/gmock.h"
#include "gtest/gtest.h"
@ -121,12 +121,12 @@ class SerializedAstGenerator : public Base {
visitor.visit(parser.tree());
std::stringstream stream;
{
boost::archive::text_oarchive out_archive(stream);
boost::archive::binary_oarchive out_archive(stream);
out_archive << *visitor.query();
}
AstTreeStorage new_ast;
{
boost::archive::text_iarchive in_archive(stream);
boost::archive::binary_iarchive in_archive(stream);
new_ast.Load(in_archive);
}
return new_ast;

View File

@ -1,6 +1,7 @@
#include <experimental/optional>
#include <memory>
#include <thread>
#include <unordered_set>
#include <vector>
#include "gtest/gtest.h"

View File

@ -7,23 +7,35 @@
#include <thread>
#include <vector>
#include "communication/messaging/distributed.hpp"
#include "boost/archive/binary_iarchive.hpp"
#include "boost/archive/binary_oarchive.hpp"
#include "boost/archive/text_iarchive.hpp"
#include "boost/archive/text_oarchive.hpp"
#include "boost/serialization/access.hpp"
#include "boost/serialization/base_object.hpp"
#include "boost/serialization/export.hpp"
#include "gtest/gtest.h"
#include "communication/messaging/distributed.hpp"
using namespace communication::messaging;
using namespace std::literals::chrono_literals;
struct MessageInt : public Message {
MessageInt() {} // cereal needs this
MessageInt(int x) : x(x) {}
int x;
template <class Archive>
void serialize(Archive &ar) {
ar(cereal::virtual_base_class<Message>(this), x);
private:
friend class boost::serialization::access;
MessageInt() {} // Needed for serialization
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &boost::serialization::base_object<Message>(*this);
ar &x;
}
};
CEREAL_REGISTER_TYPE(MessageInt);
BOOST_CLASS_EXPORT(MessageInt);
#define GET_X(p) dynamic_cast<MessageInt *>((p).get())->x

View File

@ -1,8 +1,15 @@
#include <thread>
#include "boost/archive/binary_iarchive.hpp"
#include "boost/archive/binary_oarchive.hpp"
#include "boost/archive/text_iarchive.hpp"
#include "boost/archive/text_oarchive.hpp"
#include "boost/serialization/access.hpp"
#include "boost/serialization/base_object.hpp"
#include "boost/serialization/export.hpp"
#include "gmock/gmock.h"
#include "gtest/gtest.h"
#include <thread>
#include "communication/messaging/distributed.hpp"
#include "communication/rpc/rpc.hpp"
#include "gtest/gtest.h"
@ -13,29 +20,38 @@ using namespace communication::rpc;
using namespace std::literals::chrono_literals;
struct SumReq : public Message {
SumReq() {} // cereal needs this
SumReq(int x, int y) : x(x), y(y) {}
int x;
int y;
template <class Archive>
void serialize(Archive &ar) {
ar(cereal::virtual_base_class<Message>(this), x, y);
private:
friend class boost::serialization::access;
SumReq() {} // Needed for serialization.
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &boost::serialization::base_object<Message>(*this);
ar &x;
ar &y;
}
};
CEREAL_REGISTER_TYPE(SumReq);
BOOST_CLASS_EXPORT(SumReq);
struct SumRes : public Message {
SumRes() {} // cereal needs this
SumRes(int sum) : sum(sum) {}
int sum;
template <class Archive>
void serialize(Archive &ar) {
ar(cereal::virtual_base_class<Message>(this), sum);
private:
friend class boost::serialization::access;
SumRes() {} // Needed for serialization.
template <class TArchive>
void serialize(TArchive &ar, unsigned int) {
ar &boost::serialization::base_object<Message>(*this);
ar &sum;
}
};
CEREAL_REGISTER_TYPE(SumRes);
BOOST_CLASS_EXPORT(SumRes);
using Sum = RequestResponse<SumReq, SumRes>;
TEST(Rpc, Call) {

View File

@ -0,0 +1,33 @@
#include <experimental/optional>
#include <sstream>
#include "gtest/gtest.h"
#include "boost/archive/binary_iarchive.hpp"
#include "boost/archive/binary_oarchive.hpp"
#include "utils/serialization_optional.hpp"
using std::experimental::optional;
TEST(SerializationOptionalTest, SerializeAndDeserialize) {
std::stringstream ss;
optional<int> x1 = {};
optional<int> x2 = 42;
optional<int> y1, y2;
{
boost::archive::binary_oarchive ar(ss);
ar << x1;
ar << x2;
}
{
boost::archive::binary_iarchive ar(ss);
ar >> y1;
ar >> y2;
}
EXPECT_EQ(x1, y1);
EXPECT_EQ(x2, y2);
}

View File

@ -5,6 +5,7 @@
#include "communication/messaging/distributed.hpp"
#include "io/network/network_endpoint.hpp"
#include "transactions/engine_master.hpp"
#include "transactions/engine_rpc_messages.hpp"
#include "transactions/engine_worker.hpp"
using namespace tx;

View File

@ -1,62 +0,0 @@
#!/usr/bin/env python3
# To run from vim add to your .vimrc something similar to:
# command -nargs=* Rpcgen :r !/home/mislav/code/memgraph/tools/rpcgen <args>
import sys
USAGE = "\n\nUsage:\n" \
"./rpcgen request_response_name request_args -- response_args\n" \
"Arguments should be seperated with minus sign (-).\n" \
"Example: ./rpcgen Sum int x - int y -- int sum\n\n"
assert len(sys.argv) >= 3, "Too few arguments.\n" + USAGE
request_response_name = sys.argv[1]
args_string = " ".join(sys.argv[2:])
split = args_string.split("--")
assert len(split) == 2, "Arguments should contain one -- separator.\n" + USAGE
request_args, response_args = split
def generate(message_name, args):
def process_arg(arg):
arg = arg.strip()
assert arg, "Each arg should be non empty string.\n" + USAGE
for i in range(len(arg) - 1, -1, -1):
if not arg[i].isalpha() and arg[i] != "_": break
assert i != -1 and i != len(arg), "Each string separated with - " \
"should contain type and variable name.\n" + USAGE
typ = arg[:i+1].strip()
name = arg[i+1:].strip()
return typ, name
types, names = zip(*map(process_arg, args.split("-")))
return \
"""
struct {message_name} : public Message {{
{message_name}() {{}} // cereal needs this
{message_name}({constructor_args}): {init_list} {{}}
{members}
template <class Archive>
void serialize(Archive &ar) {{
ar(cereal::virtual_base_class<Message>(this), {serialize_args});
}}
}};
CEREAL_REGISTER_TYPE({message_name});""" \
.format(message_name=message_name,
constructor_args=",".join(
map(lambda x: x[0] + " " + x[1], zip(types, names))),
init_list=", ".join(map(lambda x: "{x}({x})".format(x=x), names)),
members="\n".join(map(lambda x:
"{} {};".format(x[0], x[1]), zip(types, names))),
serialize_args=", ".join(names))
request_name = request_response_name + "Req"
response_name = request_response_name + "Res"
req_class = generate(request_name, request_args)
res_class = generate(response_name, response_args)
print(req_class)
print(res_class)
print("using {} = RequestResponse<{}, {}>;".format(
request_response_name, request_name, response_name))