Skip to content

Commit

Permalink
Merge branch 'master' into crioux-stripe/scalar-to-group-group-to-sca…
Browse files Browse the repository at this point in the history
…lar-concurrency
  • Loading branch information
crioux-stripe authored Oct 9, 2024
2 parents 38c8d4f + 0aabc16 commit 6a45a4a
Show file tree
Hide file tree
Showing 6 changed files with 74 additions and 36 deletions.
2 changes: 1 addition & 1 deletion .github/workflows/nebula-ci.yml
Original file line number Diff line number Diff line change
Expand Up @@ -40,7 +40,7 @@ jobs:
restore-keys: |
- ${{ runner.os }}-gradlewrapper-
- name: Build with Gradle
run: ./gradlew --info --stacktrace build --warning-mode=all
run: ./gradlew --info --stacktrace build akkatest --warning-mode=all
env:
CI_NAME: github_actions
CI_BUILD_NUMBER: ${{ github.sha }}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1906,20 +1906,13 @@ public void checkHeartBeats(Instant currentTime) {
for (JobWorker worker : stage.getAllWorkers()) {
IMantisWorkerMetadata workerMeta = worker.getMetadata();
if (!workerMeta.getLastHeartbeatAt().isPresent()) {
// the worker is still waiting for resource allocation and the scheduler should take care of
// the retry logic.
Instant acceptedAt = Instant.ofEpochMilli(workerMeta.getAcceptedAt());
if (Duration.between(acceptedAt, currentTime).getSeconds() > stuckInSubmitToleranceSecs) {
// worker stuck in accepted
LOGGER.info("Job {}, Worker {} stuck in accepted state for {}", this.jobMgr.getJobId(),
workerMeta.getWorkerId(), Duration.between(acceptedAt, currentTime).getSeconds());

workersToResubmit.add(worker);
eventPublisher.publishStatusEvent(new LifecycleEventsProto.WorkerStatusEvent(
WARN,
"worker stuck in Accepted state, resubmitting worker",
workerMeta.getStageNum(),
workerMeta.getWorkerId(),
workerMeta.getState()));
}
LOGGER.warn("Job {}, Worker {} stuck in accepted state since {}",
this.jobMgr.getJobId(),
workerMeta.getWorkerId(),
acceptedAt);
} else {
if (Duration.between(workerMeta.getLastHeartbeatAt().get(), currentTime).getSeconds()
> missedHeartBeatToleranceSecs) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -173,18 +173,14 @@ private void onAssignedBatchScheduleRequestEvent(AssignedBatchScheduleRequestEve

private void onFailedToBatchScheduleRequestEvent(FailedToBatchScheduleRequestEvent event) {
batchSchedulingFailures.increment();
if (event.getAttempt() >= this.maxScheduleRetries) {
log.error("Failed to submit the batch request {} because of ", event.getScheduleRequestEvent(), event.getThrowable());
} else {
Duration timeout = Duration.ofMillis(intervalBetweenRetries.toMillis());
log.error("Failed to submit the request {}; Retrying in {} because of ",
event.getScheduleRequestEvent(), timeout, event.getThrowable());

getTimers().startSingleTimer(
getBatchSchedulingQueueKeyFor(event.getScheduleRequestEvent().getJobId()),
event.onRetry(),
timeout);
}
Duration timeout = Duration.ofMillis(intervalBetweenRetries.toMillis());
log.warn("BatchScheduleRequest failed to allocate resource: {}; Retrying in {} because of ",
event.getScheduleRequestEvent(), timeout, event.getThrowable());

getTimers().startSingleTimer(
getBatchSchedulingQueueKeyFor(event.getScheduleRequestEvent().getJobId()),
event.onRetry(),
timeout);
}

private void onScheduleRequestEvent(ScheduleRequestEvent event) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -56,6 +56,7 @@
import static org.junit.Assert.assertFalse;
import static org.junit.Assert.assertTrue;
import static org.junit.Assert.fail;
import static org.mockito.ArgumentMatchers.anyInt;
import static org.mockito.Matchers.any;
import static org.mockito.Matchers.anyString;
import static org.mockito.Mockito.doAnswer;
Expand Down Expand Up @@ -114,7 +115,14 @@
import io.mantisrx.server.core.Status;
import io.mantisrx.server.core.Status.TYPE;
import io.mantisrx.server.core.domain.WorkerId;
import io.mantisrx.server.master.domain.*;
import io.mantisrx.server.master.domain.DataFormatAdapter;
import io.mantisrx.server.master.domain.IJobClusterDefinition;
import io.mantisrx.server.master.domain.JobClusterConfig;
import io.mantisrx.server.master.domain.JobClusterDefinitionImpl;
import io.mantisrx.server.master.domain.JobClusterDefinitionImpl.CompletedJob;
import io.mantisrx.server.master.domain.JobDefinition;
import io.mantisrx.server.master.domain.JobId;
import io.mantisrx.server.master.domain.SLA;
import io.mantisrx.server.master.persistence.IMantisPersistenceProvider;
import io.mantisrx.server.master.persistence.KeyValueBasedPersistenceProvider;
import io.mantisrx.server.master.persistence.MantisJobStore;
Expand All @@ -123,6 +131,7 @@
import io.mantisrx.server.master.scheduler.WorkerEvent;
import io.mantisrx.server.master.store.FileBasedStore;
import io.mantisrx.server.master.store.NamedJob;
import io.mantisrx.shaded.com.google.common.collect.ImmutableList;
import io.mantisrx.shaded.com.google.common.collect.Lists;
import java.io.File;
import java.time.Duration;
Expand Down Expand Up @@ -230,6 +239,10 @@ private JobClusterDefinitionImpl createFakeJobClusterDefn(String clusterName, Li
.withVersion("0.0.1")
.build();

if (labels.stream().noneMatch(l -> l.getName().equals("_mantis.resourceCluster"))) {
labels.add(new Label("_mantis.resourceCluster", "akkaTestCluster1"));
}

return new JobClusterDefinitionImpl.Builder()
.withJobClusterConfig(clusterConfig)
.withName(clusterName)
Expand Down Expand Up @@ -340,7 +353,7 @@ public void testJobClusterCreate() throws Exception {
assertEquals(SUCCESS, resp2.responseCode);
assertEquals(name, resp2.getJobCluster().get().getName());
assertEquals("Nick", resp2.getJobCluster().get().getOwner().getName());
assertTrue(resp2.getJobCluster().get().getLabels().isEmpty());
assertEquals(1, resp2.getJobCluster().get().getLabels().size());
assertEquals(1,resp2.getJobCluster().get().getJars().size());

jobClusterActor.tell(new JobClusterProto.DeleteJobClusterRequest(user, name, probe.getRef()), probe.getRef());
Expand Down Expand Up @@ -487,7 +500,7 @@ public void testJobClusterUpdateAndDelete() throws Exception {
System.out.println("Job cluster " + resp3.getJobCluster());
assertEquals(clusterName, resp3.getJobCluster().get().getName());
System.out.println("Updated job cluster " + resp3.getJobCluster());
assertEquals(1, resp3.getJobCluster().get().getLabels().size());
assertEquals(2, resp3.getJobCluster().get().getLabels().size());
assertEquals("labelname", resp3.getJobCluster().get().getLabels().get(0).getName());

jobClusterActor.tell(new JobClusterProto.DeleteJobClusterRequest(user, clusterName, probe.getRef()), probe.getRef());
Expand Down Expand Up @@ -568,6 +581,7 @@ public void testJobClusterDeleteFailsIfJobsActive() throws Exception {
}

@Test
@Ignore("todo: Purge logic changed")
public void testJobClusterDeletePurgesCompletedJobs() throws Exception {

TestKit probe = new TestKit(system);
Expand Down Expand Up @@ -638,6 +652,18 @@ public void testJobClusterDisable() throws InterruptedException {
.withJobDefinition(jobDefn)
.withJobState(JobState.Completed)
.build();
when(jobStoreMock.loadCompletedJobsForCluster(any(), anyInt(), any()))
// .thenReturn(ImmutableList.of());
.thenReturn(ImmutableList.of(
new CompletedJob(
completedJobMock.getClusterName(),
completedJobMock.getJobId().getId(),
"v1",
JobState.Completed,
-1L,
-1L,
completedJobMock.getUser(),
completedJobMock.getLabels())));
when(jobStoreMock.getArchivedJob(any())).thenReturn(of(completedJobMock));
doAnswer((Answer) invocation -> {
storeCompletedCalled.countDown();
Expand Down Expand Up @@ -962,13 +988,14 @@ public void testJobClusterLabelsUpdate() throws Exception {
System.out.println("Job cluster " + resp3.getJobCluster());
assertEquals(clusterName, resp3.getJobCluster().get().getName());
System.out.println("Updated job cluster " + resp3.getJobCluster());
assertEquals(0, resp3.getJobCluster().get().getLabels().size());
assertEquals(1, resp3.getJobCluster().get().getLabels().size());


// new labels
List<Label> labels = Lists.newLinkedList();
Label l = new Label("labelname","labelvalue");
labels.add(l);
labels.add(new Label("_mantis.resourceCluster","cl2"));

UpdateJobClusterLabelsRequest updateLabelsReq = new UpdateJobClusterLabelsRequest(clusterName, labels, "user");
jobClusterActor.tell(updateLabelsReq, probe.getRef());
Expand All @@ -985,7 +1012,7 @@ public void testJobClusterLabelsUpdate() throws Exception {
assertTrue(resp3.getJobCluster() != null);
assertEquals(clusterName, resp3.getJobCluster().get().getName());
//assert label list is of size 1
assertEquals(1, resp3.getJobCluster().get().getLabels().size());
assertEquals(2, resp3.getJobCluster().get().getLabels().size());
assertEquals(l, resp3.getJobCluster().get().getLabels().get(0));

verify(jobStoreMock, times(1)).createJobCluster(any());
Expand Down Expand Up @@ -1107,11 +1134,14 @@ public void testJobSubmitWithVersionAndNoSchedInfo() {
.withVersion("0.0.2")
.build();

List<Label> labels = Lists.newLinkedList();
labels.add(new Label("_mantis.resourceCluster","cl2"));

final JobClusterDefinitionImpl updatedFakeJobCluster = new JobClusterDefinitionImpl.Builder()
.withJobClusterConfig(clusterConfig)
.withName(clusterName)
.withParameters(Lists.newArrayList())

.withLabels(labels)
.withUser(user)
.withIsReadyForJobMaster(true)
.withOwner(DEFAULT_JOB_OWNER)
Expand Down Expand Up @@ -1972,7 +2002,7 @@ public void testJobSubmitWithNoSchedInfoUsesJobClusterValues() {

assertEquals(SUCCESS, detailsResp2.responseCode);
assertEquals(JobState.Accepted, detailsResp2.getJobMetadata().get().getState());
assertEquals(clusterLabels.size()+2,detailsResp2.getJobMetadata().get().getLabels().size());
assertEquals(clusterLabels.size() + 3, detailsResp2.getJobMetadata().get().getLabels().size());
// confirm that the clusters labels got inherited
//assertEquals(jobLabel, detailsResp2.getJobMetadata().get().getLabels().get(0));
assertEquals(1, detailsResp2.getJobMetadata().get()
Expand Down Expand Up @@ -2170,8 +2200,11 @@ public void testGetLastSubmittedJobSubject() {
JobTestHelper.submitJobAndVerifySuccess(probe, clusterName, jobClusterActor, jobDefn, jobId);

JobTestHelper.getJobDetailsAndVerify(probe, jobClusterActor, jobId, SUCCESS, JobState.Accepted);
JobTestHelper.sendLaunchedInitiatedStartedEventsToWorker(
probe, jobClusterActor, jobId,1, new WorkerId(clusterName,jobId,0,1));


jobIdLatch.await(1000,TimeUnit.SECONDS);
jobIdLatch.await(1,TimeUnit.SECONDS);

} catch (Exception e) {
// TODO Auto-generated catch block
Expand Down Expand Up @@ -2283,6 +2316,7 @@ public void testListArchivedWorkers() {
}

@Test
@Ignore("todo: fix")
public void testZombieWorkerKilledOnMessage() {
String clusterName = "testZombieWorkerKilledOnMessage";
TestKit probe = new TestKit(system);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -30,6 +30,7 @@
import static org.junit.Assert.assertNotNull;
import static org.junit.Assert.assertTrue;
import static org.junit.Assert.fail;
import static org.mockito.ArgumentMatchers.anyInt;
import static org.mockito.Matchers.any;
import static org.mockito.Matchers.eq;
import static org.mockito.Mockito.mock;
Expand Down Expand Up @@ -89,6 +90,7 @@
import io.mantisrx.server.master.domain.IJobClusterDefinition;
import io.mantisrx.server.master.domain.JobClusterConfig;
import io.mantisrx.server.master.domain.JobClusterDefinitionImpl;
import io.mantisrx.server.master.domain.JobClusterDefinitionImpl.CompletedJob;
import io.mantisrx.server.master.domain.JobDefinition;
import io.mantisrx.server.master.domain.JobId;
import io.mantisrx.server.master.domain.SLA;
Expand All @@ -99,6 +101,7 @@
import io.mantisrx.server.master.scheduler.WorkerEvent;
import io.mantisrx.server.master.scheduler.WorkerLaunched;
import io.mantisrx.server.master.store.FileBasedStore;
import io.mantisrx.shaded.com.google.common.collect.ImmutableList;
import io.mantisrx.shaded.com.google.common.collect.Lists;
import java.io.IOException;
import java.net.MalformedURLException;
Expand Down Expand Up @@ -1596,7 +1599,7 @@ public void testTerminalEventFromZombieWorkerIgnored() {
}

@Test
public void testNonTerminalEventFromZombieWorkerLeadsToTermination() {
public void testNonTerminalEventFromZombieWorkerLeadsToTermination() throws IOException {
TestKit probe = new TestKit(system);
String clusterName = "testNonTerminalEventFromZombieWorkerLeadsToTermination";

Expand All @@ -1612,6 +1615,18 @@ public void testNonTerminalEventFromZombieWorkerLeadsToTermination() {
assertEquals(SUCCESS_CREATED, resp.responseCode);

WorkerId zWorker1 = new WorkerId("randomCluster", "randomCluster-1", 0, 1);
when(jobStoreMock.loadCompletedJobsForCluster(any(), anyInt(), any()))
// .thenReturn(ImmutableList.of());
.thenReturn(ImmutableList.of(
new CompletedJob(
clusterName,
clusterName + "-1",
"v1",
JobState.Completed,
-1L,
-1L,
"ut",
ImmutableList.of())));
when(jobStoreMock.getArchivedJob(zWorker1.getJobId()))
.thenReturn(Optional.of(
new MantisJobMetadataImpl.Builder().withJobDefinition(mock(JobDefinition.class))
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -81,7 +81,7 @@ public Observable<ObservableConnection<HttpClientResponse<O>, HttpClientRequest<
}

protected void trackConnection(Channel channel) {
log.info("Tracking connection: {}", channel.toString());
log.debug("Tracking connection: {}", channel.toString());
synchronized (connectionTracker) {
if (isClosed.get()) {
log.info("Http client is already closed. Close the channel immediately. {}", channel);
Expand Down

0 comments on commit 6a45a4a

Please sign in to comment.