[flink] branch release-1.11 updated: [FLINK-17463][tests] Avoid concurrent directory creation and deletion
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/release-1.11 by this push: new 647f762 [FLINK-17463][tests] Avoid concurrent directory creation and deletion 647f762 is described below commit 647f76283c900048e12361cf96d26db2a184b10b Author: Gary Yao AuthorDate: Wed May 27 15:23:47 2020 +0200 [FLINK-17463][tests] Avoid concurrent directory creation and deletion BlobCacheCleanupTest#testPermanentBlobCleanup() tests that job related files are cleaned up by a background task when the job is released from the PermanentBlobCache. The tests asserts that the uploaded blobs are deleted from the filesystem. Because the scheduling of the background task cannot be controlled from outside the cache, the test polls the filesystem. More precisely, the test uses BlobUtils#getStorageLocation() to build the path on the filesystem given a blobkey and tests the existence of that path in regular intervals. As a side effect, however, BlobUtils#getStorageLocation() also creates all necessary directories to that path if they do not exist yet. This leads to a situation where directories and concurrently deleted and created, which can cause FileAlreadyExists exceptions. This commit fixes the issue. Note that the above applies to all tests that invoke BlobServerCleanupTest#checkFilesExist(). --- .../java/org/apache/flink/runtime/blob/AbstractBlobCache.java | 4 .../main/java/org/apache/flink/runtime/blob/BlobServer.java| 4 .../org/apache/flink/runtime/blob/BlobServerCleanupTest.java | 10 ++ 3 files changed, 14 insertions(+), 4 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/blob/AbstractBlobCache.java b/flink-runtime/src/main/java/org/apache/flink/runtime/blob/AbstractBlobCache.java index ce12898..8a873f1 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/blob/AbstractBlobCache.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/blob/AbstractBlobCache.java @@ -122,6 +122,10 @@ public abstract class AbstractBlobCache implements Closeable { this.serverAddress = serverAddress; } + public File getStorageDir() { + return storageDir; + } + /** * Returns local copy of the file for the BLOB with the given key. * diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/blob/BlobServer.java b/flink-runtime/src/main/java/org/apache/flink/runtime/blob/BlobServer.java index a47040c..a50f535 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/blob/BlobServer.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/blob/BlobServer.java @@ -213,6 +213,10 @@ public class BlobServer extends Thread implements BlobService, BlobWriter, Perma // Path Accessors // + public File getStorageDir() { + return storageDir; + } + /** * Returns a file handle to the file associated with the given blob key on the blob * server. diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/blob/BlobServerCleanupTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/blob/BlobServerCleanupTest.java index aafba30..04c1187 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/blob/BlobServerCleanupTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/blob/BlobServerCleanupTest.java @@ -204,20 +204,22 @@ public class BlobServerCleanupTest extends TestLogger { int numFiles = 0; for (BlobKey key : keys) { - final File blobFile; + final File storageDir; if (blobService instanceof BlobServer) { BlobServer server = (BlobServer) blobService; - blobFile = server.getStorageLocation(jobId, key); + storageDir = server.getStorageDir(); } else if (blobService instanceof PermanentBlobCache) { PermanentBlobCache cache = (PermanentBlobCache) blobService; - blobFile = cache.getStorageLocation(jobId, key); + storageDir = cache.getStorageDir(); } else if (blobService instanceof TransientBlobCache) { TransientBlobCache cache = (TransientBlobCache) blobService; - blobFile = cache.getStorageLocation(jobId, key); + storageDir = cache.getStorageDir(); } else
[flink] branch master updated: [FLINK-17463][tests] Avoid concurrent directory creation and deletion
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/master by this push: new c22d01d [FLINK-17463][tests] Avoid concurrent directory creation and deletion c22d01d is described below commit c22d01d3bfbb1384f98664361f1491b806e95798 Author: Gary Yao AuthorDate: Wed May 27 15:23:47 2020 +0200 [FLINK-17463][tests] Avoid concurrent directory creation and deletion BlobCacheCleanupTest#testPermanentBlobCleanup() tests that job related files are cleaned up by a background task when the job is released from the PermanentBlobCache. The tests asserts that the uploaded blobs are deleted from the filesystem. Because the scheduling of the background task cannot be controlled from outside the cache, the test polls the filesystem. More precisely, the test uses BlobUtils#getStorageLocation() to build the path on the filesystem given a blobkey and tests the existence of that path in regular intervals. As a side effect, however, BlobUtils#getStorageLocation() also creates all necessary directories to that path if they do not exist yet. This leads to a situation where directories and concurrently deleted and created, which can cause FileAlreadyExists exceptions. This commit fixes the issue. Note that the above applies to all tests that invoke BlobServerCleanupTest#checkFilesExist(). This closes #12376. --- .../java/org/apache/flink/runtime/blob/AbstractBlobCache.java | 4 .../main/java/org/apache/flink/runtime/blob/BlobServer.java| 4 .../org/apache/flink/runtime/blob/BlobServerCleanupTest.java | 10 ++ 3 files changed, 14 insertions(+), 4 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/blob/AbstractBlobCache.java b/flink-runtime/src/main/java/org/apache/flink/runtime/blob/AbstractBlobCache.java index ce12898..8a873f1 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/blob/AbstractBlobCache.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/blob/AbstractBlobCache.java @@ -122,6 +122,10 @@ public abstract class AbstractBlobCache implements Closeable { this.serverAddress = serverAddress; } + public File getStorageDir() { + return storageDir; + } + /** * Returns local copy of the file for the BLOB with the given key. * diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/blob/BlobServer.java b/flink-runtime/src/main/java/org/apache/flink/runtime/blob/BlobServer.java index a47040c..a50f535 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/blob/BlobServer.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/blob/BlobServer.java @@ -213,6 +213,10 @@ public class BlobServer extends Thread implements BlobService, BlobWriter, Perma // Path Accessors // + public File getStorageDir() { + return storageDir; + } + /** * Returns a file handle to the file associated with the given blob key on the blob * server. diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/blob/BlobServerCleanupTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/blob/BlobServerCleanupTest.java index aafba30..04c1187 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/blob/BlobServerCleanupTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/blob/BlobServerCleanupTest.java @@ -204,20 +204,22 @@ public class BlobServerCleanupTest extends TestLogger { int numFiles = 0; for (BlobKey key : keys) { - final File blobFile; + final File storageDir; if (blobService instanceof BlobServer) { BlobServer server = (BlobServer) blobService; - blobFile = server.getStorageLocation(jobId, key); + storageDir = server.getStorageDir(); } else if (blobService instanceof PermanentBlobCache) { PermanentBlobCache cache = (PermanentBlobCache) blobService; - blobFile = cache.getStorageLocation(jobId, key); + storageDir = cache.getStorageDir(); } else if (blobService instanceof TransientBlobCache) { TransientBlobCache cache = (TransientBlobCache) blobService; - blobFile = cache.getStorageLocation(jobId, key); + storageDir = cache.getStorageDir(); } else
[flink] branch master updated (03b6d71 -> 7e3b2cd)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 03b6d71 [FLINK-17058] Adding ProcessingTimeoutTrigger of nested triggers. add 8461066 [hotfix][tests] Remove note that E2E exactly-once is not covered add 7e3b2cd [hotfix][tests] Document flink-jepsen correctness model No new revisions were added by this update. Summary of changes: flink-jepsen/README.md | 17 +++-- 1 file changed, 11 insertions(+), 6 deletions(-)
[flink] 02/04: [hotfix][qs] Fix logging of exception in RequestWriteListener
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git commit 81c185bdeeccf9f10bf584defde740676d305646 Author: Gary Yao AuthorDate: Thu May 21 09:30:14 2020 +0200 [hotfix][qs] Fix logging of exception in RequestWriteListener Log stacktrace of exception instead of only the message. --- .../org/apache/flink/queryablestate/network/AbstractServerHandler.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/flink-queryable-state/flink-queryable-state-client-java/src/main/java/org/apache/flink/queryablestate/network/AbstractServerHandler.java b/flink-queryable-state/flink-queryable-state-client-java/src/main/java/org/apache/flink/queryablestate/network/AbstractServerHandler.java index 7941131..4115fd9 100644 --- a/flink-queryable-state/flink-queryable-state-client-java/src/main/java/org/apache/flink/queryablestate/network/AbstractServerHandler.java +++ b/flink-queryable-state/flink-queryable-state-client-java/src/main/java/org/apache/flink/queryablestate/network/AbstractServerHandler.java @@ -308,7 +308,7 @@ public abstract class AbstractServerHandler
[flink] 01/04: [hotfix][tests] Make ClientTest extend TestLogger
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git commit 1bf5b810eb73faa420bed5fdef92a2634f8b7f34 Author: Gary Yao AuthorDate: Thu May 21 09:29:52 2020 +0200 [hotfix][tests] Make ClientTest extend TestLogger --- .../test/java/org/apache/flink/queryablestate/network/ClientTest.java | 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-) diff --git a/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/ClientTest.java b/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/ClientTest.java index b2ebdaf..e19a717 100644 --- a/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/ClientTest.java +++ b/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/ClientTest.java @@ -45,6 +45,7 @@ import org.apache.flink.runtime.state.memory.MemoryStateBackend; import org.apache.flink.runtime.state.ttl.TtlTimeProvider; import org.apache.flink.util.ExceptionUtils; import org.apache.flink.util.NetUtils; +import org.apache.flink.util.TestLogger; import org.apache.flink.shaded.netty4.io.netty.bootstrap.ServerBootstrap; import org.apache.flink.shaded.netty4.io.netty.buffer.ByteBuf; @@ -97,7 +98,7 @@ import static org.junit.Assert.fail; /** * Tests for {@link Client}. */ -public class ClientTest { +public class ClientTest extends TestLogger { private static final Logger LOG = LoggerFactory.getLogger(ClientTest.class);
[flink] branch release-1.11 updated (828ba1d -> 3e18c10)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git. from 828ba1d [FLINK-17801][tests] Increase timeout of TaskExecutorTest.testHeartbeatTimeoutWithResourceManager new 1bf5b81 [hotfix][tests] Make ClientTest extend TestLogger new 81c185b [hotfix][qs] Fix logging of exception in RequestWriteListener new 7cbdd91 [FLINK-13553][qs] Add logging to AbstractServerHandler new 3e18c10 [FLINK-13553][tests] Enable TRACE logging for org.apache.flink.queryablestate The 4 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../apache/flink/queryablestate/network/AbstractServerHandler.java | 5 - .../java/org/apache/flink/queryablestate/network/ClientTest.java | 3 ++- tools/log4j-travis.properties| 5 + 3 files changed, 11 insertions(+), 2 deletions(-)
[flink] 04/04: [FLINK-13553][tests] Enable TRACE logging for org.apache.flink.queryablestate
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git commit 3e18c109051821176575a15a6b10aaa5cc2e3e12 Author: Gary Yao AuthorDate: Thu May 21 10:05:05 2020 +0200 [FLINK-13553][tests] Enable TRACE logging for org.apache.flink.queryablestate --- tools/log4j-travis.properties | 5 + 1 file changed, 5 insertions(+) diff --git a/tools/log4j-travis.properties b/tools/log4j-travis.properties index f5912d5..4550a98 100644 --- a/tools/log4j-travis.properties +++ b/tools/log4j-travis.properties @@ -68,3 +68,8 @@ logger.zkclient.level = INFO logger.zkclient.appenderRef.out.ref = ConsoleAppender logger.consumer.name = org.apache.flink.streaming.connectors.kafka.internals.SimpleConsumerThread logger.consumer.level = OFF + +# Enable TRACE logging to debug FLINK-13553 +logger.queryablestate.name = org.apache.flink.queryablestate +logger.queryablestate.level = TRACE +logger.queryablestate.appenderRef.out.ref = ConsoleAppender
[flink] 03/04: [FLINK-13553][qs] Add logging to AbstractServerHandler
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git commit 7cbdd91413ee26d00d9015581ce2fa8538fd5963 Author: Gary Yao AuthorDate: Thu May 21 09:44:33 2020 +0200 [FLINK-13553][qs] Add logging to AbstractServerHandler Log every request on trace level. Log caught exceptions in AsyncRequestTask. --- .../org/apache/flink/queryablestate/network/AbstractServerHandler.java | 3 +++ 1 file changed, 3 insertions(+) diff --git a/flink-queryable-state/flink-queryable-state-client-java/src/main/java/org/apache/flink/queryablestate/network/AbstractServerHandler.java b/flink-queryable-state/flink-queryable-state-client-java/src/main/java/org/apache/flink/queryablestate/network/AbstractServerHandler.java index 4115fd9..9011bb2 100644 --- a/flink-queryable-state/flink-queryable-state-client-java/src/main/java/org/apache/flink/queryablestate/network/AbstractServerHandler.java +++ b/flink-queryable-state/flink-queryable-state-client-java/src/main/java/org/apache/flink/queryablestate/network/AbstractServerHandler.java @@ -110,6 +110,7 @@ public abstract class AbstractServerHandler
[flink] branch master updated (a47d705 -> 0cc7aae)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from a47d705 [FLINK-17801][tests] Increase timeout of TaskExecutorTest.testHeartbeatTimeoutWithResourceManager add d184f73 [hotfix][tests] Make ClientTest extend TestLogger add 984aa0b [hotfix][qs] Fix logging of exception in RequestWriteListener add 564e880 [FLINK-13553][qs] Add logging to AbstractServerHandler add 0cc7aae [FLINK-13553][tests] Enable TRACE logging for org.apache.flink.queryablestate No new revisions were added by this update. Summary of changes: .../apache/flink/queryablestate/network/AbstractServerHandler.java | 5 - .../java/org/apache/flink/queryablestate/network/ClientTest.java | 3 ++- tools/log4j-travis.properties| 5 + 3 files changed, 11 insertions(+), 2 deletions(-)
[flink] branch release-1.11 updated: [FLINK-17794][tests] Tear down installed software in reverse order
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/release-1.11 by this push: new 6a4714f [FLINK-17794][tests] Tear down installed software in reverse order 6a4714f is described below commit 6a4714fdeff96d54db5fde5fac9b0eb355886b47 Author: Gary Yao AuthorDate: Mon May 18 10:48:34 2020 +0200 [FLINK-17794][tests] Tear down installed software in reverse order Tear down installed software in reverse order in Jepsen tests. This mitigates the issue that sometimes YARN's NodeManager directories cannot be removed using 'rm -rf' because Flink processes keep running and generate files after the YARN NodeManager is shut down. rm -r removes files recursively but if files are created in the background concurrently, the command can still fail with a non-zero exit code. --- flink-jepsen/src/jepsen/flink/db.clj | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/flink-jepsen/src/jepsen/flink/db.clj b/flink-jepsen/src/jepsen/flink/db.clj index 61743b1..71c6716 100644 --- a/flink-jepsen/src/jepsen/flink/db.clj +++ b/flink-jepsen/src/jepsen/flink/db.clj @@ -109,7 +109,7 @@ (teardown! [_ test node] (c/su (try - (doseq [db dbs] (db/teardown! db test node)) + (doseq [db (reverse dbs)] (db/teardown! db test node)) (finally (fu/stop-all-supervised-services!) db/LogFiles (log-files [_ test node]
[flink] branch master updated: [FLINK-17794][tests] Tear down installed software in reverse order
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/master by this push: new 2b2c574 [FLINK-17794][tests] Tear down installed software in reverse order 2b2c574 is described below commit 2b2c574f102689b3cde9deac0bd1bcf78ad7ebc7 Author: Gary Yao AuthorDate: Mon May 18 10:48:34 2020 +0200 [FLINK-17794][tests] Tear down installed software in reverse order Tear down installed software in reverse order in Jepsen tests. This mitigates the issue that sometimes YARN's NodeManager directories cannot be removed using 'rm -rf' because Flink processes keep running and generate files after the YARN NodeManager is shut down. rm -r removes files recursively but if files are created in the background concurrently, the command can still fail with a non-zero exit code. This closes #12249. --- flink-jepsen/src/jepsen/flink/db.clj | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/flink-jepsen/src/jepsen/flink/db.clj b/flink-jepsen/src/jepsen/flink/db.clj index 61743b1..71c6716 100644 --- a/flink-jepsen/src/jepsen/flink/db.clj +++ b/flink-jepsen/src/jepsen/flink/db.clj @@ -109,7 +109,7 @@ (teardown! [_ test node] (c/su (try - (doseq [db dbs] (db/teardown! db test node)) + (doseq [db (reverse dbs)] (db/teardown! db test node)) (finally (fu/stop-all-supervised-services!) db/LogFiles (log-files [_ test node]
[flink] 03/03: [FLINK-17687][tests] Simplify collection of Mesos logs
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git commit 2aa45cdcc88d75837df165fcde71200d796deee7 Author: Gary Yao AuthorDate: Thu May 14 17:13:28 2020 +0200 [FLINK-17687][tests] Simplify collection of Mesos logs --- flink-jepsen/src/jepsen/flink/mesos.clj | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/flink-jepsen/src/jepsen/flink/mesos.clj b/flink-jepsen/src/jepsen/flink/mesos.clj index c104158..de1ba50 100644 --- a/flink-jepsen/src/jepsen/flink/mesos.clj +++ b/flink-jepsen/src/jepsen/flink/mesos.clj @@ -197,5 +197,5 @@ db/LogFiles (log-files [_ test node] (concat -(if (cu/exists? log-dir) (cu/ls-full log-dir) []) +(fu/find-files! log-dir) (fu/find-files! slave-dir "*.log")
[flink] branch release-1.11 updated (59a65709 -> 2aa45cd)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git. from 59a65709 [FLINK-17796] Respect user specified classpath for application mode new 656d56e [FLINK-17687][tests] Enable listing files recursively by pattern new aa2a570 [FLINK-17687][tests] Collect log files before tearing down Mesos new 2aa45cd [FLINK-17687][tests] Simplify collection of Mesos logs The 3 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: flink-jepsen/src/jepsen/flink/mesos.clj | 5 - flink-jepsen/src/jepsen/flink/utils.clj | 7 --- 2 files changed, 8 insertions(+), 4 deletions(-)
[flink] 02/03: [FLINK-17687][tests] Collect log files before tearing down Mesos
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git commit aa2a5709309ef8149607cc6ac696cd990a8aef81 Author: Gary Yao AuthorDate: Thu May 14 17:12:08 2020 +0200 [FLINK-17687][tests] Collect log files before tearing down Mesos --- flink-jepsen/src/jepsen/flink/mesos.clj | 5 - 1 file changed, 4 insertions(+), 1 deletion(-) diff --git a/flink-jepsen/src/jepsen/flink/mesos.clj b/flink-jepsen/src/jepsen/flink/mesos.clj index e944089..c104158 100644 --- a/flink-jepsen/src/jepsen/flink/mesos.clj +++ b/flink-jepsen/src/jepsen/flink/mesos.clj @@ -22,6 +22,7 @@ [util :as util :refer [meh]]] [jepsen.control.util :as cu] [jepsen.os.debian :as debian] +[jepsen.flink.utils :as fu] [jepsen.flink.utils :refer [create-supervised-service! stop-supervised-service!]] [jepsen.flink.zookeeper :refer [zookeeper-uri]])) @@ -195,4 +196,6 @@ (stop-marathon! test node)) db/LogFiles (log-files [_ test node] - (if (cu/exists? log-dir) (cu/ls-full log-dir) [] + (concat +(if (cu/exists? log-dir) (cu/ls-full log-dir) []) +(fu/find-files! slave-dir "*.log")
[flink] 01/03: [FLINK-17687][tests] Enable listing files recursively by pattern
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git commit 656d56e99e3c158c7252db04bc034cce77ad39ba Author: Gary Yao AuthorDate: Thu May 14 17:10:35 2020 +0200 [FLINK-17687][tests] Enable listing files recursively by pattern --- flink-jepsen/src/jepsen/flink/utils.clj | 7 --- 1 file changed, 4 insertions(+), 3 deletions(-) diff --git a/flink-jepsen/src/jepsen/flink/utils.clj b/flink-jepsen/src/jepsen/flink/utils.clj index 5d8e712..39ee36c 100644 --- a/flink-jepsen/src/jepsen/flink/utils.clj +++ b/flink-jepsen/src/jepsen/flink/utils.clj @@ -58,16 +58,17 @@ (defn find-files! "Lists files recursively given a directory. If the directory does not exist, an empty collection is returned." - [dir] + ([dir] (find-files! dir "*")) + ([dir name] (let [files (try -(c/exec :find dir :-type :f) +(c/exec :find dir :-type :f :-name (c/lit (str "\"" name "\""))) (catch Exception e (if (.contains (.getMessage e) "No such file or directory") "" (throw e] (->> (clojure.string/split files #"\n") - (remove clojure.string/blank? + (remove clojure.string/blank?) ;;; runit process supervisor (http://smarden.org/runit/)
[flink] branch master updated (b591f90 -> ed74173)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from b591f90 [FLINK-17796] Respect user specified classpath for application mode new be8c02e [FLINK-17687][tests] Enable listing files recursively by pattern new 12d662c [FLINK-17687][tests] Collect log files before tearing down Mesos new ed74173 [FLINK-17687][tests] Simplify collection of Mesos logs The 3 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: flink-jepsen/src/jepsen/flink/mesos.clj | 5 - flink-jepsen/src/jepsen/flink/utils.clj | 7 --- 2 files changed, 8 insertions(+), 4 deletions(-)
[flink] 03/03: [FLINK-17687][tests] Simplify collection of Mesos logs
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit ed74173c087fe879f5728b810e204bafb69bdae6 Author: Gary Yao AuthorDate: Thu May 14 17:13:28 2020 +0200 [FLINK-17687][tests] Simplify collection of Mesos logs This closes #12203. --- flink-jepsen/src/jepsen/flink/mesos.clj | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/flink-jepsen/src/jepsen/flink/mesos.clj b/flink-jepsen/src/jepsen/flink/mesos.clj index c104158..de1ba50 100644 --- a/flink-jepsen/src/jepsen/flink/mesos.clj +++ b/flink-jepsen/src/jepsen/flink/mesos.clj @@ -197,5 +197,5 @@ db/LogFiles (log-files [_ test node] (concat -(if (cu/exists? log-dir) (cu/ls-full log-dir) []) +(fu/find-files! log-dir) (fu/find-files! slave-dir "*.log")
[flink] 02/03: [FLINK-17687][tests] Collect log files before tearing down Mesos
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit 12d662c9da2dc3e18fdd3d752ddeeb07df1f5945 Author: Gary Yao AuthorDate: Thu May 14 17:12:08 2020 +0200 [FLINK-17687][tests] Collect log files before tearing down Mesos --- flink-jepsen/src/jepsen/flink/mesos.clj | 5 - 1 file changed, 4 insertions(+), 1 deletion(-) diff --git a/flink-jepsen/src/jepsen/flink/mesos.clj b/flink-jepsen/src/jepsen/flink/mesos.clj index e944089..c104158 100644 --- a/flink-jepsen/src/jepsen/flink/mesos.clj +++ b/flink-jepsen/src/jepsen/flink/mesos.clj @@ -22,6 +22,7 @@ [util :as util :refer [meh]]] [jepsen.control.util :as cu] [jepsen.os.debian :as debian] +[jepsen.flink.utils :as fu] [jepsen.flink.utils :refer [create-supervised-service! stop-supervised-service!]] [jepsen.flink.zookeeper :refer [zookeeper-uri]])) @@ -195,4 +196,6 @@ (stop-marathon! test node)) db/LogFiles (log-files [_ test node] - (if (cu/exists? log-dir) (cu/ls-full log-dir) [] + (concat +(if (cu/exists? log-dir) (cu/ls-full log-dir) []) +(fu/find-files! slave-dir "*.log")
[flink] 01/03: [FLINK-17687][tests] Enable listing files recursively by pattern
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit be8c02e397943d668c4ff64e4c491a560136e2e1 Author: Gary Yao AuthorDate: Thu May 14 17:10:35 2020 +0200 [FLINK-17687][tests] Enable listing files recursively by pattern --- flink-jepsen/src/jepsen/flink/utils.clj | 7 --- 1 file changed, 4 insertions(+), 3 deletions(-) diff --git a/flink-jepsen/src/jepsen/flink/utils.clj b/flink-jepsen/src/jepsen/flink/utils.clj index 5d8e712..39ee36c 100644 --- a/flink-jepsen/src/jepsen/flink/utils.clj +++ b/flink-jepsen/src/jepsen/flink/utils.clj @@ -58,16 +58,17 @@ (defn find-files! "Lists files recursively given a directory. If the directory does not exist, an empty collection is returned." - [dir] + ([dir] (find-files! dir "*")) + ([dir name] (let [files (try -(c/exec :find dir :-type :f) +(c/exec :find dir :-type :f :-name (c/lit (str "\"" name "\""))) (catch Exception e (if (.contains (.getMessage e) "No such file or directory") "" (throw e] (->> (clojure.string/split files #"\n") - (remove clojure.string/blank? + (remove clojure.string/blank?) ;;; runit process supervisor (http://smarden.org/runit/)
[flink] branch release-1.11 updated: [FLINK-17777][tests] Set HADOOP_CLASSPATH for Mesos TaskManagers
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/release-1.11 by this push: new f46735c [FLINK-1][tests] Set HADOOP_CLASSPATH for Mesos TaskManagers f46735c is described below commit f46735cb4963af616c0e8538331bed8739a1d353 Author: Gary Yao AuthorDate: Mon May 18 10:08:49 2020 +0200 [FLINK-1][tests] Set HADOOP_CLASSPATH for Mesos TaskManagers --- flink-jepsen/src/jepsen/flink/db.clj | 1 + 1 file changed, 1 insertion(+) diff --git a/flink-jepsen/src/jepsen/flink/db.clj b/flink-jepsen/src/jepsen/flink/db.clj index df04765..61743b1 100644 --- a/flink-jepsen/src/jepsen/flink/db.clj +++ b/flink-jepsen/src/jepsen/flink/db.clj @@ -303,6 +303,7 @@ "-Djobmanager.rpc.address=$(hostname -f)" "-Djobmanager.rpc.port=6123" "-Dmesos.resourcemanager.tasks.cpus=1" +"-Dcontainerized.taskmanager.env.HADOOP_CLASSPATH=$(/opt/hadoop/bin/hadoop classpath)" "-Dtaskmanager.memory.process.size=2048m" "-Drest.bind-address=$(hostname -f)"))
[flink] branch master updated (1aee2c6 -> 81ffe8a)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 1aee2c6 [FLINK-17593] Update BucketStateSerializerTest for v2 add 81ffe8a [FLINK-1][tests] Set HADOOP_CLASSPATH for Mesos TaskManagers No new revisions were added by this update. Summary of changes: flink-jepsen/src/jepsen/flink/db.clj | 1 + 1 file changed, 1 insertion(+)
[flink] branch release-1.11 updated: [FLINK-17792][tests] Catch and log exception if jstack fails
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.11 in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/release-1.11 by this push: new d8a77cb [FLINK-17792][tests] Catch and log exception if jstack fails d8a77cb is described below commit d8a77cbf93007bf970963a4499aa06501c0d9808 Author: Gary Yao AuthorDate: Mon May 18 09:17:58 2020 +0200 [FLINK-17792][tests] Catch and log exception if jstack fails jstack can fail if the JVM process that we want to sample exits while or before we invoke jstack. Since a JVM process is free to exit at any time, we should not propagate the exception so that we do not fail the test prematurely. --- flink-jepsen/src/jepsen/flink/utils.clj | 5 - 1 file changed, 4 insertions(+), 1 deletion(-) diff --git a/flink-jepsen/src/jepsen/flink/utils.clj b/flink-jepsen/src/jepsen/flink/utils.clj index 8f6f654..5d8e712 100644 --- a/flink-jepsen/src/jepsen/flink/utils.clj +++ b/flink-jepsen/src/jepsen/flink/utils.clj @@ -133,7 +133,10 @@ (defn- write-jstack! [pid out-path] - (c/exec :jstack :-l pid :> out-path)) + (try +(c/exec :jstack :-l pid :> out-path) +(catch Exception e + (warn e "Failed to invoke jstack on pid" pid (defn dump-jstack-by-pattern! "Dumps the output of jstack for all JVMs that match one of the specified patterns."
[flink] branch master updated (43c3bfa -> 417936d)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 43c3bfa [FLINK-17730][CI] Increase 'no output timeout' to 15 minutes add 417936d [FLINK-17792][tests] Catch and log exception if jstack fails No new revisions were added by this update. Summary of changes: flink-jepsen/src/jepsen/flink/utils.clj | 5 - 1 file changed, 4 insertions(+), 1 deletion(-)
[flink] branch master updated (547c168 -> b606cba)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 547c168 [FLINK-14255][hive] Integrate hive to streaming file sink new 95f25b2 [hotfix][tests] Reformat default flink configuration map new fa5206a [hotfix][tests] Add missing space to help text new cd491ea [hotfix][tests] Add option -m PEM to ssh-keygen new b606cba [FLINK-17522][tests] Add details on how to run Jepsen tests The 4 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: flink-jepsen/README.md | 148 +--- flink-jepsen/docker/up.sh | 2 +- flink-jepsen/src/jepsen/flink/db.clj| 30 +++ flink-jepsen/src/jepsen/flink/flink.clj | 4 +- 4 files changed, 154 insertions(+), 30 deletions(-)
[flink] 01/04: [hotfix][tests] Reformat default flink configuration map
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit 95f25b278ebe21362a9eb3453c33878c5a822402 Author: Gary Yao AuthorDate: Wed May 6 11:23:14 2020 +0200 [hotfix][tests] Reformat default flink configuration map --- flink-jepsen/src/jepsen/flink/db.clj | 30 +++--- 1 file changed, 15 insertions(+), 15 deletions(-) diff --git a/flink-jepsen/src/jepsen/flink/db.clj b/flink-jepsen/src/jepsen/flink/db.clj index d4d46f3..df04765 100644 --- a/flink-jepsen/src/jepsen/flink/db.clj +++ b/flink-jepsen/src/jepsen/flink/db.clj @@ -38,21 +38,21 @@ (defn- default-flink-configuration [test node] - {:high-availability "zookeeper" - :high-availability.zookeeper.quorum(zookeeper-quorum test) - :high-availability.storageDir "hdfs:///flink/ha" - :jobmanager.memory.process.size"2048m" - :jobmanager.rpc.addressnode - :state.savepoints.dir "hdfs:///flink/savepoints" - :rest.address node - :rest.port 8081 - :rest.bind-address "0.0.0.0" - :taskmanager.numberOfTaskSlots taskmanager-slots - :yarn.application-attempts 9 - :slotmanager.taskmanager-timeout 1 - :state.backend.local-recovery "true" - :taskmanager.memory.process.size "2048m" - :taskmanager.registration.timeout "30 s"}) + {:high-availability "zookeeper" + :high-availability.zookeeper.quorum (zookeeper-quorum test) + :high-availability.storageDir "hdfs:///flink/ha" + :jobmanager.memory.process.size "2048m" + :jobmanager.rpc.address node + :state.savepoints.dir "hdfs:///flink/savepoints" + :rest.address node + :rest.port 8081 + :rest.bind-address "0.0.0.0" + :taskmanager.numberOfTaskSlots taskmanager-slots + :yarn.application-attempts 9 + :slotmanager.taskmanager-timeout1 + :state.backend.local-recovery "true" + :taskmanager.memory.process.size"2048m" + :taskmanager.registration.timeout "30 s"}) (defn flink-configuration [test node]
[flink] 02/04: [hotfix][tests] Add missing space to help text
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit fa5206ae7ad43948ba479eeba947c8b5e5ddd831 Author: Gary Yao AuthorDate: Wed May 6 11:40:10 2020 +0200 [hotfix][tests] Add missing space to help text --- flink-jepsen/src/jepsen/flink/flink.clj | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/flink-jepsen/src/jepsen/flink/flink.clj b/flink-jepsen/src/jepsen/flink/flink.clj index d70cc5e..33deed9 100644 --- a/flink-jepsen/src/jepsen/flink/flink.clj +++ b/flink-jepsen/src/jepsen/flink/flink.clj @@ -120,13 +120,13 @@ :parse-fn read-test-spec :validate [#(->> % :dbs (map dbs) (every? (complement nil?))) (str "Invalid :dbs specification. " (keys->allowed-values-help-text dbs))]] -[nil "--nemesis-gen GEN" (str "Which nemesis should be used?" +[nil "--nemesis-gen GEN" (str "Which nemesis should be used? " (keys->allowed-values-help-text fn/nemesis-generator-factories)) :parse-fn keyword :default :kill-task-managers :validate [#(fn/nemesis-generator-factories %) (keys->allowed-values-help-text fn/nemesis-generator-factories)]] -[nil "--client-gen GEN" (str "Which client should be used?" +[nil "--client-gen GEN" (str "Which client should be used? " (keys->allowed-values-help-text client-gens)) :parse-fn keyword :default :poll-job-running
[flink] 04/04: [FLINK-17522][tests] Add details on how to run Jepsen tests
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit b606cbaf36f9e206f44243dfdc7e8005e92d2d66 Author: Gary Yao AuthorDate: Tue May 5 13:17:17 2020 +0200 [FLINK-17522][tests] Add details on how to run Jepsen tests Document command line options for the 'lein run test' command. In the Docker section, add that DataStreamAllroundTestProgram must be built first. Add details about types/roles of nodes and minimum number of nodes required. This closes #12019. --- flink-jepsen/README.md | 148 + 1 file changed, 136 insertions(+), 12 deletions(-) diff --git a/flink-jepsen/README.md b/flink-jepsen/README.md index 5694b74..ddf6ad7 100644 --- a/flink-jepsen/README.md +++ b/flink-jepsen/README.md @@ -21,24 +21,68 @@ semantics. ## Usage ### Setting up the Environment -See the [Jepsen documentation](https://github.com/jepsen-io/jepsen#setting-up-a-jepsen-environment) -for details on how to set up the environment required to run the tests. +A Jepsen test is started and orchestrated by a Clojure program running on a _control node_. +That program logs into several specified _DB nodes_ to set up a Flink cluster +and submit test job(s). Afterwards, the control node will inject faults into the DB nodes +and continue to interact with the Flink cluster (for example by probing the status of the test job). +Tests in this project require at least 3 DB nodes; this is due to [how we install +software](#test-spec), such as HDFS, on the DB nodes. There is no upper limit on the number of DB nodes. + +For details on how to set up the environment required to run the tests, +see the [Jepsen documentation](https://github.com/jepsen-io/jepsen#setting-up-a-jepsen-environment). To simplify development, we have prepared Dockerfiles and a [Docker Compose](https://docs.docker.com/compose/) template so that you can run the tests locally in containers (see Section [Docker](#usage-docker)). ### Running Tests -This project does not comprise of only a single test that can be run but rather a parameterizable -test template. This allows the user to specify the cluster manager that Flink should be on, the -location of the high availability storage directory, the jobs to be submitted, etc. -The script under `docker/run-tests.sh` shows examples on how to specify and run tests. -By default, the example tests run the `DataStreamAllroundTestProgram`, which is located under -`flink-end-to-end-tests/flink-datastream-allround-test` of the Flink project root. -Before running the tests, you have to build the job first, and copy the resulting jar -(`DataStreamAllroundTestProgram.jar`) to the `./bin` directory of this project's root. -Also included in the examples is a more complicated scenario with two jobs that share a Kafka -topic. See the `run-tests.sh` script for details on how to enable and run this test. +Because tests are started from the control node, the contents of this directory +must be copied to the control node first. An example for a command that will run a test is + +```bash +lein run test --tarball http:// \ +--ssh-private-key ~/.ssh/id_rsa \ +--nodes-file nodes \ +--nemesis-gen kill-task-managers\ +--test-spec test-specs/yarn-session.edn +``` + +Here is a breakdown of the command line options and their arguments: + +* `--tarball` A URL to the Flink distribution under test +* `--ssh-private-key` Path to the private key that will be used to connect to the DB nodes (from the control node) +* `--nodes-file` Path to a file containing newline separated IPs or hostnames of the DB nodes +* `--nemesis-gen` Specifies the faults that will be injected. `kill-task-managers` means for the duration of the test (default 60s), +TaskManagers will be killed on all DB nodes with a uniform random timing noise with a mean delay of 3 seconds +(delay ranging from 0 to 6 seconds). +* `--test-spec` Path to an [edn](https://github.com/edn-format/edn) formatted file +containing a specification about which software to install and which jobs to submit. For example, by using the file below, +we would request the framework to install Hadoop (YARN, HDFS), ZooKeeper, and deploy Flink as a session cluster on YARN. +Furthermore, we instruct to submit the `DataStreamAllroundTestProgram` to Flink with the specified job arguments. + ``` + {:dbs [:hadoop :zookeeper :flink-yarn-session] + :jobs [{:job-jar "/path/to/DataStreamAllroundTestProgram.jar" + :job-args "--environment.parallelism 1 --state_backend.checkpoint_directory hdfs:///checkpoints --state_backend rocks --state_backend.rocks.incremental true"}]} + ``` + +For more details on the command line options, see Section [Command Line Options & Configuratio
[flink] 03/04: [hotfix][tests] Add option -m PEM to ssh-keygen
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit cd491ea303502475d06dfc498e3a27b2c945ee40 Author: Gary Yao AuthorDate: Thu May 7 09:46:05 2020 +0200 [hotfix][tests] Add option -m PEM to ssh-keygen Make ssh-keygen output private key in PEM format so that Jsch does not complain about wrong private key format. --- flink-jepsen/docker/up.sh | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/flink-jepsen/docker/up.sh b/flink-jepsen/docker/up.sh index 5479b3b..d3fbf05 100755 --- a/flink-jepsen/docker/up.sh +++ b/flink-jepsen/docker/up.sh @@ -23,7 +23,7 @@ dockerdir=$(dirname $0) dockerdir=$(cd ${dockerdir}; pwd) if [ ! -f ./id_rsa ]; then -ssh-keygen -t rsa -N "" -f ./id_rsa +ssh-keygen -m PEM -t rsa -N "" -f ./id_rsa fi export JEPSEN_ROOT=${dockerdir}/../
[flink] branch master updated (87c9e4d -> 56ea8d1)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 87c9e4d [hotfix][docs] Fix and improve query configuration docs. add 56ea8d1 [FLINK-17616][tests] Temporarily increase akka.ask.timeout in TPC-DS e2e test No new revisions were added by this update. Summary of changes: flink-end-to-end-tests/test-scripts/test_tpcds.sh | 1 + 1 file changed, 1 insertion(+)
[flink] branch master updated: [FLINK-17536][core] Change the config option for slot max limitation to slotmanager.number-of-slots.max
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/master by this push: new 9888a87 [FLINK-17536][core] Change the config option for slot max limitation to slotmanager.number-of-slots.max 9888a87 is described below commit 9888a87495827b619f5b49dae5ad29a34931d0a9 Author: Yangze Guo AuthorDate: Mon May 11 13:59:23 2020 +0800 [FLINK-17536][core] Change the config option for slot max limitation to slotmanager.number-of-slots.max This closes #12067. --- docs/_includes/generated/expert_scheduling_section.html | 2 +- docs/_includes/generated/resource_manager_configuration.html| 2 +- .../java/org/apache/flink/configuration/ResourceManagerOptions.java | 2 +- 3 files changed, 3 insertions(+), 3 deletions(-) diff --git a/docs/_includes/generated/expert_scheduling_section.html b/docs/_includes/generated/expert_scheduling_section.html index 5991224..9020268 100644 --- a/docs/_includes/generated/expert_scheduling_section.html +++ b/docs/_includes/generated/expert_scheduling_section.html @@ -27,7 +27,7 @@ The timeout in milliseconds for requesting a slot from Slot Pool. -slotmanager.max-number-of-slots +slotmanager.number-of-slots.max 2147483647 Integer Defines the maximum number of slots that the Flink cluster allocates. This configuration option is meant for limiting the resource consumption for batch workloads. It is not recommended to configure this option for streaming workloads, which may fail if there are not enough slots. Note that this configuration option does not take effect for standalone clusters, where how many slots are allocated is not controlled by Flink. diff --git a/docs/_includes/generated/resource_manager_configuration.html b/docs/_includes/generated/resource_manager_configuration.html index 5ab8e35..03090e9 100644 --- a/docs/_includes/generated/resource_manager_configuration.html +++ b/docs/_includes/generated/resource_manager_configuration.html @@ -33,7 +33,7 @@ The timeout for an idle task manager to be released. -slotmanager.max-number-of-slots +slotmanager.number-of-slots.max 2147483647 Integer Defines the maximum number of slots that the Flink cluster allocates. This configuration option is meant for limiting the resource consumption for batch workloads. It is not recommended to configure this option for streaming workloads, which may fail if there are not enough slots. Note that this configuration option does not take effect for standalone clusters, where how many slots are allocated is not controlled by Flink. diff --git a/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java b/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java index f6342da..1ba2699 100644 --- a/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java +++ b/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java @@ -59,7 +59,7 @@ public class ResourceManagerOptions { @Documentation.Section(Documentation.Sections.EXPERT_SCHEDULING) public static final ConfigOption MAX_SLOT_NUM = ConfigOptions - .key("slotmanager.max-number-of-slots") + .key("slotmanager.number-of-slots.max") .intType() .defaultValue(Integer.MAX_VALUE) .withDescription("Defines the maximum number of slots that the Flink cluster allocates. This configuration option " +
[flink] branch master updated (d7525ba -> 74b850cd)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from d7525ba [FLINK-16536][network][checkpointing] Implement InputChannel state recovery for unaligned checkpoint add 74b850cd [FLINK-17130][web] Enable listing JM logs and displaying logs by filename No new revisions were added by this update. Summary of changes: .../job-manager/job-manager-routing.module.ts | 16 + .../app/pages/job-manager/job-manager.component.ts | 3 +- .../app/pages/job-manager/job-manager.module.ts| 4 ++ .../job-manager-log-detail.component.html} | 18 - .../job-manager-log-detail.component.less} | 36 +++--- .../log-detail/job-manager-log-detail.component.ts | 78 ++ .../log-list/job-manager-log-list.component.html | 34 +++--- .../job-manager-log-list.component.ts} | 26 +--- .../task-manager-log-detail.component.less | 8 ++- .../src/app/services/job-manager.service.ts| 27 10 files changed, 219 insertions(+), 31 deletions(-) copy flink-runtime-web/web-dashboard/src/app/pages/job-manager/{logs/job-manager-logs.component.html => log-detail/job-manager-log-detail.component.html} (64%) copy flink-runtime-web/web-dashboard/src/app/pages/{job/timeline/job-timeline.component.less => job-manager/log-detail/job-manager-log-detail.component.less} (65%) create mode 100644 flink-runtime-web/web-dashboard/src/app/pages/job-manager/log-detail/job-manager-log-detail.component.ts copy flink-examples/flink-examples-batch/src/main/resources/logback.xml => flink-runtime-web/web-dashboard/src/app/pages/job-manager/log-list/job-manager-log-list.component.html (60%) copy flink-runtime-web/web-dashboard/src/app/pages/job-manager/{configuration/job-manager-configuration.component.ts => log-list/job-manager-log-list.component.ts} (68%)
[flink] branch master updated: [FLINK-17608][web] Add TM log and stdout page/tab back
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/master by this push: new 7cfcd33 [FLINK-17608][web] Add TM log and stdout page/tab back 7cfcd33 is described below commit 7cfcd33e983c6e07eedf8c0d5514450a565710ff Author: vthinkxie AuthorDate: Mon May 11 20:06:13 2020 +0800 [FLINK-17608][web] Add TM log and stdout page/tab back This closes #12085. --- .../task-manager-log-detail.component.ts | 10 + .../task-manager-logs.component.html} | 5 ++- .../logs/task-manager-logs.component.less | 28 ++ .../task-manager-logs.component.ts}| 44 +- .../status/task-manager-status.component.ts| 8 +++- .../task-manager-stdout.component.html}| 4 +- .../stdout/task-manager-stdout.component.less | 28 ++ .../task-manager-stdout.component.ts} | 44 +- .../task-manager/task-manager-routing.module.ts| 17 ++--- .../app/pages/task-manager/task-manager.module.ts | 6 ++- .../task-manager-thread-dump.component.html| 2 +- .../src/app/services/task-manager.service.ts | 42 - 12 files changed, 174 insertions(+), 64 deletions(-) diff --git a/flink-runtime-web/web-dashboard/src/app/pages/task-manager/log-detail/task-manager-log-detail.component.ts b/flink-runtime-web/web-dashboard/src/app/pages/task-manager/log-detail/task-manager-log-detail.component.ts index f589122..4226a87 100644 --- a/flink-runtime-web/web-dashboard/src/app/pages/task-manager/log-detail/task-manager-log-detail.component.ts +++ b/flink-runtime-web/web-dashboard/src/app/pages/task-manager/log-detail/task-manager-log-detail.component.ts @@ -37,7 +37,6 @@ export class TaskManagerLogDetailComponent implements OnInit { isLoading = false; taskManagerDetail: TaskManagerDetailInterface; isFullScreen = false; - hasLogName = false; @ViewChild(MonacoEditorComponent) monacoEditorComponent: MonacoEditorComponent; constructor( @@ -49,7 +48,7 @@ export class TaskManagerLogDetailComponent implements OnInit { reloadLog() { this.isLoading = true; this.cdr.markForCheck(); -this.taskManagerService.loadLog(this.taskManagerDetail.id, this.logName, this.hasLogName).subscribe( +this.taskManagerService.loadLog(this.taskManagerDetail.id, this.logName).subscribe( data => { this.logs = data.data; this.downloadUrl = data.url; @@ -77,12 +76,7 @@ export class TaskManagerLogDetailComponent implements OnInit { ngOnInit() { this.taskManagerService.taskManagerDetail$.pipe(first()).subscribe(data => { this.taskManagerDetail = data; - this.hasLogName = this.activatedRoute.snapshot.data.hasLogName; - if (this.hasLogName) { -this.logName = this.activatedRoute.snapshot.params.logName; - } else { -this.logName = `taskmanager_${data.id}_log`; - } + this.logName = this.activatedRoute.snapshot.params.logName; this.reloadLog(); }); } diff --git a/flink-runtime-web/web-dashboard/src/app/pages/task-manager/thread-dump/task-manager-thread-dump.component.html b/flink-runtime-web/web-dashboard/src/app/pages/task-manager/logs/task-manager-logs.component.html similarity index 75% copy from flink-runtime-web/web-dashboard/src/app/pages/task-manager/thread-dump/task-manager-thread-dump.component.html copy to flink-runtime-web/web-dashboard/src/app/pages/task-manager/logs/task-manager-logs.component.html index 096b6b9..8c07727 100644 --- a/flink-runtime-web/web-dashboard/src/app/pages/task-manager/thread-dump/task-manager-thread-dump.component.html +++ b/flink-runtime-web/web-dashboard/src/app/pages/task-manager/logs/task-manager-logs.component.html @@ -16,5 +16,6 @@ ~ limitations under the License. --> - - + + + diff --git a/flink-runtime-web/web-dashboard/src/app/pages/task-manager/logs/task-manager-logs.component.less b/flink-runtime-web/web-dashboard/src/app/pages/task-manager/logs/task-manager-logs.component.less new file mode 100644 index 000..df80525 --- /dev/null +++ b/flink-runtime-web/web-dashboard/src/app/pages/task-manager/logs/task-manager-logs.component.less @@ -0,0 +1,28 @@ +/* + * 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, softw
[flink] branch master updated (ce6b97e -> fd0ef6e)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from ce6b97e [FLINK-17567][python][release] Create a dedicated Python directory in release directory to place Python-related source and binary packages add cba4845 [FLINK-17369][tests] In RestartPipelinedRegionFailoverStrategyBuildingTest invoke PipelinedRegionComputeUtil directly add 56cc76b [FLINK-17369][tests] Rename RestartPipelinedRegionFailoverStrategyBuildingTest to PipelinedRegionComputeUtilTest add fd0ef6e [FLINK-17369][tests] Reduce visiblity of internal test methods No new revisions were added by this update. Summary of changes: ...st.java => PipelinedRegionComputeUtilTest.java} | 174 - 1 file changed, 98 insertions(+), 76 deletions(-) rename flink-runtime/src/test/java/org/apache/flink/runtime/executiongraph/failover/flip1/{RestartPipelinedRegionFailoverStrategyBuildingTest.java => PipelinedRegionComputeUtilTest.java} (65%)
[flink] branch master updated (fdd128a -> c2540e4)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from fdd128a [FLINK-17514] Harden ExceptionUtils.tryEnrichTaskManagerError, .isMetaspaceOutOfMemoryError and .isDirectOutOfMemoryError to handle null values add c2540e4 [FLINK-17501][qs] Improve logging in AbstractServerHandler#channelRead(ChannelHandlerContext, Object) No new revisions were added by this update. Summary of changes: .../flink/queryablestate/network/AbstractServerHandler.java | 9 +++-- 1 file changed, 7 insertions(+), 2 deletions(-)
[flink] branch master updated (fdd128a -> c2540e4)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from fdd128a [FLINK-17514] Harden ExceptionUtils.tryEnrichTaskManagerError, .isMetaspaceOutOfMemoryError and .isDirectOutOfMemoryError to handle null values add c2540e4 [FLINK-17501][qs] Improve logging in AbstractServerHandler#channelRead(ChannelHandlerContext, Object) No new revisions were added by this update. Summary of changes: .../flink/queryablestate/network/AbstractServerHandler.java | 9 +++-- 1 file changed, 7 insertions(+), 2 deletions(-)
[flink] branch master updated (28185fc -> d385f20)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 28185fc [FLINK-17423][python] Support Java UDTF and UDAF in Blink planner under batch mode for PyFlink add d385f20 [FLINK-17473][tests] Remove unused test utilities No new revisions were added by this update. Summary of changes: .../legacy/utils/ArchivedExecutionBuilder.java | 157 - .../utils/ArchivedExecutionJobVertexBuilder.java | 86 --- .../utils/ArchivedExecutionVertexBuilder.java | 73 -- 3 files changed, 316 deletions(-) delete mode 100644 flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/legacy/utils/ArchivedExecutionBuilder.java delete mode 100644 flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/legacy/utils/ArchivedExecutionJobVertexBuilder.java delete mode 100644 flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/legacy/utils/ArchivedExecutionVertexBuilder.java
[flink] 04/05: [FLINK-16605][runtime] Make the SlotManager respect the max limitation for slots
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit dcf9cc601f6ee1bb90a5d548043564a1a0522a25 Author: Yangze Guo AuthorDate: Tue Apr 28 16:02:14 2020 +0800 [FLINK-16605][runtime] Make the SlotManager respect the max limitation for slots This closes #11615. --- .../slotmanager/SlotManagerImpl.java | 64 +++- .../flink/runtime/taskexecutor/SlotReport.java | 4 + .../slotmanager/SlotManagerBuilder.java| 8 ++ .../slotmanager/SlotManagerImplTest.java | 88 +- 4 files changed, 156 insertions(+), 8 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerImpl.java b/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerImpl.java index 1a55df0..b02af3f 100755 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerImpl.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerImpl.java @@ -54,11 +54,13 @@ import javax.annotation.Nullable; import java.util.ArrayList; import java.util.Collections; import java.util.HashMap; +import java.util.HashSet; import java.util.Iterator; import java.util.LinkedHashMap; import java.util.Map; import java.util.Objects; import java.util.Optional; +import java.util.Set; import java.util.concurrent.CancellationException; import java.util.concurrent.CompletableFuture; import java.util.concurrent.Executor; @@ -436,6 +438,12 @@ public class SlotManagerImpl implements SlotManager { if (taskManagerRegistrations.containsKey(taskExecutorConnection.getInstanceID())) { reportSlotStatus(taskExecutorConnection.getInstanceID(), initialSlotReport); } else { + if (isMaxSlotNumExceededAfterRegistration(initialSlotReport)) { + LOG.info("The total number of slots exceeds the max limitation {}, release the excess resource.", maxSlotNum); + resourceActions.releaseResource(taskExecutorConnection.getInstanceID(), new FlinkException("The total number of slots exceeds the max limitation.")); + return; + } + // first register the TaskManager ArrayList reportedSlots = new ArrayList<>(); @@ -557,7 +565,7 @@ public class SlotManagerImpl implements SlotManager { if (pendingSlotRequest.getAssignedPendingTaskManagerSlot() != null) { continue; } - if (!isFulfillableByRegisteredSlots(pendingSlotRequest.getResourceProfile())) { + if (!isFulfillableByRegisteredOrPendingSlots(pendingSlotRequest.getResourceProfile())) { slotRequestIterator.remove(); resourceActions.notifyAllocationFailure( pendingSlotRequest.getJobId(), @@ -696,7 +704,7 @@ public class SlotManagerImpl implements SlotManager { @Nullable private PendingTaskManagerSlot findExactlyMatchingPendingTaskManagerSlot(ResourceProfile resourceProfile) { for (PendingTaskManagerSlot pendingTaskManagerSlot : pendingSlots.values()) { - if (pendingTaskManagerSlot.getResourceProfile().equals(resourceProfile)) { + if (isPendingSlotExactlyMatchingResourceProfile(pendingTaskManagerSlot, resourceProfile)) { return pendingTaskManagerSlot; } } @@ -704,6 +712,35 @@ public class SlotManagerImpl implements SlotManager { return null; } + private boolean isPendingSlotExactlyMatchingResourceProfile(PendingTaskManagerSlot pendingTaskManagerSlot, ResourceProfile resourceProfile) { + return pendingTaskManagerSlot.getResourceProfile().equals(resourceProfile); + } + + private boolean isMaxSlotNumExceededAfterRegistration(SlotReport initialSlotReport) { + // check if the total number exceed before matching pending slot. + if (!isMaxSlotNumExceededAfterAdding(initialSlotReport.getNumSlotStatus())) { + return false; + } + + // check if the total number exceed slots after consuming pending slot. + return isMaxSlotNumExceededAfterAdding(getNumNonPendingReportedNewSlots(initialSlotReport)); + } + + private int getNumNonPendingReportedNewSlots(SlotReport slotReport) { + final Set matchingP
[flink] branch master updated (cc9b6ea -> f87e120)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from cc9b6ea [FLINK-17455][table][filesystem] Move FileSystemFormatFactory to table common new 9cb1ebd [hotfix][runtime] Add sanity check to SlotManagerConfiguration new 026a2b6 [FLINK-16605][core][config] Add slotmanager.max-number-of-slots config option new 9e69b27 [FLINK-16605][runtime] Pass the slotmanager.max-number-of-slots to the SlotManagerImpl new dcf9cc6 [FLINK-16605][runtime] Make the SlotManager respect the max limitation for slots new f87e120 [hotfix][tests] Deduplicate code in SlotManagerImplTest The 5 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../generated/expert_scheduling_section.html | 6 ++ .../generated/resource_manager_configuration.html | 6 ++ .../configuration/ResourceManagerOptions.java | 11 ++ .../StandaloneResourceManagerFactory.java | 25 - .../slotmanager/SlotManagerConfiguration.java | 20 +++- .../slotmanager/SlotManagerImpl.java | 68 +++- .../flink/runtime/taskexecutor/SlotReport.java | 4 + .../resourcemanager/ResourceManagerHATest.java | 4 +- .../slotmanager/SlotManagerBuilder.java| 19 +++- .../slotmanager/SlotManagerImplTest.java | 117 - 10 files changed, 244 insertions(+), 36 deletions(-)
[flink] 02/05: [FLINK-16605][core][config] Add slotmanager.max-number-of-slots config option
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit 026a2b6d8ed3aab5bc29d998ba6e585fa5b2d9ef Author: Yangze Guo AuthorDate: Mon Mar 30 11:05:25 2020 +0800 [FLINK-16605][core][config] Add slotmanager.max-number-of-slots config option --- docs/_includes/generated/expert_scheduling_section.html | 6 ++ docs/_includes/generated/resource_manager_configuration.html | 6 ++ .../apache/flink/configuration/ResourceManagerOptions.java| 11 +++ 3 files changed, 23 insertions(+) diff --git a/docs/_includes/generated/expert_scheduling_section.html b/docs/_includes/generated/expert_scheduling_section.html index a29ad69..5991224 100644 --- a/docs/_includes/generated/expert_scheduling_section.html +++ b/docs/_includes/generated/expert_scheduling_section.html @@ -26,5 +26,11 @@ Long The timeout in milliseconds for requesting a slot from Slot Pool. + +slotmanager.max-number-of-slots +2147483647 +Integer +Defines the maximum number of slots that the Flink cluster allocates. This configuration option is meant for limiting the resource consumption for batch workloads. It is not recommended to configure this option for streaming workloads, which may fail if there are not enough slots. Note that this configuration option does not take effect for standalone clusters, where how many slots are allocated is not controlled by Flink. + diff --git a/docs/_includes/generated/resource_manager_configuration.html b/docs/_includes/generated/resource_manager_configuration.html index cb79c38..5ab8e35 100644 --- a/docs/_includes/generated/resource_manager_configuration.html +++ b/docs/_includes/generated/resource_manager_configuration.html @@ -32,5 +32,11 @@ Long The timeout for an idle task manager to be released. + +slotmanager.max-number-of-slots +2147483647 +Integer +Defines the maximum number of slots that the Flink cluster allocates. This configuration option is meant for limiting the resource consumption for batch workloads. It is not recommended to configure this option for streaming workloads, which may fail if there are not enough slots. Note that this configuration option does not take effect for standalone clusters, where how many slots are allocated is not controlled by Flink. + diff --git a/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java b/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java index 4af839f..f6342da 100644 --- a/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java +++ b/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java @@ -19,6 +19,7 @@ package org.apache.flink.configuration; import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.docs.Documentation; import org.apache.flink.configuration.description.Description; /** @@ -56,6 +57,16 @@ public class ResourceManagerOptions { " default, the port of the JobManager, because the same ActorSystem is used." + " Its not possible to use this configuration key to define port ranges."); + @Documentation.Section(Documentation.Sections.EXPERT_SCHEDULING) + public static final ConfigOption MAX_SLOT_NUM = ConfigOptions + .key("slotmanager.max-number-of-slots") + .intType() + .defaultValue(Integer.MAX_VALUE) + .withDescription("Defines the maximum number of slots that the Flink cluster allocates. This configuration option " + + "is meant for limiting the resource consumption for batch workloads. It is not recommended to configure this option " + + "for streaming workloads, which may fail if there are not enough slots. Note that this configuration option does not take " + + "effect for standalone clusters, where how many slots are allocated is not controlled by Flink."); + /** * The timeout for a slot request to be discarded, in milliseconds. * @deprecated Use {@link JobManagerOptions#SLOT_REQUEST_TIMEOUT}.
[flink] 03/05: [FLINK-16605][runtime] Pass the slotmanager.max-number-of-slots to the SlotManagerImpl
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit 9e69b270c8b192876dae128541aa73ae6e788e2f Author: Yangze Guo AuthorDate: Thu Apr 9 14:26:41 2020 +0800 [FLINK-16605][runtime] Pass the slotmanager.max-number-of-slots to the SlotManagerImpl --- .../StandaloneResourceManagerFactory.java | 25 +- .../slotmanager/SlotManagerConfiguration.java | 15 +++-- .../slotmanager/SlotManagerImpl.java | 4 .../resourcemanager/ResourceManagerHATest.java | 4 +++- .../slotmanager/SlotManagerBuilder.java| 11 +- 5 files changed, 54 insertions(+), 5 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/StandaloneResourceManagerFactory.java b/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/StandaloneResourceManagerFactory.java index c79a7dd..135276b 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/StandaloneResourceManagerFactory.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/StandaloneResourceManagerFactory.java @@ -21,6 +21,7 @@ package org.apache.flink.runtime.resourcemanager; import org.apache.flink.api.common.time.Time; import org.apache.flink.configuration.Configuration; import org.apache.flink.configuration.ConfigurationUtils; +import org.apache.flink.configuration.ResourceManagerOptions; import org.apache.flink.runtime.akka.AkkaUtils; import org.apache.flink.runtime.clusterframework.types.ResourceID; import org.apache.flink.runtime.entrypoint.ClusterInformation; @@ -32,6 +33,9 @@ import org.apache.flink.runtime.rpc.FatalErrorHandler; import org.apache.flink.runtime.rpc.RpcService; import org.apache.flink.util.ConfigurationException; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + import javax.annotation.Nullable; /** @@ -47,6 +51,8 @@ public final class StandaloneResourceManagerFactory extends ResourceManagerFacto return INSTANCE; } + private static final Logger LOG = LoggerFactory.getLogger(ConfigurationUtils.class); + @Override protected ResourceManager createResourceManager( Configuration configuration, @@ -80,6 +86,23 @@ public final class StandaloneResourceManagerFactory extends ResourceManagerFacto @Override protected ResourceManagerRuntimeServicesConfiguration createResourceManagerRuntimeServicesConfiguration( Configuration configuration) throws ConfigurationException { - return ResourceManagerRuntimeServicesConfiguration.fromConfiguration(configuration, ArbitraryWorkerResourceSpecFactory.INSTANCE); + return ResourceManagerRuntimeServicesConfiguration + .fromConfiguration(getConfigurationWithoutMaxSlotNumberIfSet(configuration), ArbitraryWorkerResourceSpecFactory.INSTANCE); + } + + /** +* Get the configuration for standalone ResourceManager, overwrite invalid configs. +* +* @param configuration configuration object +* @return the configuration for standalone ResourceManager +*/ + private static Configuration getConfigurationWithoutMaxSlotNumberIfSet(Configuration configuration) { + final Configuration copiedConfig = new Configuration(configuration); + // The max slot limit should not take effect for standalone cluster, we overwrite the configure in case user + // sets this value by mistake. + if (copiedConfig.removeConfig(ResourceManagerOptions.MAX_SLOT_NUM)) { + LOG.warn("Config option {} will be ignored in standalone mode.", ResourceManagerOptions.MAX_SLOT_NUM.key()); + } + return copiedConfig; } } diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java b/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java index 0da0f31..5647129 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java @@ -47,6 +47,7 @@ public class SlotManagerConfiguration { private final SlotMatchingStrategy slotMatchingStrategy; private final WorkerResourceSpec defaultWorkerResourceSpec; private final int numSlotsPerWorker; + private final int maxSlotNum; public SlotManagerConfiguration( Time taskManagerRequestTimeout, @@ -55,7 +56,8 @@ public class SlotManagerConfiguration { boolean waitResultConsumedBef
[flink] 01/05: [hotfix][runtime] Add sanity check to SlotManagerConfiguration
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit 9cb1ebdf8947d51134476601b215064c699127fc Author: Yangze Guo AuthorDate: Mon Apr 13 19:09:09 2020 +0800 [hotfix][runtime] Add sanity check to SlotManagerConfiguration --- .../resourcemanager/slotmanager/SlotManagerConfiguration.java| 5 +++-- 1 file changed, 3 insertions(+), 2 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java b/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java index 470e838..0da0f31 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java @@ -61,8 +61,9 @@ public class SlotManagerConfiguration { this.slotRequestTimeout = Preconditions.checkNotNull(slotRequestTimeout); this.taskManagerTimeout = Preconditions.checkNotNull(taskManagerTimeout); this.waitResultConsumedBeforeRelease = waitResultConsumedBeforeRelease; - this.slotMatchingStrategy = slotMatchingStrategy; - this.defaultWorkerResourceSpec = defaultWorkerResourceSpec; + this.slotMatchingStrategy = Preconditions.checkNotNull(slotMatchingStrategy); + this.defaultWorkerResourceSpec = Preconditions.checkNotNull(defaultWorkerResourceSpec); + Preconditions.checkState(numSlotsPerWorker > 0); this.numSlotsPerWorker = numSlotsPerWorker; }
[flink] 05/05: [hotfix][tests] Deduplicate code in SlotManagerImplTest
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit f87e12004c8fb9491a2f983414f7626fac025e3a Author: Yangze Guo AuthorDate: Tue Apr 28 16:08:51 2020 +0800 [hotfix][tests] Deduplicate code in SlotManagerImplTest --- .../slotmanager/SlotManagerImplTest.java | 29 ++ 1 file changed, 8 insertions(+), 21 deletions(-) diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerImplTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerImplTest.java index 2f4ddef..25e5220 100755 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerImplTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerImplTest.java @@ -27,7 +27,6 @@ import org.apache.flink.runtime.clusterframework.types.AllocationID; import org.apache.flink.runtime.clusterframework.types.ResourceID; import org.apache.flink.runtime.clusterframework.types.ResourceProfile; import org.apache.flink.runtime.clusterframework.types.SlotID; -import org.apache.flink.runtime.concurrent.Executors; import org.apache.flink.runtime.concurrent.FutureUtils; import org.apache.flink.runtime.concurrent.ScheduledExecutor; import org.apache.flink.runtime.instance.InstanceID; @@ -688,8 +687,7 @@ public class SlotManagerImplTest extends TestLogger { final Executor mainThreadExecutor = TestingUtils.defaultExecutor(); - try (SlotManager slotManager = SlotManagerBuilder.newBuilder() - .setDefaultWorkerResourceSpec(WORKER_RESOURCE_SPEC) + try (SlotManager slotManager = createSlotManagerBuilder() .setSlotRequestTimeout(Time.milliseconds(allocationTimeout)) .build()) { @@ -817,8 +815,7 @@ public class SlotManagerImplTest extends TestLogger { final ScheduledExecutor mainThreadExecutor = TestingUtils.defaultScheduledExecutor(); - final SlotManagerImpl slotManager = SlotManagerBuilder - .newBuilder() + final SlotManagerImpl slotManager = createSlotManagerBuilder() .setScheduledExecutor(mainThreadExecutor) .build(); @@ -909,8 +906,7 @@ public class SlotManagerImplTest extends TestLogger { final Executor mainThreadExecutor = TestingUtils.defaultExecutor(); - try (final SlotManagerImpl slotManager = SlotManagerBuilder.newBuilder() - .setDefaultWorkerResourceSpec(WORKER_RESOURCE_SPEC) + try (final SlotManagerImpl slotManager = createSlotManagerBuilder() .setTaskManagerTimeout(Time.of(taskManagerTimeout, TimeUnit.MILLISECONDS)) .build()) { @@ -978,12 +974,9 @@ public class SlotManagerImplTest extends TestLogger { final SlotStatus slotStatus = createEmptySlotStatus(new SlotID(resourceID, 0), ResourceProfile.fromResources(1.0, 1)); final SlotReport initialSlotReport = new SlotReport(slotStatus); - try (final SlotManager slotManager = SlotManagerBuilder.newBuilder() - .setDefaultWorkerResourceSpec(WORKER_RESOURCE_SPEC) + try (final SlotManager slotManager = createSlotManagerBuilder() .setTaskManagerTimeout(taskManagerTimeout) - .build()) { - - slotManager.start(resourceManagerId, Executors.directExecutor(), resourceActions); + .buildAndStartWithDirectExec(resourceManagerId, resourceActions)) { slotManager.registerTaskManager(taskExecutorConnection, initialSlotReport); @@ -1013,9 +1006,7 @@ public class SlotManagerImplTest extends TestLogger { final TestingTaskExecutorGateway taskExecutorGateway = new TestingTaskExecutorGatewayBuilder().createTestingTaskExecutorGateway(); final TaskExecutorConnection taskExecutorConnection = new TaskExecutorConnection(taskManagerId, taskExecutorGateway); - try (final SlotManagerImpl slotManager = SlotManagerBuilder.newBuilder().build()) { - - slotManager.start(ResourceManagerId.generate(), Executors.directExecutor(), resourceActions); + try (final SlotManagerImpl slotManager = createSlotManager(ResourceManagerId.generate(), resourceActions)) { // initially report a single slot as free final SlotID slotId = new SlotID(taskManagerId, 0); @@ -1510,13 +1501,9 @@ public class SlotManagerImplTest extends TestLogger { */ @Test public void testSpreadOutSlotAllocationStrategy() throws
[flink] 05/05: [hotfix][filesystems] Remove unused StopWatch
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit efd22a7cfc53d083b6c02941641d8f17f1c767ea Author: Gary Yao AuthorDate: Thu Apr 23 19:00:49 2020 +0200 [hotfix][filesystems] Remove unused StopWatch --- .../flink/runtime/fs/hdfs/HadoopRecoverableFsDataOutputStream.java| 4 1 file changed, 4 deletions(-) diff --git a/flink-filesystems/flink-hadoop-fs/src/main/java/org/apache/flink/runtime/fs/hdfs/HadoopRecoverableFsDataOutputStream.java b/flink-filesystems/flink-hadoop-fs/src/main/java/org/apache/flink/runtime/fs/hdfs/HadoopRecoverableFsDataOutputStream.java index 05e8cd0..adf123f 100644 --- a/flink-filesystems/flink-hadoop-fs/src/main/java/org/apache/flink/runtime/fs/hdfs/HadoopRecoverableFsDataOutputStream.java +++ b/flink-filesystems/flink-hadoop-fs/src/main/java/org/apache/flink/runtime/fs/hdfs/HadoopRecoverableFsDataOutputStream.java @@ -29,7 +29,6 @@ import org.apache.flink.util.FlinkRuntimeException; import org.apache.flink.util.IOUtils; import org.apache.flink.util.Preconditions; -import org.apache.commons.lang3.time.StopWatch; import org.apache.hadoop.fs.FSDataOutputStream; import org.apache.hadoop.fs.FileStatus; import org.apache.hadoop.fs.FileSystem; @@ -332,9 +331,6 @@ class HadoopRecoverableFsDataOutputStream extends RecoverableFsDataOutputStream final Deadline deadline = Deadline.now().plus(Duration.ofMillis(LEASE_TIMEOUT)); - final StopWatch sw = new StopWatch(); - sw.start(); - boolean isClosed = dfs.isFileClosed(path); while (!isClosed && deadline.hasTimeLeft()) { try {
[flink] 01/05: [hotfix][runtime] Throw exception from TestingSchedulingPipelinedRegion#getVertex() for unknown vertices
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit fd13c2c79002e436df3c430e79d010f13b567e2d Author: Gary Yao AuthorDate: Tue Apr 21 15:40:11 2020 +0200 [hotfix][runtime] Throw exception from TestingSchedulingPipelinedRegion#getVertex() for unknown vertices --- .../scheduler/strategy/TestingSchedulingPipelinedRegion.java| 6 +- 1 file changed, 5 insertions(+), 1 deletion(-) diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/strategy/TestingSchedulingPipelinedRegion.java b/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/strategy/TestingSchedulingPipelinedRegion.java index 34450a7..2166fdc 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/strategy/TestingSchedulingPipelinedRegion.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/strategy/TestingSchedulingPipelinedRegion.java @@ -52,7 +52,11 @@ public class TestingSchedulingPipelinedRegion implements SchedulingPipelinedRegi @Override public TestingSchedulingExecutionVertex getVertex(ExecutionVertexID vertexId) { - return regionVertices.get(vertexId); + final TestingSchedulingExecutionVertex executionVertex = regionVertices.get(vertexId); + if (executionVertex == null) { + throw new IllegalArgumentException(String.format("Execution vertex %s not found in pipelined region", vertexId)); + } + return executionVertex; } @Override
[flink] 03/05: [FLINK-17180][runtime] Use SchedulingPipelinedRegion in RegionPartitionReleaseStrategy
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit 95b3c955f115dacb58b9695ae4192f729f5d5662 Author: Gary Yao AuthorDate: Tue Apr 21 15:23:40 2020 +0200 [FLINK-17180][runtime] Use SchedulingPipelinedRegion in RegionPartitionReleaseStrategy Avoid re-computing pipelined regions in the RegionPartitionReleaseStrategy using the PipelinedRegionComputeUtil. Instead, rely on the pipelined regions provided by the Topology. --- .../failover/flip1/PipelinedRegionComputeUtil.java | 20 - .../flip1/partitionrelease/PipelinedRegion.java| 69 --- .../PipelinedRegionConsumedBlockingPartitions.java | 51 .../PipelinedRegionExecutionView.java | 24 +++--- .../RegionPartitionReleaseStrategy.java| 97 -- .../RegionPartitionReleaseStrategyTest.java| 38 + .../PipelinedRegionExecutionViewTest.java | 21 +++-- 7 files changed, 45 insertions(+), 275 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java index aa94841..52d96f4 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java @@ -19,9 +19,6 @@ package org.apache.flink.runtime.executiongraph.failover.flip1; -import org.apache.flink.runtime.executiongraph.failover.flip1.partitionrelease.PipelinedRegion; -import org.apache.flink.runtime.scheduler.strategy.ExecutionVertexID; -import org.apache.flink.runtime.scheduler.strategy.SchedulingExecutionVertex; import org.apache.flink.runtime.topology.BaseTopology; import org.apache.flink.runtime.topology.Result; import org.apache.flink.runtime.topology.Vertex; @@ -34,8 +31,6 @@ import java.util.HashSet; import java.util.IdentityHashMap; import java.util.Map; import java.util.Set; -import java.util.function.Function; -import java.util.stream.Collectors; /** * Utility for computing pipelined regions. @@ -44,21 +39,6 @@ public final class PipelinedRegionComputeUtil { private static final Logger LOG = LoggerFactory.getLogger(PipelinedRegionComputeUtil.class); - public static Set toPipelinedRegionsSet( - final Set> distinctRegions) { - - return distinctRegions.stream() - .map(toExecutionVertexIdSet()) - .map(PipelinedRegion::from) - .collect(Collectors.toSet()); - } - - private static Function, Set> toExecutionVertexIdSet() { - return failoverVertices -> failoverVertices.stream() - .map(SchedulingExecutionVertex::getId) - .collect(Collectors.toSet()); - } - public static , R extends Result> Set> computePipelinedRegions( final BaseTopology topology) { diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/partitionrelease/PipelinedRegion.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/partitionrelease/PipelinedRegion.java deleted file mode 100644 index 36c042e..000 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/partitionrelease/PipelinedRegion.java +++ /dev/null @@ -1,69 +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.flink.runtime.executiongraph.failover.flip1.partitionrelease; - -import org.apache.flink.runtime.scheduler.strategy.ExecutionVertexID; - -import java.util.Arrays; -import java.util.HashSet; -import java.util.Iterator; -import java.util.Set; - -import static org.apache.flink.util.Preconditions.checkNotNull; - -/** - * Set of execution vertices that are connected through pipeli
[flink] 02/05: [FLINK-17180][runtime] Implement SchedulingPipelinedRegion interface
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit 23c13bbdbfa1b538a6c9e4e9622ef4563f69cd03 Author: Gary Yao AuthorDate: Mon Apr 20 19:17:07 2020 +0200 [FLINK-17180][runtime] Implement SchedulingPipelinedRegion interface Implement interfaces - Toplogy#getAllPipelinedRegions() - Topology#getPipelinedRegionOfVertex(ExecutionVertexID) in DefaultExecutionTopology to enable retrieval of pipelined regions. --- .../adapter/DefaultExecutionTopology.java | 49 .../adapter/DefaultSchedulingPipelinedRegion.java | 85 ++ .../adapter/DefaultExecutionTopologyTest.java | 21 .../DefaultSchedulingPipelinedRegionTest.java | 127 + 4 files changed, 282 insertions(+) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/adapter/DefaultExecutionTopology.java b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/adapter/DefaultExecutionTopology.java index 63bd39f..d7eb54e 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/adapter/DefaultExecutionTopology.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/adapter/DefaultExecutionTopology.java @@ -23,16 +23,23 @@ import org.apache.flink.runtime.executiongraph.ExecutionGraph; import org.apache.flink.runtime.executiongraph.ExecutionJobVertex; import org.apache.flink.runtime.executiongraph.ExecutionVertex; import org.apache.flink.runtime.executiongraph.IntermediateResultPartition; +import org.apache.flink.runtime.executiongraph.failover.flip1.PipelinedRegionComputeUtil; import org.apache.flink.runtime.jobgraph.IntermediateResultPartitionID; import org.apache.flink.runtime.scheduler.strategy.ExecutionVertexID; import org.apache.flink.runtime.scheduler.strategy.ResultPartitionState; +import org.apache.flink.runtime.scheduler.strategy.SchedulingExecutionVertex; import org.apache.flink.runtime.scheduler.strategy.SchedulingTopology; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + import java.util.ArrayList; +import java.util.Collections; import java.util.HashMap; import java.util.List; import java.util.Map; import java.util.Objects; +import java.util.Set; import static org.apache.flink.util.Preconditions.checkNotNull; @@ -41,6 +48,8 @@ import static org.apache.flink.util.Preconditions.checkNotNull; */ public class DefaultExecutionTopology implements SchedulingTopology { + private static final Logger LOG = LoggerFactory.getLogger(DefaultExecutionTopology.class); + private final boolean containsCoLocationConstraints; private final Map executionVerticesById; @@ -49,6 +58,10 @@ public class DefaultExecutionTopology implements SchedulingTopology { private final Map resultPartitionsById; + private final Map pipelinedRegionsByVertex; + + private final List pipelinedRegions; + public DefaultExecutionTopology(ExecutionGraph graph) { checkNotNull(graph, "execution graph can not be null"); @@ -74,6 +87,28 @@ public class DefaultExecutionTopology implements SchedulingTopology { this.resultPartitionsById = tmpResultPartitionsById; connectVerticesToConsumedPartitions(executionVertexMap, tmpResultPartitionsById); + + this.pipelinedRegionsByVertex = new HashMap<>(); + this.pipelinedRegions = new ArrayList<>(); + initializePipelinedRegions(); + } + + private void initializePipelinedRegions() { + final long buildRegionsStartTime = System.nanoTime(); + + final Set> rawPipelinedRegions = PipelinedRegionComputeUtil.computePipelinedRegions(this); + for (Set rawPipelinedRegion : rawPipelinedRegions) { + //noinspection unchecked + final DefaultSchedulingPipelinedRegion pipelinedRegion = new DefaultSchedulingPipelinedRegion((Set) rawPipelinedRegion); + pipelinedRegions.add(pipelinedRegion); + + for (SchedulingExecutionVertex executionVertex : rawPipelinedRegion) { + pipelinedRegionsByVertex.put(executionVertex.getId(), pipelinedRegion); + } + } + + final long buildRegionsDuration = (System.nanoTime() - buildRegionsStartTime) / 1_000_000; + LOG.info("Built {} pipelined regions in {} ms", pipelinedRegions.size(), buildRegionsDuration); } @Override @@ -104,6 +139,20 @@ public class DefaultExecutionTopology implements SchedulingTopology { return resultPartition; } + @Override + public Iterable getAllPipelinedRegions() { + return Collections.unmo
[flink] branch master updated (8514200 -> efd22a7)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 8514200 [FLINK-17308] Add regular cleanup task for ExecutionGraphCache new fd13c2c [hotfix][runtime] Throw exception from TestingSchedulingPipelinedRegion#getVertex() for unknown vertices new 23c13bb [FLINK-17180][runtime] Implement SchedulingPipelinedRegion interface new 95b3c95 [FLINK-17180][runtime] Use SchedulingPipelinedRegion in RegionPartitionReleaseStrategy new f9c23a0 [FLINK-17180][runtime] Use SchedulingPipelinedRegion in RestartPipelinedRegionFailoverStrategy new efd22a7 [hotfix][filesystems] Remove unused StopWatch The 5 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../hdfs/HadoopRecoverableFsDataOutputStream.java | 4 - .../failover/flip1/FailoverRegion.java | 68 --- .../failover/flip1/PipelinedRegionComputeUtil.java | 20 .../RestartPipelinedRegionFailoverStrategy.java| 63 +++--- .../flip1/partitionrelease/PipelinedRegion.java| 69 --- .../PipelinedRegionConsumedBlockingPartitions.java | 51 - .../PipelinedRegionExecutionView.java | 24 ++-- .../RegionPartitionReleaseStrategy.java| 97 +++- .../adapter/DefaultExecutionTopology.java | 49 .../adapter/DefaultSchedulingPipelinedRegion.java | 85 ++ .../RegionPartitionReleaseStrategyTest.java| 38 +- ...ipelinedRegionFailoverStrategyBuildingTest.java | 125 ++-- .../PipelinedRegionExecutionViewTest.java | 21 ++-- .../adapter/DefaultExecutionTopologyTest.java | 21 .../DefaultSchedulingPipelinedRegionTest.java | 127 + .../strategy/TestingSchedulingPipelinedRegion.java | 6 +- 16 files changed, 410 insertions(+), 458 deletions(-) delete mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/FailoverRegion.java delete mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/partitionrelease/PipelinedRegion.java delete mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/partitionrelease/PipelinedRegionConsumedBlockingPartitions.java create mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/adapter/DefaultSchedulingPipelinedRegion.java create mode 100644 flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/adapter/DefaultSchedulingPipelinedRegionTest.java
[flink] 04/05: [FLINK-17180][runtime] Use SchedulingPipelinedRegion in RestartPipelinedRegionFailoverStrategy
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit f9c23a0b86121d6361df403a05f75ba4b3902735 Author: Gary Yao AuthorDate: Tue Apr 21 15:59:27 2020 +0200 [FLINK-17180][runtime] Use SchedulingPipelinedRegion in RestartPipelinedRegionFailoverStrategy Avoid re-computing pipelined regions in the RestartPipelinedRegionFailoverStrategy using the PipelinedRegionComputeUtil. Instead, rely on the pipelined regions provided by the Topology. This closes #11857. --- .../failover/flip1/FailoverRegion.java | 68 --- .../RestartPipelinedRegionFailoverStrategy.java| 63 +++ ...ipelinedRegionFailoverStrategyBuildingTest.java | 125 +++-- 3 files changed, 78 insertions(+), 178 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/FailoverRegion.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/FailoverRegion.java deleted file mode 100644 index d1efb6f..000 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/FailoverRegion.java +++ /dev/null @@ -1,68 +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.flink.runtime.executiongraph.failover.flip1; - -import org.apache.flink.runtime.scheduler.strategy.ExecutionVertexID; -import org.apache.flink.runtime.scheduler.strategy.SchedulingExecutionVertex; - -import java.util.HashSet; -import java.util.Set; - -import static org.apache.flink.util.Preconditions.checkNotNull; - -/** - * FailoverRegion is a subset of all the vertices in the job topology. - */ -public class FailoverRegion { - - /** All vertex IDs in this region. */ - private final Set executionVertexIDs; - - /** All vertices in this region. */ - private final Set executionVertices; - - /** -* Creates a new failover region containing a set of vertices. -* -* @param executionVertices to be contained in this region -*/ - public FailoverRegion(Set executionVertices) { - this.executionVertices = checkNotNull(executionVertices); - this.executionVertexIDs = new HashSet<>(); - executionVertices.forEach(v -> this.executionVertexIDs.add(v.getId())); - } - - /** -* Returns IDs of all vertices in this region. -* -* @return IDs of all vertices in this region -*/ - public Set getAllExecutionVertexIDs() { - return executionVertexIDs; - } - - /** -* Returns all vertices in this region. -* -* @return all vertices in this region -*/ - public Set getAllExecutionVertices() { - return executionVertices; - } -} diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/RestartPipelinedRegionFailoverStrategy.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/RestartPipelinedRegionFailoverStrategy.java index 3c158f0..eb8d06b 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/RestartPipelinedRegionFailoverStrategy.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/RestartPipelinedRegionFailoverStrategy.java @@ -22,6 +22,7 @@ import org.apache.flink.runtime.io.network.partition.PartitionException; import org.apache.flink.runtime.jobgraph.IntermediateResultPartitionID; import org.apache.flink.runtime.scheduler.strategy.ExecutionVertexID; import org.apache.flink.runtime.scheduler.strategy.SchedulingExecutionVertex; +import org.apache.flink.runtime.scheduler.strategy.SchedulingPipelinedRegion; import org.apache.flink.runtime.scheduler.strategy.SchedulingResultPartition; import org.apache.flink.runtime.scheduler.strategy.SchedulingTopology; import org.apache.flink.util.ExceptionUtils; @@ -31,10 +32,8 @@ import org.slf4j.LoggerFactory; import java.util.ArrayDeque; import j
[flink] 02/02: [FLINK-17181][runtime] Drop generic Types in Topology Interface
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit cc30a4ea080abda6bc58515e95fdf74f24792ec8 Author: Zhu Zhu AuthorDate: Thu Apr 9 15:58:21 2020 +0800 [FLINK-17181][runtime] Drop generic Types in Topology Interface Co-authored-by: Gary Yao --- .../runtime/executiongraph/ExecutionGraph.java | 6 ++-- .../failover/flip1/ExecutionFailureHandler.java| 4 +-- .../failover/flip1/FailoverRegion.java | 6 ++-- .../failover/flip1/FailoverStrategy.java | 2 +- .../failover/flip1/PipelinedRegionComputeUtil.java | 4 +-- .../failover/flip1/RestartAllFailoverStrategy.java | 6 ++-- .../RestartPipelinedRegionFailoverStrategy.java| 24 .../NotReleasingPartitionReleaseStrategy.java | 2 +- .../partitionrelease/PartitionReleaseStrategy.java | 2 +- .../RegionPartitionReleaseStrategy.java| 18 ++-- .../flink/runtime/scheduler/SchedulerBase.java | 4 +-- .../adapter/DefaultExecutionTopology.java | 2 +- .../scheduler/adapter/DefaultExecutionVertex.java | 2 +- .../scheduler/adapter/DefaultResultPartition.java | 2 +- .../strategy/EagerSchedulingStrategy.java | 6 ++-- .../strategy/InputDependencyConstraintChecker.java | 32 +++--- .../LazyFromSourcesSchedulingStrategy.java | 20 +++--- .../strategy/SchedulingExecutionVertex.java| 4 +-- .../strategy/SchedulingPipelinedRegion.java| 2 +- .../strategy/SchedulingResultPartition.java| 4 +-- .../strategy/SchedulingStrategyFactory.java| 2 +- .../strategy/SchedulingStrategyUtils.java | 8 +++--- .../scheduler/strategy/SchedulingTopology.java | 8 +++--- .../flink/runtime/topology/BaseTopology.java | 2 +- .../flink/runtime/topology/PipelinedRegion.java| 4 +-- .../org/apache/flink/runtime/topology/Result.java | 2 +- .../apache/flink/runtime/topology/Topology.java| 2 +- .../org/apache/flink/runtime/topology/Vertex.java | 4 +-- .../flip1/ExecutionFailureHandlerTest.java | 2 +- .../runtime/scheduler/DefaultSchedulerTest.java| 8 +++--- .../InputDependencyConstraintCheckerTest.java | 2 +- .../scheduler/strategy/TestSchedulingStrategy.java | 8 +++--- .../strategy/TestingSchedulingExecutionVertex.java | 3 +- .../strategy/TestingSchedulingResultPartition.java | 3 +- .../strategy/TestingSchedulingTopology.java| 3 +- 35 files changed, 105 insertions(+), 108 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java index 190259d..e348059 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java @@ -410,7 +410,7 @@ public class ExecutionGraph implements AccessExecutionGraph { return this.verticesInCreationOrder.size(); } - public SchedulingTopology getSchedulingTopology() { + public SchedulingTopology getSchedulingTopology() { return executionTopology; } @@ -1541,9 +1541,9 @@ public class ExecutionGraph implements AccessExecutionGraph { } ResultPartitionID createResultPartitionId(final IntermediateResultPartitionID resultPartitionId) { - final SchedulingResultPartition schedulingResultPartition = + final SchedulingResultPartition schedulingResultPartition = getSchedulingTopology().getResultPartition(resultPartitionId); - final SchedulingExecutionVertex producer = schedulingResultPartition.getProducer(); + final SchedulingExecutionVertex producer = schedulingResultPartition.getProducer(); final ExecutionVertexID producerId = producer.getId(); final JobVertexID jobVertexId = producerId.getJobVertexId(); final ExecutionJobVertex jobVertex = getJobVertex(jobVertexId); diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/ExecutionFailureHandler.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/ExecutionFailureHandler.java index c89717e..63d5e88 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/ExecutionFailureHandler.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/ExecutionFailureHandler.java @@ -38,7 +38,7 @@ import static org.apache.flink.util.Preconditions.checkNotNull; */ public class ExecutionFailureHandler { - private final SchedulingTopology schedulingTopology; + private final SchedulingTopology
[flink] 01/02: [hotfix][runtime] Fix import order in PipelinedRegionComputeUtil
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit 935313b80a1f23f827692ed0fb0ef4f6516e6265 Author: Gary Yao AuthorDate: Fri Apr 17 08:01:21 2020 +0200 [hotfix][runtime] Fix import order in PipelinedRegionComputeUtil --- .../executiongraph/failover/flip1/PipelinedRegionComputeUtil.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java index 539e20e..10a5342 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java @@ -22,8 +22,8 @@ package org.apache.flink.runtime.executiongraph.failover.flip1; import org.apache.flink.runtime.executiongraph.failover.flip1.partitionrelease.PipelinedRegion; import org.apache.flink.runtime.scheduler.strategy.ExecutionVertexID; import org.apache.flink.runtime.scheduler.strategy.SchedulingExecutionVertex; -import org.apache.flink.runtime.topology.Result; import org.apache.flink.runtime.topology.BaseTopology; +import org.apache.flink.runtime.topology.Result; import org.apache.flink.runtime.topology.Vertex; import org.slf4j.Logger;
[flink] branch master updated (b6956ff -> cc30a4e)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from b6956ff [FLINK-17132][metrics] Update licensing new 935313b [hotfix][runtime] Fix import order in PipelinedRegionComputeUtil new cc30a4e [FLINK-17181][runtime] Drop generic Types in Topology Interface The 2 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../runtime/executiongraph/ExecutionGraph.java | 6 ++-- .../failover/flip1/ExecutionFailureHandler.java| 4 +-- .../failover/flip1/FailoverRegion.java | 6 ++-- .../failover/flip1/FailoverStrategy.java | 2 +- .../failover/flip1/PipelinedRegionComputeUtil.java | 6 ++-- .../failover/flip1/RestartAllFailoverStrategy.java | 6 ++-- .../RestartPipelinedRegionFailoverStrategy.java| 24 .../NotReleasingPartitionReleaseStrategy.java | 2 +- .../partitionrelease/PartitionReleaseStrategy.java | 2 +- .../RegionPartitionReleaseStrategy.java| 18 ++-- .../flink/runtime/scheduler/SchedulerBase.java | 4 +-- .../adapter/DefaultExecutionTopology.java | 2 +- .../scheduler/adapter/DefaultExecutionVertex.java | 2 +- .../scheduler/adapter/DefaultResultPartition.java | 2 +- .../strategy/EagerSchedulingStrategy.java | 6 ++-- .../strategy/InputDependencyConstraintChecker.java | 32 +++--- .../LazyFromSourcesSchedulingStrategy.java | 20 +++--- .../strategy/SchedulingExecutionVertex.java| 4 +-- .../strategy/SchedulingPipelinedRegion.java| 2 +- .../strategy/SchedulingResultPartition.java| 4 +-- .../strategy/SchedulingStrategyFactory.java| 2 +- .../strategy/SchedulingStrategyUtils.java | 8 +++--- .../scheduler/strategy/SchedulingTopology.java | 8 +++--- .../flink/runtime/topology/BaseTopology.java | 2 +- .../flink/runtime/topology/PipelinedRegion.java| 4 +-- .../org/apache/flink/runtime/topology/Result.java | 2 +- .../apache/flink/runtime/topology/Topology.java| 2 +- .../org/apache/flink/runtime/topology/Vertex.java | 4 +-- .../flip1/ExecutionFailureHandlerTest.java | 2 +- .../runtime/scheduler/DefaultSchedulerTest.java| 8 +++--- .../InputDependencyConstraintCheckerTest.java | 2 +- .../scheduler/strategy/TestSchedulingStrategy.java | 8 +++--- .../strategy/TestingSchedulingExecutionVertex.java | 3 +- .../strategy/TestingSchedulingResultPartition.java | 3 +- .../strategy/TestingSchedulingTopology.java| 3 +- 35 files changed, 106 insertions(+), 109 deletions(-)
[flink] branch master updated (577abe7 -> 2669f5b)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 577abe7 [FLINK-15790][k8s] Make some interfaces in FlinkKubeClient asynchronous which potentially blocks the execution of RpcEndpoint's main thread add 2669f5b [FLINK-17172][tests] Enable DEBUG level logging in Jepsen tests No new revisions were added by this update. Summary of changes: flink-jepsen/src/jepsen/flink/db.clj | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
[flink] branch master updated (936cb65 -> 231bdf3)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 936cb65 [FLINK-13880][core] Correct the behavior of JobExecutionResult#getAccumulatorResult to match Javadoc add 231bdf3 [hotfix][runtime] Remove unused method ExecutionJobVertex#generateDebugString() No new revisions were added by this update. Summary of changes: .../flink/runtime/executiongraph/ExecutionJobVertex.java | 12 1 file changed, 12 deletions(-)
[flink] 01/02: [FLINK-16960][runtime] Rename LogicalPipelinedRegion to DefaultLogicalPipelinedRegion
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit 1f28494e476e81db944aea0ba152bcc92180d72c Author: Gary Yao AuthorDate: Thu Apr 9 11:55:08 2020 +0200 [FLINK-16960][runtime] Rename LogicalPipelinedRegion to DefaultLogicalPipelinedRegion --- ...gicalPipelinedRegion.java => DefaultLogicalPipelinedRegion.java} | 6 +++--- .../flink/runtime/jobgraph/topology/DefaultLogicalTopology.java | 6 +++--- .../flink/runtime/jobgraph/topology/DefaultLogicalTopologyTest.java | 2 +- .../flink/streaming/api/graph/StreamingJobGraphGenerator.java | 6 +++--- 4 files changed, 10 insertions(+), 10 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/LogicalPipelinedRegion.java b/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/DefaultLogicalPipelinedRegion.java similarity index 88% rename from flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/LogicalPipelinedRegion.java rename to flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/DefaultLogicalPipelinedRegion.java index 9b7cc8d..39d3c0d 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/LogicalPipelinedRegion.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/DefaultLogicalPipelinedRegion.java @@ -29,11 +29,11 @@ import static org.apache.flink.util.Preconditions.checkNotNull; /** * Set of {@link LogicalVertex} that are connected through pipelined {@link LogicalResult}. */ -public class LogicalPipelinedRegion { +public class DefaultLogicalPipelinedRegion { private final Set vertexIDs; - public LogicalPipelinedRegion(final Set> logicalVertices) { + public DefaultLogicalPipelinedRegion(final Set> logicalVertices) { checkNotNull(logicalVertices); this.vertexIDs = logicalVertices.stream() @@ -47,7 +47,7 @@ public class LogicalPipelinedRegion { @Override public String toString() { - return "LogicalPipelinedRegion{" + + return "DefaultLogicalPipelinedRegion{" + "vertexIDs=" + vertexIDs + '}'; } diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/DefaultLogicalTopology.java b/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/DefaultLogicalTopology.java index 00b8d62..e701d25 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/DefaultLogicalTopology.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/DefaultLogicalTopology.java @@ -102,12 +102,12 @@ public class DefaultLogicalTopology implements LogicalTopology new IllegalArgumentException("can not find result: " + resultId)); } - public Set getLogicalPipelinedRegions() { + public Set getLogicalPipelinedRegions() { final Set> regionsRaw = PipelinedRegionComputeUtil.computePipelinedRegions(this); - final Set regions = new HashSet<>(); + final Set regions = new HashSet<>(); for (Set regionVertices : regionsRaw) { - regions.add(new LogicalPipelinedRegion(regionVertices)); + regions.add(new DefaultLogicalPipelinedRegion(regionVertices)); } return regions; } diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/jobgraph/topology/DefaultLogicalTopologyTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/jobgraph/topology/DefaultLogicalTopologyTest.java index 0a17db1..3039426 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/jobgraph/topology/DefaultLogicalTopologyTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/jobgraph/topology/DefaultLogicalTopologyTest.java @@ -89,7 +89,7 @@ public class DefaultLogicalTopologyTest extends TestLogger { @Test public void testGetLogicalPipelinedRegions() { - final Set regions = logicalTopology.getLogicalPipelinedRegions(); + final Set regions = logicalTopology.getLogicalPipelinedRegions(); assertEquals(2, regions.size()); } diff --git a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/graph/StreamingJobGraphGenerator.java b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/graph/StreamingJobGraphGenerator.java index 6e22b73..611d103 100644 --- a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/graph/StreamingJobGraphGenerator.java +++ b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/graph/StreamingJobGraphGenerator.java @@ -
[flink] branch master updated (3933b04 -> b36c44e)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 3933b04 [FLINK-16995][table-common] Add new data structure interfaces in table-common new 1f28494 [FLINK-16960][runtime] Rename LogicalPipelinedRegion to DefaultLogicalPipelinedRegion new b36c44e [FLINK-16960][runtime] Add PipelinedRegion interface The 2 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../failover/flip1/PipelinedRegionComputeUtil.java | 6 +-- ...ion.java => DefaultLogicalPipelinedRegion.java} | 6 +-- .../jobgraph/topology/DefaultLogicalTopology.java | 6 +-- .../jobgraph/topology/LogicalPipelinedRegion.java | 33 ++--- .../runtime/jobgraph/topology/LogicalTopology.java | 2 +- .../strategy/SchedulingPipelinedRegion.java| 20 +++- .../scheduler/strategy/SchedulingTopology.java | 2 +- .../topology/{Topology.java => BaseTopology.java} | 2 +- .../flink/runtime/topology/PipelinedRegion.java| 57 ++ .../apache/flink/runtime/topology/Topology.java| 28 +++ .../topology/DefaultLogicalTopologyTest.java | 2 +- .../api/graph/StreamingJobGraphGenerator.java | 6 +-- 12 files changed, 103 insertions(+), 67 deletions(-) copy flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/{LogicalPipelinedRegion.java => DefaultLogicalPipelinedRegion.java} (88%) copy flink-libraries/flink-gelly/src/main/java/org/apache/flink/graph/asm/result/ResultBase.java => flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/strategy/SchedulingPipelinedRegion.java (60%) copy flink-runtime/src/main/java/org/apache/flink/runtime/topology/{Topology.java => BaseTopology.java} (95%) create mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/topology/PipelinedRegion.java
[flink] 02/02: [FLINK-16960][runtime] Add PipelinedRegion interface
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit b36c44e3bd46e0b160f0b1ae8a89446987ff44d4 Author: Gary Yao AuthorDate: Mon Apr 6 13:53:16 2020 +0200 [FLINK-16960][runtime] Add PipelinedRegion interface This closes #11647. --- .../failover/flip1/PipelinedRegionComputeUtil.java | 6 +-- .../jobgraph/topology/LogicalPipelinedRegion.java | 31 .../runtime/jobgraph/topology/LogicalTopology.java | 2 +- .../strategy/SchedulingPipelinedRegion.java| 30 .../scheduler/strategy/SchedulingTopology.java | 2 +- .../topology/{Topology.java => BaseTopology.java} | 2 +- .../flink/runtime/topology/PipelinedRegion.java| 57 ++ .../apache/flink/runtime/topology/Topology.java| 28 +++ 8 files changed, 142 insertions(+), 16 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java index 73f77a5..539e20e 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/PipelinedRegionComputeUtil.java @@ -23,7 +23,7 @@ import org.apache.flink.runtime.executiongraph.failover.flip1.partitionrelease.P import org.apache.flink.runtime.scheduler.strategy.ExecutionVertexID; import org.apache.flink.runtime.scheduler.strategy.SchedulingExecutionVertex; import org.apache.flink.runtime.topology.Result; -import org.apache.flink.runtime.topology.Topology; +import org.apache.flink.runtime.topology.BaseTopology; import org.apache.flink.runtime.topology.Vertex; import org.slf4j.Logger; @@ -60,7 +60,7 @@ public final class PipelinedRegionComputeUtil { } public static , R extends Result> Set> computePipelinedRegions( - final Topology topology) { + final BaseTopology topology) { // currently we let a job with co-location constraints fail as one region // putting co-located vertices in the same region with each other can be a future improvement @@ -115,7 +115,7 @@ public final class PipelinedRegionComputeUtil { } private static > Map> buildOneRegionForAllVertices( - final Topology topology) { + final BaseTopology topology) { LOG.warn("Cannot decompose the topology into individual failover regions due to use of " + "Co-Location constraints (iterations). Job will fail over as one holistic unit."); diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/LogicalPipelinedRegion.java b/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/LogicalPipelinedRegion.java new file mode 100644 index 000..0e19a26 --- /dev/null +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/LogicalPipelinedRegion.java @@ -0,0 +1,31 @@ +/* + * 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.flink.runtime.jobgraph.topology; + +import org.apache.flink.runtime.jobgraph.IntermediateDataSetID; +import org.apache.flink.runtime.jobgraph.JobVertex; +import org.apache.flink.runtime.jobgraph.JobVertexID; +import org.apache.flink.runtime.topology.PipelinedRegion; + +/** + * Pipelined region on logical level, i.e., {@link JobVertex} level. + */ +public interface LogicalPipelinedRegion, R extends LogicalResult> extends PipelinedRegion { +} diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/LogicalTopology.java b/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/LogicalTopology.java index 3c195cf..b8a8609 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/topology/LogicalTopology.java +++ b/flink-runtim
[flink] branch master updated (531d6bf -> 53755ec)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 531d6bf [hotfix] Extract common logic of getting CheckpointingMode as StreamingJobGraphGenerator#getCheckpointingMode add 53755ec [FLINK-16303][rest] Enable retrieval of custom JobManager log files No new revisions were added by this update. Summary of changes: docs/_includes/generated/rest_v1_dispatcher.html | 64 +- .../runtime/webmonitor/WebFrontendITCase.java | 13 ++ .../src/test/resources/rest_api_v1.snapshot| 39 +- .../runtime/resourcemanager/ResourceManager.java | 2 +- .../resourcemanager/ResourceManagerGateway.java| 2 +- .../cluster/AbstractJobManagerFileHandler.java | 83 + .../cluster/JobManagerCustomLogHandler.java| 60 ++ .../handler/cluster/JobManagerLogFileHandler.java | 55 + .../handler/cluster/JobManagerLogListHandler.java | 79 + .../rest/handler/legacy/ConstantTextHandler.java | 57 - .../legacy/files/StaticFileServerHandler.java | 9 +- .../AbstractTaskManagerFileHandler.java| 91 +- .../taskmanager/TaskManagerCustomLogHandler.java | 2 +- .../taskmanager/TaskManagerLogListHandler.java | 4 +- .../runtime/rest/handler/util/HandlerUtils.java| 80 + .../LogFileNamePathParameter.java | 6 +- .../rest/messages/{taskmanager => }/LogInfo.java | 13 +- .../messages/{taskmanager => }/LogListInfo.java| 3 +- .../FileMessageParameters.java}| 22 ++-- .../JobManagerCustomLogHeaders.java} | 27 +++-- .../cluster/JobManagerLogFileHeader.java} | 27 +++-- .../JobManagerLogListHeaders.java} | 24 ++-- .../cluster/JobManagerStdoutFileHeader.java} | 26 ++-- .../taskmanager/TaskManagerCustomLogHeaders.java | 1 + .../TaskManagerFileMessageParameters.java | 1 + .../taskmanager/TaskManagerLogsHeaders.java| 1 + .../flink/runtime/taskexecutor/TaskExecutor.java | 2 +- .../runtime/taskexecutor/TaskExecutorGateway.java | 2 +- .../runtime/webmonitor/WebMonitorEndpoint.java | 70 ++- .../flink/runtime/webmonitor/WebMonitorUtils.java | 17 ++- .../utils/TestingResourceManagerGateway.java | 2 +- .../cluster/JobManagerLogListHandlerTest.java | 131 + .../taskmanager/TaskManagerLogListHandlerTest.java | 4 +- .../rest/messages/taskmanager/LogListInfoTest.java | 2 + .../taskexecutor/TestingTaskExecutorGateway.java | 2 +- 35 files changed, 765 insertions(+), 258 deletions(-) create mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/cluster/AbstractJobManagerFileHandler.java create mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/cluster/JobManagerCustomLogHandler.java create mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/cluster/JobManagerLogFileHandler.java create mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/cluster/JobManagerLogListHandler.java delete mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/legacy/ConstantTextHandler.java rename flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/{taskmanager => }/LogFileNamePathParameter.java (85%) rename flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/{taskmanager => }/LogInfo.java (88%) rename flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/{taskmanager => }/LogListInfo.java (94%) copy flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/{taskmanager/TaskManagerFileMessageParameters.java => cluster/FileMessageParameters.java} (63%) copy flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/{taskmanager/TaskManagerCustomLogHeaders.java => cluster/JobManagerCustomLogHeaders.java} (62%) rename flink-runtime/src/main/java/org/apache/flink/runtime/rest/{handler/legacy/files/LogFileHandlerSpecification.java => messages/cluster/JobManagerLogFileHeader.java} (54%) copy flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/{taskmanager/TaskManagerLogsHeaders.java => cluster/JobManagerLogListHeaders.java} (65%) rename flink-runtime/src/main/java/org/apache/flink/runtime/rest/{handler/legacy/files/StdoutFileHandlerSpecification.java => messages/cluster/JobManagerStdoutFileHeader.java} (56%) create mode 100644 flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/cluster/JobManagerLogListHandlerTest.java
[flink] 01/01: [FLINK-17050][runtime] Rename methods getVertexOrThrow() and getResultPartitionOrThrow()
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit 39ebaae65a9a2f55f50f8714597fbc1c2eafc7c0 Author: Gary Yao AuthorDate: Thu Apr 9 10:38:46 2020 +0200 [FLINK-17050][runtime] Rename methods getVertexOrThrow() and getResultPartitionOrThrow() Rename methods - SchedulingTopology#getVertexOrThrow(ExecutionVertexID) - SchedulingTopology#getResultPartitionOrThrow(IntermediateResultPartitionID) to - SchedulingTopology#getVertex(ExecutionVertexID) - SchedulingTopology#getResultPartition(IntermediateResultPartitionID) respectively. This closes #11684. --- .../org/apache/flink/runtime/executiongraph/ExecutionGraph.java | 2 +- .../flip1/partitionrelease/RegionPartitionReleaseStrategy.java| 4 ++-- .../flink/runtime/scheduler/adapter/DefaultExecutionTopology.java | 4 ++-- .../scheduler/strategy/LazyFromSourcesSchedulingStrategy.java | 6 +++--- .../flink/runtime/scheduler/strategy/SchedulingStrategyUtils.java | 2 +- .../flink/runtime/scheduler/strategy/SchedulingTopology.java | 4 ++-- .../org/apache/flink/runtime/scheduler/DefaultSchedulerTest.java | 2 +- .../runtime/scheduler/adapter/DefaultExecutionTopologyTest.java | 8 .../runtime/scheduler/strategy/TestingSchedulingTopology.java | 4 ++-- 9 files changed, 18 insertions(+), 18 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java index d3aa0e2..4ef2bbf 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java @@ -1547,7 +1547,7 @@ public class ExecutionGraph implements AccessExecutionGraph { ResultPartitionID createResultPartitionId(final IntermediateResultPartitionID resultPartitionId) { final SchedulingResultPartition schedulingResultPartition = - getSchedulingTopology().getResultPartitionOrThrow(resultPartitionId); + getSchedulingTopology().getResultPartition(resultPartitionId); final SchedulingExecutionVertex producer = schedulingResultPartition.getProducer(); final ExecutionVertexID producerId = producer.getId(); final JobVertexID jobVertexId = producerId.getJobVertexId(); diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/partitionrelease/RegionPartitionReleaseStrategy.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/partitionrelease/RegionPartitionReleaseStrategy.java index a05a24a..d5529b7 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/partitionrelease/RegionPartitionReleaseStrategy.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/partitionrelease/RegionPartitionReleaseStrategy.java @@ -88,7 +88,7 @@ public class RegionPartitionReleaseStrategy implements PartitionReleaseStrategy final Set> allConsumedPartitionsInRegion = pipelinedRegion .getExecutionVertexIds() .stream() - .map(schedulingTopology::getVertexOrThrow) + .map(schedulingTopology::getVertex) .flatMap(vertex -> IterableUtils.toStream(vertex.getConsumedResults())) .collect(Collectors.toSet()); @@ -157,7 +157,7 @@ public class RegionPartitionReleaseStrategy implements PartitionReleaseStrategy } private boolean areConsumerRegionsFinished(final IntermediateResultPartitionID resultPartitionId) { - final SchedulingResultPartition resultPartition = schedulingTopology.getResultPartitionOrThrow(resultPartitionId); + final SchedulingResultPartition resultPartition = schedulingTopology.getResultPartition(resultPartitionId); return IterableUtils.toStream(resultPartition.getConsumers()) .map(SchedulingExecutionVertex::getId) .allMatch(this::isRegionOfVertexFinished); diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/adapter/DefaultExecutionTopology.java b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/adapter/DefaultExecutionTopology.java index 162a888..a535fdc 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/adapter/DefaultExecutionTopology.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/adapter/DefaultExecutionTopology.java @@ -87,7 +87,7 @@ public class DefaultExecutionTopology implements SchedulingTo
[flink] branch master updated (30311ec -> 39ebaae)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 30311ec [FLINK-15639][k8s] Support to set tolerations for jobmanager and taskmanger pod add b0719f0 [FLINK-17050][runtime] Replace usages of getResultPartition() with getResultPartitionOrThrow() add 3e9418c [FLINK-17050][runtime] Remove methods getVertex() and getResultPartition() new 39ebaae [FLINK-17050][runtime] Rename methods getVertexOrThrow() and getResultPartitionOrThrow() The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../runtime/executiongraph/ExecutionGraph.java | 2 +- .../RegionPartitionReleaseStrategy.java| 4 ++-- .../adapter/DefaultExecutionTopology.java | 17 + .../LazyFromSourcesSchedulingStrategy.java | 6 ++--- .../strategy/SchedulingStrategyUtils.java | 2 +- .../scheduler/strategy/SchedulingTopology.java | 28 ++ .../runtime/scheduler/DefaultSchedulerTest.java| 2 +- .../adapter/DefaultExecutionTopologyTest.java | 10 .../strategy/TestingSchedulingTopology.java| 18 +- 9 files changed, 38 insertions(+), 51 deletions(-)
[flink] branch master updated (7cf0f68 -> 37f5c24)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 7cf0f68 [FLINK-16555] Reject Enum as key type add c51b65a [FLINK-17035][runtime] Add new methods to TestingSchedulingTopology add 10af1cc [FLINK-17035][runtime] Replace FailoverTopology with SchedulingTopology add 37f5c24 [FLINK-17035][runtime] Rename method TestingSchedulingExecutionVertex#newVertex() No new revisions were added by this update. Summary of changes: .../runtime/executiongraph/ExecutionGraph.java | 5 - .../AdaptedRestartPipelinedRegionStrategyNG.java | 2 +- .../failover/flip1/ExecutionFailureHandler.java| 14 +- .../failover/flip1/FailoverRegion.java | 7 +- .../failover/flip1/FailoverResultPartition.java| 30 --- .../failover/flip1/FailoverStrategy.java | 3 +- .../failover/flip1/FailoverTopology.java | 29 --- .../failover/flip1/FailoverVertex.java | 30 --- .../failover/flip1/RestartAllFailoverStrategy.java | 10 +- .../RestartPipelinedRegionFailoverStrategy.java| 27 +-- .../flink/runtime/scheduler/DefaultScheduler.java | 4 +- .../flink/runtime/scheduler/SchedulerBase.java | 9 - .../adapter/DefaultExecutionTopology.java | 6 +- .../scheduler/adapter/DefaultExecutionVertex.java | 6 +- .../scheduler/adapter/DefaultResultPartition.java | 6 +- .../flip1/ExecutionFailureHandlerTest.java | 17 +- .../flip1/RestartAllFailoverStrategyTest.java | 16 +- ...ipelinedRegionFailoverStrategyBuildingTest.java | 222 ++--- ...RestartPipelinedRegionFailoverStrategyTest.java | 111 +-- .../failover/flip1/TestFailoverTopology.java | 174 .../strategy/TestingSchedulingTopology.java| 35 +++- 21 files changed, 251 insertions(+), 512 deletions(-) delete mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/FailoverResultPartition.java delete mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/FailoverTopology.java delete mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/failover/flip1/FailoverVertex.java delete mode 100644 flink-runtime/src/test/java/org/apache/flink/runtime/executiongraph/failover/flip1/TestFailoverTopology.java
[flink] branch master updated: [FLINK-16710][runtime] Avoid blocking TaskExecutor Main Thread
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/master by this push: new dbf0c4c [FLINK-16710][runtime] Avoid blocking TaskExecutor Main Thread dbf0c4c is described below commit dbf0c4c5914d11b6c1209f089ed014db8cd733cb Author: jrthe42 AuthorDate: Mon Mar 30 13:43:27 2020 +0800 [FLINK-16710][runtime] Avoid blocking TaskExecutor Main Thread Avoid blocking TaskExecutor main thread when uploading log files. This closes #11571. --- .../flink/runtime/taskexecutor/TaskExecutor.java | 62 -- .../runtime/taskexecutor/TaskManagerServices.java | 23 +++- .../taskexecutor/TaskManagerServicesBuilder.java | 12 - 3 files changed, 66 insertions(+), 31 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/TaskExecutor.java b/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/TaskExecutor.java index 663213c..fd5a884 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/TaskExecutor.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/TaskExecutor.java @@ -149,6 +149,7 @@ import java.util.Set; import java.util.UUID; import java.util.concurrent.CompletableFuture; import java.util.concurrent.CompletionException; +import java.util.concurrent.Executor; import java.util.concurrent.TimeoutException; import java.util.function.BiConsumer; import java.util.stream.Collectors; @@ -197,6 +198,8 @@ public class TaskExecutor extends RpcEndpoint implements TaskExecutorGateway { /** The kvState registration service in the task manager. */ private final KvStateService kvStateService; + private final Executor ioExecutor; + // - job manager connections --- private final Map jobManagerConnections; @@ -277,6 +280,7 @@ public class TaskExecutor extends RpcEndpoint implements TaskExecutorGateway { this.localStateStoresManager = taskExecutorServices.getTaskManagerStateStore(); this.shuffleEnvironment = taskExecutorServices.getShuffleEnvironment(); this.kvStateService = taskExecutorServices.getKvStateService(); + this.ioExecutor = taskExecutorServices.getIOExecutor(); this.resourceManagerLeaderRetriever = haServices.getResourceManagerLeaderRetriever(); this.jobManagerConnections = new HashMap<>(4); @@ -315,21 +319,22 @@ public class TaskExecutor extends RpcEndpoint implements TaskExecutorGateway { @Override public CompletableFuture> requestLogList(Time timeout) { - final String logDir = taskManagerConfiguration.getTaskManagerLogDir(); - if (logDir != null) { - final File[] logFiles = new File(logDir).listFiles(); + return CompletableFuture.supplyAsync(() -> { + final String logDir = taskManagerConfiguration.getTaskManagerLogDir(); + if (logDir != null) { + final File[] logFiles = new File(logDir).listFiles(); - if (logFiles == null) { - return FutureUtils.completedExceptionally(new FlinkException(String.format("There isn't a log file in TaskExecutor’s log dir %s.", logDir))); - } + if (logFiles == null) { + throw new CompletionException(new FlinkException(String.format("There isn't a log file in TaskExecutor’s log dir %s.", logDir))); + } - final List logsWithLength = Arrays.stream(logFiles) - .filter(File::isFile) - .map(logFile -> new LogInfo(logFile.getName(), logFile.length())) - .collect(Collectors.toList()); - return CompletableFuture.completedFuture(logsWithLength); - } - return CompletableFuture.completedFuture(Collections.emptyList()); + return Arrays.stream(logFiles) + .filter(File::isFile) + .map(logFile -> new LogInfo(logFile.getName(), logFile.length())) + .collect(Collectors.toList()); + } + return Collections.emptyList(); + }, ioExecutor); } // @@ -1666,22 +1671,23 @@ public class TaskExecutor extends RpcEndpoint implements TaskExecutorGateway { private CompletableFut
[flink] branch master updated (3fa4a7e5 -> f1c91cc)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 3fa4a7e5 [FLINK-16939][rest] Declare field taskManagerIdParameter final add f1c91cc [FLINK-16940][runtime] Create currentRegion HashSet with default capacity No new revisions were added by this update. Summary of changes: .../executiongraph/failover/flip1/PipelinedRegionComputeUtil.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
[flink] branch master updated: [FLINK-16939][rest] Declare field taskManagerIdParameter final
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/master by this push: new 3fa4a7e5 [FLINK-16939][rest] Declare field taskManagerIdParameter final 3fa4a7e5 is described below commit 3fa4a7e5ae9befc5a01df4202942090b63b60a69 Author: Gary Yao AuthorDate: Thu Apr 2 10:20:01 2020 +0200 [FLINK-16939][rest] Declare field taskManagerIdParameter final This closes #11612. --- .../runtime/rest/messages/taskmanager/TaskManagerMessageParameters.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/taskmanager/TaskManagerMessageParameters.java b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/taskmanager/TaskManagerMessageParameters.java index 4a76121..aafe4ee 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/taskmanager/TaskManagerMessageParameters.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/taskmanager/TaskManagerMessageParameters.java @@ -31,7 +31,7 @@ import java.util.Collections; */ public class TaskManagerMessageParameters extends MessageParameters { - public TaskManagerIdPathParameter taskManagerIdParameter = new TaskManagerIdPathParameter(); + public final TaskManagerIdPathParameter taskManagerIdParameter = new TaskManagerIdPathParameter(); @Override public Collection> getPathParameters() {
[flink] branch master updated (750ef68 -> da30827)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 750ef68 [hotfix][operators] Rename InputSelectable inputSelector field to inputSelectable add da30827 [FLINK-16741][web] Add TM log list and TM log detail page No new revisions were added by this update. Summary of changes: .../src/app/interfaces/task-manager.ts | 4 + .../logs/job-manager-logs.component.html | 2 +- .../stdout/job-manager-stdout.component.html | 2 +- .../task-manager-log-detail.component.html | 36 + .../task-manager-log-detail.component.less | 54 + .../task-manager-log-detail.component.ts | 89 ++ .../log-list/task-manager-log-list.component.html | 39 ++ .../log-list/task-manager-log-list.component.ts| 50 .../logs/task-manager-logs.component.html | 21 - .../logs/task-manager-logs.component.less | 28 --- .../logs/task-manager-logs.component.ts| 60 --- .../status/task-manager-status.component.ts| 6 +- .../stdout/task-manager-stdout.component.html | 20 - .../stdout/task-manager-stdout.component.less | 28 --- .../stdout/task-manager-stdout.component.ts| 60 --- .../task-manager/task-manager-routing.module.ts| 25 -- .../app/pages/task-manager/task-manager.module.ts | 8 +- .../src/app/services/task-manager.service.ts | 39 ++ .../common/navigation/navigation.component.ts | 28 +++ .../refresh-download.component.html| 21 - .../refresh-download.component.less| 4 + .../refresh-download/refresh-download.component.ts | 9 +++ 22 files changed, 370 insertions(+), 263 deletions(-) create mode 100644 flink-runtime-web/web-dashboard/src/app/pages/task-manager/log-detail/task-manager-log-detail.component.html create mode 100644 flink-runtime-web/web-dashboard/src/app/pages/task-manager/log-detail/task-manager-log-detail.component.less create mode 100644 flink-runtime-web/web-dashboard/src/app/pages/task-manager/log-detail/task-manager-log-detail.component.ts create mode 100644 flink-runtime-web/web-dashboard/src/app/pages/task-manager/log-list/task-manager-log-list.component.html create mode 100644 flink-runtime-web/web-dashboard/src/app/pages/task-manager/log-list/task-manager-log-list.component.ts delete mode 100644 flink-runtime-web/web-dashboard/src/app/pages/task-manager/logs/task-manager-logs.component.html delete mode 100644 flink-runtime-web/web-dashboard/src/app/pages/task-manager/logs/task-manager-logs.component.less delete mode 100644 flink-runtime-web/web-dashboard/src/app/pages/task-manager/logs/task-manager-logs.component.ts delete mode 100644 flink-runtime-web/web-dashboard/src/app/pages/task-manager/stdout/task-manager-stdout.component.html delete mode 100644 flink-runtime-web/web-dashboard/src/app/pages/task-manager/stdout/task-manager-stdout.component.less delete mode 100644 flink-runtime-web/web-dashboard/src/app/pages/task-manager/stdout/task-manager-stdout.component.ts
[flink] branch master updated (37d1043 -> eec8fea)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 37d1043 [hotfix][metrics] Do not log reporter config to prevent credentials leak add eec8fea [FLINK-16594][runtime] Fix typo in GlobalAggregateManager Javadoc No new revisions were added by this update. Summary of changes: .../org/apache/flink/runtime/taskexecutor/GlobalAggregateManager.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
[flink] branch master updated (b752bfb -> eb489cc)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from b752bfb [FLINK-16821][e2e] Use constant minikube version instead of latest add eb489cc [FLINK-11404][web] Add load more feature to exception page No new revisions were added by this update. Summary of changes: .../job/exceptions/job-exceptions.component.html | 5 + .../job/exceptions/job-exceptions.component.ts | 26 -- .../web-dashboard/src/app/services/job.service.ts | 7 -- 3 files changed, 29 insertions(+), 9 deletions(-)
[flink] branch master updated (a2b669f -> 505a808)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from a2b669f [hotfix][doc] Document union list metadata gotcha add c125c04 [FLINK-16302][rest] Enable retrieval of custom TaskManager log files add 505a808 [FLINK-16302][rest] Add tests for (un)marshalling LogListInfo No new revisions were added by this update. Summary of changes: docs/_includes/generated/rest_v1_dispatcher.html | 66 .../src/test/resources/rest_api_v1.snapshot| 37 +++ .../runtime/resourcemanager/ResourceManager.java | 34 +- .../resourcemanager/ResourceManagerGateway.java| 23 +++- .../AbstractTaskManagerFileHandler.java| 29 -- .../taskmanager/AbstractTaskManagerHandler.java| 14 ++- ...ndler.java => TaskManagerCustomLogHandler.java} | 24 +++-- .../taskmanager/TaskManagerLogFileHandler.java | 5 +- .../taskmanager/TaskManagerLogListHandler.java | 88 .../taskmanager/TaskManagerStdoutFileHandler.java | 5 +- ...arameter.java => LogFileNamePathParameter.java} | 20 ++-- .../{TaskManagersInfo.java => LogInfo.java}| 36 --- .../{TaskManagersInfo.java => LogListInfo.java}| 33 +++--- ...aders.java => TaskManagerCustomLogHeaders.java} | 18 ++-- .../TaskManagerFileMessageParameters.java} | 17 +-- ...ilsHeaders.java => TaskManagerLogsHeaders.java} | 40 +++ .../flink/runtime/taskexecutor/TaskExecutor.java | 82 ++- .../runtime/taskexecutor/TaskExecutorGateway.java | 20 +++- .../taskexecutor/TaskManagerConfiguration.java | 15 +++ .../runtime/webmonitor/WebMonitorEndpoint.java | 23 .../utils/TestingResourceManagerGateway.java | 45 +++- .../AbstractTaskManagerFileHandlerTest.java| 8 +- .../taskmanager/TaskManagerLogListHandlerTest.java | 116 + .../rest/messages/taskmanager/LogListInfoTest.java | 30 +++--- .../runtime/taskexecutor/TaskExecutorTest.java | 2 +- .../taskexecutor/TestingTaskExecutorGateway.java | 14 ++- 26 files changed, 682 insertions(+), 162 deletions(-) copy flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/taskmanager/{TaskManagerStdoutFileHandler.java => TaskManagerCustomLogHandler.java} (71%) create mode 100644 flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/taskmanager/TaskManagerLogListHandler.java copy flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/taskmanager/{TaskManagerIdPathParameter.java => LogFileNamePathParameter.java} (64%) copy flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/taskmanager/{TaskManagersInfo.java => LogInfo.java} (62%) copy flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/taskmanager/{TaskManagersInfo.java => LogListInfo.java} (64%) copy flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/taskmanager/{TaskManagerLogFileHeaders.java => TaskManagerCustomLogHeaders.java} (72%) copy flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/{job/metrics/AggregatedJobMetricsParameters.java => taskmanager/TaskManagerFileMessageParameters.java} (66%) copy flink-runtime/src/main/java/org/apache/flink/runtime/rest/messages/taskmanager/{TaskManagerDetailsHeaders.java => TaskManagerLogsHeaders.java} (72%) create mode 100644 flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/taskmanager/TaskManagerLogListHandlerTest.java copy flink-tests/src/test/java/org/apache/flink/test/util/InfiniteIntegerSource.java => flink-runtime/src/test/java/org/apache/flink/runtime/rest/messages/taskmanager/LogListInfoTest.java (57%)
[flink] branch release-1.10 updated: [FLINK-16718][tests] Fix ByteBuf leak in KvStateServerHandlerTest
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.10 in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/release-1.10 by this push: new 00f27c6 [FLINK-16718][tests] Fix ByteBuf leak in KvStateServerHandlerTest 00f27c6 is described below commit 00f27c6b42cc7db29d5a71ad9872ef3ee27272e0 Author: Gary Yao AuthorDate: Thu Mar 19 10:55:21 2020 +0100 [FLINK-16718][tests] Fix ByteBuf leak in KvStateServerHandlerTest --- .../queryablestate/network/KvStateServerHandlerTest.java | 12 1 file changed, 12 insertions(+) diff --git a/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/KvStateServerHandlerTest.java b/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/KvStateServerHandlerTest.java index d38cca8..f6f396e 100644 --- a/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/KvStateServerHandlerTest.java +++ b/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/KvStateServerHandlerTest.java @@ -170,6 +170,7 @@ public class KvStateServerHandlerTest extends TestLogger { assertEquals(MessageType.REQUEST_RESULT, MessageSerializer.deserializeHeader(buf)); long deserRequestId = MessageSerializer.getRequestId(buf); KvStateResponse response = serializer.deserializeResponse(buf); + buf.release(); assertEquals(requestId, deserRequestId); @@ -217,6 +218,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.REQUEST_FAILURE, MessageSerializer.deserializeHeader(buf)); RequestFailure response = MessageSerializer.deserializeRequestFailure(buf); + buf.release(); assertEquals(requestId, response.getRequestId()); @@ -278,6 +280,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.REQUEST_FAILURE, MessageSerializer.deserializeHeader(buf)); RequestFailure response = MessageSerializer.deserializeRequestFailure(buf); + buf.release(); assertEquals(requestId, response.getRequestId()); @@ -363,6 +366,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.REQUEST_FAILURE, MessageSerializer.deserializeHeader(buf)); RequestFailure response = MessageSerializer.deserializeRequestFailure(buf); + buf.release(); assertTrue(response.getCause().getMessage().contains("Expected test Exception")); @@ -392,6 +396,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.SERVER_FAILURE, MessageSerializer.deserializeHeader(buf)); Throwable response = MessageSerializer.deserializeServerFailure(buf); + buf.release(); assertTrue(response.getMessage().contains("Expected test Exception")); @@ -454,6 +459,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.REQUEST_FAILURE, MessageSerializer.deserializeHeader(buf)); RequestFailure response = MessageSerializer.deserializeRequestFailure(buf); + buf.release(); assertTrue(response.getCause().getMessage().contains("RejectedExecutionException")); @@ -490,6 +496,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.SERVER_FAILURE, MessageSerializer.deserializeHeader(buf)); Throwable response = MessageSerializer.deserializeServerFailure(buf); + buf.release(); assertEquals(0L, stats.getNumRequests()); assertEquals(0L, stats.getNumFailed()); @@ -505,6 +512,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.SERVER_FAILURE, MessageSerializer.deserializeHeader(buf)); response = MessageSerializer.deserializeServerFailure(buf); + buf.release(); assertTrue("Unexpected failure cause " + response.getClass().getName(), response instanceof IllegalArgumentException); @@ -544,6 +552,7 @@ public class KvStateServerHandlerTest extends TestLogger { channel.writeInbound(unexpected); assertEquals(&quo
[flink] branch master updated: [FLINK-16718][tests] Fix ByteBuf leak in KvStateServerHandlerTest
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/master by this push: new 50ee655 [FLINK-16718][tests] Fix ByteBuf leak in KvStateServerHandlerTest 50ee655 is described below commit 50ee6554dffee784f6dbfeaba7b18a18bdba5659 Author: Gary Yao AuthorDate: Thu Mar 19 10:55:21 2020 +0100 [FLINK-16718][tests] Fix ByteBuf leak in KvStateServerHandlerTest This closes #11453. --- .../queryablestate/network/KvStateServerHandlerTest.java | 12 1 file changed, 12 insertions(+) diff --git a/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/KvStateServerHandlerTest.java b/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/KvStateServerHandlerTest.java index d38cca8..f6f396e 100644 --- a/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/KvStateServerHandlerTest.java +++ b/flink-queryable-state/flink-queryable-state-runtime/src/test/java/org/apache/flink/queryablestate/network/KvStateServerHandlerTest.java @@ -170,6 +170,7 @@ public class KvStateServerHandlerTest extends TestLogger { assertEquals(MessageType.REQUEST_RESULT, MessageSerializer.deserializeHeader(buf)); long deserRequestId = MessageSerializer.getRequestId(buf); KvStateResponse response = serializer.deserializeResponse(buf); + buf.release(); assertEquals(requestId, deserRequestId); @@ -217,6 +218,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.REQUEST_FAILURE, MessageSerializer.deserializeHeader(buf)); RequestFailure response = MessageSerializer.deserializeRequestFailure(buf); + buf.release(); assertEquals(requestId, response.getRequestId()); @@ -278,6 +280,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.REQUEST_FAILURE, MessageSerializer.deserializeHeader(buf)); RequestFailure response = MessageSerializer.deserializeRequestFailure(buf); + buf.release(); assertEquals(requestId, response.getRequestId()); @@ -363,6 +366,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.REQUEST_FAILURE, MessageSerializer.deserializeHeader(buf)); RequestFailure response = MessageSerializer.deserializeRequestFailure(buf); + buf.release(); assertTrue(response.getCause().getMessage().contains("Expected test Exception")); @@ -392,6 +396,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.SERVER_FAILURE, MessageSerializer.deserializeHeader(buf)); Throwable response = MessageSerializer.deserializeServerFailure(buf); + buf.release(); assertTrue(response.getMessage().contains("Expected test Exception")); @@ -454,6 +459,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.REQUEST_FAILURE, MessageSerializer.deserializeHeader(buf)); RequestFailure response = MessageSerializer.deserializeRequestFailure(buf); + buf.release(); assertTrue(response.getCause().getMessage().contains("RejectedExecutionException")); @@ -490,6 +496,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.SERVER_FAILURE, MessageSerializer.deserializeHeader(buf)); Throwable response = MessageSerializer.deserializeServerFailure(buf); + buf.release(); assertEquals(0L, stats.getNumRequests()); assertEquals(0L, stats.getNumFailed()); @@ -505,6 +512,7 @@ public class KvStateServerHandlerTest extends TestLogger { // Verify the response assertEquals(MessageType.SERVER_FAILURE, MessageSerializer.deserializeHeader(buf)); response = MessageSerializer.deserializeServerFailure(buf); + buf.release(); assertTrue("Unexpected failure cause " + response.getClass().getName(), response instanceof IllegalArgumentException); @@ -544,6 +552,7 @@ public class KvStateServerHandlerTest extends TestLogger { channel.writeInbound(unexpected); assertEquals(&quo
[flink] branch release-1.10 updated: [FLINK-15953][web] Fix color of RESTARTING status in Web UI
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.10 in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/release-1.10 by this push: new 742d420 [FLINK-15953][web] Fix color of RESTARTING status in Web UI 742d420 is described below commit 742d420dbe41c387515f8f87cee79e0073f78da3 Author: vthinkxie AuthorDate: Wed Mar 18 15:18:49 2020 +0800 [FLINK-15953][web] Fix color of RESTARTING status in Web UI --- flink-runtime-web/web-dashboard/src/app/app.config.ts | 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-) diff --git a/flink-runtime-web/web-dashboard/src/app/app.config.ts b/flink-runtime-web/web-dashboard/src/app/app.config.ts index 66224e3..6d41b67 100644 --- a/flink-runtime-web/web-dashboard/src/app/app.config.ts +++ b/flink-runtime-web/web-dashboard/src/app/app.config.ts @@ -29,6 +29,7 @@ export const COLOR_MAP = { RECONCILING: '#eb2f96', IN_PROGRESS: '#faad14', SCHEDULED: '#722ed1', - COMPLETED : '#1890ff' + COMPLETED: '#1890ff', + RESTARTING: '#13c2c2' }; export const LONG_MIN_VALUE = -9223372036854776000;
[flink] branch master updated (6c1f94c -> 64abbdd)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 6c1f94c [FLINK-15991][docs-zh] Translate the "Migration Guide" page into Chinese. add 64abbdd [FLINK-15953][web] Fix color of RESTARTING status in Web UI No new revisions were added by this update. Summary of changes: flink-runtime-web/web-dashboard/src/app/app.config.ts | 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-)
[flink] branch master updated (c83bc65 -> 0477368)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from c83bc65 [FLINK-11088] Add unit test for Kerberos parsing in YarnEntrypointUtilsTest add 0477368 [FLINK-14121] Update commons-compress because of CVE-2019-12402 No new revisions were added by this update. Summary of changes: flink-dist/src/main/resources/META-INF/NOTICE | 2 +- .../flink-oss-fs-hadoop/src/main/resources/META-INF/NOTICE | 2 +- .../flink-swift-fs-hadoop/src/main/resources/META-INF/NOTICE| 2 +- pom.xml | 2 +- 4 files changed, 4 insertions(+), 4 deletions(-)
[flink] branch master updated: [FLINK-16445][build] Set property japicmp.referenceVersion to 1.10.0
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/master by this push: new 809eb2a [FLINK-16445][build] Set property japicmp.referenceVersion to 1.10.0 809eb2a is described below commit 809eb2ab292ad2916d74f4f761b45ec4aa2f5404 Author: Gary Yao AuthorDate: Thu Mar 5 16:47:22 2020 +0100 [FLINK-16445][build] Set property japicmp.referenceVersion to 1.10.0 This closes #11324. --- pom.xml | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/pom.xml b/pom.xml index 90de99b..21e89ad 100644 --- a/pom.xml +++ b/pom.xml @@ -152,7 +152,7 @@ under the License. For Hadoop 2.7, the minor Hadoop version supported for flink-shaded-hadoop-2-uber is 2.7.5 --> 2.7.5 - 1.9.1 + 1.10.0 tools/japicmp-output
[flink] branch master updated (8c063d3 -> 525f164)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 8c063d3 [FLINK-16288] Document how to keep the task managers around longer in Kubernetes add 525f164 [FLINK-15329][runtime] Fix incorrect Javadoc in MemoryManager No new revisions were added by this update. Summary of changes: .../src/main/java/org/apache/flink/runtime/memory/MemoryManager.java | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-)
[flink] branch release-1.9 updated: [hotfix][docs] Remove legacy mode from config options docs
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.9 in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/release-1.9 by this push: new c7501ae [hotfix][docs] Remove legacy mode from config options docs c7501ae is described below commit c7501ae03556244024616d6ed9109a687a94f832 Author: Gary Yao AuthorDate: Tue Feb 25 11:45:09 2020 +0100 [hotfix][docs] Remove legacy mode from config options docs --- docs/ops/config.md| 4 docs/ops/config.zh.md | 4 2 files changed, 8 deletions(-) diff --git a/docs/ops/config.md b/docs/ops/config.md index 8e05c03..1d6167d 100644 --- a/docs/ops/config.md +++ b/docs/ops/config.md @@ -209,10 +209,6 @@ You have to configure `jobmanager.archive.fs.dir` in order to archive terminated {% include generated/history_server_configuration.html %} -## Legacy - -- `mode`: Execution mode of Flink. Possible values are `legacy` and `new`. In order to start the legacy components, you have to specify `legacy` (DEFAULT: `new`). - ## Background ### Configuring the Network Buffers diff --git a/docs/ops/config.zh.md b/docs/ops/config.zh.md index c99d413..58fe182 100644 --- a/docs/ops/config.zh.md +++ b/docs/ops/config.zh.md @@ -209,10 +209,6 @@ You have to configure `jobmanager.archive.fs.dir` in order to archive terminated {% include generated/history_server_configuration.html %} -## Legacy - -- `mode`: Execution mode of Flink. Possible values are `legacy` and `new`. In order to start the legacy components, you have to specify `legacy` (DEFAULT: `new`). - ## Background
[flink] branch master updated (e591947 -> f001021)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from e591947 [FLINK-15914][tests] Fix the fragile tests caused by race condition of multiple threads add f001021 [FLINK-15743][docs] Extend Flink 1.10 release notes No new revisions were added by this update. Summary of changes: docs/release-notes/flink-1.10.md| 92 +++-- docs/release-notes/flink-1.10.zh.md | 92 +++-- 2 files changed, 178 insertions(+), 6 deletions(-)
[flink] branch release-1.10 updated (220fc5b -> 7ee7fef)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch release-1.10 in repository https://gitbox.apache.org/repos/asf/flink.git. from 220fc5b [FLINK-15933][catalog][doc] Update content of how generic table schema is stored in hive via HiveCatalog add 7ee7fef [FLINK-15743][docs] Extend Flink 1.10 release notes No new revisions were added by this update. Summary of changes: docs/release-notes/flink-1.10.md| 92 +++-- docs/release-notes/flink-1.10.zh.md | 92 +++-- 2 files changed, 178 insertions(+), 6 deletions(-)
[flink] annotated tag release-1.10.0 updated (aa4eb8f -> 88d6c95)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to annotated tag release-1.10.0 in repository https://gitbox.apache.org/repos/asf/flink.git. *** WARNING: tag release-1.10.0 was modified! *** from aa4eb8f (commit) to 88d6c95 (tag) tagging b81cbd7ea34f45441612ac53abe7d6ee0099986c (tag) length 1012 bytes by Gary Yao on Tue Feb 11 14:10:48 2020 +0100 - Log - Apache Flink 1.10.0 -BEGIN PGP SIGNATURE- iQJEBAABCAAuFiEEuxN4B877590mFlVnELEqH4nBFegFAl5Cp+EQHGdhcnlAYXBh Y2hlLm9yZwAKCRAQsSoficEV6HhzD/4vVLmbRVrsetuzXR5X1fqvZLlIN4LFsAY+ ojT+cbFDocy9CMGJlu29Rgc6PvrNURF0vCln4TqjuEMOjcSl8wpHvQLsKiH+vs/V e9a4WMD1rnGl1Q/0lZl+k71BI0pD5ZvD67izbKu8avGarIVEKRbIIB62+Jy8ZAmq G+EAn30i7i7p4Yxej12EuvC0SB/ym8+OxQ8BkJE00IY1ss1JU9KiVkUA2LifWYh0 8ujnoI5KqItniJQT6kUnq65RE3OACrpVSus+CKgatQDWqlZhjoppBFcs5iF1ONT8 Ac7gmi4+a4hYI4zOkmJtCYTMl2LoRoNJYe7N4MJekxj50L+CIBefoOfDe8PjKxHO 2zFlR/+RGlvqX80ZacOENL2l1P1pi//0QeOFmXEd6Y5tJvOykQxBcSFDwsv7wk5l lUWeydgE/FTGRwZeVZYKQsSce0yq4nF085JZLEuS4tiR9PJValxNq+4ZihzkxwBe G0ZLKdbwrvvmzkXj4KMiN/gkyXhUxjnZ1ta7hDva2V6XXOWB3axVHdn0cjlp2imI 644cipaJYNJDkX9ab9/1h9QIUjY5eAHjCk4Nyqn+dXECZRpmUaEJNXR8DYh8KRdF 7v8cLlPRo/2Ant7P4J9nA351o2waYe2BthwHeUGq2fO2prpg/NIOjqQLjfmlYT2i vcXL+ZNP1A== =QsKA -END PGP SIGNATURE- --- No new revisions were added by this update. Summary of changes:
[flink] 01/02: [FLINK-15917][runtime] Fix that the root exception is not shown in Web UI
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit ba0e2e0b367a7b81f0c3a018a73fd8cc97592c06 Author: Gary Yao AuthorDate: Fri Feb 7 16:32:42 2020 +0100 [FLINK-15917][runtime] Fix that the root exception is not shown in Web UI --- .../flink/runtime/scheduler/DefaultScheduler.java| 3 ++- .../flink/runtime/scheduler/SchedulerBase.java | 8 ++-- .../runtime/scheduler/DefaultSchedulerTest.java | 20 3 files changed, 28 insertions(+), 3 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java index c107174..9779dac1 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java @@ -182,7 +182,8 @@ public class DefaultScheduler extends SchedulerBase implements SchedulerOperatio } } - private void handleTaskFailure(final ExecutionVertexID executionVertexId, final Throwable error) { + private void handleTaskFailure(final ExecutionVertexID executionVertexId, @Nullable final Throwable error) { + setGlobalFailureCause(error); final FailureHandlingResult failureHandlingResult = executionFailureHandler.getFailureHandlingResult(executionVertexId, error); maybeRestartTasks(failureHandlingResult); } diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java index d31e5e4..bd62c1e 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java @@ -94,6 +94,8 @@ import org.apache.flink.util.function.FunctionUtils; import org.slf4j.Logger; +import javax.annotation.Nullable; + import java.io.IOException; import java.net.InetSocketAddress; import java.util.Collection; @@ -348,8 +350,10 @@ public abstract class SchedulerBase implements SchedulerNG { .transitionState(ExecutionState.SCHEDULED)); } - protected void setGlobalFailureCause(final Throwable cause) { - getExecutionGraph().initFailureCause(cause); + protected void setGlobalFailureCause(@Nullable final Throwable cause) { + if (cause != null) { + getExecutionGraph().initFailureCause(cause); + } } protected ComponentMainThreadExecutor getMainThreadExecutor() { diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/DefaultSchedulerTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/DefaultSchedulerTest.java index 79b0c3b..7b54992 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/DefaultSchedulerTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/DefaultSchedulerTest.java @@ -34,6 +34,7 @@ import org.apache.flink.runtime.concurrent.ManuallyTriggeredScheduledExecutor; import org.apache.flink.runtime.execution.ExecutionState; import org.apache.flink.runtime.executiongraph.AccessExecutionJobVertex; import org.apache.flink.runtime.executiongraph.ArchivedExecutionVertex; +import org.apache.flink.runtime.executiongraph.ErrorInfo; import org.apache.flink.runtime.executiongraph.ExecutionAttemptID; import org.apache.flink.runtime.executiongraph.failover.flip1.RestartPipelinedRegionFailoverStrategy; import org.apache.flink.runtime.executiongraph.failover.flip1.TestRestartBackoffTimeStrategy; @@ -91,10 +92,12 @@ import static org.apache.flink.util.ExceptionUtils.findThrowable; import static org.apache.flink.util.ExceptionUtils.findThrowableWithMessage; import static org.hamcrest.Matchers.contains; import static org.hamcrest.Matchers.containsInAnyOrder; +import static org.hamcrest.Matchers.containsString; import static org.hamcrest.Matchers.equalTo; import static org.hamcrest.Matchers.hasSize; import static org.hamcrest.Matchers.is; import static org.hamcrest.Matchers.lessThan; +import static org.hamcrest.Matchers.notNullValue; import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertFalse; import static org.junit.Assert.assertThat; @@ -607,6 +610,23 @@ public class DefaultSchedulerTest extends TestLogger { assertTrue(executionVertexVersioner.isModified(executionVertexVersion)); } + @Test + public void failureInfoIsSetAfterTaskFailure() { + final JobGraph jobGraph = singleNonParallelJobVertexJobGraph(); + final JobID jobId = jobGraph.getJobID(); + final
[flink] 02/02: [FLINK-15918][runtime] Fix that 'uptime' metric is not reset after restart
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git commit b902ed507e2586c322f33e5612ace4222b3d9d26 Author: Gary Yao AuthorDate: Fri Feb 7 16:34:00 2020 +0100 [FLINK-15918][runtime] Fix that 'uptime' metric is not reset after restart The 'uptime' metric is the time difference between 'now' and the timestamp when the job transitioned to state RUNNING. The new scheduler until now never transitioned out of status RUNNING when restarting tasks which had the effect that the 'uptime' metric was not reset after a restart. This introduces new state transitions to the job. We transition the job status to RESTARTING if at least one ExecutionVertex is waiting to be restarted, and we transition from RESTARTING immediately to RUNNING again after the restart. This closes #11032. --- .../runtime/executiongraph/ExecutionGraph.java | 14 +-- .../flink/runtime/scheduler/DefaultScheduler.java | 21 ++ .../flink/runtime/scheduler/SchedulerBase.java | 4 ++ .../ManuallyTriggeredScheduledExecutor.java| 13 ++ .../runtime/scheduler/DefaultSchedulerTest.java| 47 ++ 5 files changed, 87 insertions(+), 12 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java index 7312b78..f8d79fa 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java @@ -897,7 +897,7 @@ public class ExecutionGraph implements AccessExecutionGraph { while (true) { JobStatus current = state; - if (current == JobStatus.RUNNING || current == JobStatus.CREATED) { + if (current == JobStatus.RUNNING || current == JobStatus.CREATED || current == JobStatus.RESTARTING) { if (transitionState(current, JobStatus.CANCELLING)) { // make sure no concurrent local actions interfere with the cancellation @@ -937,16 +937,6 @@ public class ExecutionGraph implements AccessExecutionGraph { return; } } - // All vertices have been cancelled and it's safe to directly go - // into the canceled state. - else if (current == JobStatus.RESTARTING) { - if (transitionState(current, JobStatus.CANCELED)) { - onTerminalState(JobStatus.CANCELED); - - LOG.info("Canceled during restart."); - return; - } - } else { // no need to treat other states return; @@ -1219,7 +1209,7 @@ public class ExecutionGraph implements AccessExecutionGraph { // State Transitions // - private boolean transitionState(JobStatus current, JobStatus newState) { + public boolean transitionState(JobStatus current, JobStatus newState) { return transitionState(current, newState, null); } diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java index 9779dac1..2fe8340 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java @@ -19,6 +19,7 @@ package org.apache.flink.runtime.scheduler; +import org.apache.flink.api.common.JobStatus; import org.apache.flink.api.common.time.Time; import org.apache.flink.configuration.Configuration; import org.apache.flink.runtime.blob.BlobWriter; @@ -90,6 +91,8 @@ public class DefaultScheduler extends SchedulerBase implements SchedulerOperatio private final ExecutionVertexOperations executionVertexOperations; + private final Set verticesWaitingForRestart; + public DefaultScheduler( final Logger log, final JobGraph jobGraph, @@ -151,6 +154,8 @@ public class DefaultScheduler extends SchedulerBase implements SchedulerOperatio restartBackoffTimeStrategy); this.schedulingStrategy = schedulingStrategyFactory.createInstance(this, getSchedulingT
[flink] branch master updated (edfbbb7 -> b902ed5)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from edfbbb7 [hotfix][example] Copy StreamWindowSQLExample.jar to dist/examples/table new ba0e2e0 [FLINK-15917][runtime] Fix that the root exception is not shown in Web UI new b902ed5 [FLINK-15918][runtime] Fix that 'uptime' metric is not reset after restart The 2 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../runtime/executiongraph/ExecutionGraph.java | 14 + .../flink/runtime/scheduler/DefaultScheduler.java | 24 +++- .../flink/runtime/scheduler/SchedulerBase.java | 12 +++- .../ManuallyTriggeredScheduledExecutor.java| 13 + .../runtime/scheduler/DefaultSchedulerTest.java| 67 ++ 5 files changed, 115 insertions(+), 15 deletions(-)
[flink] branch release-1.10 updated: [FLINK-15417][e2e] Change the access permission of files modified in mesos constainer
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.10 in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/release-1.10 by this push: new e237f80 [FLINK-15417][e2e] Change the access permission of files modified in mesos constainer e237f80 is described below commit e237f809ae04723502aea617421b748318867aae Author: Yangze Guo AuthorDate: Mon Feb 3 15:51:37 2020 +0800 [FLINK-15417][e2e] Change the access permission of files modified in mesos constainer --- flink-end-to-end-tests/test-scripts/common_mesos_docker.sh | 1 + 1 file changed, 1 insertion(+) diff --git a/flink-end-to-end-tests/test-scripts/common_mesos_docker.sh b/flink-end-to-end-tests/test-scripts/common_mesos_docker.sh index 08a72f7..96dd2d5 100644 --- a/flink-end-to-end-tests/test-scripts/common_mesos_docker.sh +++ b/flink-end-to-end-tests/test-scripts/common_mesos_docker.sh @@ -33,6 +33,7 @@ start_time=$(date +%s) # make sure we stop our cluster at the end function cluster_shutdown { + docker exec mesos-master bash -c "chmod -R ogu+rw ${FLINK_DIR}/log/ ${TEST_DATA_DIR}" docker-compose -f $END_TO_END_DIR/test-scripts/docker-mesos-cluster/docker-compose.yml down } on_exit cluster_shutdown
svn commit: r37947 - /dev/flink/flink-1.10.0-rc3/
Author: gary Date: Fri Feb 7 21:27:51 2020 New Revision: 37947 Log: Add flink-1.10.0-rc3 Added: dev/flink/flink-1.10.0-rc3/ dev/flink/flink-1.10.0-rc3/apache-flink-1.10.0.tar.gz (with props) dev/flink/flink-1.10.0-rc3/apache-flink-1.10.0.tar.gz.asc dev/flink/flink-1.10.0-rc3/apache-flink-1.10.0.tar.gz.sha512 dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.11.tgz (with props) dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.11.tgz.asc dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.11.tgz.sha512 dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.12.tgz (with props) dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.12.tgz.asc dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.12.tgz.sha512 dev/flink/flink-1.10.0-rc3/flink-1.10.0-src.tgz (with props) dev/flink/flink-1.10.0-rc3/flink-1.10.0-src.tgz.asc dev/flink/flink-1.10.0-rc3/flink-1.10.0-src.tgz.sha512 Added: dev/flink/flink-1.10.0-rc3/apache-flink-1.10.0.tar.gz == Binary file - no diff available. Propchange: dev/flink/flink-1.10.0-rc3/apache-flink-1.10.0.tar.gz -- svn:mime-type = application/octet-stream Added: dev/flink/flink-1.10.0-rc3/apache-flink-1.10.0.tar.gz.asc == --- dev/flink/flink-1.10.0-rc3/apache-flink-1.10.0.tar.gz.asc (added) +++ dev/flink/flink-1.10.0-rc3/apache-flink-1.10.0.tar.gz.asc Fri Feb 7 21:27:51 2020 @@ -0,0 +1,16 @@ +-BEGIN PGP SIGNATURE- + +iQIzBAABCAAdFiEEuxN4B877590mFlVnELEqH4nBFegFAl49sqoACgkQELEqH4nB +FegLvxAAuJaIJSRfiLTaUnVR43NXZ2g/MZsdGMg1Q1EbHs08LpTJuo7zm9VjC2DG +Y1jrnyILz5HfYW/fwtj5CMKrB3Revr3jMMS01I4BM0q/m66K8Lel6asoaulLclnY +yZCxFRqAwQP/InVBzY6YAJ9TI1A9or4Ydtm7BPj2qLte9XviwGub6pupDEWeyBGF +UOf8f7qgoZOGCl+GX73l8RRM7Y78RfkU1b9KC5Mt+SN5vIgueW4DKcqOA/BZa6gc +KHA1BsPzK6PgzJb0Np5j0ZIavkqJg/lhdrc/utwvGeYlfRrM3luM+79Q6b2HknDY +dv7D/p9tWEJD5OJtSWiG/sRXSTboZb4nK/OgtBC2UpmMqb4auwh7WO514IeheBEq +29Gr0iEy6zFp3KMjPAlAEYc0xE6av9a8mO574gS20bAddHmMvlNaHoB4wM6Ss5cC +OfBKXgsqWpTC2bUSRzclqQxEQr09/M6TAvEYAq2YGwgq3A4MKtRbFRzBNa2njSGQ +ob35NKDVK+Vh0QqxXgb4l+1UmjBiVKWm/FVla+lkd1Z/BDak0Hwfp3l31O1zRg+I +nNs4dS+ta84iPU5/q4UV8yGNwyIvj4lOGZImYTbfYosOCRtja7K8m29aMX4IjaME +4FBbRFiCL+Dnia2Sp70eR38K95mxpr2NsSYIgf/YzEvEjKqnlFY= +=0BT0 +-END PGP SIGNATURE- Added: dev/flink/flink-1.10.0-rc3/apache-flink-1.10.0.tar.gz.sha512 == --- dev/flink/flink-1.10.0-rc3/apache-flink-1.10.0.tar.gz.sha512 (added) +++ dev/flink/flink-1.10.0-rc3/apache-flink-1.10.0.tar.gz.sha512 Fri Feb 7 21:27:51 2020 @@ -0,0 +1 @@ +855216686362c24543ccd868f0488f1d345dfe1c3d524edefc2d33cc242826d850ffed4ba73d1345d98c416fc2dff40ffdbd4a2f5b907c04ca99747a2d8925c0 apache-flink-1.10.0.tar.gz Added: dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.11.tgz == Binary file - no diff available. Propchange: dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.11.tgz -- svn:mime-type = application/octet-stream Added: dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.11.tgz.asc == --- dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.11.tgz.asc (added) +++ dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.11.tgz.asc Fri Feb 7 21:27:51 2020 @@ -0,0 +1,16 @@ +-BEGIN PGP SIGNATURE- + +iQIzBAABCAAdFiEEuxN4B877590mFlVnELEqH4nBFegFAl49spgACgkQELEqH4nB +Feg0YxAAg89ZJer6izwuH3uV9sXifpRYXgGJhUiWZWavR0qBZQ0njW0bcYWrXHsi +vkNIavh62gJhfl72Zw+bmnQvQn8sXDa+ixm8l51YD2lTjRHH0heHRab+FIC72ld4 +02BFKJA8/fDYFjfAlCwsh6GujMDpTybuMr18tOzN87XgJ1dFa5hrDuOagr0aakzm +W4nU+If0uDnqPXI6bsJVnYaXmy9p8VcS4qEimOiKNNJViOV1S9yNNzDZJIq8RWo9 +ZjyGpJu0PFi9HFUu/01iD2YoV3eJhDbUqMeo0dgsoU8iEoCWCpj6tPUimpX27Opn +XrKr+zEPW/p/MWE/xTrzodxu9LndQ0Op5adfPcQ9MmQLa2Ukx3ubmsfFEgAHB/KG +pQQwEOChLHdCBNuGK2sxcYeAScPiTp90b2Meyy1ngFoGuM0v7JZbZ9USlxCkDehv +4NdFtQ/5V1AWDSlTI6BKqZl3YwapXJ3voBWiNMvM0sUtkkLl4bxuxYILGORsVINX +VC6DnqYtxFKeiqv2OLJMyo4RaH4gx1D4AOMlc3xW06K+DuxnvGDWVKiAm7fHTHdY +SuBPkxIhKs3T/sfpY/ScmunGyqel/DVTNGR3zpxIDw95SIwm6d5saZdd9SkM8UKi +fuSZSek/CyAwhd9jhOkAFwIqufBjQ1rhengS1v6JV20D+BzuoXA= +=Y78M +-END PGP SIGNATURE- Added: dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.11.tgz.sha512 == --- dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.11.tgz.sha512 (added) +++ dev/flink/flink-1.10.0-rc3/flink-1.10.0-bin-scala_2.11.tgz.sha512 Fri Feb 7 21:27:51 2020 @@ -0,0 +1 @@ +feb80a1a64a0ef50ff7c1dec07071c6c22cdb37fac53961c9ebd1f1e15c3b7530893b9f90ead1d23951487bb5ebf0fb7bb02f3055fadfa7d36fae0bdfc0a1502
[flink] annotated tag release-1.10.0-rc3 updated (aa4eb8f -> b81cbd7)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to annotated tag release-1.10.0-rc3 in repository https://gitbox.apache.org/repos/asf/flink.git. *** WARNING: tag release-1.10.0-rc3 was modified! *** from aa4eb8f (commit) to b81cbd7 (tag) tagging aa4eb8f0c9ce74e6b92c3d9be5dc8e8cb536239d (commit) replaces pre-apache-rename by Gary Yao on Fri Feb 7 19:18:20 2020 +0100 - Log - release-1.10.0-rc3 -BEGIN PGP SIGNATURE- iQJEBAABCAAuFiEEuxN4B877590mFlVnELEqH4nBFegFAl49qewQHGdhcnlAYXBh Y2hlLm9yZwAKCRAQsSoficEV6DqKD/wN5/I1rshvqxuaxF/wHVxogWf3mrWyIU91 EvsmLAP3+v8+FG/GPHpHmf989neEU7nUO4rjG997VuM2CYr3jZhNTpOlY05OvmkJ DHipusR7BShlJ9WsHuG32a1e0KxSbUjEg8VlOq3Ch78hqbfL2RIDaPh2g8ozAW01 ftvdhd6eNYzCsk76OlDFyDAY2f2tdIDR6Lhhl0GOEPTxEMyG+40SGinar+vOM405 ZzZ/4eGnryoUkFaccj/m2Gp2UuHKw1k96p3/Krcc03eb/lSbYJKYVR0R2Lql7vd/ 15IuMlEnrDap9ZbKZ805Bzm2HxQMkM2UqNc4q2kviBEQ6YMmKyDMGFcMFFFdhaRG vkA2lbkgpfC7FVFhjXybo9cV+DCAS0ssP9i2Dl/pEzSNQio3VdMvimVNK1UtTms2 Waz2cEFLgd8isUQnoQCXn3lPGNBWAU7Dglz+1y4AWZI/8D7bKEv9hLB2vG7RXpXY 8FcO/vyrQhkY8RNIZgbAlQN4m82TTZo8qDDuZtSLvgqjeihEyQGSioviRXnLjalf r+vE60YVxr8C5qadpq6Y4DmtS/Hfthv/tw5056o+W1esYkGEiX1ovro46Ic98e38 azKAcd1xJNfsm8A2sPIPzU2e/jgkibIRYdyho2Zm/4Y64/1NQpXJ9lxWoO3eNo6+ YNG0QiCiTg== =FnWz -END PGP SIGNATURE- --- No new revisions were added by this update. Summary of changes:
[flink] branch release-1.10 updated (cea09a8 -> 1268a7b)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch release-1.10 in repository https://gitbox.apache.org/repos/asf/flink.git. from cea09a8 [FLINK-15942][tests] Simplify ResourceProfileTest tests new 97960e2 [FLINK-15917][runtime] Fix that the root exception is not shown in Web UI new 1268a7b [FLINK-15918][runtime] Fix that 'uptime' metric is not reset after restart The 2 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../runtime/executiongraph/ExecutionGraph.java | 14 + .../flink/runtime/scheduler/DefaultScheduler.java | 24 +++- .../flink/runtime/scheduler/SchedulerBase.java | 12 +++- .../ManuallyTriggeredScheduledExecutor.java| 13 + .../runtime/scheduler/DefaultSchedulerTest.java| 67 ++ 5 files changed, 115 insertions(+), 15 deletions(-)
[flink] 01/02: [FLINK-15917][runtime] Fix that the root exception is not shown in Web UI
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.10 in repository https://gitbox.apache.org/repos/asf/flink.git commit 97960e28efb500e90d4618ba6492bca5f3da08cc Author: Gary Yao AuthorDate: Fri Feb 7 16:32:42 2020 +0100 [FLINK-15917][runtime] Fix that the root exception is not shown in Web UI --- .../flink/runtime/scheduler/DefaultScheduler.java| 3 ++- .../flink/runtime/scheduler/SchedulerBase.java | 8 ++-- .../runtime/scheduler/DefaultSchedulerTest.java | 20 3 files changed, 28 insertions(+), 3 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java index c107174..9779dac1 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java @@ -182,7 +182,8 @@ public class DefaultScheduler extends SchedulerBase implements SchedulerOperatio } } - private void handleTaskFailure(final ExecutionVertexID executionVertexId, final Throwable error) { + private void handleTaskFailure(final ExecutionVertexID executionVertexId, @Nullable final Throwable error) { + setGlobalFailureCause(error); final FailureHandlingResult failureHandlingResult = executionFailureHandler.getFailureHandlingResult(executionVertexId, error); maybeRestartTasks(failureHandlingResult); } diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java index d31e5e4..bd62c1e 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java @@ -94,6 +94,8 @@ import org.apache.flink.util.function.FunctionUtils; import org.slf4j.Logger; +import javax.annotation.Nullable; + import java.io.IOException; import java.net.InetSocketAddress; import java.util.Collection; @@ -348,8 +350,10 @@ public abstract class SchedulerBase implements SchedulerNG { .transitionState(ExecutionState.SCHEDULED)); } - protected void setGlobalFailureCause(final Throwable cause) { - getExecutionGraph().initFailureCause(cause); + protected void setGlobalFailureCause(@Nullable final Throwable cause) { + if (cause != null) { + getExecutionGraph().initFailureCause(cause); + } } protected ComponentMainThreadExecutor getMainThreadExecutor() { diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/DefaultSchedulerTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/DefaultSchedulerTest.java index 75ff382..40126cf 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/DefaultSchedulerTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/scheduler/DefaultSchedulerTest.java @@ -34,6 +34,7 @@ import org.apache.flink.runtime.concurrent.ManuallyTriggeredScheduledExecutor; import org.apache.flink.runtime.execution.ExecutionState; import org.apache.flink.runtime.executiongraph.AccessExecutionJobVertex; import org.apache.flink.runtime.executiongraph.ArchivedExecutionVertex; +import org.apache.flink.runtime.executiongraph.ErrorInfo; import org.apache.flink.runtime.executiongraph.ExecutionAttemptID; import org.apache.flink.runtime.executiongraph.failover.flip1.RestartPipelinedRegionStrategy; import org.apache.flink.runtime.executiongraph.failover.flip1.TestRestartBackoffTimeStrategy; @@ -89,10 +90,12 @@ import static org.apache.flink.util.ExceptionUtils.findThrowable; import static org.apache.flink.util.ExceptionUtils.findThrowableWithMessage; import static org.hamcrest.Matchers.contains; import static org.hamcrest.Matchers.containsInAnyOrder; +import static org.hamcrest.Matchers.containsString; import static org.hamcrest.Matchers.equalTo; import static org.hamcrest.Matchers.hasSize; import static org.hamcrest.Matchers.is; import static org.hamcrest.Matchers.lessThan; +import static org.hamcrest.Matchers.notNullValue; import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertFalse; import static org.junit.Assert.assertThat; @@ -596,6 +599,23 @@ public class DefaultSchedulerTest extends TestLogger { assertTrue(executionVertexVersioner.isModified(executionVertexVersion)); } + @Test + public void failureInfoIsSetAfterTaskFailure() { + final JobGraph jobGraph = singleNonParallelJobVertexJobGraph(); + final JobID jobId = jobGraph.getJobID(); + final
[flink] 02/02: [FLINK-15918][runtime] Fix that 'uptime' metric is not reset after restart
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.10 in repository https://gitbox.apache.org/repos/asf/flink.git commit 1268a7b9a3a3d07f76ea1fe78a0b1a6a7d0ef7eb Author: Gary Yao AuthorDate: Fri Feb 7 16:34:00 2020 +0100 [FLINK-15918][runtime] Fix that 'uptime' metric is not reset after restart The 'uptime' metric is the time difference between 'now' and the timestamp when the job transitioned to state RUNNING. The new scheduler until now never transitioned out of status RUNNING when restarting tasks which had the effect that the 'uptime' metric was not reset after a restart. This introduces new state transitions to the job. We transition the job status to RESTARTING if at least one ExecutionVertex is waiting to be restarted, and we transition from RESTARTING immediately to RUNNING again after the restart. This closes #11032. --- .../runtime/executiongraph/ExecutionGraph.java | 14 +-- .../flink/runtime/scheduler/DefaultScheduler.java | 21 ++ .../flink/runtime/scheduler/SchedulerBase.java | 4 ++ .../ManuallyTriggeredScheduledExecutor.java| 13 ++ .../runtime/scheduler/DefaultSchedulerTest.java| 47 ++ 5 files changed, 87 insertions(+), 12 deletions(-) diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java index 5803f8d..4447b1e 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionGraph.java @@ -898,7 +898,7 @@ public class ExecutionGraph implements AccessExecutionGraph { while (true) { JobStatus current = state; - if (current == JobStatus.RUNNING || current == JobStatus.CREATED) { + if (current == JobStatus.RUNNING || current == JobStatus.CREATED || current == JobStatus.RESTARTING) { if (transitionState(current, JobStatus.CANCELLING)) { // make sure no concurrent local actions interfere with the cancellation @@ -938,16 +938,6 @@ public class ExecutionGraph implements AccessExecutionGraph { return; } } - // All vertices have been cancelled and it's safe to directly go - // into the canceled state. - else if (current == JobStatus.RESTARTING) { - if (transitionState(current, JobStatus.CANCELED)) { - onTerminalState(JobStatus.CANCELED); - - LOG.info("Canceled during restart."); - return; - } - } else { // no need to treat other states return; @@ -1220,7 +1210,7 @@ public class ExecutionGraph implements AccessExecutionGraph { // State Transitions // - private boolean transitionState(JobStatus current, JobStatus newState) { + public boolean transitionState(JobStatus current, JobStatus newState) { return transitionState(current, newState, null); } diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java index 9779dac1..2fe8340 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/DefaultScheduler.java @@ -19,6 +19,7 @@ package org.apache.flink.runtime.scheduler; +import org.apache.flink.api.common.JobStatus; import org.apache.flink.api.common.time.Time; import org.apache.flink.configuration.Configuration; import org.apache.flink.runtime.blob.BlobWriter; @@ -90,6 +91,8 @@ public class DefaultScheduler extends SchedulerBase implements SchedulerOperatio private final ExecutionVertexOperations executionVertexOperations; + private final Set verticesWaitingForRestart; + public DefaultScheduler( final Logger log, final JobGraph jobGraph, @@ -151,6 +154,8 @@ public class DefaultScheduler extends SchedulerBase implements SchedulerOperatio restartBackoffTimeStrategy); this.schedulingStrategy = schedulingStr
svn commit: r37886 - /dev/flink/flink-1.10.0-rc2/
Author: gary Date: Wed Feb 5 18:55:24 2020 New Revision: 37886 Log: Add flink-1.10.0-rc2 Added: dev/flink/flink-1.10.0-rc2/ dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz (with props) dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.asc dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.sha512 dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz (with props) dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.asc dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.sha512 dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.12.tgz (with props) dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.12.tgz.asc dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.12.tgz.sha512 dev/flink/flink-1.10.0-rc2/flink-1.10.0-src.tgz (with props) dev/flink/flink-1.10.0-rc2/flink-1.10.0-src.tgz.asc dev/flink/flink-1.10.0-rc2/flink-1.10.0-src.tgz.sha512 Added: dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz == Binary file - no diff available. Propchange: dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz -- svn:mime-type = application/octet-stream Added: dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.asc == --- dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.asc (added) +++ dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.asc Wed Feb 5 18:55:24 2020 @@ -0,0 +1,16 @@ +-BEGIN PGP SIGNATURE- + +iQIzBAABCAAdFiEEuxN4B877590mFlVnELEqH4nBFegFAl47BiAACgkQELEqH4nB +FeiA+Q/+PqcdNINJnF2DZIaid8QxNdxcyJc4x5ppumPeRja4UCOtXf+Y7FPqpcM+ +e/jmp9DHP/rYosovJQz6PKN+GTRuxaKik/OnO3mNrBHq830/hYXSqRCMYAPSG6vx +ErDqphFsfpKaTilN1BN3t2XfKZFyHk97jsML6uw7HZTpbtXq+kLObQPOhdklIndE +pkKlnSXSiwIMQWeQ/Q7UxqTkhkfwoenqcWZ5c8xKOeIDZo4fj+tR1emH0z6zeDWZ +aJAIzJ03qCHOYqw8toc7CORWedCNIJx/7Nw0xDZy1AyJJ5LRuMAtMqbsGb9epkbu +v45BTmLG0nFQSV0XMUNuYdUZZmu1CCcMm7pvMdKLjlsYexc2TytOXRcJXcnDSFDj +7dp9nb44WzQIWubiDjgcoV/YIqPtLHmfdUtA95VW8WSl0YTGSOg+c9VkGgF8XtjI +jk93vS6Qgou7oKvzm7HKuNANMO1dtGJP9Wlgz6qNilaSE67V5tu/QXmCr5jfEoTx +utgCmnMLODNBR4V92viIBfemscmj22IcJEsgNxbml26O8axQGbDGC8QulmsKzhw4 +4yEl7izOJnvb0M9NMP2XBAAeJbQ/J06e8koBalXG9xthqVLzGipEa3B5nimxtQ0O +U71qIyq+3/R0C673R8gnYqT4hoNDRXUum9YW+RjLV7aP+EN5SFY= +=b6Ty +-END PGP SIGNATURE- Added: dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.sha512 == --- dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.sha512 (added) +++ dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.sha512 Wed Feb 5 18:55:24 2020 @@ -0,0 +1 @@ +0bedeb9a84fdfc5ec85bf19632c3af4ff9cd06e874a42cbb929b0faa18f038c99ef7c9b8b11fc0f55391afbec72115c32605a7244adf5a8f468d1fed1e7667ea apache-flink-1.10.0.tar.gz Added: dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz == Binary file - no diff available. Propchange: dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz -- svn:mime-type = application/octet-stream Added: dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.asc == --- dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.asc (added) +++ dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.asc Wed Feb 5 18:55:24 2020 @@ -0,0 +1,16 @@ +-BEGIN PGP SIGNATURE- + +iQIzBAABCAAdFiEEuxN4B877590mFlVnELEqH4nBFegFAl47BgsACgkQELEqH4nB +Fei5rRAAlcp89iw0DBO6/6+ar3Nqj6sEWlZCBWfva98qrq/Cv0yFqdRZF0Nv6w66 +y65pNNg299obqn5p/Ggx9uUlSTYiejdoX4WYO27fi59JD5GI9f+ExMjXGXic/fpE +8LKLxmVRcLgVCw0vDULuj6P+UztgsnxoKNbkykM+q3TdxWslQ+i+aBlf5Tzh47jO +msfbBClRiizZCNuU2xjhy4+XSmZLf0SFkfyE/82xPlOrrdE9bjX8ljlkFOX6M9S1 +KZll6LhOIN08zcDKMse8E+/yjDWVoz72Od4hwQkR521UVdOc4me3JeWXy8xosw+L +l8+a3kfXDCWcSO5U5vOVBFabqWAW9hNOlCGm3+B9YSaeRN4U9loizNXQVKQ4YpVp +x3uYPdgyA3Uq0iWot96XRkPAAgR+JrCz1wBpJ2J9jhixkDX4ueHqqMG2seoNuhhd +xJXXEMMEsOlsAaCFtnDy/FwrDy+lLOyagc5Sp90rbxomRRu/+SUxONFX4v5OQuy5 +0q2i51dTnPyVYQ1wvjbiNTFg8oybYoxQKE+L82jvl12hFTBwkOmxD+jJPubKM5K3 +s5EnOeiwBaMWhU1CLi7ynvPtNzuHM7xV+Jn5cxTK55FXfFg10kJh7ljR/HWvXukV +KlPOizOUKyyfhvJt1Pa5AWI7tsXL+S4TeVB8GVfYw6EOJKvFCaM= +=bOw+ +-END PGP SIGNATURE- Added: dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.sha512 == --- dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.sha512 (added) +++ dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.sha512 Wed Feb 5 18:55:24 2020 @@ -0,0 +1 @@ +c4f84593aac96fc7bc093adad5e3db0a8c8e0eb80f99d013ed3910958f38d89d3477e101130750254a876e4b5fd5ccf0cd0751b91d316f428f199d3b42a0bc9b
[flink] annotated tag release-1.10.0-rc2 updated (a44066b -> af4a9ee)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to annotated tag release-1.10.0-rc2 in repository https://gitbox.apache.org/repos/asf/flink.git. *** WARNING: tag release-1.10.0-rc2 was modified! *** from a44066b (commit) to af4a9ee (tag) tagging a44066b602169227e924be77c8b96ccc40a304f9 (commit) replaces pre-apache-rename by Gary Yao on Wed Feb 5 15:59:12 2020 +0100 - Log - release-1.10.0-rc2 -BEGIN PGP SIGNATURE- iQJEBAABCAAuFiEEuxN4B877590mFlVnELEqH4nBFegFAl462EAQHGdhcnlAYXBh Y2hlLm9yZwAKCRAQsSoficEV6KHTD/0YJJca7kLkcSF8AnGS99rc6wMbLEea5Zov g7kVNJXlJM2LgAvQrjdMTVQfJy956Q10eofA1JmNu3lP3/Cazs4Ei8r7p5BDsj+J N5BOZ0P+0okKWunwAsy6qZgDwg8ybtOpbmE7ITa9iT2sDVrW2Gk/7cHhNv8XZ7b1 Y8p2bA1KsDlLBJe9cf87/dZDakuv29OukznoTTJe+1LdSh8j5u4gk4237HLqgmkZ KgISrW+3OJOrTflxDOaGpX5TPbvQeCvEdbEG9FZfGR4eKM7Qq+7W5kesNXhdMqjB uMlsiBCaIq2YMnkOldghB/VPwzhpjc8MM9GUG635zzhCbt/q+/AGAy/zUZ4N40sh qTbw33aMqsm3ag+7hHDlO5bBGSHqd0lyy3lyYoGerCB6tEHfKxO5PksyhHhfbIJg x2Bvm+KD+YaGfMerhoKHtyDVmgS1GJPwYB5aRTgAdRAl6O3k0QH4SmN/5ZKsgX7b f+oCAkdNTtgneF15vclXSInUD5EBcgg+OBaUTSefFpo2CBzT4+io7x9v2JoT+Hpt Jcn6Q3gw6REfN7sjwXgLyS3qvP1u4ZMaLQ/3RnWnfHIE1WVON7hkZEsPUja92t6p YMCmkR0MezkC5w84i+0qtsWDuOXsqqqVvjDn6IDZflHKnoahxSPyVd8HFl1dBjjN gN6jn9/l9g== =Xr8E -END PGP SIGNATURE- --- No new revisions were added by this update. Summary of changes:
svn commit: r37845 - /dev/flink/flink-1.10.0-rc2/
Author: gary Date: Mon Feb 3 16:29:13 2020 New Revision: 37845 Log: Remove wrong files for flink-1.10.0-rc2 Removed: dev/flink/flink-1.10.0-rc2/
[flink] annotated tag release-1.10.0-rc2 updated (5ac1b15 -> adb03e2)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to annotated tag release-1.10.0-rc2 in repository https://gitbox.apache.org/repos/asf/flink.git. *** WARNING: tag release-1.10.0-rc2 was modified! *** from 5ac1b15 (commit) to adb03e2 (tag) tagging 5ac1b15fd28c8f6e62545d6f85e9cad5dd1ca18b (commit) replaces pre-apache-rename by Gary Yao on Mon Feb 3 14:13:51 2020 +0100 - Log - release-1.10.0-rc2 -BEGIN PGP SIGNATURE- iQJEBAABCAAuFiEEuxN4B877590mFlVnELEqH4nBFegFAl44HI8QHGdhcnlAYXBh Y2hlLm9yZwAKCRAQsSoficEV6AsfEADn3soamR/goABvzP87wJAB0xFLh54ph11u ps0AqDNEXMLS55WsiJtbT1M1Rw37NIiaNsa01R/Y+rqLeZZoxSuN7HURy3mbc/gu X2b2wUAzZDHbk5wv+h3uXpFhJ4lcWLtS6y5VAH1ymDoXJW7ceZyii9Uq2yzynW0n 5WKShqs03vNWiX53iTgSsR2y8l0wQtNqEJcbJOtoqlsVA3A5VYYeiNpYYEWEUQn/ y0NFXd1ccnmZzUg5igf4ldhB7PH+SMyiItEV+ZKG0gDO+Ltr2KFcNqLwSmEWewIT h2jv1aW2gxt0yZPI1X5o3hXYyhNR+FXIUKa1TUNzAabvlFS5wv8Iz93JGwcp3Axj aG3Mugm8/D1e7ESGtauZAO/xmYEzkaa93bDstFh/1/J0rdbQYN1FTaA535PVIHuP k497/NJmJo0VzoqfSpDmuSm7Joyh72IVKj9iUXwRYoPyvf4PuRazQrAbmJH7xltJ khdxKAVHExl6Q8oSFwLsXQ6u7vyINgcLkhk8JrC6UVpmJeOv3OY1b01QXDcpDcaj lpPDZFOyrFTcIAQ20s8dRjokVyEyzQ8WbUQ5K4gfCMLqCG543d+Iux3PBUJ4M/ts e3pJ5viRdNJ1XezGfjfqmyFzODyowzshC4PQEMIuR+/GljVLQcjKjj8uNMsLrD+i eb58dQE5hg== =4ByT -END PGP SIGNATURE- --- No new revisions were added by this update. Summary of changes:
svn commit: r37844 - /dev/flink/flink-1.10.0-rc2/
Author: gary Date: Mon Feb 3 16:14:56 2020 New Revision: 37844 Log: Add flink-1.10.0-rc2 Added: dev/flink/flink-1.10.0-rc2/ dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz (with props) dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.asc dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.sha512 dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz (with props) dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.asc dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.sha512 dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.12.tgz (with props) dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.12.tgz.asc dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.12.tgz.sha512 dev/flink/flink-1.10.0-rc2/flink-1.10.0-src.tgz (with props) dev/flink/flink-1.10.0-rc2/flink-1.10.0-src.tgz.asc dev/flink/flink-1.10.0-rc2/flink-1.10.0-src.tgz.sha512 Added: dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz == Binary file - no diff available. Propchange: dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz -- svn:mime-type = application/octet-stream Added: dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.asc == --- dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.asc (added) +++ dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.asc Mon Feb 3 16:14:56 2020 @@ -0,0 +1,16 @@ +-BEGIN PGP SIGNATURE- + +iQIzBAABCAAdFiEEuxN4B877590mFlVnELEqH4nBFegFAl44RXgACgkQELEqH4nB +Feg/Xw//XNLdaZApIIEyOZG/QV2/19Z96u3O9Sk2ZkBjGRy8f9DuLsQOngysSUht +TW/6T1lLdmazTt1oNiisuD+fCAxZpwiip2Tu16ZXfrsVGVHkBxJuXPblSM5gyYgq +TbOC6xrJ+aBrZASvekzgYr0gcowDaiKEVStYvZtXM40qIFMfTof9tMsEi6Hz4wyG +057smmm4wAgUuXgwVuB1qScSqV7af6AVG/v95SZn3ffoZ76NnsUvN9QivK4Lmyr1 +pmj8IZvQMSORp2fHAhYGO+F23xcnBL/A1m67/Iv/ftxm4xSQ6uEWvl/7cMcFlwKj +Y4bIHHi3RVIJBNKEhBTNmFw8//3Nk1RlMy6C84whGDbiuPgtfST028GFG6TNAU6d +C0Ee7ezvUjlCrBqxrxCmWkog88wh2NygeNOzYC21ulMMK8VGTJ0bOqxkRxlbNCJe +4d343ojJhBP/cLvdxwxD7g5nRxBVUYlbB5CPwoK3gHJRzPVlA40NbnRxbnN8kmlk +niwNMK+BZjTaPKBtiWEy62T+0qli0L+juaMzdF43WPfxaXsoW7VI0SatkDFrvJB4 +s0vzX0+s63oiqgKbOfpGihI/zayMQfJeIHueMXlV65CF1JCpFgiJxNj9QsRTShFk +q26J/pFhJuHJPP/7jdq+vSyb093QDpMh20HJ+hpW1L4cWEopPsQ= +=rkti +-END PGP SIGNATURE- Added: dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.sha512 == --- dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.sha512 (added) +++ dev/flink/flink-1.10.0-rc2/apache-flink-1.10.0.tar.gz.sha512 Mon Feb 3 16:14:56 2020 @@ -0,0 +1 @@ +f2180811fad92b6d9088e081969b5ebbef3109aa74eb26d1bca69ed5e15b8f59aad5d58737b0646957ef1db214969921ce1c2977a4b2ebf9ef7e92441cc53c67 apache-flink-1.10.0.tar.gz Added: dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz == Binary file - no diff available. Propchange: dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz -- svn:mime-type = application/octet-stream Added: dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.asc == --- dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.asc (added) +++ dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.asc Mon Feb 3 16:14:56 2020 @@ -0,0 +1,16 @@ +-BEGIN PGP SIGNATURE- + +iQIzBAABCAAdFiEEuxN4B877590mFlVnELEqH4nBFegFAl44RWIACgkQELEqH4nB +Fegk6A/+NrLZUC2HJxEfCIA8lJI2yTnjg5SZkZ/z/2Gjzr93+g3YEUBcurT7BUsH +3lImYngbVVXIYAxpN7R/EJanlAoTuUBqZv4OAGSzaXpLJ0pEXKfBwKLwk8H/rupC +heOotznZYKxEXJPcRnV1EYT+HZ8k05NIzxfhM3oavBrt3xmaMBjzRlrIWyVNPxEn +zAKdj+F7t56YNonPn7TeOFrdPC56TLEK7VxsDnZoQVtw3PQ/p6HCRW6F8Sq5+RCn +NjZjv5Rn/vAgOegLT1Vh3vP4BirrC4RasDU6gNEZmWsEXNB6kiPCRzZo1oZjIq4S +9HdqMNLg/RzZ4SSOhE3DLgUgvFxhTYhZXBfSJsk7VqNMADJ+T1TNQaPxkBPA+qjB ++u7gspuTsxab3IuXULeSGCdE8LqMcER4GfEraL5Ip2dqVr0jOqW22L3f5ghR4/1r +TKtXsBqBcqRSjqNbQ1EmnBtT/XvcX39XGREamWpjqoj9nbksTeCtCw6rlegbqbj1 +jVLB2o+5GOdZSOpjpQjaLvGFJWPeMqULBRkCvGPjj+QtLdfdZvnUaDq/m6SjEEB2 +rYlC9tPlHjgTBYr7uYHYt7nKlZXQMVJiY8n1VL0bmbC+X9voX60Vu6SAT/urcGAs ++sx3VgHRiCswRNvZ8kOpRcwjMu1uAJCKONXtmAVW+bUMZgaxH0s= +=9Sal +-END PGP SIGNATURE- Added: dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.sha512 == --- dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.sha512 (added) +++ dev/flink/flink-1.10.0-rc2/flink-1.10.0-bin-scala_2.11.tgz.sha512 Mon Feb 3 16:14:56 2020 @@ -0,0 +1 @@ +54f649fdc90ea7c59d5aad02cc26e2533c8028d6ebcd92a02e626eafa06ee621ff7f0f97d1b9c48180e60c554235d9f8a3eb1c1be2a5945f35d13d367942ff52
[flink] branch master updated (5619307 -> 631722d)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from 5619307 [hotfix][runtime] Add missing space to exception message add 631722d [FLINK-15743][docs] Add release notes for Flink 1.10 No new revisions were added by this update. Summary of changes: docs/index.md | 1 + docs/release-notes/flink-1.10.md| 378 docs/release-notes/flink-1.10.zh.md | 378 3 files changed, 757 insertions(+) create mode 100644 docs/release-notes/flink-1.10.md create mode 100644 docs/release-notes/flink-1.10.zh.md
[flink] branch release-1.10 updated: [hotfix][docs] Fix divergence between flink-1.10.md and flink-1.10.zh.md
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.10 in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/release-1.10 by this push: new e4e1bf9 [hotfix][docs] Fix divergence between flink-1.10.md and flink-1.10.zh.md e4e1bf9 is described below commit e4e1bf9de554c7aa9d6cf66bf023155298829d3b Author: Gary Yao AuthorDate: Wed Jan 29 13:30:21 2020 +0100 [hotfix][docs] Fix divergence between flink-1.10.md and flink-1.10.zh.md --- docs/release-notes/flink-1.10.zh.md | 81 +++-- 1 file changed, 41 insertions(+), 40 deletions(-) diff --git a/docs/release-notes/flink-1.10.zh.md b/docs/release-notes/flink-1.10.zh.md index 3cfa13b..cec6353 100644 --- a/docs/release-notes/flink-1.10.zh.md +++ b/docs/release-notes/flink-1.10.zh.md @@ -31,13 +31,11 @@ these notes carefully if you are planning to upgrade your Flink version to 1.10. ### Clusters & Deployment FileSystems should be loaded via Plugin Architecture ([FLINK-11956](https://issues.apache.org/jira/browse/FLINK-11956)) -In the s3-hadoop and s3-presto filesystems, classes from external -dependencies, such as the AWS SDK, are no longer relocated. In the past, class -relocation turned out to be problematic in combination with custom -implementations of the `AWSCredentialsProvider` interface. As a consequence of -removing class relocation, s3-hadoop and s3-presto filesystems can only be -used as [plugins]({{ site.baseurl }}/ops/filesystems/#pluggable-file-systems). -Other filesystems are strongly recommended to be only used as plugins. +s3-hadoop and s3-presto filesystems do no longer use class relocations and need +to be loaded through [plugins]({{ site.baseurl }}/ops/filesystems/#pluggable-file-systems) +but now seamlessly integrate with all credential providers. Other filesystems +are strongly recommended to be only used as plugins as we will continue to +remove relocations. Flink Client respects Classloading Policy ([FLINK-13749](https://issues.apache.org/jira/browse/FLINK-13749)) The Flink client now also respects the configured classloading policy, i.e., @@ -48,14 +46,14 @@ which case they should configure the classloading policy explicitly to use `parent-first` classloading, which was the previous (hard-coded) behaviour. Enable spreading out Tasks evenly across all TaskManagers ([FLINK-12122](https://issues.apache.org/jira/browse/FLINK-12122)) -When [FLIP-6](https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=65147077) -was rolled out with Flink 1.5.0, we changed how slots are allocated -from TaskManagers (TMs). Instead of evenly allocating the slots from all -registered TMs, we had the tendency to exhaust a TM before using another one. -To use a scheduling strategy that is more similar to the pre-FLIP-6 -behaviour, where Flink tries to spread out the workload across all available -TMs, one can set `cluster.evenly-spread-out-slots: true` in the -`flink-conf.yaml`. +When +[FLIP-6](https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=65147077) +was rolled out with Flink 1.5.0, we changed how slots are allocated from +TaskManagers (TMs). Instead of evenly allocating the slots from all registered +TMs, we had the tendency to exhaust a TM before using another one. To use a +scheduling strategy that is more similar to the pre-FLIP-6 behaviour, where +Flink tries to spread out the workload across all currently available TMs, one +can set `cluster.evenly-spread-out-slots: true` in the `flink-conf.yaml`. Directory Structure Change for highly available Artifacts ([FLINK-13633](https://issues.apache.org/jira/browse/FLINK-13633)) All highly available artifacts stored by Flink will now be stored under @@ -102,8 +100,6 @@ If you try to reuse your previous Flink configuration without any adjustments, the new memory model can result in differently computed memory parameters for the JVM and, thus, performance changes. -Please check the user documentation for more details. - # Deprecation and breaking changes The following options have been removed and have no effect anymore: @@ -126,7 +122,7 @@ The following options have been removed and have no effect anymore: taskmanager.memory.off-heap - On-heap managed memory is no longer supported + Support for on-heap managed memory has been removed, leaving off-heap managed memory as the only possibility taskmanager.memory.preallocate @@ -180,13 +176,18 @@ The container cut-off configuration options, `containerized.heap-cutoff-ratio` and `containerized.heap-cutoff-min`, have no effect for task executor processes anymore but they still have the same semantics for the JobManager process. - Fine Grained Operator Resource Management ([FLINK-14058](https://issues.apache.org/jira/browse/FLINK-14058)) - + Fine-grained Oper
[flink] branch release-1.10 updated (14a29a4 -> 5b8a3c2)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch release-1.10 in repository https://gitbox.apache.org/repos/asf/flink.git. from 14a29a4 [FLINK-15768] Consolidate executor-related classes in flink-client add 5b8a3c2 [FLINK-15743][docs] Add release notes for Flink 1.10 No new revisions were added by this update. Summary of changes: docs/index.md | 1 + docs/release-notes/flink-1.10.md| 378 docs/release-notes/flink-1.10.zh.md | 377 +++ 3 files changed, 756 insertions(+) create mode 100644 docs/release-notes/flink-1.10.md create mode 100644 docs/release-notes/flink-1.10.zh.md
[flink] branch master updated (fcd2834 -> 4aae548)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/flink.git. from fcd2834 [FLINK-15768] Consolidate executor-related classes in flink-client add 4aae548 [FLINK-15628][rest] Create webSumissionHandlers List with default capacity No new revisions were added by this update. Summary of changes: .../org/apache/flink/runtime/webmonitor/WebSubmissionExtension.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
svn commit: r37758 - /dev/flink/flink-1.10.0-rc1/
Author: gary Date: Mon Jan 27 20:45:52 2020 New Revision: 37758 Log: Add flink-1.10.0-rc1 Added: dev/flink/flink-1.10.0-rc1/ dev/flink/flink-1.10.0-rc1/apache-flink-1.10.0.tar.gz (with props) dev/flink/flink-1.10.0-rc1/apache-flink-1.10.0.tar.gz.asc dev/flink/flink-1.10.0-rc1/apache-flink-1.10.0.tar.gz.sha512 dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.11.tgz (with props) dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.11.tgz.asc dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.11.tgz.sha512 dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.12.tgz (with props) dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.12.tgz.asc dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.12.tgz.sha512 dev/flink/flink-1.10.0-rc1/flink-1.10.0-src.tgz (with props) dev/flink/flink-1.10.0-rc1/flink-1.10.0-src.tgz.asc dev/flink/flink-1.10.0-rc1/flink-1.10.0-src.tgz.sha512 Added: dev/flink/flink-1.10.0-rc1/apache-flink-1.10.0.tar.gz == Binary file - no diff available. Propchange: dev/flink/flink-1.10.0-rc1/apache-flink-1.10.0.tar.gz -- svn:mime-type = application/octet-stream Added: dev/flink/flink-1.10.0-rc1/apache-flink-1.10.0.tar.gz.asc == --- dev/flink/flink-1.10.0-rc1/apache-flink-1.10.0.tar.gz.asc (added) +++ dev/flink/flink-1.10.0-rc1/apache-flink-1.10.0.tar.gz.asc Mon Jan 27 20:45:52 2020 @@ -0,0 +1,16 @@ +-BEGIN PGP SIGNATURE- + +iQIzBAABCAAdFiEEuxN4B877590mFlVnELEqH4nBFegFAl4vRgEACgkQELEqH4nB +Fej/0w/+MvWXogNVhoSqHG3rLADwctQKrxCkA0HXNXGne0zxDmEpeGh5tH7pqlF6 +ink8KjMO3Na8/StSzSDGEuh+KTsTCRLbwCdr1NUgKvMUi8hh+1qOCNOrll6gppUW +JpFPf/CDnSEmaBe0MYZD3tD6YKUaA8Hf79ZRaR5ICCTNgKqcnjy0fRfdToVd+NrL +1FxcpBDVXJRfoJURc/r96oNDEPiBqbtpR8dwXBtY+dqKrpeH/ECR4LtqS0J1dDHW +oTPgz0T6JFHxcZb3C6ZL4KxrLjELrTmgwNfQRrlrvjzYfwtfB+ZdnApdzImjwqiF +7u+2GoTfOJjXKwKXZDF6VBjCS6qe3KvtHYtmtoP5RTWJJ3WWR3a7t8K1Wkn7g/ND +nQkggKY/7dYxSfz2AUCLCzAgNIXG/oocKOZszjlqdUxWBcOeYdlw1kZincgIX0fT +rXtSNjmDZqLRYoW2a2yqUiRCoZ/FI7GReC0uBbDJ8guJV9ETMJ1NGmFgAumNTSKg +HR6VSowLYvlQG3TxxdIVofXU24CyvrP2r5Cjv7aLGnLh4II+NhTyHBUtE8xkyn50 +0iHnJT+8pfH93Tstim2ZSqPUWr7qW6UD/luPgS+UCLn3RXsbdzy9lYPIqJgq99s6 +hC75syLIL7g215rWA+Ce0ZdBJco9jpJFkL6JVyX0Y6NkBf4p0N8= +=V7Z2 +-END PGP SIGNATURE- Added: dev/flink/flink-1.10.0-rc1/apache-flink-1.10.0.tar.gz.sha512 == --- dev/flink/flink-1.10.0-rc1/apache-flink-1.10.0.tar.gz.sha512 (added) +++ dev/flink/flink-1.10.0-rc1/apache-flink-1.10.0.tar.gz.sha512 Mon Jan 27 20:45:52 2020 @@ -0,0 +1 @@ +f415a008726836002f349490537c1ddc03bd6c85946dbf5f447b2983fc8ab2a85c200be40aeb73be7542dfd6026b0c53763e019fedab2452f1eb20a9b327c378 apache-flink-1.10.0.tar.gz Added: dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.11.tgz == Binary file - no diff available. Propchange: dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.11.tgz -- svn:mime-type = application/octet-stream Added: dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.11.tgz.asc == --- dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.11.tgz.asc (added) +++ dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.11.tgz.asc Mon Jan 27 20:45:52 2020 @@ -0,0 +1,16 @@ +-BEGIN PGP SIGNATURE- + +iQIzBAABCAAdFiEEuxN4B877590mFlVnELEqH4nBFegFAl4vRb8ACgkQELEqH4nB +FegdvRAAsxFTfKGqi6+OIkV7TFkO/iWuFlJud0Z/rKem9hiLVQshE9zVgC/2Ve0p +yvqCf1f+Q7K+bu95mEEB961RJhK3Y7ZyZCzLWcuc+ig9mRkrzfQ/GtXZqsyjz8N6 +JIsAiBymk3octQFHyZ4ykbTEquHeaTcaAZsF8Ton7UxvBPHTsvSXjXePh2PuCKCM +jCzVjR5++9hUIg+99JzLHHKge5P9pLs1s3p4vf5hfcU2ELDOMU91v8vj2GKNY/dX +hAAU++e0XrlH6eIhqTCJDxQlz58IMa/zrqk5JTsC9JBzUTjYZQGA05aa4IEVaG+m +IqbXbHcAB8KKK/keecIS8BaFJoe0LWFKxOYzQSBM0PtNYoh4o4aIFVgrilnUo6LU +oTyyc/5nt2xbK922MV3qbSGPXrzKimUCwZeDeG1qdR0FHKkDc0VsziwAMs9rvFHQ ++Q9fCoqLz2jDpDI/VKa3Mf90D9dg+gP5XgObiQZn+AWdF6qv2nfgfZLS30ujdkBZ +cvwuvwd5JR1narNFK4zIKmnXWlwG2eeLWe7ilrX5UKKqILygufCzt/OuL+WpQROw +tyamGygaktGbiz3vUuSRGFsrNIg03rWfn/LQZElvh8Gw7usQUPqTxUCPFAIcTRRG +M1JG5sH+4pN+9qTkX4BeV+pnzi6VLCCy4sb0g0pbM/JeUDijfNY= +=sbtb +-END PGP SIGNATURE- Added: dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.11.tgz.sha512 == --- dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.11.tgz.sha512 (added) +++ dev/flink/flink-1.10.0-rc1/flink-1.10.0-bin-scala_2.11.tgz.sha512 Mon Jan 27 20:45:52 2020 @@ -0,0 +1 @@ +1b09bb08f9f12a4b2ef066083fcb7697fbb7bad783b1b88b23601a755fd8ccaed7bbf2040bd0de4ad41f2e9e0900e45dead5222b9c684f2312435af73690
[flink] annotated tag release-1.10.0-rc1 updated (96992c0 -> 6af3e55)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to annotated tag release-1.10.0-rc1 in repository https://gitbox.apache.org/repos/asf/flink.git. *** WARNING: tag release-1.10.0-rc1 was modified! *** from 96992c0 (commit) to 6af3e55 (tag) tagging 96992c0a23c2eb0d5bc7dc462b407950c2ae0833 (commit) replaces pre-apache-rename by Gary Yao on Mon Jan 27 16:48:02 2020 +0100 - Log - release-1.10.0-rc1 -BEGIN PGP SIGNATURE- iQJEBAABCAAuFiEEuxN4B877590mFlVnELEqH4nBFegFAl4vBjIQHGdhcnlAYXBh Y2hlLm9yZwAKCRAQsSoficEV6BEoEADePuCXUvQluZdsVqdCAfLtB7bvwv/2VFDV C5/jOMNqGEVC1cEgoKM7Vs67PJe5nVdcMiGzQG6QSXPBqscdHw4+fz8Hqqwln6xB DKzgb/o8djWa1hotMJIh31DuHLPebVnXzhWpCiFPjsBDoz0juBtdbXK/qbKO3bjZ vbV50Axg+NZf+CYIc83VxspmWbI2ZjQDIDYapH00CINDOl8oBgmPwIr2rWOzDFwU LUMgcQF7a9asozgFG6SbpbaicMV9qofXTJDCubj0P0Rq7/+yiHHvFoYEkh6sO3R+ 7++sRE7hg5u08ChKFpq5hCHBWnDH41EjVLLBK998WIrCKQwrU+XfI7A0NF0zBfCZ o/7qepXJDJMi21TjIGFh/Q8jazkeCnT20mTOMO0zra10C3rGkieCARDPNVo1JcoN vYyR1ekivtx77QvoMUND3HD25c2I3skdxAxZuO7PmNiXO51GiQgw91fqOtraO8oP h1YXKIe0fTTJuWGJD+r3/NIpvt6FEAySshg7uNX7h+Dh/W7boN/M6/foTKvH/6cj lD9jR1rIOm6FAIKacLfuC+PatTc2M8FOxR1WSk5E2kl1F3CrKJ8nG3r6qMYs+qiz wEDKntG+8xD5f3gTuqtRNXqjzUPmVynFbuC8Z3NudqzgwKRhsJE3yq/lRjOysxXQ tKPwK0J8sQ== =Qw7Y -END PGP SIGNATURE- --- No new revisions were added by this update. Summary of changes:
[flink] branch release-1.10 updated: [hotfix][docs] Add missing endhighlight tag
This is an automated email from the ASF dual-hosted git repository. gary pushed a commit to branch release-1.10 in repository https://gitbox.apache.org/repos/asf/flink.git The following commit(s) were added to refs/heads/release-1.10 by this push: new 10d1821 [hotfix][docs] Add missing endhighlight tag 10d1821 is described below commit 10d1821dcbe4d6d1d445bdf97f65c3f5032ebda5 Author: Gary Yao AuthorDate: Thu Jan 23 17:13:41 2020 +0100 [hotfix][docs] Add missing endhighlight tag --- docs/dev/table/catalogs.md | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/docs/dev/table/catalogs.md b/docs/dev/table/catalogs.md index 79972e1..6fb2b47 100644 --- a/docs/dev/table/catalogs.md +++ b/docs/dev/table/catalogs.md @@ -135,7 +135,7 @@ catalog.createTable( ); List tables = catalog.listTables("mydb); // tables should contain "mytable" - +{% endhighlight %}
[flink] branch shaded-hadoop-s3a created (now a4f2271)
This is an automated email from the ASF dual-hosted git repository. gary pushed a change to branch shaded-hadoop-s3a in repository https://gitbox.apache.org/repos/asf/flink.git. at a4f2271 Loop Shaded Hadoop S3A with credentials provider This branch includes the following new commits: new a4f2271 Loop Shaded Hadoop S3A with credentials provider The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference.