[3/4] flink git commit: [FLINK-1771] Add support for submitting single jobs to a detached YARN session
http://git-wip-us.apache.org/repos/asf/flink/blob/6b0d4076/flink-optimizer/src/test/java/org/apache/flink/optimizer/PropertyDataSourceTest.java -- diff --git a/flink-optimizer/src/test/java/org/apache/flink/optimizer/PropertyDataSourceTest.java b/flink-optimizer/src/test/java/org/apache/flink/optimizer/PropertyDataSourceTest.java index 3cf081f..d6b9444 100644 --- a/flink-optimizer/src/test/java/org/apache/flink/optimizer/PropertyDataSourceTest.java +++ b/flink-optimizer/src/test/java/org/apache/flink/optimizer/PropertyDataSourceTest.java @@ -35,6 +35,7 @@ import org.apache.flink.optimizer.plan.NAryUnionPlanNode; import org.apache.flink.optimizer.plan.OptimizedPlan; import org.apache.flink.optimizer.plan.SinkPlanNode; import org.apache.flink.optimizer.plan.SourcePlanNode; +import org.apache.flink.optimizer.util.CompilerTestBase; import org.junit.Assert; import org.junit.Test; http://git-wip-us.apache.org/repos/asf/flink/blob/6b0d4076/flink-optimizer/src/test/java/org/apache/flink/optimizer/ReduceAllTest.java -- diff --git a/flink-optimizer/src/test/java/org/apache/flink/optimizer/ReduceAllTest.java b/flink-optimizer/src/test/java/org/apache/flink/optimizer/ReduceAllTest.java index f6885c5..3af64fc 100644 --- a/flink-optimizer/src/test/java/org/apache/flink/optimizer/ReduceAllTest.java +++ b/flink-optimizer/src/test/java/org/apache/flink/optimizer/ReduceAllTest.java @@ -26,6 +26,7 @@ import org.apache.flink.api.java.record.operators.FileDataSource; import org.apache.flink.api.java.record.operators.ReduceOperator; import org.apache.flink.optimizer.plan.OptimizedPlan; import org.apache.flink.optimizer.plantranslate.JobGraphGenerator; +import org.apache.flink.optimizer.util.CompilerTestBase; import org.apache.flink.optimizer.util.DummyInputFormat; import org.apache.flink.optimizer.util.DummyOutputFormat; import org.apache.flink.optimizer.util.IdentityReduce; http://git-wip-us.apache.org/repos/asf/flink/blob/6b0d4076/flink-optimizer/src/test/java/org/apache/flink/optimizer/ReplicatingDataSourceTest.java -- diff --git a/flink-optimizer/src/test/java/org/apache/flink/optimizer/ReplicatingDataSourceTest.java b/flink-optimizer/src/test/java/org/apache/flink/optimizer/ReplicatingDataSourceTest.java index 230cc6b..25643a4 100644 --- a/flink-optimizer/src/test/java/org/apache/flink/optimizer/ReplicatingDataSourceTest.java +++ b/flink-optimizer/src/test/java/org/apache/flink/optimizer/ReplicatingDataSourceTest.java @@ -39,6 +39,7 @@ import org.apache.flink.optimizer.plan.SinkPlanNode; import org.apache.flink.core.fs.FileInputSplit; import org.apache.flink.core.fs.Path; import org.apache.flink.runtime.operators.shipping.ShipStrategyType; +import org.apache.flink.optimizer.util.CompilerTestBase; import org.apache.flink.util.Collector; import org.junit.Assert; import org.junit.Test; http://git-wip-us.apache.org/repos/asf/flink/blob/6b0d4076/flink-optimizer/src/test/java/org/apache/flink/optimizer/SemanticPropertiesAPIToPlanTest.java -- diff --git a/flink-optimizer/src/test/java/org/apache/flink/optimizer/SemanticPropertiesAPIToPlanTest.java b/flink-optimizer/src/test/java/org/apache/flink/optimizer/SemanticPropertiesAPIToPlanTest.java index 1fe16bb..3a24ce1 100644 --- a/flink-optimizer/src/test/java/org/apache/flink/optimizer/SemanticPropertiesAPIToPlanTest.java +++ b/flink-optimizer/src/test/java/org/apache/flink/optimizer/SemanticPropertiesAPIToPlanTest.java @@ -38,6 +38,7 @@ import org.apache.flink.optimizer.plan.OptimizedPlan; import org.apache.flink.optimizer.plan.PlanNode; import org.apache.flink.optimizer.plan.SingleInputPlanNode; import org.apache.flink.runtime.operators.shipping.ShipStrategyType; +import org.apache.flink.optimizer.util.CompilerTestBase; import org.apache.flink.util.Visitor; import org.junit.Assert; import org.junit.Test; http://git-wip-us.apache.org/repos/asf/flink/blob/6b0d4076/flink-optimizer/src/test/java/org/apache/flink/optimizer/SortPartialReuseTest.java -- diff --git a/flink-optimizer/src/test/java/org/apache/flink/optimizer/SortPartialReuseTest.java b/flink-optimizer/src/test/java/org/apache/flink/optimizer/SortPartialReuseTest.java index 40b54e0..65e5025 100644 --- a/flink-optimizer/src/test/java/org/apache/flink/optimizer/SortPartialReuseTest.java +++ b/flink-optimizer/src/test/java/org/apache/flink/optimizer/SortPartialReuseTest.java @@ -20,6 +20,7 @@ package org.apache.flink.optimizer; import static org.junit.Assert.*; +import org.apache.flink.optimizer.util.CompilerTestBase; import org.junit.Test; import org.apache.flink.api.common.Plan; import
[4/4] flink git commit: [FLINK-1771] Add support for submitting single jobs to a detached YARN session
[FLINK-1771] Add support for submitting single jobs to a detached YARN session With this change, users can submit a Flink job to a YARN cluster without having a local client monitoring the Application Master or job. You can basically fire and forget a Flink job to YARN. For supporting this, the ApplicationMaster can now monitor the status of a job and shutdown itself once it is in a terminal state. The change also verifies that various ways of setting the parallelism on YARN are passed through the system correctly (per job, session). There was a bug in YARN container creation which made the configuration values for the heap offset useless. This change fixes this error. All mentioned features and bugs are covered by the flink-yarn-tests. This closes #542 Project: http://git-wip-us.apache.org/repos/asf/flink/repo Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/6b0d4076 Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/6b0d4076 Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/6b0d4076 Branch: refs/heads/master Commit: 6b0d40764da9dce2e2d21882e9a03a21c6783ff0 Parents: 121a5a0 Author: Robert Metzger rmetz...@apache.org Authored: Fri Mar 13 15:53:51 2015 +0100 Committer: Robert Metzger rmetz...@apache.org Committed: Tue Mar 31 08:56:39 2015 +0200 -- .../org/apache/flink/client/CliFrontend.java| 116 ++--- .../flink/client/FlinkYarnSessionCli.java | 12 +- .../org/apache/flink/client/LocalExecutor.java | 19 +- .../org/apache/flink/client/RemoteExecutor.java | 27 ++- .../org/apache/flink/client/program/Client.java | 85 --- .../client/program/ContextEnvironment.java | 39 ++- .../flink/client/CliFrontendInfoTest.java | 4 +- .../flink/client/CliFrontendListCancelTest.java | 2 +- .../apache/flink/client/CliFrontendRunTest.java | 2 +- .../client/program/ClientConnectionTest.java| 2 +- .../apache/flink/client/program/ClientTest.java | 9 +- .../ExecutionPlanAfterExecutionTest.java| 3 +- .../program/ExecutionPlanCreationTest.java | 2 +- .../flink/api/common/JobExecutionResult.java| 7 +- .../java/org/apache/flink/api/common/JobID.java | 60 + .../flink/api/common/JobSubmissionResult.java | 39 +++ .../common/operators/CollectionExecutor.java| 2 +- .../main/flink-bin/conf/log4j-cli.properties| 1 + .../examples/java/wordcount/WordCount.java | 1 - flink-optimizer/pom.xml | 19 +- .../org/apache/flink/optimizer/Optimizer.java | 20 +- .../optimizer/AdditionalOperatorsTest.java | 1 + .../optimizer/BranchingPlansCompilerTest.java | 1 + .../BroadcastVariablePipelinebreakerTest.java | 1 + .../CachedMatchStrategyCompilerTest.java| 1 + .../optimizer/CoGroupSolutionSetFirstTest.java | 1 + .../flink/optimizer/CompilerTestBase.java | 229 -- .../flink/optimizer/DisjointDataFlowsTest.java | 1 + .../optimizer/DistinctCompilationTest.java | 1 + .../apache/flink/optimizer/GroupOrderTest.java | 1 + .../optimizer/HardPlansCompilationTest.java | 1 + .../flink/optimizer/IterationsCompilerTest.java | 1 + .../flink/optimizer/NestedIterationsTest.java | 1 + .../flink/optimizer/ParallelismChangeTest.java | 1 + .../flink/optimizer/PartitionPushdownTest.java | 1 + .../optimizer/PartitioningReusageTest.java | 1 + .../flink/optimizer/PipelineBreakerTest.java| 1 + .../flink/optimizer/PropertyDataSourceTest.java | 1 + .../apache/flink/optimizer/ReduceAllTest.java | 1 + .../optimizer/ReplicatingDataSourceTest.java| 1 + .../SemanticPropertiesAPIToPlanTest.java| 1 + .../flink/optimizer/SortPartialReuseTest.java | 1 + .../UnionBetweenDynamicAndStaticPathTest.java | 1 + .../optimizer/UnionPropertyPropagationTest.java | 1 + .../flink/optimizer/UnionReplacementTest.java | 1 + .../WorksetIterationCornerCasesTest.java| 1 + .../WorksetIterationsRecordApiCompilerTest.java | 1 + ...naryCustomPartitioningCompatibilityTest.java | 2 +- .../CoGroupCustomPartitioningTest.java | 2 +- ...ustomPartitioningGlobalOptimizationTest.java | 2 +- .../custompartition/CustomPartitioningTest.java | 2 +- .../GroupingKeySelectorTranslationTest.java | 2 +- .../GroupingPojoTranslationTest.java| 2 +- .../GroupingTupleTranslationTest.java | 2 +- .../JoinCustomPartitioningTest.java | 2 +- .../DataExchangeModeClosedBranchingTest.java| 2 +- .../DataExchangeModeForwardTest.java| 2 +- .../DataExchangeModeOpenBranchingTest.java | 2 +- .../java/DeltaIterationDependenciesTest.java| 2 +- .../java/DistinctAndGroupingOptimizerTest.java | 2 +- .../java/GroupReduceCompilationTest.java| 2 +- .../optimizer/java/IterationCompilerTest.java | 2 +-
flink git commit: [FLINK-1806] Improve error message when no S3 access/secret key configured
Repository: flink Updated Branches: refs/heads/master 6b0d40764 - c0b297573 [FLINK-1806] Improve error message when no S3 access/secret key configured Project: http://git-wip-us.apache.org/repos/asf/flink/repo Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/c0b29757 Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/c0b29757 Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/c0b29757 Branch: refs/heads/master Commit: c0b29757390225d49fccd1f680c87501e7714e32 Parents: 6b0d407 Author: Ufuk Celebi u...@apache.org Authored: Tue Mar 31 11:33:12 2015 +0200 Committer: Ufuk Celebi u...@apache.org Committed: Tue Mar 31 11:33:12 2015 +0200 -- .../java/org/apache/flink/runtime/fs/s3/S3FileSystem.java| 8 ++-- 1 file changed, 6 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/flink/blob/c0b29757/flink-runtime/src/main/java/org/apache/flink/runtime/fs/s3/S3FileSystem.java -- diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/fs/s3/S3FileSystem.java b/flink-runtime/src/main/java/org/apache/flink/runtime/fs/s3/S3FileSystem.java index d3eb0e2..522d90e 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/fs/s3/S3FileSystem.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/fs/s3/S3FileSystem.java @@ -204,7 +204,9 @@ public final class S3FileSystem extends FileSystem { LOG.debug(Provided URI does not provide an access key to Amazon S3, using configuration...); awsAccessKey = GlobalConfiguration.getString(S3_ACCESS_KEY_KEY, null); if (awsAccessKey == null) { - throw new IOException(Cannot determine access key to Amazon S3); + throw new IOException(Cannot determine access key to Amazon S3. Please make + + sure to configure it by setting the configuration key ' + + S3_ACCESS_KEY_KEY + '.); } } @@ -212,7 +214,9 @@ public final class S3FileSystem extends FileSystem { LOG.debug(Provided URI does not provide a secret key to Amazon S3, using configuration...); awsSecretKey = GlobalConfiguration.getString(S3_SECRET_KEY_KEY, null); if (awsSecretKey == null) { - throw new IOException(Cannot determine secret key to Amazon S3); + throw new IOException(Cannot determine secret key to Amazon S3. Please make + + sure to configure it by setting the configuration key ' + + S3_SECRET_KEY_KEY + '.); } }
flink git commit: [FLINK-1805]The class IOManagerAsync should use its own Log instance
Repository: flink Updated Branches: refs/heads/master b8aa49ce7 - 224fa94c6 [FLINK-1805]The class IOManagerAsync should use its own Log instance Although class 'IOManagerAsync' is extended from 'IOManager' in package 'org.apache.flink.runtime.io.disk.iomanager', but I think it should has its own Log instance. Author: hongsibao hongsi...@huawei.com Closes #546 from matadorhong/FLINK-1805 and squashes the following commits: 89c0dd7 [hongsibao] Use private instead of protected 56b9dab [hongsibao] The class IOManagerAsync(in org.apache.flink.runtime.io.disk.iomanager) should use its own Log instance Project: http://git-wip-us.apache.org/repos/asf/flink/repo Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/224fa94c Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/224fa94c Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/224fa94c Branch: refs/heads/master Commit: 224fa94c620491b364c715b10d7f627bb5c739dc Parents: b8aa49c Author: hongsibao hongsi...@huawei.com Authored: Tue Mar 31 09:54:25 2015 -0700 Committer: Henry Saputra henry.sapu...@gmail.com Committed: Tue Mar 31 09:54:25 2015 -0700 -- .../flink/runtime/io/disk/iomanager/IOManagerAsync.java | 7 ++- 1 file changed, 6 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/flink/blob/224fa94c/flink-runtime/src/main/java/org/apache/flink/runtime/io/disk/iomanager/IOManagerAsync.java -- diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/io/disk/iomanager/IOManagerAsync.java b/flink-runtime/src/main/java/org/apache/flink/runtime/io/disk/iomanager/IOManagerAsync.java index e615913..4401d03 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/io/disk/iomanager/IOManagerAsync.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/io/disk/iomanager/IOManagerAsync.java @@ -21,6 +21,8 @@ package org.apache.flink.runtime.io.disk.iomanager; import org.apache.flink.core.memory.MemorySegment; import org.apache.flink.runtime.io.network.buffer.Buffer; import org.apache.flink.runtime.util.EnvironmentInformation; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; import java.io.IOException; import java.lang.Thread.UncaughtExceptionHandler; @@ -43,7 +45,10 @@ public class IOManagerAsync extends IOManager implements UncaughtExceptionHandle /** Flag to signify that the IOManager has been shut down already */ private final AtomicBoolean isShutdown = new AtomicBoolean(); - + + /** Logging */ + private static final Logger LOG = LoggerFactory.getLogger(IOManagerAsync.class); + // - // Constructors / Destructors // -
flink git commit: [FLINK-1766]Fix the bug of equals function of FSKey
Repository: flink Updated Branches: refs/heads/master 224fa94c6 - d8c74d209 [FLINK-1766]Fix the bug of equals function of FSKey The equals function in org.apache.flink.core.fs.FileSystem.FSKey should first confirm whether obj == this, if obj is the same object.It should return true Author: hongsibao hongsi...@huawei.com Closes #511 from matadorhong/Bug_Flink_1776 and squashes the following commits: 1ad06d7 [hongsibao] Fix the code format problem 431fc4b [hongsibao] Fix the code format problem ab0ae5e [hongsibao] [FLINK-1766]Fix the bug of equals function of FSKey Project: http://git-wip-us.apache.org/repos/asf/flink/repo Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/d8c74d20 Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/d8c74d20 Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/d8c74d20 Branch: refs/heads/master Commit: d8c74d20970e6cf379165ffafe8550a5b98d7c4d Parents: 224fa94 Author: hongsibao hongsi...@huawei.com Authored: Tue Mar 31 10:54:27 2015 -0700 Committer: Henry Saputra henry.sapu...@gmail.com Committed: Tue Mar 31 10:54:27 2015 -0700 -- .../src/main/java/org/apache/flink/core/fs/FileSystem.java | 4 +++- 1 file changed, 3 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/flink/blob/d8c74d20/flink-core/src/main/java/org/apache/flink/core/fs/FileSystem.java -- diff --git a/flink-core/src/main/java/org/apache/flink/core/fs/FileSystem.java b/flink-core/src/main/java/org/apache/flink/core/fs/FileSystem.java index 7e8c667..0e38d8a 100644 --- a/flink-core/src/main/java/org/apache/flink/core/fs/FileSystem.java +++ b/flink-core/src/main/java/org/apache/flink/core/fs/FileSystem.java @@ -106,6 +106,9 @@ public abstract class FileSystem { */ @Override public boolean equals(final Object obj) { + if (obj == this) { + return true; + } if (obj instanceof FSKey) { final FSKey key = (FSKey) obj; @@ -127,7 +130,6 @@ public abstract class FileSystem { */ @Override public int hashCode() { - if (this.scheme != null) { return this.scheme.hashCode(); }
flink git commit: [FLINK-1794] [test-utils] Adds test base for scala tests and adapts existing flink-ml tests
Repository: flink Updated Branches: refs/heads/master c0b297573 - b8aa49ce7 [FLINK-1794] [test-utils] Adds test base for scala tests and adapts existing flink-ml tests [FLINK-1794] [test-utils] Adds scala docs to FlinkTestBase This closes #540. Project: http://git-wip-us.apache.org/repos/asf/flink/repo Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/b8aa49ce Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/b8aa49ce Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/b8aa49ce Branch: refs/heads/master Commit: b8aa49ce7d806c52183ab79c17c6024b60d0eaaf Parents: c0b2975 Author: Till Rohrmann trohrm...@apache.org Authored: Fri Mar 27 19:40:42 2015 +0100 Committer: Till Rohrmann trohrm...@apache.org Committed: Tue Mar 31 13:40:53 2015 +0200 -- .../org/apache/flink/client/program/Client.java | 2 +- flink-staging/flink-ml/pom.xml | 25 ++- .../flink/ml/feature/PolynomialBaseITCase.scala | 132 .../ml/feature/PolynomialBaseITSuite.scala | 126 +++ .../flink/ml/recommendation/ALSITCase.scala | 153 --- .../flink/ml/recommendation/ALSITSuite.scala| 149 ++ .../MultipleLinearRegressionITCase.scala| 115 -- .../MultipleLinearRegressionITSuite.scala | 109 + flink-test-utils/pom.xml| 6 + .../apache/flink/test/util/FlinkTestBase.scala | 68 + 10 files changed, 479 insertions(+), 406 deletions(-) -- http://git-wip-us.apache.org/repos/asf/flink/blob/b8aa49ce/flink-clients/src/main/java/org/apache/flink/client/program/Client.java -- diff --git a/flink-clients/src/main/java/org/apache/flink/client/program/Client.java b/flink-clients/src/main/java/org/apache/flink/client/program/Client.java index 6dff9e2..dc1a7ab 100644 --- a/flink-clients/src/main/java/org/apache/flink/client/program/Client.java +++ b/flink-clients/src/main/java/org/apache/flink/client/program/Client.java @@ -70,7 +70,7 @@ public class Client { private final Optimizer compiler; // the compiler to compile the jobs - private boolean printStatusDuringExecution = false; + private boolean printStatusDuringExecution = true; /** * If != -1, this field specifies the total number of available slots on the cluster http://git-wip-us.apache.org/repos/asf/flink/blob/b8aa49ce/flink-staging/flink-ml/pom.xml -- diff --git a/flink-staging/flink-ml/pom.xml b/flink-staging/flink-ml/pom.xml index 123770f..4f251e5 100644 --- a/flink-staging/flink-ml/pom.xml +++ b/flink-staging/flink-ml/pom.xml @@ -41,11 +41,6 @@ /dependency dependency - groupIdorg.scalatest/groupId - artifactIdscalatest_${scala.binary.version}/artifactId - /dependency - - dependency groupIdcom.github.fommil.netlib/groupId artifactIdcore/artifactId version1.1.2/version @@ -65,6 +60,13 @@ typetest-jar/type scopetest/scope /dependency + + dependency + groupIdorg.apache.flink/groupId + artifactIdflink-test-utils/artifactId + version${project.version}/version + scopetest/scope + /dependency /dependencies build @@ -105,6 +107,19 @@ goals goaltest/goal /goals + configuration + suffixes(?lt;!(IT|Integration))(Test|Suite|Case)/suffixes + /configuration + /execution + execution + idintegration-test/id + phaseintegration-test/phase + goals + goaltest/goal + /goals + configuration + suffixes(IT|Integration)(Test|Suite|Case)/suffixes + /configuration /execution /executions