You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@giraph.apache.org by ma...@apache.org on 2017/09/15 18:11:08 UTC

[2/5] git commit: updated refs/heads/trunk to 5ccd00f

Adding code review feedback


Project: http://git-wip-us.apache.org/repos/asf/giraph/repo
Commit: http://git-wip-us.apache.org/repos/asf/giraph/commit/93634eaf
Tree: http://git-wip-us.apache.org/repos/asf/giraph/tree/93634eaf
Diff: http://git-wip-us.apache.org/repos/asf/giraph/diff/93634eaf

Branch: refs/heads/trunk
Commit: 93634eafabfe609d1dd1085fab00de90520d93f4
Parents: e3e288b
Author: Yuksel Akinci <yu...@fb.com>
Authored: Thu Sep 14 11:57:07 2017 -0700
Committer: Yuksel Akinci <yu...@fb.com>
Committed: Thu Sep 14 11:57:07 2017 -0700

----------------------------------------------------------------------
 .../giraph/job/CombinedWorkerProgress.java      | 48 ++++++----
 .../job/DefaultJobProgressTrackerService.java   | 19 ++--
 .../apache/giraph/job/SuperstepProgress.java    | 92 --------------------
 3 files changed, 43 insertions(+), 116 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/giraph/blob/93634eaf/giraph-core/src/main/java/org/apache/giraph/job/CombinedWorkerProgress.java
----------------------------------------------------------------------
diff --git a/giraph-core/src/main/java/org/apache/giraph/job/CombinedWorkerProgress.java b/giraph-core/src/main/java/org/apache/giraph/job/CombinedWorkerProgress.java
index 567156e..2921d60 100644
--- a/giraph-core/src/main/java/org/apache/giraph/job/CombinedWorkerProgress.java
+++ b/giraph-core/src/main/java/org/apache/giraph/job/CombinedWorkerProgress.java
@@ -143,6 +143,38 @@ public class CombinedWorkerProgress extends WorkerProgressStats {
   }
 
   /**
+   * Get Current superstep
+   * @return Current superstep
+   */
+  public long getCurrentSuperstep() {
+    return currentSuperstep;
+  }
+
+  /**
+   * Get workers in superstep
+   * @return Workers in superstep.
+   */
+  public long getWorkerseInSuperstep() {
+    return workersInSuperstep;
+  }
+
+  /**
+   * Get vertices computed
+   * @return Vertices computed
+   */
+  public long getVerticesComputed() {
+    return verticesComputed;
+  }
+
+  /**
+   * Get vertices to compute
+   * @return Vertices to compute
+   */
+  public long getVerticesToCompute() {
+    return verticesToCompute;
+  }
+
+  /**
    * Is the application done
    *
    * @param expectedWorkersDone Number of workers which should be done in
@@ -154,22 +186,6 @@ public class CombinedWorkerProgress extends WorkerProgressStats {
   }
 
   /**
-   * Get the current superstep progress.
-   * @param workerCount total worker count.
-   * @return current superstep progress
-   */
-  public SuperstepProgress getCurrentComputeSuperstepProgress(int workerCount) {
-    if (workersInSuperstep != workerCount) {
-      return null;
-    }
-    if (isComputeSuperstep()) {
-      return new SuperstepProgress(currentSuperstep, verticesComputed,
-          verticesToCompute);
-    }
-    return null;
-  }
-
-  /**
    * Get string describing total job progress
    *
    * @return String describing total job progress

http://git-wip-us.apache.org/repos/asf/giraph/blob/93634eaf/giraph-core/src/main/java/org/apache/giraph/job/DefaultJobProgressTrackerService.java
----------------------------------------------------------------------
diff --git a/giraph-core/src/main/java/org/apache/giraph/job/DefaultJobProgressTrackerService.java b/giraph-core/src/main/java/org/apache/giraph/job/DefaultJobProgressTrackerService.java
index 9920750..a824107 100644
--- a/giraph-core/src/main/java/org/apache/giraph/job/DefaultJobProgressTrackerService.java
+++ b/giraph-core/src/main/java/org/apache/giraph/job/DefaultJobProgressTrackerService.java
@@ -107,10 +107,12 @@ public class DefaultJobProgressTrackerService
               break;
             }
 
-            if (!canFinishInTime(conf, job,
-                combinedWorkerProgress.getCurrentComputeSuperstepProgress(
-                    mappersStarted - 1))) {
-              killSlowJob();
+            if (!canFinishInTime(conf, job, mappersStarted - 1,
+                combinedWorkerProgress)) {
+              killJobWithMessage("Killing the job because it won't " +
+                "complete in max allotted time: " +
+                GiraphConstants.MAX_ALLOWED_JOB_TIME_MS.get(conf) / 1000 +
+                "s");
             }
 
             if (lastProgress == null ||
@@ -135,14 +137,15 @@ public class DefaultJobProgressTrackerService
   }
 
   /**
-   * Determine if the job will finish in allotted time.
+   * Determine if the job will finish in allotted time
    * @param conf Giraph configuration
    * @param job Job
-   * @param progress Current superstep progress.
-   * @return true it the job can finish in allotted time, false otherwise.
+   * @param workerCount Worker count
+   * @param progress Combined worker progress
+   * @return true it the job can finish in allotted time, false otherwise
    */
   protected boolean canFinishInTime(GiraphConfiguration conf, Job job,
-      SuperstepProgress progress) {
+      int workerCount, CombinedWorkerProgress progress) {
     // No defaut implementation.
     return true;
   }

http://git-wip-us.apache.org/repos/asf/giraph/blob/93634eaf/giraph-core/src/main/java/org/apache/giraph/job/SuperstepProgress.java
----------------------------------------------------------------------
diff --git a/giraph-core/src/main/java/org/apache/giraph/job/SuperstepProgress.java b/giraph-core/src/main/java/org/apache/giraph/job/SuperstepProgress.java
deleted file mode 100644
index 05e9077..0000000
--- a/giraph-core/src/main/java/org/apache/giraph/job/SuperstepProgress.java
+++ /dev/null
@@ -1,92 +0,0 @@
-/*
- * 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.giraph.job;
-
-/**
- * Class which contains worker progress for a given superstep
- *
- */
-public class SuperstepProgress {
-  /**
-  * Current superstep
-  */
-  private long currentSuperstep;
-
-  /**
-  * Total computed vertex count
-  */
-  private long computedVertexCount;
-
-  /**
-  * Total vertex count
-  */
-  private long totalVertexCount;
-
-  /**
-  * Timestamp when this progress was created
-  */
-  private long timestamp;
-
-  /**
-   * Constructor
-   * @param currentSuperstep Current superstep
-   * @param computedVertexCount Computed vertex count
-   * @param totalVertexCount Total vertex count
-   */
-  public SuperstepProgress(long currentSuperstep, long computedVertexCount,
-      long totalVertexCount) {
-    this.currentSuperstep = currentSuperstep;
-    this.computedVertexCount = computedVertexCount;
-    this.totalVertexCount = totalVertexCount;
-    this.timestamp = System.currentTimeMillis();
-  }
-
-  /**
-   * Get current superstep
-   * @return current superstep
-   */
-  public long getCurrentSuperstep() {
-    return currentSuperstep;
-  }
-
-  /**
-   * Get computed vertex count
-   * @return Computed vertex count
-   */
-  public long getComputedVertexCount() {
-    return computedVertexCount;
-  }
-
-  /**
-   * Get total vertex count.
-   * @return total vertex count
-   */
-  public long getTotalVertexCount() {
-    return totalVertexCount;
-  }
-
-  /**
-   * Get timestamp
-   * @return timestamp of this progress
-   */
-  public long getTimestamp() {
-    return timestamp;
-  }
-}
-