Skip to content

Commit

Permalink
Refactor ID Serial 1: Separate ObjectID and TaskID from UniqueID (#4776)
Browse files Browse the repository at this point in the history
* Enable BaseId.

* Change TaskID and make python test pass

* Remove unnecessary functions and fix test failure and change TaskID to
16 bytes.

* Java code change draft

* Refine

* Lint

* Update java/api/src/main/java/org/ray/api/id/TaskId.java

Co-Authored-By: Hao Chen <chenh1024@gmail.com>

* Update java/api/src/main/java/org/ray/api/id/BaseId.java

Co-Authored-By: Hao Chen <chenh1024@gmail.com>

* Update java/api/src/main/java/org/ray/api/id/BaseId.java

Co-Authored-By: Hao Chen <chenh1024@gmail.com>

* Update java/api/src/main/java/org/ray/api/id/ObjectId.java

Co-Authored-By: Hao Chen <chenh1024@gmail.com>

* Address comment

* Lint

* Fix SINGLE_PROCESS

* Fix comments

* Refine code

* Refine test

* Resolve conflict
  • Loading branch information
guoyuhong authored May 22, 2019
1 parent 259cdfa commit 1a39fee
Show file tree
Hide file tree
Showing 57 changed files with 1,076 additions and 644 deletions.
5 changes: 3 additions & 2 deletions java/api/src/main/java/org/ray/api/Ray.java
Original file line number Diff line number Diff line change
@@ -1,6 +1,7 @@
package org.ray.api;

import java.util.List;
import org.ray.api.id.ObjectId;
import org.ray.api.id.UniqueId;
import org.ray.api.runtime.RayRuntime;
import org.ray.api.runtime.RayRuntimeFactory;
Expand Down Expand Up @@ -65,7 +66,7 @@ public static <T> RayObject<T> put(T obj) {
* @param objectId The ID of the object to get.
* @return The Java object.
*/
public static <T> T get(UniqueId objectId) {
public static <T> T get(ObjectId objectId) {
return runtime.get(objectId);
}

Expand All @@ -75,7 +76,7 @@ public static <T> T get(UniqueId objectId) {
* @param objectIds The list of object IDs.
* @return A list of Java objects.
*/
public static <T> List<T> get(List<UniqueId> objectIds) {
public static <T> List<T> get(List<ObjectId> objectIds) {
return runtime.get(objectIds);
}

Expand Down
4 changes: 2 additions & 2 deletions java/api/src/main/java/org/ray/api/RayObject.java
Original file line number Diff line number Diff line change
@@ -1,6 +1,6 @@
package org.ray.api;

import org.ray.api.id.UniqueId;
import org.ray.api.id.ObjectId;

/**
* Represents an object in the object store.
Expand All @@ -17,7 +17,7 @@ public interface RayObject<T> {
/**
* Get the object id.
*/
UniqueId getId();
ObjectId getId();

}

Original file line number Diff line number Diff line change
@@ -1,6 +1,6 @@
package org.ray.api.exception;

import org.ray.api.id.UniqueId;
import org.ray.api.id.ObjectId;

/**
* Indicates that an object is lost (either evicted or explicitly deleted) and cannot be
Expand All @@ -11,9 +11,9 @@
*/
public class UnreconstructableException extends RayException {

public final UniqueId objectId;
public final ObjectId objectId;

public UnreconstructableException(UniqueId objectId) {
public UnreconstructableException(ObjectId objectId) {
super(String.format(
"Object %s is lost (either evicted or explicitly deleted) and cannot be reconstructed.",
objectId));
Expand Down
99 changes: 99 additions & 0 deletions java/api/src/main/java/org/ray/api/id/BaseId.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,99 @@
package org.ray.api.id;

import java.io.Serializable;
import java.nio.ByteBuffer;
import java.util.Arrays;
import javax.xml.bind.DatatypeConverter;

public abstract class BaseId implements Serializable {
private static final long serialVersionUID = 8588849129675565761L;
private final byte[] id;
private int hashCodeCache = 0;
private Boolean isNilCache = null;

/**
* Create a BaseId instance according to the input byte array.
*/
public BaseId(byte[] id) {
if (id.length != size()) {
throw new IllegalArgumentException("Failed to construct BaseId, expect " + size()
+ " bytes, but got " + id.length + " bytes.");
}
this.id = id;
}

/**
* Get the byte data of this id.
*/
public byte[] getBytes() {
return id;
}

/**
* Convert the byte data to a ByteBuffer.
*/
public ByteBuffer toByteBuffer() {
return ByteBuffer.wrap(id);
}

/**
* @return True if this id is nil.
*/
public boolean isNil() {
if (isNilCache == null) {
isNilCache = true;
for (int i = 0; i < size(); ++i) {
if (id[i] != (byte) 0xff) {
isNilCache = false;
break;
}
}
}
return isNilCache;
}

/**
* Derived class should implement this function.
* @return The length of this id in bytes.
*/
public abstract int size();

@Override
public int hashCode() {
// Lazy evaluation.
if (hashCodeCache == 0) {
hashCodeCache = Arrays.hashCode(id);
}
return hashCodeCache;
}

@Override
public boolean equals(Object obj) {
if (obj == null) {
return false;
}

if (!this.getClass().equals(obj.getClass())) {
return false;
}

BaseId r = (BaseId) obj;
return Arrays.equals(id, r.id);
}

@Override
public String toString() {
return DatatypeConverter.printHexBinary(id).toLowerCase();
}

protected static byte[] hexString2Bytes(String hex) {
return DatatypeConverter.parseHexBinary(hex);
}

protected static byte[] byteBuffer2Bytes(ByteBuffer bb) {
byte[] id = new byte[bb.remaining()];
bb.get(id);
return id;
}

}
62 changes: 62 additions & 0 deletions java/api/src/main/java/org/ray/api/id/ObjectId.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,62 @@
package org.ray.api.id;

import java.io.Serializable;
import java.nio.ByteBuffer;
import java.util.Arrays;
import java.util.Random;

/**
* Represents the id of a Ray object.
*/
public class ObjectId extends BaseId implements Serializable {

public static final int LENGTH = 20;
public static final ObjectId NIL = genNil();

/**
* Create an ObjectId from a hex string.
*/
public static ObjectId fromHexString(String hex) {
return new ObjectId(hexString2Bytes(hex));
}

/**
* Create an ObjectId from a ByteBuffer.
*/
public static ObjectId fromByteBuffer(ByteBuffer bb) {
return new ObjectId(byteBuffer2Bytes(bb));
}

/**
* Generate a nil ObjectId.
*/
private static ObjectId genNil() {
byte[] b = new byte[LENGTH];
Arrays.fill(b, (byte) 0xFF);
return new ObjectId(b);
}

/**
* Generate an ObjectId with random value.
*/
public static ObjectId randomId() {
byte[] b = new byte[LENGTH];
new Random().nextBytes(b);
return new ObjectId(b);
}

public ObjectId(byte[] id) {
super(id);
}

@Override
public int size() {
return LENGTH;
}

public TaskId getTaskId() {
byte[] taskIdBytes = Arrays.copyOf(getBytes(), TaskId.LENGTH);
return new TaskId(taskIdBytes);
}

}
56 changes: 56 additions & 0 deletions java/api/src/main/java/org/ray/api/id/TaskId.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,56 @@
package org.ray.api.id;

import java.io.Serializable;
import java.nio.ByteBuffer;
import java.util.Arrays;
import java.util.Random;

/**
* Represents the id of a Ray task.
*/
public class TaskId extends BaseId implements Serializable {

public static final int LENGTH = 16;
public static final TaskId NIL = genNil();

/**
* Create a TaskId from a hex string.
*/
public static TaskId fromHexString(String hex) {
return new TaskId(hexString2Bytes(hex));
}

/**
* Creates a TaskId from a ByteBuffer.
*/
public static TaskId fromByteBuffer(ByteBuffer bb) {
return new TaskId(byteBuffer2Bytes(bb));
}

/**
* Generate a nil TaskId.
*/
private static TaskId genNil() {
byte[] b = new byte[LENGTH];
Arrays.fill(b, (byte) 0xFF);
return new TaskId(b);
}

/**
* Generate an TaskId with random value.
*/
public static TaskId randomId() {
byte[] b = new byte[LENGTH];
new Random().nextBytes(b);
return new TaskId(b);
}

public TaskId(byte[] id) {
super(id);
}

@Override
public int size() {
return LENGTH;
}
}
76 changes: 8 additions & 68 deletions java/api/src/main/java/org/ray/api/id/UniqueId.java
Original file line number Diff line number Diff line change
Expand Up @@ -4,41 +4,34 @@
import java.nio.ByteBuffer;
import java.util.Arrays;
import java.util.Random;
import javax.xml.bind.DatatypeConverter;

/**
* Represents a unique id of all Ray concepts, including
* objects, tasks, workers, actors, etc.
* workers, actors, checkpoints, etc.
*/
public class UniqueId implements Serializable {
public class UniqueId extends BaseId implements Serializable {

public static final int LENGTH = 20;
public static final UniqueId NIL = genNil();
private static final long serialVersionUID = 8588849129675565761L;
private final byte[] id;

/**
* Create a UniqueId from a hex string.
*/
public static UniqueId fromHexString(String hex) {
byte[] bytes = DatatypeConverter.parseHexBinary(hex);
return new UniqueId(bytes);
return new UniqueId(hexString2Bytes(hex));
}

/**
* Creates a UniqueId from a ByteBuffer.
*/
public static UniqueId fromByteBuffer(ByteBuffer bb) {
byte[] id = new byte[bb.remaining()];
bb.get(id);

return new UniqueId(id);
return new UniqueId(byteBuffer2Bytes(bb));
}

/**
* Generate a nil UniqueId.
*/
public static UniqueId genNil() {
private static UniqueId genNil() {
byte[] b = new byte[LENGTH];
Arrays.fill(b, (byte) 0xFF);
return new UniqueId(b);
Expand All @@ -54,64 +47,11 @@ public static UniqueId randomId() {
}

public UniqueId(byte[] id) {
if (id.length != LENGTH) {
throw new IllegalArgumentException("Illegal argument for UniqueId, expect " + LENGTH
+ " bytes, but got " + id.length + " bytes.");
}

this.id = id;
}

/**
* Get the byte data of this UniqueId.
*/
public byte[] getBytes() {
return id;
}

/**
* Convert the byte data to a ByteBuffer.
*/
public ByteBuffer toByteBuffer() {
return ByteBuffer.wrap(id);
}

/**
* Create a copy of this UniqueId.
*/
public UniqueId copy() {
byte[] nid = Arrays.copyOf(id, id.length);
return new UniqueId(nid);
}

/**
* Returns true if this id is nil.
*/
public boolean isNil() {
return this.equals(NIL);
}

@Override
public int hashCode() {
return Arrays.hashCode(id);
}

@Override
public boolean equals(Object obj) {
if (obj == null) {
return false;
}

if (!(obj instanceof UniqueId)) {
return false;
}

UniqueId r = (UniqueId) obj;
return Arrays.equals(id, r.id);
super(id);
}

@Override
public String toString() {
return DatatypeConverter.printHexBinary(id).toLowerCase();
public int size() {
return LENGTH;
}
}
Loading

0 comments on commit 1a39fee

Please sign in to comment.