Skip to content

Commit

Permalink
Merge pull request ClickHouse#56354 from chhetripradeep/pchhetri/yiel…
Browse files Browse the repository at this point in the history
…d-leadership-keeper

Add 4-letter command for yielding/resigning leadership
  • Loading branch information
antonio2368 authored Nov 7, 2023
2 parents 0437b57 + 4e8ff99 commit eb0e4a0
Show file tree
Hide file tree
Showing 9 changed files with 66 additions and 1 deletion.
1 change: 1 addition & 0 deletions docs/en/operations/utilities/clickhouse-keeper-client.md
Original file line number Diff line number Diff line change
Expand Up @@ -12,6 +12,7 @@ A client application to interact with clickhouse-keeper by its native protocol.
- `-q QUERY`, `--query=QUERY` — Query to execute. If this parameter is not passed, `clickhouse-keeper-client` will start in interactive mode.
- `-h HOST`, `--host=HOST` — Server host. Default value: `localhost`.
- `-p N`, `--port=N` — Server port. Default value: 9181
- `-c FILE_PATH`, `--config-file=FILE_PATH` — Set path of config file to get the connection string. Default value: `config.xml`.
- `--connection-timeout=TIMEOUT` — Set connection timeout in seconds. Default value: 10s.
- `--session-timeout=TIMEOUT` — Set session timeout in seconds. Default value: 10s.
- `--operation-timeout=TIMEOUT` — Set operation timeout in seconds. Default value: 10s.
Expand Down
2 changes: 1 addition & 1 deletion src/Coordination/CoordinationSettings.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -36,7 +36,7 @@ void CoordinationSettings::loadFromConfig(const String & config_elem, const Poco
}


const String KeeperConfigurationAndSettings::DEFAULT_FOUR_LETTER_WORD_CMD = "conf,cons,crst,envi,ruok,srst,srvr,stat,wchs,dirs,mntr,isro,rcvr,apiv,csnp,lgif,rqld,rclc,clrs,ftfl";
const String KeeperConfigurationAndSettings::DEFAULT_FOUR_LETTER_WORD_CMD = "conf,cons,crst,envi,ruok,srst,srvr,stat,wchs,dirs,mntr,isro,rcvr,apiv,csnp,lgif,rqld,rclc,clrs,ftfl,ydld";

KeeperConfigurationAndSettings::KeeperConfigurationAndSettings()
: server_id(NOT_EXIST)
Expand Down
9 changes: 9 additions & 0 deletions src/Coordination/FourLetterCommand.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -172,6 +172,9 @@ void FourLetterCommandFactory::registerCommands(KeeperDispatcher & keeper_dispat
FourLetterCommandPtr feature_flags_command = std::make_shared<FeatureFlagsCommand>(keeper_dispatcher);
factory.registerCommand(feature_flags_command);

FourLetterCommandPtr yield_leadership_command = std::make_shared<YieldLeadershipCommand>(keeper_dispatcher);
factory.registerCommand(yield_leadership_command);

factory.initializeAllowList(keeper_dispatcher);
factory.setInitialize(true);
}
Expand Down Expand Up @@ -579,4 +582,10 @@ String FeatureFlagsCommand::run()
return ret.str();
}

String YieldLeadershipCommand::run()
{
keeper_dispatcher.yieldLeadership();
return "Sent yield leadership request to leader.";
}

}
13 changes: 13 additions & 0 deletions src/Coordination/FourLetterCommand.h
Original file line number Diff line number Diff line change
Expand Up @@ -415,4 +415,17 @@ struct FeatureFlagsCommand : public IFourLetterCommand
~FeatureFlagsCommand() override = default;
};

/// Yield leadership and become follower.
struct YieldLeadershipCommand : public IFourLetterCommand
{
explicit YieldLeadershipCommand(KeeperDispatcher & keeper_dispatcher_)
: IFourLetterCommand(keeper_dispatcher_)
{
}

String name() override { return "ydld"; }
String run() override;
~YieldLeadershipCommand() override = default;
};

}
6 changes: 6 additions & 0 deletions src/Coordination/KeeperDispatcher.h
Original file line number Diff line number Diff line change
Expand Up @@ -237,6 +237,12 @@ class KeeperDispatcher
return server->requestLeader();
}

/// Yield leadership and become follower.
void yieldLeadership()
{
return server->yieldLeadership();
}

void recalculateStorageStats()
{
return server->recalculateStorageStats();
Expand Down
6 changes: 6 additions & 0 deletions src/Coordination/KeeperServer.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -1087,6 +1087,12 @@ bool KeeperServer::requestLeader()
return isLeader() || raft_instance->request_leadership();
}

void KeeperServer::yieldLeadership()
{
if (isLeader())
raft_instance->yield_leadership();
}

void KeeperServer::recalculateStorageStats()
{
state_machine->recalculateStorageStats();
Expand Down
2 changes: 2 additions & 0 deletions src/Coordination/KeeperServer.h
Original file line number Diff line number Diff line change
Expand Up @@ -141,6 +141,8 @@ class KeeperServer

bool requestLeader();

void yieldLeadership();

void recalculateStorageStats();
};

Expand Down
5 changes: 5 additions & 0 deletions tests/integration/helpers/keeper_utils.py
Original file line number Diff line number Diff line change
Expand Up @@ -248,6 +248,11 @@ def is_leader(cluster, node, port=9181):
return "Mode: leader" in stat


def is_follower(cluster, node, port=9181):
stat = send_4lw_cmd(cluster, node, "stat", port)
return "Mode: follower" in stat


def get_leader(cluster, nodes):
for node in nodes:
if is_leader(cluster, node):
Expand Down
23 changes: 23 additions & 0 deletions tests/integration/test_keeper_four_word_command/test.py
Original file line number Diff line number Diff line change
Expand Up @@ -718,3 +718,26 @@ def get_memory_purges():

finally:
destroy_zk_client(zk)


def test_cmd_ydld(started_cluster):
wait_nodes()
for node in [node1, node3]:
data = keeper_utils.send_4lw_cmd(cluster, node, cmd="ydld")
assert data == "Sent yield leadership request to leader."

print("ydld output -------------------------------------")
print(data)

if keeper_utils.is_leader(cluster, node):
# wait for it to yield leadership
retry = 0
while keeper_utils.is_leader(cluster, node) and retry < 30:
time.sleep(1)
retry += 1
if retry == 30:
print(
node.name
+ " did not yield leadership after 30s, maybe there is something wrong."
)
assert keeper_utils.is_follower(cluster, node)

0 comments on commit eb0e4a0

Please sign in to comment.