Skip to content

Commit

Permalink
Fix actor ID collision in local mode (#5863)
Browse files Browse the repository at this point in the history
* Fixed local mode actor id

* Update python/ray/actor.py

Co-Authored-By: Edward Oakes <[email protected]>

* Added hyphen to match comments

* Added tests to test_local_mode

* Helloworld

* Better test naming

* lint
  • Loading branch information
edoakes authored Oct 8, 2019
1 parent 375852a commit 42dd0fa
Show file tree
Hide file tree
Showing 3 changed files with 42 additions and 13 deletions.
4 changes: 1 addition & 3 deletions python/ray/actor.py
Original file line number Diff line number Diff line change
Expand Up @@ -369,9 +369,7 @@ def _remote(self,
# Instead, instantiate the actor locally and add it to the worker's
# dictionary
if worker.mode == ray.LOCAL_MODE:
actor_id = ActorID.of(worker.current_job_id,
worker.current_task_id,
worker.task_context.task_index + 1)
actor_id = ActorID.from_random()
worker.actors[actor_id] = meta.modified_class(
*copy.deepcopy(args), **copy.deepcopy(kwargs))
core_handle = ray._raylet.ActorHandle(
Expand Down
29 changes: 19 additions & 10 deletions python/ray/includes/unique_ids.pxi
Original file line number Diff line number Diff line change
Expand Up @@ -211,30 +211,35 @@ cdef class TaskID(BaseID):

@classmethod
def for_driver_task(cls, job_id):
return cls(CTaskID.ForDriverTask(CJobID.FromBinary(job_id.binary())).Binary())
return cls(CTaskID.ForDriverTask(
CJobID.FromBinary(job_id.binary())).Binary())

@classmethod
def for_actor_creation_task(cls, actor_id):
assert isinstance(actor_id, ActorID)
return cls(CTaskID.ForActorCreationTask(CActorID.FromBinary(actor_id.binary())).Binary())
return cls(CTaskID.ForActorCreationTask(
CActorID.FromBinary(actor_id.binary())).Binary())

@classmethod
def for_actor_task(cls, job_id, parent_task_id, parent_task_counter, actor_id):
def for_actor_task(cls, job_id, parent_task_id,
parent_task_counter, actor_id):
assert isinstance(job_id, JobID)
assert isinstance(parent_task_id, TaskID)
assert isinstance(actor_id, ActorID)
return cls(CTaskID.ForActorTask(CJobID.FromBinary(job_id.binary()),
CTaskID.FromBinary(parent_task_id.binary()),
parent_task_counter,
CActorID.FromBinary(actor_id.binary())).Binary())
return cls(CTaskID.ForActorTask(
CJobID.FromBinary(job_id.binary()),
CTaskID.FromBinary(parent_task_id.binary()),
parent_task_counter,
CActorID.FromBinary(actor_id.binary())).Binary())

@classmethod
def for_normal_task(cls, job_id, parent_task_id, parent_task_counter):
assert isinstance(job_id, JobID)
assert isinstance(parent_task_id, TaskID)
return cls(CTaskID.ForNormalTask(CJobID.FromBinary(job_id.binary()),
CTaskID.FromBinary(parent_task_id.binary()),
parent_task_counter).Binary())
return cls(CTaskID.ForNormalTask(
CJobID.FromBinary(job_id.binary()),
CTaskID.FromBinary(parent_task_id.binary()),
parent_task_counter).Binary())

cdef class ClientID(UniqueID):

Expand Down Expand Up @@ -314,6 +319,10 @@ cdef class ActorID(BaseID):
def nil(cls):
return cls(CActorID.Nil().Binary())

@classmethod
def from_random(cls):
return cls(os.urandom(CActorID.Size()))

@classmethod
def size(cls):
return CActorID.Size()
Expand Down
22 changes: 22 additions & 0 deletions python/ray/tests/test_basic.py
Original file line number Diff line number Diff line change
Expand Up @@ -1766,6 +1766,28 @@ def returns_multiple_throws():
with pytest.raises(Exception, match=exception_str):
ray.get(obj2)

# Check that Actors are not overwritten by remote calls from different
# classes.
@ray.remote
class RemoteActor1(object):
def __init__(self):
pass

def function1(self):
return 0

@ray.remote
class RemoteActor2(object):
def __init__(self):
pass

def function2(self):
return 1

actor1 = RemoteActor1.remote()
_ = RemoteActor2.remote()
assert ray.get(actor1.function1.remote()) == 0


def test_resource_constraints(shutdown_only):
num_workers = 20
Expand Down

0 comments on commit 42dd0fa

Please sign in to comment.