Skip to content

Commit

Permalink
Allow copy constructor for raft_params (#10)
Browse files Browse the repository at this point in the history
* `raft_params` is just a simple structure and there is no reason to
block copy operation.

* Modified related APIs.
  • Loading branch information
greensky00 committed Aug 8, 2019
1 parent 3beeefc commit 84ecf5c
Show file tree
Hide file tree
Showing 12 changed files with 51 additions and 78 deletions.
6 changes: 3 additions & 3 deletions docs/custom_quorum_size.md
Original file line number Diff line number Diff line change
Expand Up @@ -20,9 +20,9 @@ The default value of both parameters is 0, which follows the original algorithm.
Those parameters are dynamically adjustable; you can change the size of quorum without shutting down Raft server:

```C++
ptr<raft_params> params = server->get_current_params();
params->custom_commit_quorum_size_ = 2;
params->custom_election_quorum_size_ = 4;
raft_params params = server->get_current_params();
params.custom_commit_quorum_size_ = 2;
params.custom_election_quorum_size_ = 4;
server->update_params(params);
```
Expand Down
4 changes: 2 additions & 2 deletions include/libnuraft/context.hxx
Original file line number Diff line number Diff line change
Expand Up @@ -45,14 +45,14 @@ public:
ptr<logger>& l,
ptr<rpc_client_factory>& cli_factory,
ptr<delayed_task_scheduler>& scheduler,
raft_params* params = nilptr )
const raft_params& params )
: state_mgr_(mgr)
, state_machine_(m)
, rpc_listener_(listener)
, logger_(l)
, rpc_cli_factory_(cli_factory)
, scheduler_(scheduler)
, params_(params == nilptr ? new raft_params() : params)
, params_( cs_new<raft_params>(params) )
{}

void set_cb_func(cb_func::func_type func) {
Expand Down
26 changes: 0 additions & 26 deletions include/libnuraft/raft_params.hxx
Original file line number Diff line number Diff line change
Expand Up @@ -29,7 +29,6 @@ limitations under the License.
namespace nuraft {

struct raft_params {
public:
enum return_method_type {
// `append_entries()` will be a blocking call,
// and will return after it is committed in leader node.
Expand Down Expand Up @@ -62,9 +61,6 @@ public:
, return_method_(blocking)
{}

__nocopy__(raft_params);

public:
/**
* Election timeout upper bound in milliseconds
*
Expand Down Expand Up @@ -296,28 +292,6 @@ public:
election_timeout_lower_bound_ - (heart_beat_interval_ / 2) );
}

raft_params* copy_to() const {
raft_params* ret = new raft_params();
ret->election_timeout_upper_bound_ = election_timeout_upper_bound_;
ret->election_timeout_lower_bound_ = election_timeout_lower_bound_;
ret->heart_beat_interval_ = heart_beat_interval_;
ret->rpc_failure_backoff_ = rpc_failure_backoff_;
ret->log_sync_batch_size_ = log_sync_batch_size_;
ret->log_sync_stop_gap_ = log_sync_stop_gap_;
ret->snapshot_distance_ = snapshot_distance_;
ret->snapshot_block_size_ = snapshot_block_size_;
ret->max_append_size_ = max_append_size_;
ret->reserved_log_items_ = reserved_log_items_;
ret->client_req_timeout_ = client_req_timeout_;
ret->fresh_log_gap_ = fresh_log_gap_;
ret->stale_log_gap_ = stale_log_gap_;
ret->custom_commit_quorum_size_ = custom_commit_quorum_size_;
ret->custom_election_quorum_size_ = custom_election_quorum_size_;
ret->auto_forwarding_ = auto_forwarding_;
ret->return_method_ = return_method_;
return ret;
}

public:
// Upper bound of election timer, in millisecond.
int32 election_timeout_upper_bound_;
Expand Down
4 changes: 2 additions & 2 deletions include/libnuraft/raft_server.hxx
Original file line number Diff line number Diff line change
Expand Up @@ -327,7 +327,7 @@ public:
*
* @param new_params Parameters to set.
*/
void update_params(ptr<raft_params>& new_params);
void update_params(const raft_params& new_params);

/**
* Get the current Raft parameters.
Expand All @@ -336,7 +336,7 @@ public:
*
* @return Clone of Raft parameters.
*/
ptr<raft_params> get_current_params() const;
raft_params get_current_params() const;

protected:
typedef std::unordered_map<int32, ptr<peer>>::const_iterator peer_itor;
Expand Down
3 changes: 1 addition & 2 deletions src/launcher.cxx
Original file line number Diff line number Diff line change
Expand Up @@ -37,14 +37,13 @@ ptr<raft_server> raft_launcher::init(ptr<state_machine> sm,
ptr<delayed_task_scheduler> scheduler = asio_svc_;
ptr<rpc_client_factory> rpc_cli_factory = asio_svc_;

raft_params* params = params_given.copy_to();
context* ctx = new context( smgr,
sm,
asio_listener_,
lg,
rpc_cli_factory,
scheduler,
params );
params_given );
raft_instance_ = cs_new<raft_server>(ctx);
asio_listener_->listen( raft_instance_ );
return raft_instance_;
Expand Down
12 changes: 7 additions & 5 deletions src/raft_server.cxx
Original file line number Diff line number Diff line change
Expand Up @@ -251,9 +251,11 @@ void raft_server::update_rand_timeout() {
params->election_timeout_upper_bound_);
}

void raft_server::update_params(ptr<raft_params>& new_params) {
void raft_server::update_params(const raft_params& new_params) {
recur_lock(lock_);
ctx_->set_params(new_params);

ptr<raft_params> clone = cs_new<raft_params>(new_params);
ctx_->set_params(clone);
log_current_params();

update_rand_timeout();
Expand All @@ -262,7 +264,7 @@ void raft_server::update_params(ptr<raft_params>& new_params) {
}
for (auto& entry: peers_) {
peer* p = entry.second.get();
p->set_hb_interval(new_params->heart_beat_interval_);
p->set_hb_interval(clone->heart_beat_interval_);
p->resume_hb_speed();
}
}
Expand Down Expand Up @@ -293,8 +295,8 @@ void raft_server::log_current_params() {
params->custom_election_quorum_size_ );
}

ptr<raft_params> raft_server::get_current_params() const {
return ptr<raft_params>( ctx_->get_params()->copy_to() );
raft_params raft_server::get_current_params() const {
return *ctx_->get_params();
}

void raft_server::stop_server() {
Expand Down
16 changes: 8 additions & 8 deletions tests/bench/raft_bench.cxx
Original file line number Diff line number Diff line change
Expand Up @@ -192,14 +192,14 @@ int init_raft(server_stuff& stuff) {
ptr<rpc_client_factory> rpc_cli_factory = stuff.asio_svc_;

// Set parameters and start Raft server.
raft_params* params = new raft_params();
params->heart_beat_interval_ = 500;
params->election_timeout_lower_bound_ = 1000;
params->election_timeout_upper_bound_ = 2000;
params->reserved_log_items_ = 10000000;
params->snapshot_distance_ = 100000;
params->client_req_timeout_ = 4000;
params->return_method_ = raft_params::blocking;
raft_params params;
params.heart_beat_interval_ = 500;
params.election_timeout_lower_bound_ = 1000;
params.election_timeout_upper_bound_ = 2000;
params.reserved_log_items_ = 10000000;
params.snapshot_distance_ = 100000;
params.client_req_timeout_ = 4000;
params.return_method_ = raft_params::blocking;
context* ctx = new context( stuff.smgr_,
stuff.sm_,
stuff.asio_listener_,
Expand Down
4 changes: 2 additions & 2 deletions tests/unit/asio_service_test.cxx
Original file line number Diff line number Diff line change
Expand Up @@ -524,8 +524,8 @@ int async_append_handler_test() {
// Set async.
for (auto& entry: pkgs) {
RaftAsioPkg* pp = entry;
ptr<raft_params> param = pp->raftServer->get_current_params();
param->return_method_ = raft_params::async_handler;
raft_params param = pp->raftServer->get_current_params();
param.return_method_ = raft_params::async_handler;
pp->raftServer->update_params(param);
}

Expand Down
4 changes: 2 additions & 2 deletions tests/unit/failure_test.cxx
Original file line number Diff line number Diff line change
Expand Up @@ -46,8 +46,8 @@ int simple_conflict_test() {

for (auto& entry: pkgs) {
RaftPkg* pp = entry;
ptr<raft_params> param = pp->raftServer->get_current_params();
param->return_method_ = raft_params::async_handler;
raft_params param = pp->raftServer->get_current_params();
param.return_method_ = raft_params::async_handler;
pp->raftServer->update_params(param);
}

Expand Down
14 changes: 7 additions & 7 deletions tests/unit/raft_package_asio.hxx
Original file line number Diff line number Diff line change
Expand Up @@ -114,13 +114,13 @@ public:
ptr<delayed_task_scheduler> scheduler = asioSvc;
ptr<rpc_client_factory> rpc_cli_factory = asioSvc;

raft_params* params( new raft_params() );
params->with_hb_interval(HEARTBEAT_MS);
params->with_election_timeout_lower(HEARTBEAT_MS * 2);
params->with_election_timeout_upper(HEARTBEAT_MS * 4);
params->with_reserved_log_items(10);
params->with_snapshot_enabled(5);
params->with_client_req_timeout(10000);
raft_params params;
params.with_hb_interval(HEARTBEAT_MS);
params.with_election_timeout_lower(HEARTBEAT_MS * 2);
params.with_election_timeout_upper(HEARTBEAT_MS * 4);
params.with_reserved_log_items(10);
params.with_snapshot_enabled(5);
params.with_client_req_timeout(10000);
context* ctx( new context( sMgr, sm, listener, myLog,
rpc_cli_factory, scheduler, params ) );
raftServer = cs_new<raft_server>(ctx);
Expand Down
20 changes: 9 additions & 11 deletions tests/unit/raft_package_fake.hxx
Original file line number Diff line number Diff line change
Expand Up @@ -39,7 +39,6 @@ public:
, listener(nullptr)
, rpcCliFactory(nullptr)
, scheduler(nullptr)
, params(nullptr)
, ctx(nullptr)
, raftServer(nullptr)
{}
Expand Down Expand Up @@ -67,16 +66,15 @@ public:
scheduler = fTimer;

if (!given_params) {
params = new raft_params();
params->with_election_timeout_lower(0);
params->with_election_timeout_upper(10000);
params->with_hb_interval(5000);
params->with_client_req_timeout(1000000);
params->with_reserved_log_items(0);
params->with_snapshot_enabled(5);
params->with_log_sync_stopping_gap(1);
params.with_election_timeout_lower(0);
params.with_election_timeout_upper(10000);
params.with_hb_interval(5000);
params.with_client_req_timeout(1000000);
params.with_reserved_log_items(0);
params.with_snapshot_enabled(5);
params.with_log_sync_stopping_gap(1);
} else {
params = given_params;
params = *given_params;
}

ctx = new context( sMgr, sm, listener, myLog,
Expand Down Expand Up @@ -116,7 +114,7 @@ public:
ptr<rpc_listener> listener;
ptr<rpc_client_factory> rpcCliFactory;
ptr<delayed_task_scheduler> scheduler;
raft_params* params;
raft_params params;
context* ctx;
ptr<raft_server> raftServer;
};
Expand Down
16 changes: 8 additions & 8 deletions tests/unit/raft_server_test.cxx
Original file line number Diff line number Diff line change
Expand Up @@ -185,13 +185,13 @@ int update_params_test() {

for (auto& entry: pkgs) {
RaftPkg* pp = entry;
ptr<raft_params> param = pp->raftServer->get_current_params();
int old_value = param->election_timeout_upper_bound_;
param->with_election_timeout_upper( old_value + 1 );
raft_params param = pp->raftServer->get_current_params();
int old_value = param.election_timeout_upper_bound_;
param.with_election_timeout_upper( old_value + 1 );
pp->raftServer->update_params(param);

param = pp->raftServer->get_current_params();
CHK_EQ( old_value + 1, param->election_timeout_upper_bound_ );
CHK_EQ( old_value + 1, param.election_timeout_upper_bound_ );
}

print_stats(pkgs);
Expand Down Expand Up @@ -809,8 +809,8 @@ int async_append_handler_test() {

for (auto& entry: pkgs) {
RaftPkg* pp = entry;
ptr<raft_params> param = pp->raftServer->get_current_params();
param->return_method_ = raft_params::async_handler;
raft_params param = pp->raftServer->get_current_params();
param.return_method_ = raft_params::async_handler;
pp->raftServer->update_params(param);
}

Expand Down Expand Up @@ -897,8 +897,8 @@ int async_append_handler_cancel_test() {

for (auto& entry: pkgs) {
RaftPkg* pp = entry;
ptr<raft_params> param = pp->raftServer->get_current_params();
param->return_method_ = raft_params::async_handler;
raft_params param = pp->raftServer->get_current_params();
param.return_method_ = raft_params::async_handler;
pp->raftServer->update_params(param);
}

Expand Down

0 comments on commit 84ecf5c

Please sign in to comment.