Repository: incubator-reef
Updated Branches:
  refs/heads/master 0181e2cca -> 2c994ec97


http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/VortexWorkerConf.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/VortexWorkerConf.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/VortexWorkerConf.java
new file mode 100644
index 0000000..4706b96
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/VortexWorkerConf.java
@@ -0,0 +1,53 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.driver;
+
+import org.apache.reef.annotations.Unstable;
+import org.apache.reef.annotations.audience.DriverSide;
+import org.apache.reef.tang.annotations.Name;
+import org.apache.reef.tang.annotations.NamedParameter;
+import org.apache.reef.tang.formats.ConfigurationModule;
+import org.apache.reef.tang.formats.ConfigurationModuleBuilder;
+import org.apache.reef.tang.formats.RequiredParameter;
+
+/**
+ *  Vortex Worker configuration.
+ */
+@Unstable
+@DriverSide
+public final class VortexWorkerConf extends ConfigurationModuleBuilder {
+  /**
+   * Worker Threads.
+   */
+  @NamedParameter(doc = "Number of Worker Threads")
+  public final class NumOfThreads implements Name<Integer> {
+  }
+
+  /**
+   * Worker Threads.
+   */
+  public static final RequiredParameter<Integer> NUM_OF_THREADS = new 
RequiredParameter<>();
+
+  /**
+   * Vortex Worker configuration.
+   */
+  public static final ConfigurationModule CONF = new VortexWorkerConf()
+      .bindNamedParameter(NumOfThreads.class, NUM_OF_THREADS)
+      .build();
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/VortexWorkerManager.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/VortexWorkerManager.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/VortexWorkerManager.java
new file mode 100644
index 0000000..5aa4759
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/VortexWorkerManager.java
@@ -0,0 +1,95 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.driver;
+
+import net.jcip.annotations.ThreadSafe;
+import org.apache.reef.annotations.audience.DriverSide;
+import org.apache.reef.driver.task.RunningTask;
+import org.apache.reef.vortex.common.TaskletExecutionRequest;
+
+import java.io.Serializable;
+import java.util.Collection;
+import java.util.concurrent.ConcurrentHashMap;
+
+/**
+ * Representation of a VortexWorkerManager in Driver.
+ */
+@ThreadSafe
+@DriverSide
+class VortexWorkerManager {
+  private final VortexRequestor vortexRequestor;
+  private final RunningTask reefTask;
+  private final ConcurrentHashMap<Integer, Tasklet> runningTasklets = new 
ConcurrentHashMap<>();
+
+  VortexWorkerManager(final VortexRequestor vortexRequestor, final RunningTask 
reefTask) {
+    this.vortexRequestor = vortexRequestor;
+    this.reefTask = reefTask;
+  }
+
+  <TInput extends Serializable, TOutput extends Serializable>
+      void launchTasklet(final Tasklet<TInput, TOutput> tasklet) {
+    assert(!runningTasklets.containsKey(tasklet.getId()));
+    runningTasklets.put(tasklet.getId(), tasklet);
+    final TaskletExecutionRequest<TInput, TOutput> taskletExecutionRequest
+        = new TaskletExecutionRequest<>(tasklet.getId(), 
tasklet.getUserFunction(), tasklet.getInput());
+    vortexRequestor.send(reefTask, taskletExecutionRequest);
+  }
+
+  <TOutput extends Serializable>
+      void taskletCompleted(final Integer taskletId, final TOutput result) {
+    final Tasklet<?, TOutput> tasklet = runningTasklets.remove(taskletId);
+    if (tasklet != null) { // Tasklet should complete/error only once
+      tasklet.completed(result);
+    }
+  }
+
+  void taskletThrewException(final Integer taskletId, final Exception 
exception) {
+    final Tasklet tasklet = runningTasklets.remove(taskletId);
+    if (tasklet != null) { // Tasklet should complete/error only once
+      tasklet.threwException(exception);
+    }
+  }
+
+  Collection<Tasklet> removed() {
+    return runningTasklets.values();
+  }
+
+  void terminate() {
+    reefTask.close();
+  }
+
+  String getId() {
+    return reefTask.getId();
+  }
+
+  /**
+   * @return the description of this worker in string.
+   */
+  @Override
+  public String toString() {
+    return "VortexWorkerManager: " + getId();
+  }
+
+  /**
+   * For unit tests only.
+   */
+  boolean containsTasklet(final Integer taskletId) {
+    return runningTasklets.containsKey(taskletId);
+  }
+}
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/package-info.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/package-info.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/package-info.java
new file mode 100644
index 0000000..6deaf0e
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/driver/package-info.java
@@ -0,0 +1,22 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+/**
+ * Vortex Code that runs as part of REEF Driver.
+ */
+package org.apache.reef.vortex.driver;

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/evaluator/VortexWorker.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/evaluator/VortexWorker.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/evaluator/VortexWorker.java
new file mode 100644
index 0000000..86906c1
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/evaluator/VortexWorker.java
@@ -0,0 +1,160 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.evaluator;
+
+import org.apache.commons.lang.SerializationUtils;
+import org.apache.reef.annotations.Unstable;
+import org.apache.reef.annotations.audience.TaskSide;
+import org.apache.reef.tang.annotations.Parameter;
+import org.apache.reef.tang.annotations.Unit;
+import org.apache.reef.task.HeartBeatTriggerManager;
+import org.apache.reef.task.Task;
+import org.apache.reef.task.TaskMessage;
+import org.apache.reef.task.TaskMessageSource;
+import org.apache.reef.task.events.CloseEvent;
+import org.apache.reef.task.events.DriverMessage;
+import org.apache.reef.util.Optional;
+import org.apache.reef.vortex.common.*;
+import org.apache.reef.vortex.driver.VortexWorkerConf;
+import org.apache.reef.wake.EventHandler;
+
+import javax.inject.Inject;
+import java.io.Serializable;
+import java.util.concurrent.*;
+
+/**
+ * Receives commands from VortexMaster, executes them, and returns the results.
+ * TODO[REEF-503]: Basic Vortex profiling.
+ */
+@Unstable
+@Unit
+@TaskSide
+public final class VortexWorker implements Task, TaskMessageSource {
+  private static final String MESSAGE_SOURCE_ID = ""; // empty string as there 
is no use for it
+
+  private final BlockingDeque<byte[]> pendingRequests = new 
LinkedBlockingDeque<>();
+  private final BlockingDeque<byte[]> workerReports = new 
LinkedBlockingDeque<>();
+
+  private final HeartBeatTriggerManager heartBeatTriggerManager;
+  private final int numOfThreads;
+  private final CountDownLatch terminated = new CountDownLatch(1);
+
+  @Inject
+  private VortexWorker(final HeartBeatTriggerManager heartBeatTriggerManager,
+                      @Parameter(VortexWorkerConf.NumOfThreads.class) final 
int numOfThreads) {
+    this.heartBeatTriggerManager = heartBeatTriggerManager;
+    this.numOfThreads = numOfThreads;
+  }
+
+  /**
+   * Starts the scheduler & executor and waits until termination.
+   */
+  @Override
+  public byte[] call(final byte[] memento) throws Exception {
+    final ExecutorService schedulerThread = 
Executors.newSingleThreadExecutor();
+    final ExecutorService commandExecutor = 
Executors.newFixedThreadPool(numOfThreads);
+
+    // Scheduling thread starts
+    schedulerThread.execute(new Runnable() {
+      @Override
+      public void run() {
+        while (true) {
+          // Scheduler Thread: Pick a command to execute (For now, simple FIFO 
order)
+          final byte[] message;
+          try {
+            message = pendingRequests.takeFirst();
+          } catch (InterruptedException e) {
+            throw new RuntimeException(e);
+          }
+
+          // Scheduler Thread: Pass the command to the worker thread pool to 
be executed
+          commandExecutor.execute(new Runnable() {
+            @Override
+            public void run() {
+              // Command Executor: Deserialize the command
+              final VortexRequest vortexRequest = (VortexRequest) 
SerializationUtils.deserialize(message);
+              switch (vortexRequest.getType()) {
+                case ExecuteTasklet:
+                  final TaskletExecutionRequest taskletExecutionRequest = 
(TaskletExecutionRequest) vortexRequest;
+                  try {
+                    // Command Executor: Execute the command
+                    final Serializable result = 
taskletExecutionRequest.execute();
+
+                    // Command Executor: Tasklet successfully returns result
+                    final WorkerReport report =
+                        new 
TaskletResultReport<>(taskletExecutionRequest.getTaskletId(), result);
+                    
workerReports.addLast(SerializationUtils.serialize(report));
+                  } catch (Exception e) {
+                    // Command Executor: Tasklet throws an exception
+                    final WorkerReport report =
+                        new 
TaskletFailureReport(taskletExecutionRequest.getTaskletId(), e);
+                    
workerReports.addLast(SerializationUtils.serialize(report));
+                  }
+
+                  heartBeatTriggerManager.triggerHeartBeat();
+                  break;
+                default:
+                  throw new RuntimeException("Unknown Command");
+              }
+            }
+          });
+
+        }
+      }
+    });
+
+    terminated.await();
+    return null;
+  }
+
+  /**
+   * @return the workerReport the worker wishes to send.
+   */
+  @Override
+  public Optional<TaskMessage> getMessage() {
+    final byte[] msg = workerReports.pollFirst();
+    if (msg != null) {
+      return Optional.of(TaskMessage.from(MESSAGE_SOURCE_ID, msg));
+    } else {
+      return Optional.empty();
+    }
+  }
+
+  /**
+   * Handle requests from Vortex Master.
+   */
+  public final class DriverMessageHandler implements 
EventHandler<DriverMessage> {
+    @Override
+    public void onNext(final DriverMessage message) {
+      if (message.get().isPresent()) {
+        pendingRequests.addLast(message.get().get());
+      }
+    }
+  }
+
+  /**
+   * Shut down this worker.
+   */
+  public final class TaskCloseHandler implements EventHandler<CloseEvent> {
+    @Override
+    public void onNext(final CloseEvent closeEvent) {
+      terminated.countDown();
+    }
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/evaluator/package-info.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/evaluator/package-info.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/evaluator/package-info.java
new file mode 100644
index 0000000..49f31d5
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/evaluator/package-info.java
@@ -0,0 +1,22 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+/**
+ * Vortex Code that runs as part of REEF Evaluator.
+ */
+package org.apache.reef.vortex.evaluator;

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOne.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOne.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOne.java
new file mode 100644
index 0000000..4e27ba9
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOne.java
@@ -0,0 +1,36 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.examples.addone;
+
+import org.apache.reef.vortex.driver.VortexLauncher;
+
+/**
+ * User's main function.
+ */
+final class AddOne {
+  private AddOne() {
+  }
+
+  /**
+   * Launch the vortex job, passing appropriate arguments.
+   */
+  public static void main(final String[] args) {
+    VortexLauncher.launchLocal("Vortex_Example_AddOne", AddOneStart.class, 2, 
1024, 4);
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOneFunction.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOneFunction.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOneFunction.java
new file mode 100644
index 0000000..299a31a
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOneFunction.java
@@ -0,0 +1,34 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.examples.addone;
+
+import org.apache.reef.vortex.api.VortexFunction;
+
+/**
+ * Outputs input + 1.
+ */
+final class AddOneFunction implements VortexFunction<Integer, Integer> {
+  /**
+   * Outputs input + 1.
+   */
+  @Override
+  public Integer call(final Integer input) throws Exception {
+    return input + 1;
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOneStart.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOneStart.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOneStart.java
new file mode 100644
index 0000000..510cb14
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/AddOneStart.java
@@ -0,0 +1,67 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.examples.addone;
+
+import org.apache.reef.vortex.api.VortexFuture;
+import org.apache.reef.vortex.api.VortexThreadPool;
+import org.apache.reef.vortex.api.VortexStart;
+
+import javax.inject.Inject;
+import java.util.ArrayList;
+import java.util.List;
+import java.util.Vector;
+import java.util.concurrent.ExecutionException;
+
+/**
+ * AddOne User Code Example.
+ */
+final class AddOneStart implements VortexStart {
+  @Inject
+  private AddOneStart() {
+  }
+
+  /**
+   * Perform a simple vector calculation on Vortex.
+   */
+  @Override
+  public void start(final VortexThreadPool vortexThreadPool) {
+    final Vector<Integer> inputVector = new Vector<>();
+    for (int i = 0; i < 1000; i++) {
+      inputVector.add(i);
+    }
+
+    final List<VortexFuture<Integer>> futures = new ArrayList<>();
+    final AddOneFunction addOneFunction = new AddOneFunction();
+    for (final int i : inputVector) {
+      futures.add(vortexThreadPool.submit(addOneFunction, i));
+    }
+
+    final Vector<Integer> outputVector = new Vector<>();
+    for (final VortexFuture<Integer> future : futures) {
+      try {
+        outputVector.add(future.get());
+      } catch (InterruptedException | ExecutionException e) {
+        throw new RuntimeException(e);
+      }
+    }
+
+    System.out.println("RESULT:");
+    System.out.println(outputVector);
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/package-info.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/package-info.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/package-info.java
new file mode 100644
index 0000000..4bbc452
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/addone/package-info.java
@@ -0,0 +1,22 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+/**
+ * A Simple Vortex vector calculation example.
+ */
+package org.apache.reef.vortex.examples.addone;

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortex.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortex.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortex.java
new file mode 100644
index 0000000..565bce2
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortex.java
@@ -0,0 +1,36 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.examples.hello;
+
+import org.apache.reef.vortex.driver.VortexLauncher;
+
+/**
+ * User's main function.
+ */
+final class HelloVortex {
+  private HelloVortex() {
+  }
+
+  /**
+   * Launch the vortex job, passing appropriate arguments.
+   */
+  public static void main(final String[] args) {
+    VortexLauncher.launchLocal("Vortex_Example_HelloVortex", 
HelloVortexStart.class, 1, 1024, 1);
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortexFunction.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortexFunction.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortexFunction.java
new file mode 100644
index 0000000..52f7d21
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortexFunction.java
@@ -0,0 +1,37 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.examples.hello;
+
+import org.apache.reef.vortex.api.VortexFunction;
+
+import java.io.Serializable;
+
+/**
+ * Prints to stdout.
+ */
+final class HelloVortexFunction implements VortexFunction {
+  /**
+   * Prints to stdout.
+   */
+  @Override
+  public Serializable call(final Serializable serializable) throws Exception {
+    System.out.println("Hello, Vortex!");
+    return null;
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortexStart.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortexStart.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortexStart.java
new file mode 100644
index 0000000..6317c08
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/HelloVortexStart.java
@@ -0,0 +1,48 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.examples.hello;
+
+import org.apache.reef.vortex.api.VortexFuture;
+import org.apache.reef.vortex.api.VortexStart;
+import org.apache.reef.vortex.api.VortexThreadPool;
+
+import javax.inject.Inject;
+import java.util.concurrent.ExecutionException;
+
+/**
+ * HelloVortex User Code Example.
+ */
+final class HelloVortexStart implements VortexStart {
+  @Inject
+  private HelloVortexStart() {
+  }
+
+  /**
+   * Run the function.
+   */
+  @Override
+  public void start(final VortexThreadPool vortexThreadPool) {
+    final VortexFuture future = vortexThreadPool.submit(new 
HelloVortexFunction(), null);
+    try {
+      future.get();
+    } catch (InterruptedException | ExecutionException e) {
+      throw new RuntimeException(e);
+    }
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/package-info.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/package-info.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/package-info.java
new file mode 100644
index 0000000..d66e1a1
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/examples/hello/package-info.java
@@ -0,0 +1,22 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+/**
+ * Vortex Hello World Example.
+ */
+package org.apache.reef.vortex.examples.hello;

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/package-info.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/package-info.java
 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/package-info.java
index 5f264b4..87c9792 100644
--- 
a/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/package-info.java
+++ 
b/lang/java/reef-applications/reef-vortex/src/main/java/org/apache/reef/vortex/package-info.java
@@ -17,6 +17,6 @@
  * under the License.
  */
 /**
- * TODO: Document.
+ * Vortex, a distributed runtime that makes efficient use of unreliable 
resources.
  */
 package org.apache.reef.vortex;

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/DefaultVortexMasterTest.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/DefaultVortexMasterTest.java
 
b/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/DefaultVortexMasterTest.java
new file mode 100644
index 0000000..f2c2f92
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/DefaultVortexMasterTest.java
@@ -0,0 +1,158 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.driver;
+
+import org.apache.reef.vortex.api.VortexFunction;
+import org.apache.reef.vortex.api.VortexFuture;
+import org.junit.Test;
+
+import java.util.ArrayList;
+import java.util.HashSet;
+import java.util.List;
+
+import static org.junit.Assert.*;
+
+/**
+ * Test whether DefaultVortexMaster correctly handles (simulated) events.
+ */
+public class DefaultVortexMasterTest {
+  private TestUtil testUtil = new TestUtil();
+
+  /**
+   * Test handling of single tasklet execution without failure.
+   */
+  @Test(timeout = 10000)
+  public void testSingleTaskletNoFailure() throws Exception {
+    final VortexFunction vortexFunction = testUtil.newFunction();
+    final VortexWorkerManager vortexWorkerManager1 = testUtil.newWorker();
+    final RunningWorkers runningWorkers = new RunningWorkers();
+    final PendingTasklets pendingTasklets = new PendingTasklets();
+    final DefaultVortexMaster vortexMaster = new 
DefaultVortexMaster(runningWorkers, pendingTasklets);
+
+    vortexMaster.workerAllocated(vortexWorkerManager1);
+    final VortexFuture future = vortexMaster.enqueueTasklet(vortexFunction, 
null);
+    final ArrayList<Integer> taskletIds = scheduleTasklets(runningWorkers, 
pendingTasklets, 1);
+    for (final int taskletId : taskletIds) {
+      vortexMaster.taskletCompleted(vortexWorkerManager1.getId(), taskletId, 
null);
+    }
+
+    assertTrue("The VortexFuture should be done", future.isDone());
+  }
+
+  /**
+   * Test handling of single tasklet execution with a failure.
+   */
+  @Test(timeout = 10000)
+  public void testSingleTaskletFailure() throws Exception {
+    final VortexFunction vortexFunction = testUtil.newFunction();
+    final VortexWorkerManager vortexWorkerManager1 = testUtil.newWorker();
+    final VortexWorkerManager vortexWorkerManager2 = testUtil.newWorker();
+    final RunningWorkers runningWorkers = new RunningWorkers();
+    final PendingTasklets pendingTasklets = new PendingTasklets();
+    final DefaultVortexMaster vortexMaster = new 
DefaultVortexMaster(runningWorkers, pendingTasklets);
+
+    // Allocate worker & tasklet and schedule
+    vortexMaster.workerAllocated(vortexWorkerManager1);
+    final VortexFuture future = vortexMaster.enqueueTasklet(vortexFunction, 
null);
+    final ArrayList<Integer> taskletIds1 = scheduleTasklets(runningWorkers, 
pendingTasklets, 1);
+
+    // Preemption!
+    vortexMaster.workerPreempted(vortexWorkerManager1.getId());
+    assertFalse("The VortexFuture should not be done", future.isDone());
+
+    // New resource allocation and scheduling
+    vortexMaster.workerAllocated(vortexWorkerManager2);
+    final ArrayList<Integer> taskletIds2 = scheduleTasklets(runningWorkers, 
pendingTasklets, 1);
+    assertEquals("Both lists need to contain the same single tasklet id", 
taskletIds1, taskletIds2);
+
+    // Completed?
+    for (final int taskletId : taskletIds2) {
+      vortexMaster.taskletCompleted(vortexWorkerManager2.getId(), taskletId, 
null);
+    }
+    assertTrue("The VortexFuture should be done", future.isDone());
+  }
+
+  /**
+   * Test handling of multiple tasklet execution with failures.
+   */
+  @Test(timeout = 10000)
+  public void testMultipleTaskletsFailure() throws Exception {
+    // The tasklets that need to be executed
+    final ArrayList<VortexFuture> vortexFutures = new ArrayList<>();
+    final RunningWorkers runningWorkers = new RunningWorkers();
+    final PendingTasklets pendingTasklets = new PendingTasklets();
+    final DefaultVortexMaster vortexMaster = new 
DefaultVortexMaster(runningWorkers, pendingTasklets);
+
+    // Allocate iniital evaluators (will all be preempted later...)
+    final List<VortexWorkerManager> initialWorkers = new ArrayList<>();
+    final int numOfWorkers = 10;
+    for (int i = 0; i < numOfWorkers; i++) {
+      final VortexWorkerManager vortexWorkerManager = testUtil.newWorker();
+      initialWorkers.add(vortexWorkerManager);
+      vortexMaster.workerAllocated(vortexWorkerManager);
+    }
+
+    // Schedule tasklets
+    final int numOfTasklets = 100;
+    for (int i = 0; i < numOfTasklets; i++) {
+      vortexFutures.add(vortexMaster.enqueueTasklet(testUtil.newFunction(), 
null));
+    }
+    final ArrayList<Integer> taskletIds1 = scheduleTasklets(runningWorkers, 
pendingTasklets, numOfTasklets);
+
+    // Preempt all evaluators
+    for (int i = 0; i < numOfWorkers; i++) {
+      vortexMaster.workerPreempted(initialWorkers.get(i).getId());
+    }
+
+    // Allocate new evaluators and reschedule
+    for (int i = 0; i < numOfWorkers; i++) {
+      vortexMaster.workerAllocated(testUtil.newWorker());
+    }
+    final ArrayList<Integer> taskletIds2 = scheduleTasklets(runningWorkers, 
pendingTasklets, numOfTasklets);
+    assertEquals("Must contain same tasklet ids", new HashSet<>(taskletIds1), 
new HashSet<>(taskletIds2));
+
+    // Completed?
+    for (final int taskletId : taskletIds2) {
+      final String workerId = 
runningWorkers.getWhereTaskletWasScheduledTo(taskletId);
+      assertNotNull("The tasklet must have been scheduled", workerId);
+      vortexMaster.taskletCompleted(workerId, taskletId, null);
+    }
+    for (final VortexFuture vortexFuture : vortexFutures) {
+      assertTrue("The VortexFuture should be done", vortexFuture.isDone());
+    }
+  }
+
+  /**
+   * Schedule specified number of tasklets.
+   * @return ids of scheduled tasklets
+   */
+  private ArrayList<Integer> scheduleTasklets(final RunningWorkers 
runningWorkers,
+                                              final PendingTasklets 
pendingTasklets,
+                                              final int numOfTasklets) throws 
InterruptedException {
+    final ArrayList<Integer> taskletIds = new ArrayList<>();
+    for (int i = 0; i < numOfTasklets; i++) {
+      final Tasklet tasklet = pendingTasklets.takeFirst(); // blocks when no 
tasklet exists
+      assertNotNull("Tasklet should exist in the pending queue", tasklet);
+
+      taskletIds.add(tasklet.getId());
+      runningWorkers.launchTasklet(tasklet); // blocks when no worker exists
+    }
+    return taskletIds;
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/RunningWorkersTest.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/RunningWorkersTest.java
 
b/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/RunningWorkersTest.java
new file mode 100644
index 0000000..03c671e
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/RunningWorkersTest.java
@@ -0,0 +1,64 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.driver;
+
+import org.junit.Test;
+
+import java.util.Collection;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertTrue;
+
+/**
+ * Test Possible Race Conditions.
+ */
+public class RunningWorkersTest {
+  private final RunningWorkers runningWorkers = new RunningWorkers();
+  private final TestUtil testUtil = new TestUtil();
+
+  /**
+   * Test executor preemption -> executor allocation.
+   * Possible scenario: RM preempts Evaluator before the Evaluator sends its 
first heartbeat to Driver.
+   */
+  @Test
+  public void removeExecutorAndAddExecutor() throws Exception {
+    final VortexWorkerManager vortexWorkerManager = testUtil.newWorker();
+    assertEquals("Must be no running tasklets", 0, 
runningWorkers.removeWorker(vortexWorkerManager.getId()).size());
+    runningWorkers.addWorker(vortexWorkerManager);
+    assertFalse("Executor should not be running", 
runningWorkers.isWorkerRunning(vortexWorkerManager.getId()));
+  }
+
+  /**
+   * Test executor allocation -> tasklet launch -> executor preemption -> 
tasklet complete
+   * Possible scenario: Tasklet completion message from Evaluator comes in 
slowly, after preemption message from RM.
+   */
+  @Test(timeout = 10000)
+  public void removeExecutorAndCompleteTasklet() throws Exception {
+    final VortexWorkerManager vortexWorkerManager = testUtil.newWorker();
+    final Tasklet tasklet = testUtil.newTasklet();
+    runningWorkers.addWorker(vortexWorkerManager);
+    runningWorkers.launchTasklet(tasklet); // blocks when no worker exists
+    final Collection<Tasklet> tasklets = 
runningWorkers.removeWorker(vortexWorkerManager.getId());
+    assertEquals("Only 1 Tasklet must have been running", 1, tasklets.size());
+    assertTrue("This Tasklet must have been running", 
tasklets.contains(tasklet));
+    runningWorkers.completeTasklet(vortexWorkerManager.getId(), 
tasklet.getId(), null);
+    assertFalse("Tasklet must not have been completed", tasklet.isCompleted());
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/TestUtil.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/TestUtil.java
 
b/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/TestUtil.java
new file mode 100644
index 0000000..5a7da97
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/TestUtil.java
@@ -0,0 +1,66 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.vortex.driver;
+
+import org.apache.reef.driver.task.RunningTask;
+import org.apache.reef.vortex.api.VortexFunction;
+import org.apache.reef.vortex.api.VortexFuture;
+
+import java.io.Serializable;
+import java.util.concurrent.atomic.AtomicInteger;
+
+import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.when;
+
+/**
+ * Utility methods for tests.
+ */
+public class TestUtil {
+  private final AtomicInteger taskletId = new AtomicInteger(0);
+  private final AtomicInteger workerId = new AtomicInteger(0);
+
+  /**
+   * @return a new mocked worker.
+   */
+  public VortexWorkerManager newWorker() {
+    final RunningTask reefTask = mock(RunningTask.class);
+    when(reefTask.getId()).thenReturn("worker" + 
String.valueOf(workerId.getAndIncrement()));
+    final VortexRequestor vortexRequestor = mock(VortexRequestor.class);
+    return new VortexWorkerManager(vortexRequestor, reefTask);
+  }
+
+  /**
+   * @return a new dummy tasklet.
+   */
+  public Tasklet newTasklet() {
+    return new Tasklet(taskletId.getAndIncrement(), null, null, new 
VortexFuture());
+  }
+
+  /**
+   * @return a new dummy function.
+   */
+  public VortexFunction newFunction() {
+    return new VortexFunction() {
+      @Override
+      public Serializable call(final Serializable serializable) throws 
Exception {
+        return null;
+      }
+    };
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/package-info.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/package-info.java
 
b/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/package-info.java
new file mode 100644
index 0000000..95c8c0e
--- /dev/null
+++ 
b/lang/java/reef-applications/reef-vortex/src/test/java/org/apache/reef/vortex/driver/package-info.java
@@ -0,0 +1,22 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+/**
+ * Vortex unit tests.
+ */
+package org.apache.reef.vortex.driver;

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-tests/pom.xml
----------------------------------------------------------------------
diff --git a/lang/java/reef-tests/pom.xml b/lang/java/reef-tests/pom.xml
index 039c082..ef0ecb2 100644
--- a/lang/java/reef-tests/pom.xml
+++ b/lang/java/reef-tests/pom.xml
@@ -65,6 +65,11 @@ under the License.
             <version>${project.version}</version>
         </dependency>
         <dependency>
+            <groupId>${project.groupId}</groupId>
+            <artifactId>vortex</artifactId>
+            <version>${project.version}</version>
+        </dependency>
+        <dependency>
             <groupId>junit</groupId>
             <artifactId>junit</artifactId>
         </dependency>

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-tests/src/test/java/org/apache/reef/tests/AllTestsSuite.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-tests/src/test/java/org/apache/reef/tests/AllTestsSuite.java 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/AllTestsSuite.java
index 8f58134..82d1829 100644
--- 
a/lang/java/reef-tests/src/test/java/org/apache/reef/tests/AllTestsSuite.java
+++ 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/AllTestsSuite.java
@@ -18,6 +18,7 @@
  */
 package org.apache.reef.tests;
 
+import org.apache.reef.tests.applications.ApplicationTestSuite;
 import org.apache.reef.tests.close_eval.CloseEvaluatorTest;
 import org.apache.reef.tests.configurationproviders.ConfigurationProviderTest;
 import org.apache.reef.tests.driver.DriverTest;
@@ -50,7 +51,8 @@ import org.junit.runners.Suite;
     CloseEvaluatorTest.class,
     EvaluatorFailureTest.class,
     ExamplesTestSuite.class,
-    ConfigurationProviderTest.class
+    ConfigurationProviderTest.class,
+    ApplicationTestSuite.class
     })
 public final class AllTestsSuite {
 }

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/ApplicationTestSuite.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/ApplicationTestSuite.java
 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/ApplicationTestSuite.java
new file mode 100644
index 0000000..ac302a1
--- /dev/null
+++ 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/ApplicationTestSuite.java
@@ -0,0 +1,30 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.tests.applications;
+
+import org.apache.reef.tests.applications.vortex.VortexTestSuite;
+import org.junit.runner.RunWith;
+import org.junit.runners.Suite;
+
+@RunWith(Suite.class)
[email protected]({
+    VortexTestSuite.class
+    })
+public final class ApplicationTestSuite {
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/package-info.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/package-info.java
 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/package-info.java
new file mode 100644
index 0000000..5877c7a
--- /dev/null
+++ 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/package-info.java
@@ -0,0 +1,22 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+/**
+ * Tests for REEF Applications.
+ */
+package org.apache.reef.tests.applications;

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/VortexTestSuite.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/VortexTestSuite.java
 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/VortexTestSuite.java
new file mode 100644
index 0000000..d47f427
--- /dev/null
+++ 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/VortexTestSuite.java
@@ -0,0 +1,30 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.tests.applications.vortex;
+
+import org.apache.reef.tests.applications.vortex.addone.AddOneTest;
+import org.junit.runner.RunWith;
+import org.junit.runners.Suite;
+
+@RunWith(Suite.class)
[email protected]({
+    AddOneTest.class
+    })
+public final class VortexTestSuite {
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneFunction.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneFunction.java
 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneFunction.java
new file mode 100644
index 0000000..e064976
--- /dev/null
+++ 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneFunction.java
@@ -0,0 +1,34 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.tests.applications.vortex.addone;
+
+import org.apache.reef.vortex.api.VortexFunction;
+
+/**
+ * Outputs Input+1.
+ */
+public final class AddOneFunction implements VortexFunction<Integer, Integer> {
+  /**
+   * Outputs Input+1.
+   */
+  @Override
+  public Integer call(final Integer input) throws Exception {
+    return input + 1;
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneTest.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneTest.java
 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneTest.java
new file mode 100644
index 0000000..f935298
--- /dev/null
+++ 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneTest.java
@@ -0,0 +1,63 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.tests.applications.vortex.addone;
+
+import org.apache.reef.client.LauncherStatus;
+import org.apache.reef.tang.Configuration;
+import org.apache.reef.tests.TestEnvironment;
+import org.apache.reef.tests.TestEnvironmentFactory;
+import org.apache.reef.vortex.driver.VortexConfHelper;
+import org.junit.After;
+import org.junit.Assert;
+import org.junit.Before;
+import org.junit.Test;
+
+/**
+ * Launch the AddOne Vortex test.
+ */
+public final class AddOneTest {
+  private final TestEnvironment testEnvironment = 
TestEnvironmentFactory.getNewTestEnvironment();
+
+  /**
+   * Set up the test environment.
+   */
+  @Before
+  public void setUp() throws Exception {
+    this.testEnvironment.setUp();
+  }
+
+  /**
+   * Tear down the test environment.
+   */
+  @After
+  public void tearDown() throws Exception {
+    this.testEnvironment.tearDown();
+  }
+
+  /**
+   * Run the AddOne test.
+   */
+  @Test
+  public void testVortexAddOne() {
+    final Configuration conf =
+        VortexConfHelper.getVortexConf("TEST_Vortex_AddOneTest", 
AddOneTestStart.class, 2, 1024, 4);
+    final LauncherStatus status = this.testEnvironment.run(conf);
+    Assert.assertTrue("Job state after execution: " + status, 
status.isSuccess());
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneTestStart.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneTestStart.java
 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneTestStart.java
new file mode 100644
index 0000000..cbe84cf
--- /dev/null
+++ 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/AddOneTestStart.java
@@ -0,0 +1,67 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.reef.tests.applications.vortex.addone;
+
+import org.apache.reef.vortex.api.VortexFuture;
+import org.apache.reef.vortex.api.VortexStart;
+import org.apache.reef.vortex.api.VortexThreadPool;
+
+import javax.inject.Inject;
+import java.util.ArrayList;
+import java.util.List;
+import java.util.Vector;
+import java.util.concurrent.ExecutionException;
+
+/**
+ * Test correctness of a simple vector calculation on Vortex.
+ */
+public final class AddOneTestStart implements VortexStart {
+  @Inject
+  private AddOneTestStart() {
+  }
+
+  /**
+   * Test correctness of a simple vector calculation on Vortex.
+   */
+  @Override
+  public void start(final VortexThreadPool vortexThreadPool) {
+    final Vector<Integer> inputVector = new Vector<>();
+    for (int i = 0; i < 1000; i++) {
+      inputVector.add(i);
+    }
+
+    final List<VortexFuture<Integer>> futures = new ArrayList<>();
+    final AddOneFunction addOneFunction = new AddOneFunction();
+    for (final int i : inputVector) {
+      futures.add(vortexThreadPool.submit(addOneFunction, i));
+    }
+
+    int j = 0;
+    for (final VortexFuture<Integer> future : futures) {
+      try {
+        if (j+1 != future.get()) {
+          throw new RuntimeException(); // throw an exception if the result is 
not what we expected
+        }
+        j++;
+      } catch (InterruptedException | ExecutionException e) {
+        throw new RuntimeException(e);
+      }
+    }
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/package-info.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/package-info.java
 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/package-info.java
new file mode 100644
index 0000000..caafecc
--- /dev/null
+++ 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/addone/package-info.java
@@ -0,0 +1,22 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+/**
+ * Vortex AddOne test.
+ */
+package org.apache.reef.tests.applications.vortex.addone;

http://git-wip-us.apache.org/repos/asf/incubator-reef/blob/2c994ec9/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/package-info.java
----------------------------------------------------------------------
diff --git 
a/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/package-info.java
 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/package-info.java
new file mode 100644
index 0000000..62a9333
--- /dev/null
+++ 
b/lang/java/reef-tests/src/test/java/org/apache/reef/tests/applications/vortex/package-info.java
@@ -0,0 +1,22 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+/**
+ * Tests for Vortex.
+ */
+package org.apache.reef.tests.applications.vortex;

Reply via email to