buildbot success in on flink-docs-release-0.10

2016-09-24 Thread buildbot
The Buildbot has detected a restored build on builder flink-docs-release-0.10 
while building . Full details are available at:
https://ci.apache.org/builders/flink-docs-release-0.10/builds/341

Buildbot URL: https://ci.apache.org/

Buildslave for this Build: bb_slave3_ubuntu

Build Reason: The Nightly scheduler named 'flink-nightly-docs-release-0.10' 
triggered this build
Build Source Stamp: [branch release-0.10] HEAD
Blamelist: 

Build succeeded!

Sincerely,
 -The Buildbot





flink git commit: [FLINK-4485] close and remove user class loader after job completion [Forced Update!]

2016-09-24 Thread mxm
Repository: flink
Updated Branches:
  refs/heads/release-1.1 06496439a -> 62c666f57 (forced update)


[FLINK-4485] close and remove user class loader after job completion

Keeping the user class loader around after job completion may lead to
excessive temp space usage because all user jars are kept until the
class loader is garbage collected. Tests showed that garbage collection
can be delayed for a long time after the class loader is not referenced
anymore. Note that for the class loader to not be referenced anymore,
its job has to be removed from the archive.

The fastest way to minimize temp space usage is to close and remove the
URLClassloader after job completion. This requires us to keep a
serializable copy of all data which needs the user class loader after
job completion, e.g. to display data on the web interface.


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

Branch: refs/heads/release-1.1
Commit: 62c666f5794fa211bf570874b1b77044fd6840ac
Parents: 8fd08bf
Author: Maximilian Michels 
Authored: Wed Sep 14 11:00:58 2016 +0200
Committer: Maximilian Michels 
Committed: Sat Sep 24 14:21:07 2016 +0200

--
 .../flink/client/program/JobWithJars.java   |  4 +-
 .../webmonitor/handlers/JobConfigHandler.java   | 47 +---
 .../librarycache/BlobLibraryCacheManager.java   | 42 ++-
 .../librarycache/FlinkUserCodeClassLoader.java  | 35 +
 .../runtime/executiongraph/ExecutionGraph.java  | 34 +
 .../archive/ExecutionConfigSummary.java | 75 
 .../apache/flink/runtime/taskmanager/Task.java  |  3 +-
 .../flink/test/web/WebFrontendITCase.java   |  2 +-
 8 files changed, 191 insertions(+), 51 deletions(-)
--


http://git-wip-us.apache.org/repos/asf/flink/blob/62c666f5/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
--
diff --git 
a/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java 
b/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
index ef02527..d5a3014 100644
--- 
a/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
+++ 
b/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
@@ -22,12 +22,12 @@ import java.io.File;
 import java.io.IOException;
 import java.net.URISyntaxException;
 import java.net.URL;
-import java.net.URLClassLoader;
 import java.util.ArrayList;
 import java.util.Collections;
 import java.util.List;
 
 import org.apache.flink.api.common.Plan;
+import 
org.apache.flink.runtime.execution.librarycache.FlinkUserCodeClassLoader;
 
 /**
  * A JobWithJars is a Flink dataflow plan, together with a bunch of JAR files 
that contain
@@ -134,6 +134,6 @@ public class JobWithJars {
for (int i = 0; i < classpaths.size(); i++) {
urls[i + jars.size()] = classpaths.get(i);
}
-   return new URLClassLoader(urls, parent);
+   return new FlinkUserCodeClassLoader(urls, parent);
}
 }

http://git-wip-us.apache.org/repos/asf/flink/blob/62c666f5/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
--
diff --git 
a/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
 
b/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
index cd63630..75389b1 100644
--- 
a/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
+++ 
b/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
@@ -22,8 +22,8 @@ import java.io.StringWriter;
 import java.util.Map;
 
 import com.fasterxml.jackson.core.JsonGenerator;
-import org.apache.flink.api.common.ExecutionConfig;
 import org.apache.flink.runtime.executiongraph.ExecutionGraph;
+import org.apache.flink.runtime.executiongraph.archive.ExecutionConfigSummary;
 import org.apache.flink.runtime.webmonitor.ExecutionGraphHolder;
 
 /**
@@ -45,37 +45,28 @@ public class JobConfigHandler extends 
AbstractExecutionGraphRequestHandler {
gen.writeStringField("jid", graph.getJobID().toString());
gen.writeStringField("name", graph.getJobName());
 
-   ExecutionConfig ec;
-   try {
-   ec = 
graph.getSerializedExecutionConfig().deserializeValue(graph.getUserClassLoader());
-   } catch (Exception e) {
-   throw new RuntimeExceptio

flink git commit: [FLINK-4485] close classloader in absence of reference holders

2016-09-24 Thread mxm
Repository: flink
Updated Branches:
  refs/heads/master cd43dd59e -> 4a8e94403


[FLINK-4485] close classloader in absence of reference holders


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

Branch: refs/heads/master
Commit: 4a8e94403fb48318561a3cf2da57ba9da280949e
Parents: cd43dd5
Author: Maximilian Michels 
Authored: Fri Sep 23 14:14:09 2016 +0200
Committer: Maximilian Michels 
Committed: Sat Sep 24 14:19:09 2016 +0200

--
 .../runtime/execution/librarycache/BlobLibraryCacheManager.java  | 4 +++-
 1 file changed, 3 insertions(+), 1 deletion(-)
--


http://git-wip-us.apache.org/repos/asf/flink/blob/4a8e9440/flink-runtime/src/main/java/org/apache/flink/runtime/execution/librarycache/BlobLibraryCacheManager.java
--
diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/execution/librarycache/BlobLibraryCacheManager.java
 
b/flink-runtime/src/main/java/org/apache/flink/runtime/execution/librarycache/BlobLibraryCacheManager.java
index 21c6b4d..c94768d 100644
--- 
a/flink-runtime/src/main/java/org/apache/flink/runtime/execution/librarycache/BlobLibraryCacheManager.java
+++ 
b/flink-runtime/src/main/java/org/apache/flink/runtime/execution/librarycache/BlobLibraryCacheManager.java
@@ -157,10 +157,12 @@ public final class BlobLibraryCacheManager extends 
TimerTask implements LibraryC
Preconditions.checkNotNull(task, "The task execution id must 
not be null.");
 
synchronized (lockObject) {
-   LibraryCacheEntry entry = cacheEntries.remove(jobId);
+   LibraryCacheEntry entry = cacheEntries.get(jobId);
 
if (entry != null) {
if (entry.unregister(task)) {
+   cacheEntries.remove(jobId);
+
entry.releaseClassLoader();
 
for (BlobKey key : 
entry.getLibraries()) {



flink git commit: [FLINK-4485] close and remove user class loader after job completion

2016-09-24 Thread mxm
Repository: flink
Updated Branches:
  refs/heads/release-1.1 8fd08bf29 -> 06496439a


[FLINK-4485] close and remove user class loader after job completion

Keeping the user class loader around after job completion may lead to
excessive temp space usage because all user jars are kept until the
class loader is garbage collected. Tests showed that garbage collection
can be delayed for a long time after the class loader is not referenced
anymore. Note that for the class loader to not be referenced anymore,
its job has to be removed from the archive.

The fastest way to minimize temp space usage is to close and remove the
URLClassloader after job completion. This requires us to keep a
serializable copy of all data which needs the user class loader after
job completion, e.g. to display data on the web interface.

This closes #2499


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

Branch: refs/heads/release-1.1
Commit: 06496439a845820ff190edcf09d2ebbd28b0f0a5
Parents: 8fd08bf
Author: Maximilian Michels 
Authored: Wed Sep 14 11:00:58 2016 +0200
Committer: Maximilian Michels 
Committed: Sat Sep 24 13:55:59 2016 +0200

--
 .../flink/client/program/JobWithJars.java   |  4 +-
 .../webmonitor/handlers/JobConfigHandler.java   | 47 +---
 .../librarycache/BlobLibraryCacheManager.java   | 44 ++--
 .../librarycache/FlinkUserCodeClassLoader.java  | 35 +
 .../runtime/executiongraph/ExecutionGraph.java  | 34 +
 .../archive/ExecutionConfigSummary.java | 75 
 .../apache/flink/runtime/taskmanager/Task.java  |  3 +-
 .../flink/test/web/WebFrontendITCase.java   |  2 +-
 8 files changed, 191 insertions(+), 53 deletions(-)
--


http://git-wip-us.apache.org/repos/asf/flink/blob/06496439/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
--
diff --git 
a/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java 
b/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
index ef02527..d5a3014 100644
--- 
a/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
+++ 
b/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
@@ -22,12 +22,12 @@ import java.io.File;
 import java.io.IOException;
 import java.net.URISyntaxException;
 import java.net.URL;
-import java.net.URLClassLoader;
 import java.util.ArrayList;
 import java.util.Collections;
 import java.util.List;
 
 import org.apache.flink.api.common.Plan;
+import 
org.apache.flink.runtime.execution.librarycache.FlinkUserCodeClassLoader;
 
 /**
  * A JobWithJars is a Flink dataflow plan, together with a bunch of JAR files 
that contain
@@ -134,6 +134,6 @@ public class JobWithJars {
for (int i = 0; i < classpaths.size(); i++) {
urls[i + jars.size()] = classpaths.get(i);
}
-   return new URLClassLoader(urls, parent);
+   return new FlinkUserCodeClassLoader(urls, parent);
}
 }

http://git-wip-us.apache.org/repos/asf/flink/blob/06496439/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
--
diff --git 
a/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
 
b/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
index cd63630..75389b1 100644
--- 
a/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
+++ 
b/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
@@ -22,8 +22,8 @@ import java.io.StringWriter;
 import java.util.Map;
 
 import com.fasterxml.jackson.core.JsonGenerator;
-import org.apache.flink.api.common.ExecutionConfig;
 import org.apache.flink.runtime.executiongraph.ExecutionGraph;
+import org.apache.flink.runtime.executiongraph.archive.ExecutionConfigSummary;
 import org.apache.flink.runtime.webmonitor.ExecutionGraphHolder;
 
 /**
@@ -45,37 +45,28 @@ public class JobConfigHandler extends 
AbstractExecutionGraphRequestHandler {
gen.writeStringField("jid", graph.getJobID().toString());
gen.writeStringField("name", graph.getJobName());
 
-   ExecutionConfig ec;
-   try {
-   ec = 
graph.getSerializedExecutionConfig().deserializeValue(graph.getUserClassLoader());
-   } catch (Exception e) {
-   throw new RuntimeExce

[2/2] flink git commit: [FLINK-4555] wait for ResourceManager to cleanly unregister application

2016-09-24 Thread mxm
[FLINK-4555] wait for ResourceManager to cleanly unregister application

This ensures that the ResourceManager has enough time to unregister the
application before shutting down.

This closes #2514


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

Branch: refs/heads/master
Commit: 1e4b7ebd4aa7dba9aa87122fec9db561df198160
Parents: 40c978b
Author: Maximilian Michels 
Authored: Wed Sep 21 18:45:37 2016 +0200
Committer: Maximilian Michels 
Committed: Sat Sep 24 13:45:50 2016 +0200

--
 .../runtime/clusterframework/FlinkResourceManager.java   |  8 +---
 .../org/apache/flink/runtime/jobmanager/JobManager.scala | 11 +--
 2 files changed, 14 insertions(+), 5 deletions(-)
--


http://git-wip-us.apache.org/repos/asf/flink/blob/1e4b7ebd/flink-runtime/src/main/java/org/apache/flink/runtime/clusterframework/FlinkResourceManager.java
--
diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/clusterframework/FlinkResourceManager.java
 
b/flink-runtime/src/main/java/org/apache/flink/runtime/clusterframework/FlinkResourceManager.java
index 7ea286d..911c1f6 100644
--- 
a/flink-runtime/src/main/java/org/apache/flink/runtime/clusterframework/FlinkResourceManager.java
+++ 
b/flink-runtime/src/main/java/org/apache/flink/runtime/clusterframework/FlinkResourceManager.java
@@ -33,16 +33,17 @@ import org.apache.flink.runtime.akka.FlinkUntypedActor;
 import 
org.apache.flink.runtime.clusterframework.messages.CheckAndAllocateContainers;
 import org.apache.flink.runtime.clusterframework.messages.FatalErrorOccurred;
 import org.apache.flink.runtime.clusterframework.messages.InfoMessage;
-import 
org.apache.flink.runtime.clusterframework.messages.RegisterInfoMessageListenerSuccessful;
-import 
org.apache.flink.runtime.clusterframework.messages.NotifyResourceStarted;
-import 
org.apache.flink.runtime.clusterframework.messages.RegisterResourceManagerSuccessful;
 import org.apache.flink.runtime.clusterframework.messages.NewLeaderAvailable;
+import 
org.apache.flink.runtime.clusterframework.messages.NotifyResourceStarted;
 import 
org.apache.flink.runtime.clusterframework.messages.RegisterInfoMessageListener;
+import 
org.apache.flink.runtime.clusterframework.messages.RegisterInfoMessageListenerSuccessful;
 import 
org.apache.flink.runtime.clusterframework.messages.RegisterResourceManager;
+import 
org.apache.flink.runtime.clusterframework.messages.RegisterResourceManagerSuccessful;
 import org.apache.flink.runtime.clusterframework.messages.RemoveResource;
 import org.apache.flink.runtime.clusterframework.messages.ResourceRemoved;
 import org.apache.flink.runtime.clusterframework.messages.SetWorkerPoolSize;
 import org.apache.flink.runtime.clusterframework.messages.StopCluster;
+import 
org.apache.flink.runtime.clusterframework.messages.StopClusterSuccessful;
 import 
org.apache.flink.runtime.clusterframework.messages.TriggerRegistrationAtJobManager;
 import 
org.apache.flink.runtime.clusterframework.messages.UnRegisterInfoMessageListener;
 import org.apache.flink.runtime.clusterframework.types.ResourceID;
@@ -253,6 +254,7 @@ public abstract class FlinkResourceManagerhttp://git-wip-us.apache.org/repos/asf/flink/blob/1e4b7ebd/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 fd96f86..a733943 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
@@ -1024,8 +1024,15 @@ class JobManager(
   // send resource manager the ok
   currentResourceManager match {
 case Some(rm) =>
-  // inform rm
-  rm ! decorateMessage(msg)
+  try {
+// inform rm and wait for it to confirm
+val waitTime = FiniteDuration(5, TimeUnit.SECONDS)
+val answer = (rm ? decorateMessage(msg))(waitTime)
+Await.ready(answer, waitTime)
+  } catch {
+case e: TimeoutException =>
+case e: InterruptedException =>
+  }
 case None =>
   // ResourceManager not available
   // we choose not to wait here beacuse it might block the shutdown 
forever



[1/2] flink git commit: [FLINK-4485] close and remove user class loader after job completion

2016-09-24 Thread mxm
Repository: flink
Updated Branches:
  refs/heads/master 40c978b04 -> cd43dd59e


[FLINK-4485] close and remove user class loader after job completion

Keeping the user class loader around after job completion may lead to
excessive temp space usage because all user jars are kept until the
class loader is garbage collected. Tests showed that garbage collection
can be delayed for a long time after the class loader is not referenced
anymore. Note that for the class loader to not be referenced anymore,
its job has to be removed from the archive.

The fastest way to minimize temp space usage is to close and remove the
URLClassloader after job completion. This requires us to keep a
serializable copy of all data which needs the user class loader after
job completion, e.g. to display data on the web interface.

This closes #2499


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

Branch: refs/heads/master
Commit: cd43dd59e248766627c35f90038b2202ed9e52dc
Parents: 1e4b7eb
Author: Maximilian Michels 
Authored: Wed Sep 14 11:00:58 2016 +0200
Committer: Maximilian Michels 
Committed: Sat Sep 24 13:45:50 2016 +0200

--
 .../flink/client/program/JobWithJars.java   |  4 +-
 .../webmonitor/handlers/JobConfigHandler.java   | 47 +---
 .../apache/flink/runtime/client/JobClient.java  |  4 +-
 .../librarycache/BlobLibraryCacheManager.java   | 44 ++--
 .../librarycache/FlinkUserCodeClassLoader.java  | 35 +
 .../runtime/executiongraph/ExecutionGraph.java  | 34 +
 .../archive/ExecutionConfigSummary.java | 75 
 .../apache/flink/runtime/taskmanager/Task.java  |  3 +-
 .../flink/test/web/WebFrontendITCase.java   |  2 +-
 9 files changed, 193 insertions(+), 55 deletions(-)
--


http://git-wip-us.apache.org/repos/asf/flink/blob/cd43dd59/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
--
diff --git 
a/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java 
b/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
index ef02527..d5a3014 100644
--- 
a/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
+++ 
b/flink-clients/src/main/java/org/apache/flink/client/program/JobWithJars.java
@@ -22,12 +22,12 @@ import java.io.File;
 import java.io.IOException;
 import java.net.URISyntaxException;
 import java.net.URL;
-import java.net.URLClassLoader;
 import java.util.ArrayList;
 import java.util.Collections;
 import java.util.List;
 
 import org.apache.flink.api.common.Plan;
+import 
org.apache.flink.runtime.execution.librarycache.FlinkUserCodeClassLoader;
 
 /**
  * A JobWithJars is a Flink dataflow plan, together with a bunch of JAR files 
that contain
@@ -134,6 +134,6 @@ public class JobWithJars {
for (int i = 0; i < classpaths.size(); i++) {
urls[i + jars.size()] = classpaths.get(i);
}
-   return new URLClassLoader(urls, parent);
+   return new FlinkUserCodeClassLoader(urls, parent);
}
 }

http://git-wip-us.apache.org/repos/asf/flink/blob/cd43dd59/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
--
diff --git 
a/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
 
b/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
index cd63630..75389b1 100644
--- 
a/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
+++ 
b/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/handlers/JobConfigHandler.java
@@ -22,8 +22,8 @@ import java.io.StringWriter;
 import java.util.Map;
 
 import com.fasterxml.jackson.core.JsonGenerator;
-import org.apache.flink.api.common.ExecutionConfig;
 import org.apache.flink.runtime.executiongraph.ExecutionGraph;
+import org.apache.flink.runtime.executiongraph.archive.ExecutionConfigSummary;
 import org.apache.flink.runtime.webmonitor.ExecutionGraphHolder;
 
 /**
@@ -45,37 +45,28 @@ public class JobConfigHandler extends 
AbstractExecutionGraphRequestHandler {
gen.writeStringField("jid", graph.getJobID().toString());
gen.writeStringField("name", graph.getJobName());
 
-   ExecutionConfig ec;
-   try {
-   ec = 
graph.getSerializedExecutionConfig().deserializeValue(graph.getUserClassLoader());
-   } catch (Exception e)