tillrohrmann commented on a change in pull request #7613: 
[FLINK-11370][test]Check and port ZooKeeperLeaderElectionITCase to ne…
URL: https://github.com/apache/flink/pull/7613#discussion_r252713595
 
 

 ##########
 File path: 
flink-tests/src/test/java/org/apache/flink/test/runtime/leaderelection/ZooKeeperLeaderElectionITCase.java
 ##########
 @@ -1,301 +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.test.runtime.leaderelection;
-
-import org.apache.flink.api.common.JobExecutionResult;
-import org.apache.flink.configuration.AkkaOptions;
-import org.apache.flink.configuration.ConfigConstants;
-import org.apache.flink.configuration.Configuration;
-import org.apache.flink.configuration.HighAvailabilityOptions;
-import org.apache.flink.configuration.TaskManagerOptions;
-import org.apache.flink.runtime.akka.AkkaUtils;
-import org.apache.flink.runtime.client.JobClient;
-import org.apache.flink.runtime.instance.ActorGateway;
-import org.apache.flink.runtime.io.network.partition.ResultPartitionType;
-import org.apache.flink.runtime.jobgraph.DistributionPattern;
-import org.apache.flink.runtime.jobgraph.JobGraph;
-import org.apache.flink.runtime.jobgraph.JobVertex;
-import org.apache.flink.runtime.jobmanager.Tasks;
-import org.apache.flink.runtime.jobmanager.scheduler.SlotSharingGroup;
-import org.apache.flink.runtime.messages.JobManagerMessages;
-import org.apache.flink.runtime.minicluster.LocalFlinkMiniCluster;
-import org.apache.flink.runtime.testingUtils.TestingCluster;
-import org.apache.flink.runtime.testingUtils.TestingJobManagerMessages;
-import 
org.apache.flink.runtime.testingUtils.TestingJobManagerMessages.WaitForAllVerticesToBeRunningOrFinished;
-import org.apache.flink.runtime.testingUtils.TestingUtils;
-import org.apache.flink.runtime.testutils.ZooKeeperTestUtils;
-import org.apache.flink.util.TestLogger;
-
-import akka.actor.ActorSystem;
-import akka.actor.Kill;
-import akka.actor.PoisonPill;
-import org.apache.curator.test.TestingServer;
-import org.junit.AfterClass;
-import org.junit.BeforeClass;
-import org.junit.Rule;
-import org.junit.Test;
-import org.junit.rules.TemporaryFolder;
-import org.slf4j.Logger;
-import org.slf4j.LoggerFactory;
-
-import java.io.File;
-import java.util.UUID;
-
-import scala.concurrent.Await;
-import scala.concurrent.Future;
-import scala.concurrent.duration.Deadline;
-import scala.concurrent.duration.FiniteDuration;
-import scala.concurrent.impl.Promise;
-
-import static org.junit.Assert.assertEquals;
-import static org.junit.Assert.fail;
-
-/**
- * Test the election of a new JobManager leader.
- */
-public class ZooKeeperLeaderElectionITCase extends TestLogger {
-
-       private static final FiniteDuration timeout = 
TestingUtils.TESTING_DURATION();
-
-       private static TestingServer zkServer;
-
-       @Rule
-       public TemporaryFolder tempFolder = new TemporaryFolder();
-
-       @BeforeClass
-       public static void setup() throws Exception {
-               zkServer = new TestingServer(true);
-       }
-
-       @AfterClass
-       public static void tearDown() throws Exception {
-               if (zkServer != null) {
-                       zkServer.close();
-                       zkServer = null;
-               }
-       }
-
-       /**
-        * Tests that the TaskManagers successfully register at the new leader 
once the old leader
-        * is terminated.
-        */
-       @Test
-       public void testTaskManagerRegistrationAtReelectedLeader() throws 
Exception {
-               File rootFolder = tempFolder.getRoot();
-
-               Configuration configuration = 
ZooKeeperTestUtils.createZooKeeperHAConfig(
-                       zkServer.getConnectString(),
-                       rootFolder.getPath());
-               configuration.setString(HighAvailabilityOptions.HA_CLUSTER_ID, 
UUID.randomUUID().toString());
-
-               int numJMs = 10;
-               int numTMs = 3;
-
-               
configuration.setInteger(ConfigConstants.LOCAL_NUMBER_JOB_MANAGER, numJMs);
-               
configuration.setInteger(ConfigConstants.LOCAL_NUMBER_TASK_MANAGER, numTMs);
-
-               TestingCluster cluster = new TestingCluster(configuration);
-
-               try {
-                       cluster.start();
-
-                       for (int i = 0; i < numJMs; i++) {
-                               ActorGateway leadingJM = 
cluster.getLeaderGateway(timeout);
-
-                               
cluster.waitForTaskManagersToBeRegisteredAtJobManager(leadingJM.actor());
-
-                               Future<Object> registeredTMs = leadingJM.ask(
-                                       
JobManagerMessages.getRequestNumberRegisteredTaskManager(),
-                                       timeout);
-
-                               int numRegisteredTMs = (Integer) 
Await.result(registeredTMs, timeout);
-
-                               assertEquals(numTMs, numRegisteredTMs);
-
-                               cluster.clearLeader();
-                               leadingJM.tell(PoisonPill.getInstance());
-                       }
-               } finally {
-                       cluster.stop();
-               }
-       }
-
-       /**
-        * Tests that a job can be executed after a new leader has been 
elected. For all except for the
-        * last leader, the job is blocking. The JobManager will be terminated 
while executing the
-        * blocking job. Once only one JobManager is left, it is checked that a 
non-blocking can be
-        * successfully executed.
-        */
-       @Test
-       public void testJobExecutionOnClusterWithLeaderReelection() throws 
Exception {
 
 Review comment:
   I'm not sure whether we can simply drop this test because it tests a bit 
more than that all components will reconnect to the leader. It also tests that 
the `Dispatcher` will recover the submitted job from the 
`ZooKeeperSubmittedJobGraphStore` and then executes this job. I think it still 
makes sense to keep this test.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
us...@infra.apache.org


With regards,
Apache Git Services

Reply via email to