Skip to content

Commit 1e41918

Browse files
committed
comments
Signed-off-by: Edward Oakes <[email protected]>
1 parent 88655df commit 1e41918

File tree

2 files changed

+43
-12
lines changed

2 files changed

+43
-12
lines changed

src/ray/raylet/node_manager.cc

Lines changed: 29 additions & 8 deletions
Original file line numberDiff line numberDiff line change
@@ -224,7 +224,6 @@ NodeManager::NodeManager(
224224
cluster_lease_manager_(cluster_lease_manager),
225225
record_metrics_period_ms_(config.record_metrics_period_ms),
226226
placement_group_resource_manager_(placement_group_resource_manager),
227-
next_resource_seq_no_(0),
228227
ray_syncer_(io_service_, self_node_id_.Binary()),
229228
worker_killing_policy_(std::make_shared<GroupByOwnerIdWorkerKillingPolicy>()),
230229
memory_monitor_(std::make_unique<MemoryMonitor>(
@@ -322,16 +321,29 @@ void NodeManager::RegisterGcs() {
322321
auto on_node_change_subscribe_done = [this](Status status) {
323322
RAY_CHECK_OK(status);
324323

325-
// Register resource manager and scheduler
324+
// RESOURCE_VIEW is used to synchronize available resources across Raylets.
325+
//
326+
// LocalResourceManager::CreateSyncMessage will be called periodically to collect
327+
// the local Raylet's usage to broadcast to others (via the GCS). The updates are
328+
// versioned inside of `LocalResourceManager` to avoid unnecessary broadcasts.
329+
//
330+
// NodeManager::ConsumeSyncMessage will be called when a sync message containing
331+
// other Raylets' resource usage is received.
326332
ray_syncer_.Register(
327333
/* message_type */ syncer::MessageType::RESOURCE_VIEW,
328334
/* reporter */ &cluster_resource_scheduler_.GetLocalResourceManager(),
329335
/* receiver */ this,
330336
/* pull_from_reporter_interval_ms */
331337
report_resources_period_ms_);
332338

333-
// Register a commands channel.
334-
// It's only used for GC right now.
339+
// COMMANDS is used only to broadcast a global request to call the Python garbage
340+
// collector on all Raylets when the cluster is under memory pressure.
341+
//
342+
// Periodic collection is disabled, so this command is only broadcasted via
343+
// `OnDemandBroadcasting` (which will call NodeManager::CreateSyncMessage).
344+
//
345+
// NodeManager::ConsumeSyncMessage is called to execute the GC command from other
346+
// Raylets.
335347
ray_syncer_.Register(
336348
/* message_type */ syncer::MessageType::COMMANDS,
337349
/* reporter */ this,
@@ -346,6 +358,9 @@ void NodeManager::RegisterGcs() {
346358
// If plasma store is under high pressure, we should try to schedule a global
347359
// gc.
348360
if (triggered_by_global_gc) {
361+
// Always increment the sync message version number so that all GC commands
362+
// are sent indiscriminately.
363+
gc_command_sync_version_++;
349364
ray_syncer_.OnDemandBroadcasting(syncer::MessageType::COMMANDS);
350365
}
351366
},
@@ -3032,19 +3047,25 @@ void NodeManager::ConsumeSyncMessage(
30323047

30333048
std::optional<syncer::RaySyncMessage> NodeManager::CreateSyncMessage(
30343049
int64_t after_version, syncer::MessageType message_type) const {
3050+
// This method is only called for the COMMANDS channel, as the RESOURCE_VIEW
3051+
// channel goes through the LocalResourceManager.
30353052
RAY_CHECK_EQ(message_type, syncer::MessageType::COMMANDS);
30363053

3054+
// Serialize the COMMANDS message to a byte string to be nested inside the sync message.
3055+
std::string serialized_commands_sync_msg;
30373056
syncer::CommandsSyncMessage commands_sync_message;
30383057
commands_sync_message.set_should_global_gc(true);
30393058
commands_sync_message.set_cluster_full_of_actors_detected(resource_deadlock_warned_ >=
30403059
1);
3060+
RAY_CHECK(commands_sync_message.SerializeToString(&serialized_commands_sync_msg));
3061+
3062+
// Populate the sync message.
30413063
syncer::RaySyncMessage msg;
3042-
msg.set_version(absl::GetCurrentTimeNanos());
3064+
msg.set_version(gc_command_sync_version_);
30433065
msg.set_node_id(self_node_id_.Binary());
30443066
msg.set_message_type(syncer::MessageType::COMMANDS);
3045-
std::string serialized_msg;
3046-
RAY_CHECK(commands_sync_message.SerializeToString(&serialized_msg));
3047-
msg.set_sync_message(std::move(serialized_msg));
3067+
msg.set_sync_message(std::move(serialized_commands_sync_msg));
3068+
30483069
return std::make_optional(std::move(msg));
30493070
}
30503071

src/ray/raylet/node_manager.h

Lines changed: 14 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -202,8 +202,18 @@ class NodeManager : public rpc::NodeManagerServiceHandler,
202202
/// Get the port of the node manager rpc server.
203203
int GetServerPort() const { return node_manager_server_.GetPort(); }
204204

205+
// Consume a RaySyncer sync message from another Raylet.
206+
//
207+
// The two types of messages that are received are:
208+
// - RESOURCE_VIEW: an update of the resources available on another Raylet.
209+
// - COMMANDS: a request to run the Python garbage collector globally across Raylets.
205210
void ConsumeSyncMessage(std::shared_ptr<const syncer::RaySyncMessage> message) override;
206211

212+
// Generate a RaySyncer sync message to be sent to other Raylets.
213+
//
214+
// This is currently only used to generate messages for the COMMANDS channel to request
215+
// other Raylets to call the Python garbage collector, and is only called on demand
216+
// (not periodically polled by the RaySyncer code).
207217
std::optional<syncer::RaySyncMessage> CreateSyncMessage(
208218
int64_t after_version, syncer::MessageType message_type) const override;
209219

@@ -890,13 +900,13 @@ class NodeManager : public rpc::NodeManagerServiceHandler,
890900
/// Managers all bundle-related operations.
891901
PlacementGroupResourceManager &placement_group_resource_manager_;
892902

893-
/// Next resource broadcast seq no. Non-incrementing sequence numbers
894-
/// indicate network issues (dropped/duplicated/ooo packets, etc).
895-
int64_t next_resource_seq_no_;
896-
897903
/// Ray syncer for synchronization
898904
syncer::RaySyncer ray_syncer_;
899905

906+
/// `version` for the RaySyncer COMMANDS channel. Monotonically incremented each time
907+
/// we issue a GC command so that none of the messages are dropped.
908+
int64_t gc_command_sync_version_ = 0;
909+
900910
/// The Policy for selecting the worker to kill when the node runs out of memory.
901911
std::shared_ptr<WorkerKillingPolicy> worker_killing_policy_;
902912

0 commit comments

Comments
 (0)