Skip to content

Commit 99f76fe

Browse files
liuyang-mychuhan.ly
andauthored
Serve Java API Improvement (#38961)
This is about Serve Java Improvement. The goal of this PR is to make the Java API consistent with Python. The design of the user API is discussed in this proposal: ray-project/enhancements#42. This PR only covers the Java programming API part, including Deployment.bind, Serve.run, DeploymentHandle, and so on, ensuring that these APIs can be used properly. The alignment of some internal core logic, documentation enhancements, and support for the config file will be submitted in subsequent PRs. Signed-off-by: chuhan.ly <[email protected]> Co-authored-by: chuhan.ly <[email protected]>
1 parent 1562330 commit 99f76fe

Some content is hidden

Large Commits have some content hidden by default. Use the searchbox below for content that may be hidden.

55 files changed

+1756
-760
lines changed

java/serve/src/main/java/io/ray/serve/api/Serve.java

Lines changed: 172 additions & 50 deletions
Original file line numberDiff line numberDiff line change
@@ -11,20 +11,27 @@
1111
import io.ray.api.options.ActorLifetime;
1212
import io.ray.serve.common.Constants;
1313
import io.ray.serve.config.RayServeConfig;
14+
import io.ray.serve.dag.Graph;
15+
import io.ray.serve.deployment.Application;
1416
import io.ray.serve.deployment.Deployment;
1517
import io.ray.serve.deployment.DeploymentCreator;
1618
import io.ray.serve.deployment.DeploymentRoute;
1719
import io.ray.serve.exception.RayServeException;
1820
import io.ray.serve.generated.ActorNameList;
21+
import io.ray.serve.handle.DeploymentHandle;
1922
import io.ray.serve.poll.LongPollClientFactory;
2023
import io.ray.serve.replica.ReplicaContext;
2124
import io.ray.serve.util.CollectionUtil;
22-
import io.ray.serve.util.LogUtil;
25+
import io.ray.serve.util.MessageFormatter;
2326
import io.ray.serve.util.ServeProtoUtil;
27+
import java.util.Arrays;
2428
import java.util.Collections;
2529
import java.util.HashMap;
30+
import java.util.List;
2631
import java.util.Map;
2732
import java.util.Optional;
33+
import org.apache.commons.lang3.RandomStringUtils;
34+
import org.apache.commons.lang3.StringUtils;
2835
import org.slf4j.Logger;
2936
import org.slf4j.LoggerFactory;
3037

@@ -43,25 +50,30 @@ public class Serve {
4350
* @return
4451
*/
4552
public static synchronized ServeControllerClient start(Map<String, String> config) {
46-
// Initialize ray if needed.
47-
if (!Ray.isInitialized()) {
48-
System.setProperty("ray.job.namespace", Constants.SERVE_NAMESPACE);
49-
Ray.init();
50-
}
53+
return serveStart(config);
54+
}
55+
56+
private static synchronized ServeControllerClient serveStart(Map<String, String> config) {
5157

5258
try {
5359
ServeControllerClient client = getGlobalClient(true);
5460
LOGGER.info("Connecting to existing Serve app in namespace {}", Constants.SERVE_NAMESPACE);
5561
return client;
5662
} catch (RayServeException | IllegalStateException e) {
57-
LOGGER.info("There is no instance running on this Ray cluster. A new one will be started.");
63+
LOGGER.info(
64+
"There is no Serve instance running on this Ray cluster. A new one will be started.");
65+
}
66+
67+
// Initialize ray if needed.
68+
if (!Ray.isInitialized()) {
69+
init();
5870
}
5971

6072
int httpPort =
6173
Optional.ofNullable(config)
6274
.map(m -> m.get(RayServeConfig.PROXY_HTTP_PORT))
6375
.map(Integer::parseInt)
64-
.orElse(8000);
76+
.orElse(Integer.valueOf(System.getProperty(RayServeConfig.PROXY_HTTP_PORT, "8000")));
6577
PyActorHandle controllerAvatar =
6678
Ray.actor(
6779
PyActorClass.of("ray.serve._private.controller", "ServeControllerAvatar"),
@@ -95,7 +107,8 @@ public static synchronized ServeControllerClient start(Map<String, String> confi
95107
}
96108
} catch (RayTimeoutException e) {
97109
String errMsg =
98-
LogUtil.format("Proxies not available after {}s.", Constants.PROXY_TIMEOUT_S);
110+
MessageFormatter.format(
111+
"HTTP proxies not available after {}s.", Constants.PROXY_TIMEOUT_S);
99112
LOGGER.error(errMsg, e);
100113
throw new RayServeException(errMsg, e);
101114
}
@@ -108,24 +121,6 @@ public static synchronized ServeControllerClient start(Map<String, String> confi
108121
return client;
109122
}
110123

111-
public static synchronized ServeControllerClient start(
112-
boolean detached, boolean dedicatedCpu, Map<String, String> config) {
113-
114-
if (!detached) {
115-
throw new IllegalArgumentException(
116-
"`detached=false` is no longer supported. "
117-
+ "In a future release, it will be removed altogether.");
118-
}
119-
120-
if (dedicatedCpu) {
121-
throw new IllegalArgumentException(
122-
"`dedicatedCpu=true` is no longer supported. "
123-
+ "In a future release, it will be removed altogether.");
124-
}
125-
126-
return start(config);
127-
}
128-
129124
/**
130125
* Completely shut down the connected Serve instance.
131126
*
@@ -142,7 +137,7 @@ public static void shutdown() {
142137
}
143138

144139
LongPollClientFactory.stop();
145-
client.shutdown();
140+
client.shutdown(null);
146141
clearContext();
147142
}
148143

@@ -181,6 +176,11 @@ public static void setInternalReplicaContext(
181176
deploymentName, replicaTag, controllerName, servableObject, config, appName);
182177
}
183178

179+
/**
180+
* Set replica information to global context.
181+
*
182+
* @param replicaContext
183+
*/
184184
public static void setInternalReplicaContext(ReplicaContext replicaContext) {
185185
INTERNAL_REPLICA_CONTEXT = replicaContext;
186186
}
@@ -206,7 +206,8 @@ public static ReplicaContext getReplicaContext() {
206206
*
207207
* @param healthCheckController If True, run a health check on the cached controller if it exists.
208208
* If the check fails, try reconnecting to the controller.
209-
* @return
209+
* @return ServeControllerClient to the running Serve controller. If there is no running
210+
* controller and raise_if_no_controller_running is set to False, returns None.
210211
*/
211212
public static ServeControllerClient getGlobalClient(boolean healthCheckController) {
212213
try {
@@ -222,14 +223,15 @@ public static ServeControllerClient getGlobalClient(boolean healthCheckControlle
222223
LOGGER.info("The cached controller has died. Reconnecting.");
223224
setGlobalClient(null);
224225
}
225-
synchronized (ServeControllerClient.class) {
226-
if (GLOBAL_CLIENT != null) {
227-
return GLOBAL_CLIENT;
228-
}
229-
return connect();
230-
}
226+
return connect();
231227
}
232228

229+
/**
230+
* Gets the global client, which stores the controller's handle.
231+
*
232+
* @return ServeControllerClient to the running Serve controller. If there is no running
233+
* controller and raise_if_no_controller_running is set to False, returns None.
234+
*/
233235
public static ServeControllerClient getGlobalClient() {
234236
return getGlobalClient(false);
235237
}
@@ -249,13 +251,19 @@ private static void setGlobalClient(ServeControllerClient client) {
249251
*
250252
* @return
251253
*/
252-
public static ServeControllerClient connect() {
254+
private static synchronized ServeControllerClient connect() {
255+
256+
if (GLOBAL_CLIENT != null) {
257+
return GLOBAL_CLIENT;
258+
}
259+
253260
// Initialize ray if needed.
254261
if (!Ray.isInitialized()) {
255-
System.setProperty("ray.job.namespace", Constants.SERVE_NAMESPACE);
256-
Ray.init();
262+
init();
257263
}
258264

265+
// When running inside of a replica, _INTERNAL_REPLICA_CONTEXT is set to ensure that the correct
266+
// instance is connected to.
259267
String controllerName =
260268
INTERNAL_REPLICA_CONTEXT != null
261269
? INTERNAL_REPLICA_CONTEXT.getInternalControllerName()
@@ -264,7 +272,7 @@ public static ServeControllerClient connect() {
264272
Optional<BaseActorHandle> optional = Ray.getActor(controllerName, Constants.SERVE_NAMESPACE);
265273
Preconditions.checkState(
266274
optional.isPresent(),
267-
LogUtil.format(
275+
MessageFormatter.format(
268276
"There is no instance running on this Ray cluster. "
269277
+ "Please call `serve.start() to start one."));
270278
LOGGER.info(
@@ -286,24 +294,25 @@ public static ServeControllerClient connect() {
286294
*
287295
* @param name name of the deployment. This must have already been deployed.
288296
* @return Deployment
297+
* @deprecated {@value Constants#MIGRATION_MESSAGE}
289298
*/
299+
@Deprecated
290300
public static Deployment getDeployment(String name) {
301+
LOGGER.warn(Constants.MIGRATION_MESSAGE);
291302
DeploymentRoute deploymentRoute = getGlobalClient().getDeploymentInfo(name);
292303
if (deploymentRoute == null) {
293304
throw new RayServeException(
294-
LogUtil.format("Deployment {} was not found. Did you call Deployment.deploy?", name));
305+
MessageFormatter.format(
306+
"Deployment {} was not found. Did you call Deployment.deploy?", name));
295307
}
296308

297309
// TODO use DeploymentCreator
298310
return new Deployment(
299-
deploymentRoute.getDeploymentInfo().getReplicaConfig().getDeploymentDef(),
300311
name,
301312
deploymentRoute.getDeploymentInfo().getDeploymentConfig(),
313+
deploymentRoute.getDeploymentInfo().getReplicaConfig(),
302314
deploymentRoute.getDeploymentInfo().getVersion(),
303-
null,
304-
deploymentRoute.getDeploymentInfo().getReplicaConfig().getInitArgs(),
305-
deploymentRoute.getRoute(),
306-
deploymentRoute.getDeploymentInfo().getReplicaConfig().getRayActorOptions());
315+
deploymentRoute.getRoute());
307316
}
308317

309318
/**
@@ -312,8 +321,11 @@ public static Deployment getDeployment(String name) {
312321
* <p>Dictionary maps deployment name to Deployment objects.
313322
*
314323
* @return
324+
* @deprecated {@value Constants#MIGRATION_MESSAGE}
315325
*/
326+
@Deprecated
316327
public static Map<String, Deployment> listDeployments() {
328+
LOGGER.warn(Constants.MIGRATION_MESSAGE);
317329
Map<String, DeploymentRoute> infos = getGlobalClient().listDeployments();
318330
if (infos == null || infos.size() == 0) {
319331
return Collections.emptyMap();
@@ -323,15 +335,125 @@ public static Map<String, Deployment> listDeployments() {
323335
deployments.put(
324336
entry.getKey(),
325337
new Deployment(
326-
entry.getValue().getDeploymentInfo().getReplicaConfig().getDeploymentDef(),
327338
entry.getKey(),
328339
entry.getValue().getDeploymentInfo().getDeploymentConfig(),
340+
entry.getValue().getDeploymentInfo().getReplicaConfig(),
329341
entry.getValue().getDeploymentInfo().getVersion(),
330-
null,
331-
entry.getValue().getDeploymentInfo().getReplicaConfig().getInitArgs(),
332-
entry.getValue().getRoute(),
333-
entry.getValue().getDeploymentInfo().getReplicaConfig().getRayActorOptions()));
342+
entry.getValue().getRoute()));
334343
}
335344
return deployments;
336345
}
346+
347+
/**
348+
* Run an application and return a handle to its ingress deployment.
349+
*
350+
* @param target A Serve application returned by `Deployment.bind()`.
351+
* @return A handle that can be used to call the application.
352+
*/
353+
public static Optional<DeploymentHandle> run(Application target) {
354+
return run(target, true, Constants.SERVE_DEFAULT_APP_NAME, null, null);
355+
}
356+
357+
/**
358+
* Run an application and return a handle to its ingress deployment.
359+
*
360+
* @param target A Serve application returned by `Deployment.bind()`.
361+
* @param blocking
362+
* @param name Application name. If not provided, this will be the only application running on the
363+
* cluster (it will delete all others).
364+
* @param routePrefix Route prefix for HTTP requests. If not provided, it will use route_prefix of
365+
* the ingress deployment. If specified neither as an argument nor in the ingress deployment,
366+
* the route prefix will default to '/'.
367+
* @param config
368+
* @return A handle that can be used to call the application.
369+
*/
370+
public static Optional<DeploymentHandle> run(
371+
Application target,
372+
boolean blocking,
373+
String name,
374+
String routePrefix,
375+
Map<String, String> config) {
376+
377+
if (StringUtils.isBlank(name)) {
378+
throw new RayServeException("Application name must a non-empty string.");
379+
}
380+
381+
ServeControllerClient client = serveStart(config);
382+
383+
List<Deployment> deployments = Graph.build(target.getInternalDagNode(), name);
384+
Deployment ingress = Graph.getAndValidateIngressDeployment(deployments);
385+
386+
for (Deployment deployment : deployments) {
387+
// Overwrite route prefix
388+
if (StringUtils.isNotBlank(deployment.getRoutePrefix())
389+
&& StringUtils.isNotBlank(routePrefix)) {
390+
Preconditions.checkArgument(
391+
routePrefix.startsWith("/"), "The route_prefix must start with a forward slash ('/')");
392+
deployment.setRoutePrefix(routePrefix);
393+
}
394+
deployment
395+
.getDeploymentConfig()
396+
.setVersion(
397+
StringUtils.isNotBlank(deployment.getVersion())
398+
? deployment.getVersion()
399+
: RandomStringUtils.randomAlphabetic(6));
400+
}
401+
402+
client.deployApplication(name, deployments, blocking);
403+
404+
return Optional.ofNullable(ingress)
405+
.map(
406+
ingressDeployment ->
407+
client.getDeploymentHandle(ingressDeployment.getName(), name, true));
408+
}
409+
410+
private static void init() {
411+
System.setProperty("ray.job.namespace", Constants.SERVE_NAMESPACE);
412+
Ray.init();
413+
}
414+
415+
/**
416+
* Get a handle to the application's ingress deployment by name.
417+
*
418+
* @param name application name
419+
* @return
420+
*/
421+
public static DeploymentHandle getAppHandle(String name) {
422+
ServeControllerClient client = getGlobalClient();
423+
String ingress =
424+
(String)
425+
((PyActorHandle) client.getController())
426+
.task(PyActorMethod.of("get_ingress_deployment_name"), name)
427+
.remote()
428+
.get();
429+
430+
if (StringUtils.isBlank(ingress)) {
431+
throw new RayServeException(
432+
MessageFormatter.format("Application '{}' does not exist.", ingress));
433+
}
434+
return client.getDeploymentHandle(ingress, name, false);
435+
}
436+
437+
/**
438+
* Delete an application by its name.
439+
*
440+
* <p>Deletes the app with all corresponding deployments.
441+
*
442+
* @param name application name
443+
*/
444+
public static void delete(String name) {
445+
delete(name, true);
446+
}
447+
448+
/**
449+
* Delete an application by its name.
450+
*
451+
* <p>Deletes the app with all corresponding deployments.
452+
*
453+
* @param name application name
454+
* @param blocking Wait for the application to be deleted or not.
455+
*/
456+
public static void delete(String name, boolean blocking) {
457+
getGlobalClient().deleteApps(Arrays.asList(name), blocking);
458+
}
337459
}

0 commit comments

Comments
 (0)