Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

Fix Client Table #2880

Merged
merged 7 commits into from
Sep 17, 2018
Merged
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
22 changes: 17 additions & 5 deletions python/ray/experimental/state.py
Original file line number Diff line number Diff line change
Expand Up @@ -508,10 +508,12 @@ def client_table(self):
if message is None:
return []

node_info = []
node_info = {}
gcs_entry = ray.gcs_utils.GcsTableEntry.GetRootAsGcsTableEntry(
message, 0)

# Since GCS entries are append-only, we override so that
# only the latest entries are kept.
for i in range(gcs_entry.EntriesLength()):
client = (
ray.gcs_utils.ClientTableData.GetRootAsClientTableData(
Expand All @@ -522,8 +524,18 @@ def client_table(self):
client.ResourcesTotalCapacity(i)
for i in range(client.ResourcesTotalLabelLength())
}
node_info.append({
"ClientID": ray.utils.binary_to_hex(client.ClientId()),
client_id = ray.utils.binary_to_hex(client.ClientId())

This comment was marked as resolved.


# If this client is being removed, then it must
# have previously been inserted, and
# it cannot have previously been removed.
if not client.IsInsertion():
assert client_id in node_info, "Client removed not found!"
assert node_info[client_id]["IsInsertion"], (
"Unexpected duplicate removal of client.")

node_info[client_id] = {
"ClientID": client_id,
"IsInsertion": client.IsInsertion(),
"NodeManagerAddress": decode(client.NodeManagerAddress()),
"NodeManagerPort": client.NodeManagerPort(),
Expand All @@ -532,8 +544,8 @@ def client_table(self):
client.ObjectStoreSocketName()),
"RayletSocketName": decode(client.RayletSocketName()),
"Resources": resources
})
return node_info
}
return list(node_info.values())
Copy link
Collaborator

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I think a list is probably the right structure to return, though I'm not sure. A map from client_id to info doesn't make sense because client_ids are not really meaningful to users. A map from node_ip_address to list of clients could make sense, though in the vast majority of cases each list will contain a single element.

I think this is fine for now. This should work in the setting where we have multiple raylets per node, right?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yes, this works in the setting where we have multiple raylets per node (although it doesn't work when the raylets per node are removed ;) #2878)

I think the client_id -> info mapping is actually fine, but I'm also OK with leaving as is. In calling client_table, I don't expect a list of node_ip_addresses; I expect some data structure that has len() == num clients.


def log_files(self):
"""Fetch and return a dictionary of log file names to outputs.
Expand Down