Skip to content

Commit 4026b38

Browse files
authored
[Java] Remove RayRuntimeInternal class (#25016)
Due to we have already removed the multiple workers in one process, remove RayRuntimeInternal for purpose.
1 parent 7cf4233 commit 4026b38

21 files changed

+49
-97
lines changed

java/runtime/src/main/java/io/ray/runtime/AbstractRayRuntime.java

Lines changed: 9 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -21,6 +21,7 @@
2121
import io.ray.api.options.PlacementGroupCreationOptions;
2222
import io.ray.api.parallelactor.ParallelActorContext;
2323
import io.ray.api.placementgroup.PlacementGroup;
24+
import io.ray.api.runtime.RayRuntime;
2425
import io.ray.api.runtimecontext.RuntimeContext;
2526
import io.ray.api.runtimeenv.RuntimeEnv;
2627
import io.ray.runtime.config.RayConfig;
@@ -31,6 +32,7 @@
3132
import io.ray.runtime.functionmanager.FunctionManager;
3233
import io.ray.runtime.functionmanager.PyFunctionDescriptor;
3334
import io.ray.runtime.functionmanager.RayFunction;
35+
import io.ray.runtime.gcs.GcsClient;
3436
import io.ray.runtime.generated.Common.Language;
3537
import io.ray.runtime.object.ObjectRefImpl;
3638
import io.ray.runtime.object.ObjectStore;
@@ -50,7 +52,7 @@
5052
import org.slf4j.LoggerFactory;
5153

5254
/** Core functionality to implement Ray APIs. */
53-
public abstract class AbstractRayRuntime implements RayRuntimeInternal {
55+
public abstract class AbstractRayRuntime implements RayRuntime {
5456

5557
private static final Logger LOGGER = LoggerFactory.getLogger(AbstractRayRuntime.class);
5658
public static final String PYTHON_INIT_METHOD_NAME = "__init__";
@@ -82,6 +84,12 @@ public <T> ObjectRef<T> put(T obj) {
8284
/*skipAddingLocalRef=*/ true);
8385
}
8486

87+
public abstract GcsClient getGcsClient();
88+
89+
public abstract void start();
90+
91+
public abstract void run();
92+
8593
@Override
8694
public <T> ObjectRef<T> put(T obj, BaseActorHandle ownerActor) {
8795
if (LOGGER.isDebugEnabled()) {
@@ -355,27 +363,22 @@ private BaseActorHandle createActorImpl(
355363

356364
abstract List<ObjectId> getCurrentReturnIds(int numReturns, ActorId actorId);
357365

358-
@Override
359366
public WorkerContext getWorkerContext() {
360367
return workerContext;
361368
}
362369

363-
@Override
364370
public ObjectStore getObjectStore() {
365371
return objectStore;
366372
}
367373

368-
@Override
369374
public TaskExecutor getTaskExecutor() {
370375
return taskExecutor;
371376
}
372377

373-
@Override
374378
public FunctionManager getFunctionManager() {
375379
return functionManager;
376380
}
377381

378-
@Override
379382
public RayConfig getRayConfig() {
380383
return rayConfig;
381384
}

java/runtime/src/main/java/io/ray/runtime/ConcurrencyGroupImpl.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -24,7 +24,7 @@ public ConcurrencyGroupImpl(String name, int maxConcurrency, List<RayFunc> funcs
2424
funcs.forEach(
2525
func -> {
2626
RayFunction rayFunc =
27-
((RayRuntimeInternal) Ray.internal()).getFunctionManager().getFunction(func);
27+
((AbstractRayRuntime) Ray.internal()).getFunctionManager().getFunction(func);
2828
functionDescriptors.add(rayFunc.getFunctionDescriptor());
2929
});
3030
}

java/runtime/src/main/java/io/ray/runtime/DefaultRayRuntimeFactory.java

Lines changed: 1 addition & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -28,11 +28,10 @@ public RayRuntime createRayRuntime() {
2828

2929
try {
3030
logger.debug("Initializing runtime with config: {}", rayConfig);
31-
AbstractRayRuntime innerRuntime =
31+
AbstractRayRuntime runtime =
3232
rayConfig.runMode == RunMode.LOCAL
3333
? new RayDevRuntime(rayConfig)
3434
: new RayNativeRuntime(rayConfig);
35-
RayRuntimeInternal runtime = innerRuntime;
3635
runtime.start();
3736
return runtime;
3837
} catch (Exception e) {

java/runtime/src/main/java/io/ray/runtime/RayNativeRuntime.java

Lines changed: 0 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -288,8 +288,6 @@ private static native void nativeInitialize(
288288

289289
private static native byte[] nativeGetActorIdOfNamedActor(String actorName, String namespace);
290290

291-
private static native void nativeSetCoreWorker(byte[] workerId);
292-
293291
private static native Map<String, List<ResourceValue>> nativeGetResourceIds();
294292

295293
private static native String nativeGetNamespace();

java/runtime/src/main/java/io/ray/runtime/RayRuntimeInternal.java

Lines changed: 0 additions & 30 deletions
This file was deleted.

java/runtime/src/main/java/io/ray/runtime/actor/NativeActorHandle.java

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -8,7 +8,7 @@
88
import io.ray.api.Ray;
99
import io.ray.api.id.ActorId;
1010
import io.ray.api.id.ObjectId;
11-
import io.ray.runtime.RayRuntimeInternal;
11+
import io.ray.runtime.AbstractRayRuntime;
1212
import io.ray.runtime.generated.Common.Language;
1313
import java.io.Externalizable;
1414
import java.io.IOException;
@@ -122,7 +122,7 @@ private static final class NativeActorHandleReference
122122
public NativeActorHandleReference(NativeActorHandle handle) {
123123
super(handle, REFERENCE_QUEUE);
124124
this.actorId = handle.actorId;
125-
RayRuntimeInternal runtime = (RayRuntimeInternal) Ray.internal();
125+
AbstractRayRuntime runtime = (AbstractRayRuntime) Ray.internal();
126126
this.workerId = runtime.getWorkerContext().getCurrentWorkerId().getBytes();
127127
this.removed = new AtomicBoolean(false);
128128
REFERENCES.add(this);

java/runtime/src/main/java/io/ray/runtime/context/RuntimeContextImpl.java

Lines changed: 3 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -8,7 +8,7 @@
88
import io.ray.api.runtimecontext.NodeInfo;
99
import io.ray.api.runtimecontext.ResourceValue;
1010
import io.ray.api.runtimecontext.RuntimeContext;
11-
import io.ray.runtime.RayRuntimeInternal;
11+
import io.ray.runtime.AbstractRayRuntime;
1212
import io.ray.runtime.config.RunMode;
1313
import io.ray.runtime.util.ResourceUtil;
1414
import java.util.ArrayList;
@@ -21,9 +21,9 @@
2121

2222
public class RuntimeContextImpl implements RuntimeContext {
2323

24-
private RayRuntimeInternal runtime;
24+
private AbstractRayRuntime runtime;
2525

26-
public RuntimeContextImpl(RayRuntimeInternal runtime) {
26+
public RuntimeContextImpl(AbstractRayRuntime runtime) {
2727
this.runtime = runtime;
2828
}
2929

java/runtime/src/main/java/io/ray/runtime/object/NativeObjectStore.java

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -6,7 +6,7 @@
66
import io.ray.api.id.BaseId;
77
import io.ray.api.id.ObjectId;
88
import io.ray.api.id.UniqueId;
9-
import io.ray.runtime.RayRuntimeInternal;
9+
import io.ray.runtime.AbstractRayRuntime;
1010
import io.ray.runtime.context.WorkerContext;
1111
import io.ray.runtime.generated.Common.Address;
1212
import java.util.HashMap;
@@ -40,7 +40,7 @@ public ObjectId putRaw(NativeRayObject obj) {
4040
@Override
4141
public ObjectId putRaw(NativeRayObject obj, ActorId ownerActorId) {
4242
byte[] serializedOwnerAddressBytes =
43-
((RayRuntimeInternal) Ray.internal()).getGcsClient().getActorAddress(ownerActorId);
43+
((AbstractRayRuntime) Ray.internal()).getGcsClient().getActorAddress(ownerActorId);
4444
return new ObjectId(nativePut(obj, serializedOwnerAddressBytes));
4545
}
4646

java/runtime/src/main/java/io/ray/runtime/object/ObjectRefImpl.java

Lines changed: 5 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -8,7 +8,7 @@
88
import io.ray.api.Ray;
99
import io.ray.api.id.ObjectId;
1010
import io.ray.api.id.UniqueId;
11-
import io.ray.runtime.RayRuntimeInternal;
11+
import io.ray.runtime.AbstractRayRuntime;
1212
import java.io.Externalizable;
1313
import java.io.IOException;
1414
import java.io.ObjectInput;
@@ -60,7 +60,7 @@ public ObjectRefImpl(ObjectId id, Class<T> type) {
6060
public void init(ObjectId id, Class<?> type, boolean skipAddingLocalRef) {
6161
this.id = id;
6262
this.type = (Class<T>) type;
63-
RayRuntimeInternal runtime = (RayRuntimeInternal) Ray.internal();
63+
AbstractRayRuntime runtime = (AbstractRayRuntime) Ray.internal();
6464
Preconditions.checkState(workerId == null);
6565
workerId = runtime.getWorkerContext().getCurrentWorkerId();
6666

@@ -106,7 +106,7 @@ public String toString() {
106106
public void writeExternal(ObjectOutput out) throws IOException {
107107
out.writeObject(this.getId());
108108
out.writeObject(this.getType());
109-
RayRuntimeInternal runtime = (RayRuntimeInternal) Ray.internal();
109+
AbstractRayRuntime runtime = (AbstractRayRuntime) Ray.internal();
110110
byte[] ownerAddress = runtime.getObjectStore().getOwnershipInfo(this.getId());
111111
out.writeInt(ownerAddress.length);
112112
out.write(ownerAddress);
@@ -121,7 +121,7 @@ public void readExternal(ObjectInput in) throws IOException, ClassNotFoundExcept
121121
byte[] ownerAddress = new byte[len];
122122
in.readFully(ownerAddress);
123123

124-
RayRuntimeInternal runtime = (RayRuntimeInternal) Ray.internal();
124+
AbstractRayRuntime runtime = (AbstractRayRuntime) Ray.internal();
125125
Preconditions.checkState(workerId == null);
126126
workerId = runtime.getWorkerContext().getCurrentWorkerId();
127127
runtime.getObjectStore().addLocalReference(workerId, id);
@@ -156,7 +156,7 @@ public void finalizeReferent() {
156156
REFERENCES.remove(this);
157157
// It's possible that GC is executed after the runtime is shutdown.
158158
if (Ray.isInitialized()) {
159-
((RayRuntimeInternal) (Ray.internal()))
159+
((AbstractRayRuntime) (Ray.internal()))
160160
.getObjectStore()
161161
.removeLocalReference(workerId, objectId);
162162
allObjects.remove(objectId);

java/runtime/src/main/java/io/ray/runtime/runner/worker/DefaultWorker.java

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -1,7 +1,7 @@
11
package io.ray.runtime.runner.worker;
22

33
import io.ray.api.Ray;
4-
import io.ray.runtime.RayRuntimeInternal;
4+
import io.ray.runtime.AbstractRayRuntime;
55

66
/** Default implementation of the worker process. */
77
public class DefaultWorker {
@@ -12,6 +12,6 @@ public static void main(String[] args) {
1212
System.setProperty("ray.run-mode", "CLUSTER");
1313
System.setProperty("ray.worker.mode", "WORKER");
1414
Ray.init();
15-
((RayRuntimeInternal) Ray.internal()).run();
15+
((AbstractRayRuntime) Ray.internal()).run();
1616
}
1717
}

0 commit comments

Comments
 (0)