You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by mb...@apache.org on 2015/03/25 13:35:37 UTC

flink git commit: [FLINK-1756] [streaming] Rename Stream Monitoring to Checkpointing in JobGraph.

Repository: flink
Updated Branches:
  refs/heads/master 597d8b862 -> ae04025f1


[FLINK-1756] [streaming] Rename Stream Monitoring to Checkpointing in JobGraph.

This is improves #506

This closes #531


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

Branch: refs/heads/master
Commit: ae04025f16a46dda5dfb0b25e1ea9ab327dc70bc
Parents: 597d8b8
Author: Sascha Wolke <de...@users.noreply.github.com>
Authored: Tue Mar 24 23:30:04 2015 +0100
Committer: mbalassi <mb...@apache.org>
Committed: Wed Mar 25 10:30:43 2015 +0100

----------------------------------------------------------------------
 .../apache/flink/runtime/jobgraph/JobGraph.java | 20 ++++++++++----------
 .../flink/runtime/jobmanager/JobManager.scala   |  4 ++--
 .../api/StreamingJobGraphGenerator.java         |  6 +++---
 3 files changed, 15 insertions(+), 15 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/ae04025f/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/JobGraph.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/JobGraph.java b/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/JobGraph.java
index 4b398e5..f8f22b0 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/JobGraph.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/JobGraph.java
@@ -79,9 +79,9 @@ public class JobGraph implements Serializable {
 	
 	private JobType jobType = JobType.BATCH;
 	
-	private boolean monitoringEnabled = false;
+	private boolean checkpointingEnabled = false;
 	
-	private long monitorInterval = 10000;
+	private long checkpointingInterval = 10000;
 	
 	// --------------------------------------------------------------------------------------------
 	
@@ -270,20 +270,20 @@ public class JobGraph implements Serializable {
 		return jobType;
 	}
 
-	public void setMonitoringEnabled(boolean monitoringEnabled) {
-		this.monitoringEnabled = monitoringEnabled;
+	public void setCheckpointingEnabled(boolean checkpointingEnabled) {
+		this.checkpointingEnabled = checkpointingEnabled;
 	}
 
-	public boolean isMonitoringEnabled() {
-		return monitoringEnabled;
+	public boolean isCheckpointingEnabled() {
+		return checkpointingEnabled;
 	}
 
-	public void setMonitorInterval(long monitorInterval) {
-		this.monitorInterval = monitorInterval;
+	public void setCheckpointingInterval(long checkpointingInterval) {
+		this.checkpointingInterval = checkpointingInterval;
 	}
 
-	public long getMonitorInterval() {
-		return monitorInterval;
+	public long getCheckpointingInterval() {
+		return checkpointingInterval;
 	}
 
 	/**

http://git-wip-us.apache.org/repos/asf/flink/blob/ae04025f/flink-runtime/src/main/scala/org/apache/flink/runtime/jobmanager/JobManager.scala
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/scala/org/apache/flink/runtime/jobmanager/JobManager.scala b/flink-runtime/src/main/scala/org/apache/flink/runtime/jobmanager/JobManager.scala
index 61a0aea..2c39a0a 100644
--- a/flink-runtime/src/main/scala/org/apache/flink/runtime/jobmanager/JobManager.scala
+++ b/flink-runtime/src/main/scala/org/apache/flink/runtime/jobmanager/JobManager.scala
@@ -487,8 +487,8 @@ class JobManager(val configuration: Configuration,
         executionGraph.setScheduleMode(jobGraph.getScheduleMode)
         executionGraph.setQueuedSchedulingAllowed(jobGraph.getAllowQueuedScheduling)
         
-        executionGraph.setCheckpointingEnabled(jobGraph.isMonitoringEnabled)
-        executionGraph.setCheckpointingInterval(jobGraph.getMonitorInterval)
+        executionGraph.setCheckpointingEnabled(jobGraph.isCheckpointingEnabled)
+        executionGraph.setCheckpointingInterval(jobGraph.getCheckpointingInterval)
 
         // initialize the vertices that have a master initialization hook
         // file output formats create directories here, input formats create splits

http://git-wip-us.apache.org/repos/asf/flink/blob/ae04025f/flink-staging/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/StreamingJobGraphGenerator.java
----------------------------------------------------------------------
diff --git a/flink-staging/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/StreamingJobGraphGenerator.java b/flink-staging/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/StreamingJobGraphGenerator.java
index ad744d2..a9232c6 100644
--- a/flink-staging/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/StreamingJobGraphGenerator.java
+++ b/flink-staging/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/StreamingJobGraphGenerator.java
@@ -78,10 +78,10 @@ public class StreamingJobGraphGenerator {
 		// Turn lazy scheduling off
 		jobGraph.setScheduleMode(ScheduleMode.ALL);
 		jobGraph.setJobType(JobGraph.JobType.STREAMING);
-		jobGraph.setMonitoringEnabled(streamGraph.isCheckpointingEnabled());
-		jobGraph.setMonitorInterval(streamGraph.getCheckpointingInterval());
+		jobGraph.setCheckpointingEnabled(streamGraph.isCheckpointingEnabled());
+		jobGraph.setCheckpointingInterval(streamGraph.getCheckpointingInterval());
 
-		if(jobGraph.isMonitoringEnabled()) {
+		if(jobGraph.isCheckpointingEnabled()) {
 			int executionRetries = streamGraph.getExecutionConfig().getNumberOfExecutionRetries();
 			if(executionRetries != -1) {
 				jobGraph.setNumberOfExecutionRetries(executionRetries);