Skip to content

Commit

Permalink
renaming project, halo -> ray (ray-project#95)
Browse files Browse the repository at this point in the history
  • Loading branch information
robertnishihara authored and pcmoritz committed Jun 10, 2016
1 parent 44ae178 commit 4cc024a
Show file tree
Hide file tree
Showing 41 changed files with 453 additions and 481 deletions.
34 changes: 17 additions & 17 deletions CMakeLists.txt
Original file line number Diff line number Diff line change
@@ -1,6 +1,6 @@
cmake_minimum_required(VERSION 2.8)

project(halo)
project(ray)

set(THIRDPARTY_DIR "${CMAKE_SOURCE_DIR}/thirdparty")

Expand All @@ -23,35 +23,35 @@ include_directories("${NUMPY_INCLUDE_DIR}")

set(PROTO_PATH "${CMAKE_SOURCE_DIR}/protos")

set(HALO_PROTO "${PROTO_PATH}/halo.proto")
set(RAY_PROTO "${PROTO_PATH}/ray.proto")
set(TYPES_PROTO "${PROTO_PATH}/types.proto")
set(GENERATED_PROTOBUF_PATH "${CMAKE_BINARY_DIR}/generated")
file(MAKE_DIRECTORY ${GENERATED_PROTOBUF_PATH})

set(HALO_PB_CPP_FILE "${GENERATED_PROTOBUF_PATH}/halo.pb.cc")
set(HALO_PB_H_FILE "${GENERATED_PROTOBUF_PATH}/halo.pb.h")
set(HALO_GRPC_PB_CPP_FILE "${GENERATED_PROTOBUF_PATH}/halo.grpc.pb.cc")
set(HALO_GRPC_PB_H_FILE "${GENERATED_PROTOBUF_PATH}/halo.grpc.pb.h")
set(RAY_PB_CPP_FILE "${GENERATED_PROTOBUF_PATH}/ray.pb.cc")
set(RAY_PB_H_FILE "${GENERATED_PROTOBUF_PATH}/ray.pb.h")
set(RAY_GRPC_PB_CPP_FILE "${GENERATED_PROTOBUF_PATH}/ray.grpc.pb.cc")
set(RAY_GRPC_PB_H_FILE "${GENERATED_PROTOBUF_PATH}/ray.grpc.pb.h")

set(TYPES_PB_CPP_FILE "${GENERATED_PROTOBUF_PATH}/types.pb.cc")
set(TYPES_PB_H_FILE "${GENERATED_PROTOBUF_PATH}/types.pb.h")
set(TYPES_GRPC_PB_CPP_FILE "${GENERATED_PROTOBUF_PATH}/types.grpc.pb.cc")
set(TYPES_GRPC_PB_H_FILE "${GENERATED_PROTOBUF_PATH}/types.grpc.pb.h")

add_custom_command(
OUTPUT "${HALO_PB_H_FILE}"
"${HALO_PB_CPP_FILE}"
"${HALO_GRPC_PB_H_FILE}"
"${HALO_GRPC_PB_CPP_FILE}"
OUTPUT "${RAY_PB_H_FILE}"
"${RAY_PB_CPP_FILE}"
"${RAY_GRPC_PB_H_FILE}"
"${RAY_GRPC_PB_CPP_FILE}"
COMMAND ${CMAKE_SOURCE_DIR}/thirdparty/grpc/bins/opt/protobuf/protoc
ARGS "--proto_path=${PROTO_PATH}"
"--cpp_out=${GENERATED_PROTOBUF_PATH}"
"${HALO_PROTO}"
"${RAY_PROTO}"
COMMAND ${CMAKE_SOURCE_DIR}/thirdparty/grpc/bins/opt/protobuf/protoc
ARGS "--proto_path=${PROTO_PATH}"
"--grpc_out=${GENERATED_PROTOBUF_PATH}"
"--plugin=protoc-gen-grpc=${CMAKE_SOURCE_DIR}/thirdparty/grpc/bins/opt/grpc_cpp_plugin"
"${HALO_PROTO}"
"${RAY_PROTO}"
)

add_custom_command(
Expand All @@ -70,8 +70,8 @@ add_custom_command(
"${TYPES_PROTO}"
)

set(GENERATED_PROTOBUF_FILES ${HALO_PB_H_FILE} ${HALO_PB_CPP_FILE}
${HALO_GRPC_PB_H_FILE} ${HALO_GRPC_PB_CPP_FILE}
set(GENERATED_PROTOBUF_FILES ${RAY_PB_H_FILE} ${RAY_PB_CPP_FILE}
${RAY_GRPC_PB_H_FILE} ${RAY_GRPC_PB_CPP_FILE}
${TYPES_PB_H_FILE} ${TYPES_PB_CPP_FILE}
${TYPES_GRPC_PB_H_FILE} ${TYPES_GRPC_PB_CPP_FILE})

Expand Down Expand Up @@ -110,7 +110,7 @@ target_link_libraries(pynumbuf ${ARROW_LIB} ${PYTHON_LIBRARIES})
add_executable(objstore src/objstore.cc src/ipc.cc ${GENERATED_PROTOBUF_FILES})
target_link_libraries(objstore ${ARROW_LIB} pynumbuf)
add_executable(scheduler src/scheduler.cc src/computation_graph.cc ${GENERATED_PROTOBUF_FILES})
add_library(halolib SHARED src/halolib.cc src/worker.cc src/ipc.cc ${GENERATED_PROTOBUF_FILES})
target_link_libraries(halolib ${ARROW_LIB} pynumbuf)
add_library(raylib SHARED src/raylib.cc src/worker.cc src/ipc.cc ${GENERATED_PROTOBUF_FILES})
target_link_libraries(raylib ${ARROW_LIB} pynumbuf)

install(TARGETS objstore scheduler halolib DESTINATION ${CMAKE_SOURCE_DIR}/lib/python/halo)
install(TARGETS objstore scheduler raylib DESTINATION ${CMAKE_SOURCE_DIR}/lib/python/ray)
8 changes: 4 additions & 4 deletions README.md
Original file line number Diff line number Diff line change
@@ -1,6 +1,6 @@
# Halo
# Ray

Halo is a distributed execution framework with a Python-like programming model.
Ray is a distributed execution framework with a Python-like programming model.

## Design Decisions

Expand All @@ -14,6 +14,6 @@ For a description of our design decisions, see

1. sudo apt-get update
2. sudo apt-get install git
3. git clone https://github.com/amplab/halo.git
4. cd halo
3. git clone https://github.com/amplab/ray.git
4. cd ray
5. bash setup.sh
8 changes: 4 additions & 4 deletions doc/aliasing.md
Original file line number Diff line number Diff line change
@@ -1,6 +1,6 @@
# Aliasing

An important feature of Halo is that a remote call sent to the scheduler
An important feature of Ray is that a remote call sent to the scheduler
immediately returns object references to the outputs of the task, and the actual
outputs of the task are only associated with the relevant object references
after the task has been executed and the outputs have been computed. This allows
Expand All @@ -10,15 +10,15 @@ However, to provide a more flexible API, we allow tasks to not only return
values, but to also return object references to values. As an examples, consider
the following code.
```python
@halo.remote([], [np.ndarray])
@ray.remote([], [np.ndarray])
def f()
return np.zeros(5)

@halo.remote([], [np.ndarray])
@ray.remote([], [np.ndarray])
def g()
return f()

@halo.remote([], [np.ndarray])
@ray.remote([], [np.ndarray])
def h()
return g()
```
Expand Down
10 changes: 5 additions & 5 deletions doc/reference-counting.md
Original file line number Diff line number Diff line change
@@ -1,6 +1,6 @@
# Reference Counting

In Halo, each object is assigned a globally unique object reference by the
In Ray, each object is assigned a globally unique object reference by the
scheduler (starting with 0 and incrementing upward). The objects are stored in
object stores. In order to avoid running out of memory, the object stores must
know when it is ok to deallocate an object. Since a worker on one node may have
Expand All @@ -11,7 +11,7 @@ information.
## Reference Counting

Two approaches to reclaiming memory are garbage collection and reference
counting. We choose to use a reference counting approach in Halo. There are a
counting. We choose to use a reference counting approach in Ray. There are a
couple of reasons for this. Reference counting allows us to reclaim memory as
early as possible. It also avoids pausing the system for garbage collection. We
also note that implementing reference counting at the cluster level plays nicely
Expand Down Expand Up @@ -77,13 +77,13 @@ because they must be passed into `AliasObjRefs` at some point).
The following problem has not yet been resolved. In the following code, the
result `x` will be garbage.
```python
x = halo.pull(ra.zeros([10, 10], "float"))
x = ray.pull(ra.zeros([10, 10], "float"))
```
When `ra.zeros` is called, a worker will create an array of zeros and store
it in an object store. An object reference to the output is returned. The call
to `halo.pull` will not copy data from the object store process to the worker
to `ray.pull` will not copy data from the object store process to the worker
process, but will instead give the worker process a pointer to shared memory.
After the `halo.pull` call completes, the object reference returned by
After the `ray.pull` call completes, the object reference returned by
`ra.zeros` will go out of scope, and the object it refers to will be
deallocated from the object store. This will cause the memory that `x` points to
to be garbage.
Expand Down
2 changes: 1 addition & 1 deletion doc/scheduler.md
Original file line number Diff line number Diff line change
@@ -1,6 +1,6 @@
# Scheduler

The scheduling strategies currently implemented in Halo are fairly basic and
The scheduling strategies currently implemented in Ray are fairly basic and
all use a central scheduler.

* The naive scheduler assigns tasks to workers just taking into account
Expand Down
24 changes: 12 additions & 12 deletions include/halo/halo.h → include/ray/ray.h
Original file line number Diff line number Diff line change
@@ -1,5 +1,5 @@
#ifndef HALO_INCLUDE_HALO_H
#define HALO_INCLUDE_HALO_H
#ifndef RAY_INCLUDE_RAY_H
#define RAY_INCLUDE_RAY_H

#include <vector>
#include <unordered_map>
Expand Down Expand Up @@ -34,20 +34,20 @@ class FnInfo {
typedef std::vector<std::vector<ObjStoreId> > ObjTable;
typedef std::unordered_map<std::string, FnInfo> FnTable;

#define HALO_VERBOSE -1
#define HALO_INFO 0
#define HALO_DEBUG 1
#define HALO_FATAL 2
#define HALO_REFCOUNT HALO_VERBOSE
#define HALO_ALIAS HALO_VERBOSE
#define RAY_VERBOSE -1
#define RAY_INFO 0
#define RAY_DEBUG 1
#define RAY_FATAL 2
#define RAY_REFCOUNT RAY_VERBOSE
#define RAY_ALIAS RAY_VERBOSE

#define HALO_LOG(LEVEL, MESSAGE) \
if (LEVEL == HALO_VERBOSE) { \
#define RAY_LOG(LEVEL, MESSAGE) \
if (LEVEL == RAY_VERBOSE) { \
\
} else if (LEVEL == HALO_FATAL) { \
} else if (LEVEL == RAY_FATAL) { \
std::cerr << "fatal error occured: " << MESSAGE << std::endl; \
std::exit(1); \
} else if (LEVEL == HALO_DEBUG) { \
} else if (LEVEL == RAY_DEBUG) { \
\
} else { \
std::cout << MESSAGE << std::endl; \
Expand Down
2 changes: 1 addition & 1 deletion lib/python/halo/__init__.py → lib/python/ray/__init__.py
Original file line number Diff line number Diff line change
@@ -1,3 +1,3 @@
import libhalolib as lib
import libraylib as lib
import serialization
from worker import scheduler_info, register_module, connect, disconnect, pull, push, remote
File renamed without changes.
Original file line number Diff line number Diff line change
@@ -1,7 +1,7 @@
from typing import List
import numpy as np
import halo.arrays.remote as ra
import halo
import ray.arrays.remote as ra
import ray

__all__ = ["BLOCK_SIZE", "DistArray", "assemble", "zeros", "ones", "copy",
"eye", "triu", "tril", "blockwise_dot", "dot", "transpose", "add", "subtract", "numpy_to_dist", "subblocks"]
Expand Down Expand Up @@ -55,56 +55,56 @@ def compute_num_blocks(shape):

def assemble(self):
"""Assemble an array on this node from a distributed array object reference."""
first_block = halo.pull(self.objrefs[(0,) * self.ndim])
first_block = ray.pull(self.objrefs[(0,) * self.ndim])
dtype = first_block.dtype
result = np.zeros(self.shape, dtype=dtype)
for index in np.ndindex(*self.num_blocks):
lower = DistArray.compute_block_lower(index, self.shape)
upper = DistArray.compute_block_upper(index, self.shape)
result[[slice(l, u) for (l, u) in zip(lower, upper)]] = halo.pull(self.objrefs[index])
result[[slice(l, u) for (l, u) in zip(lower, upper)]] = ray.pull(self.objrefs[index])
return result

def __getitem__(self, sliced):
# TODO(rkn): fix this, this is just a placeholder that should work but is inefficient
a = self.assemble()
return a[sliced]

@halo.remote([DistArray], [np.ndarray])
@ray.remote([DistArray], [np.ndarray])
def assemble(a):
return a.assemble()

# TODO(rkn): what should we call this method
@halo.remote([np.ndarray], [DistArray])
@ray.remote([np.ndarray], [DistArray])
def numpy_to_dist(a):
result = DistArray(a.shape)
for index in np.ndindex(*result.num_blocks):
lower = DistArray.compute_block_lower(index, a.shape)
upper = DistArray.compute_block_upper(index, a.shape)
result.objrefs[index] = halo.push(a[[slice(l, u) for (l, u) in zip(lower, upper)]])
result.objrefs[index] = ray.push(a[[slice(l, u) for (l, u) in zip(lower, upper)]])
return result

@halo.remote([List[int], str], [DistArray])
@ray.remote([List[int], str], [DistArray])
def zeros(shape, dtype_name="float"):
result = DistArray(shape)
for index in np.ndindex(*result.num_blocks):
result.objrefs[index] = ra.zeros(DistArray.compute_block_shape(index, shape), dtype_name=dtype_name)
return result

@halo.remote([List[int], str], [DistArray])
@ray.remote([List[int], str], [DistArray])
def ones(shape, dtype_name="float"):
result = DistArray(shape)
for index in np.ndindex(*result.num_blocks):
result.objrefs[index] = ra.ones(DistArray.compute_block_shape(index, shape), dtype_name=dtype_name)
return result

@halo.remote([DistArray], [DistArray])
@ray.remote([DistArray], [DistArray])
def copy(a):
result = DistArray(a.shape)
for index in np.ndindex(*result.num_blocks):
result.objrefs[index] = a.objrefs[index] # We don't need to actually copy the objects because cluster-level objects are assumed to be immutable.
return result

@halo.remote([int, int, str], [DistArray])
@ray.remote([int, int, str], [DistArray])
def eye(dim1, dim2=-1, dtype_name="float"):
dim2 = dim1 if dim2 == -1 else dim2
shape = [dim1, dim2]
Expand All @@ -117,7 +117,7 @@ def eye(dim1, dim2=-1, dtype_name="float"):
result.objrefs[i, j] = ra.zeros(block_shape, dtype_name=dtype_name)
return result

@halo.remote([DistArray], [DistArray])
@ray.remote([DistArray], [DistArray])
def triu(a):
if a.ndim != 2:
raise Exception("Input must have 2 dimensions, but a.ndim is " + str(a.ndim))
Expand All @@ -131,7 +131,7 @@ def triu(a):
result.objrefs[i, j] = ra.zeros_like(a.objrefs[i, j])
return result

@halo.remote([DistArray], [DistArray])
@ray.remote([DistArray], [DistArray])
def tril(a):
if a.ndim != 2:
raise Exception("Input must have 2 dimensions, but a.ndim is " + str(a.ndim))
Expand All @@ -145,7 +145,7 @@ def tril(a):
result.objrefs[i, j] = ra.zeros_like(a.objrefs[i, j])
return result

@halo.remote([np.ndarray], [np.ndarray])
@ray.remote([np.ndarray], [np.ndarray])
def blockwise_dot(*matrices):
n = len(matrices)
if n % 2 != 0:
Expand All @@ -156,7 +156,7 @@ def blockwise_dot(*matrices):
result += np.dot(matrices[i], matrices[n / 2 + i])
return result

@halo.remote([DistArray, DistArray], [DistArray])
@ray.remote([DistArray, DistArray], [DistArray])
def dot(a, b):
if a.ndim != 2:
raise Exception("dot expects its arguments to be 2-dimensional, but a.ndim = {}.".format(a.ndim))
Expand All @@ -171,7 +171,7 @@ def dot(a, b):
result.objrefs[i, j] = blockwise_dot(*args)
return result

@halo.remote([DistArray, List[int]], [DistArray])
@ray.remote([DistArray, List[int]], [DistArray])
def subblocks(a, *ranges):
"""
This function produces a distributed array from a subset of the blocks in the `a`. The result and `a` will have the same number of dimensions.For example,
Expand Down Expand Up @@ -202,7 +202,7 @@ def subblocks(a, *ranges):
result.objrefs[index] = a.objrefs[tuple([ranges[i][index[i]] for i in range(a.ndim)])]
return result

@halo.remote([DistArray], [DistArray])
@ray.remote([DistArray], [DistArray])
def transpose(a):
if a.ndim != 2:
raise Exception("transpose expects its argument to be 2-dimensional, but a.ndim = {}, a.shape = {}.".format(a.ndim, a.shape))
Expand All @@ -213,7 +213,7 @@ def transpose(a):
return result

# TODO(rkn): support broadcasting?
@halo.remote([DistArray, DistArray], [DistArray])
@ray.remote([DistArray, DistArray], [DistArray])
def add(x1, x2):
if x1.shape != x2.shape:
raise Exception("add expects arguments `x1` and `x2` to have the same shape, but x1.shape = {}, and x2.shape = {}.".format(x1.shape, x2.shape))
Expand All @@ -223,7 +223,7 @@ def add(x1, x2):
return result

# TODO(rkn): support broadcasting?
@halo.remote([DistArray, DistArray], [DistArray])
@ray.remote([DistArray, DistArray], [DistArray])
def subtract(x1, x2):
if x1.shape != x2.shape:
raise Exception("subtract expects arguments `x1` and `x2` to have the same shape, but x1.shape = {}, and x2.shape = {}.".format(x1.shape, x2.shape))
Expand Down
Loading

0 comments on commit 4cc024a

Please sign in to comment.