diff --git a/python/ray/_private/import_thread.py b/python/ray/_private/import_thread.py index c16413926..9ae460624 100644 --- a/python/ray/_private/import_thread.py +++ b/python/ray/_private/import_thread.py @@ -159,10 +159,10 @@ class ImportThread: run_on_other_drivers) = self.redis_client.hmget( key, ["job_id", "function", "run_on_other_drivers"]) - if (ray._private.utils.decode(run_on_other_drivers) == "False" - and self.worker.mode == ray.SCRIPT_MODE - and job_id != self.worker.current_job_id.binary()): - return + if self.worker.mode == ray.SCRIPT_MODE: + if (run_on_other_drivers == b"False" + or job_id == self.worker.current_job_id.binary()): + return try: # FunctionActorManager may call pickle.loads at the same time. diff --git a/python/ray/tests/test_advanced.py b/python/ray/tests/test_advanced.py index 38eef09c9..28cce4f27 100644 --- a/python/ray/tests/test_advanced.py +++ b/python/ray/tests/test_advanced.py @@ -145,6 +145,11 @@ def test_running_function_on_all_workers(ray_start_regular): assert "fake_directory" == ray.get(get_path1.remote())[-1] + # the function should only run on the current driver once. + assert sys.path[-1] == "fake_directory" + if len(sys.path) > 1: + assert sys.path[-2] != "fake_directory" + def f(worker_info): sys.path.pop(-1)