2 # -*- mode: perl; perl-indent-level: 2; indent-tabs-mode: nil; -*-
3 # Copyright (C) The Arvados Authors. All rights reserved.
5 # SPDX-License-Identifier: AGPL-3.0
9 crunch-job: Execute job steps, save snapshots as requested, collate output.
13 Obtain job details from Arvados, run tasks on compute nodes (typically
14 invoked by scheduler on controller):
16 crunch-job --job x-y-z --git-dir /path/to/repo/.git
18 Obtain job details from command line, run tasks on local machine
19 (typically invoked by application or developer on VM):
21 crunch-job --job '{"script_version":"/path/to/working/tree","script":"scriptname",...}'
23 crunch-job --job '{"repository":"https://github.com/curoverse/arvados.git","script_version":"master","script":"scriptname",...}'
31 If the job is already locked, steal the lock and run it anyway.
35 Path to a .git directory (or a git URL) where the commit given in the
36 job's C<script_version> attribute is to be found. If this is I<not>
37 given, the job's C<repository> attribute will be used.
41 Arvados API authorization token to use during the course of the job.
45 Do not clear per-job/task temporary directories during initial job
46 setup. This can speed up development and debugging when running jobs
51 UUID of the job to run, or a JSON-encoded job resource without a
52 UUID. If the latter is given, a new job object will be created.
56 =head1 RUNNING JOBS LOCALLY
58 crunch-job's log messages appear on stderr along with the job tasks'
59 stderr streams. The log is saved in Keep at each checkpoint and when
62 If the job succeeds, the job's output locator is printed on stdout.
64 While the job is running, the following signals are accepted:
68 =item control-C, SIGINT, SIGQUIT
70 Save a checkpoint, terminate any job tasks that are running, and stop.
74 Save a checkpoint and continue.
78 Refresh node allocation (i.e., check whether any nodes have been added
79 or unallocated) and attributes of the Job record that should affect
80 behavior (e.g., cancel job if cancelled_at becomes non-nil).
88 use POSIX ':sys_wait_h';
89 use POSIX qw(strftime);
90 use Fcntl qw(F_GETFL F_SETFL O_NONBLOCK);
94 use Digest::MD5 qw(md5_hex);
100 use File::Path qw( make_path remove_tree );
102 use constant TASK_TEMPFAIL => 111;
103 use constant EX_TEMPFAIL => 75;
104 use constant EX_RETRY_UNLOCKED => 93;
106 $ENV{"TMPDIR"} ||= "/tmp";
107 unless (defined $ENV{"CRUNCH_TMP"}) {
108 $ENV{"CRUNCH_TMP"} = $ENV{"TMPDIR"} . "/crunch-job";
109 if ($ENV{"USER"} ne "crunch" && $< != 0) {
110 # use a tmp dir unique for my uid
111 $ENV{"CRUNCH_TMP"} .= "-$<";
115 # Create the tmp directory if it does not exist
116 if ( ! -d $ENV{"CRUNCH_TMP"} ) {
117 make_path $ENV{"CRUNCH_TMP"} or die "Failed to create temporary working directory: " . $ENV{"CRUNCH_TMP"};
120 $ENV{"JOB_WORK"} = $ENV{"CRUNCH_TMP"} . "/work";
121 $ENV{"CRUNCH_INSTALL"} = "$ENV{CRUNCH_TMP}/opt";
122 $ENV{"CRUNCH_WORK"} = $ENV{"JOB_WORK"}; # deprecated
123 mkdir ($ENV{"JOB_WORK"});
132 my $cgroup_root = "/sys/fs/cgroup";
133 my $docker_bin = "docker.io";
134 my $docker_run_args = "";
135 my $srun_sync_timeout = 15*60;
136 GetOptions('force-unlock' => \$force_unlock,
137 'git-dir=s' => \$git_dir,
138 'job=s' => \$jobspec,
139 'job-api-token=s' => \$job_api_token,
140 'no-clear-tmp' => \$no_clear_tmp,
141 'resume-stash=s' => \$resume_stash,
142 'cgroup-root=s' => \$cgroup_root,
143 'docker-bin=s' => \$docker_bin,
144 'docker-run-args=s' => \$docker_run_args,
145 'srun-sync-timeout=i' => \$srun_sync_timeout,
148 if (defined $job_api_token) {
149 $ENV{ARVADOS_API_TOKEN} = $job_api_token;
152 my $have_slurm = exists $ENV{SLURM_JOB_ID} && exists $ENV{SLURM_NODELIST};
157 $main::ENV{CRUNCH_DEBUG} = 1;
161 $main::ENV{CRUNCH_DEBUG} = 0;
164 my $arv = Arvados->new('apiVersion' => 'v1');
173 if ($jobspec =~ /^[-a-z\d]+$/)
175 # $jobspec is an Arvados UUID, not a JSON job specification
176 $Job = api_call("jobs/get", uuid => $jobspec);
181 $local_job = JSON::decode_json($jobspec);
185 # Make sure our workers (our slurm nodes, localhost, or whatever) are
186 # at least able to run basic commands: they aren't down or severely
189 if (($Job || $local_job)->{docker_image_locator}) {
190 $cmd = [$docker_bin, 'ps', '-q'];
192 Log(undef, "Sanity check is `@$cmd`");
193 my ($exited, $stdout, $stderr, $tempfail) = srun_sync(
194 ["srun", "--nodes=\Q$ENV{SLURM_NNODES}\E", "--ntasks-per-node=1"],
196 {label => "sanity check"});
198 Log(undef, "Sanity check failed: ".exit_status_s($exited));
201 Log(undef, "Sanity check OK");
204 my $User = api_call("users/current");
207 if (!$force_unlock) {
208 # Claim this job, and make sure nobody else does
209 eval { api_call("jobs/lock", uuid => $Job->{uuid}); };
211 Log(undef, "Error while locking job, exiting ".EX_TEMPFAIL);
220 map { croak ("No $_ specified") unless $local_job->{$_} }
221 qw(script script_version script_parameters);
224 $local_job->{'is_locked_by_uuid'} = $User->{'uuid'};
225 $local_job->{'started_at'} = gmtime;
226 $local_job->{'state'} = 'Running';
228 $Job = api_call("jobs/create", job => $local_job);
230 $job_id = $Job->{'uuid'};
232 my $keep_logfile = $job_id . '.log.txt';
233 log_writer_start($keep_logfile);
235 $Job->{'runtime_constraints'} ||= {};
236 $Job->{'runtime_constraints'}->{'max_tasks_per_node'} ||= 0;
237 my $max_ncpus = $Job->{'runtime_constraints'}->{'max_tasks_per_node'};
239 my $gem_versions = `gem list --quiet arvados-cli 2>/dev/null`;
241 $gem_versions =~ s/^arvados-cli \(/ with arvados-cli Gem version(s) /;
242 chomp($gem_versions);
243 chop($gem_versions); # Closing parentheses
248 "running from " . ((-e $0) ? realpath($0) : "stdin") . $gem_versions);
250 Log (undef, "check slurm allocation");
253 # Should use $ENV{SLURM_TASKS_PER_NODE} instead of sinfo? (eg. "4(x3),2,4(x2)")
257 my $localcpus = 0 + `grep -cw ^processor /proc/cpuinfo` || 1;
258 push @sinfo, "$localcpus localhost";
260 if (exists $ENV{SLURM_NODELIST})
262 push @sinfo, `sinfo -h --format='%c %N' --nodes=\Q$ENV{SLURM_NODELIST}\E`;
266 my ($ncpus, $slurm_nodelist) = split;
267 $ncpus = $max_ncpus if $max_ncpus && $ncpus > $max_ncpus;
270 while ($slurm_nodelist =~ s/^([^\[,]+?(\[.*?\])?)(,|$)//)
273 if ($nodelist =~ /\[((\d+)(-(\d+))?(,(\d+)(-(\d+))?)*)\]/)
276 foreach (split (",", $ranges))
289 push @nodelist, map {
291 $n =~ s/\[[-,\d]+\]/$_/;
298 push @nodelist, $nodelist;
301 foreach my $nodename (@nodelist)
303 Log (undef, "node $nodename - $ncpus slots");
304 my $node = { name => $nodename,
306 # The number of consecutive times a task has been dispatched
307 # to this node and failed.
309 # The number of consecutive times that SLURM has reported
310 # a node failure since the last successful task.
312 # Don't dispatch work to this node until this time
313 # (in seconds since the epoch) has passed.
315 foreach my $cpu (1..$ncpus)
317 push @slot, { node => $node,
321 push @node, @nodelist;
326 # Ensure that we get one jobstep running on each allocated node before
327 # we start overloading nodes with concurrent steps
329 @slot = sort { $a->{cpu} <=> $b->{cpu} } @slot;
332 $Job->update_attributes(
333 'tasks_summary' => { 'failed' => 0,
338 Log (undef, "start");
339 $SIG{'INT'} = sub { $main::please_freeze = 1; };
340 $SIG{'QUIT'} = sub { $main::please_freeze = 1; };
341 $SIG{'TERM'} = \&croak;
342 $SIG{'TSTP'} = sub { $main::please_freeze = 1; };
343 $SIG{'ALRM'} = sub { $main::please_info = 1; };
344 $SIG{'CONT'} = sub { $main::please_continue = 1; };
345 $SIG{'HUP'} = sub { $main::please_refresh = 1; };
347 $main::please_freeze = 0;
348 $main::please_info = 0;
349 $main::please_continue = 0;
350 $main::please_refresh = 0;
351 my $jobsteps_must_output_keys = 0; # becomes 1 when any task outputs a key
353 grep { $ENV{$1} = $2 if /^(NOCACHE.*?)=(.*)/ } split ("\n", $$Job{knobs});
354 $ENV{"CRUNCH_JOB_UUID"} = $job_id;
355 $ENV{"JOB_UUID"} = $job_id;
358 my @jobstep_todo = ();
359 my @jobstep_done = ();
360 my @jobstep_tomerge = ();
361 my $jobstep_tomerge_level = 0;
362 my $squeue_checked = 0;
363 my $sinfo_checked = 0;
364 my $latest_refresh = scalar time;
368 if (defined $Job->{thawedfromkey})
370 thaw ($Job->{thawedfromkey});
374 my $first_task = api_call("job_tasks/create", job_task => {
375 'job_uuid' => $Job->{'uuid'},
380 push @jobstep, { 'level' => 0,
382 'arvados_task' => $first_task,
384 push @jobstep_todo, 0;
390 must_lock_now("$ENV{CRUNCH_TMP}/.lock", "a job is already running here.");
393 my $build_script = handle_readall(\*DATA);
394 my $nodelist = join(",", @node);
395 my $git_tar_count = 0;
397 if (!defined $no_clear_tmp) {
398 # Find FUSE mounts under $CRUNCH_TMP and unmount them. Then clean
399 # up work directories crunch_tmp/work, crunch_tmp/opt,
401 my ($exited, $stdout, $stderr, $tempfail) = srun_sync(
402 ["srun", "--nodelist=$nodelist", "-D", $ENV{'TMPDIR'}],
404 arv-mount --unmount-timeout 10 --unmount-all ${CRUNCH_TMP}
405 rm -rf ${JOB_WORK} ${CRUNCH_INSTALL} ${CRUNCH_TMP}/task ${CRUNCH_TMP}/src* ${CRUNCH_TMP}/*.cid
407 {label => "clean work dirs"});
409 exit_retry_unlocked();
413 # If this job requires a Docker image, install that.
414 my ($docker_locator, $docker_stream, $docker_hash, $docker_limitmem, $dockeruserarg);
415 if ($docker_locator = $Job->{docker_image_locator}) {
416 Log (undef, "Install docker image $docker_locator");
417 ($docker_stream, $docker_hash) = find_docker_image($docker_locator);
420 croak("No Docker image hash found from locator $docker_locator");
422 Log (undef, "docker image hash is $docker_hash");
423 $docker_stream =~ s/^\.//;
424 my $docker_install_script = qq{
426 id=\$($docker_bin inspect --format="{{.ID}}" \Q$docker_hash\E) || return 1
427 echo "image ID is \$id"
428 [[ \${id} = \Q$docker_hash\E ]]
430 if loaded >&2 2>/dev/null; then
431 echo >&2 "image is already present"
434 echo >&2 "docker image is not present; loading"
435 arv-get \Q$docker_locator$docker_stream/$docker_hash.tar\E | $docker_bin load
436 if ! loaded >&2; then
437 echo >&2 "`docker load` exited 0, but image is not found (!)"
440 echo >&2 "image loaded successfully"
443 my ($exited, $stdout, $stderr, $tempfail) = srun_sync(
444 ["srun", "--nodelist=" . join(',', @node)],
445 ["/bin/bash", "-o", "pipefail", "-ec", $docker_install_script],
446 {label => "load docker image"});
449 exit_retry_unlocked();
452 # Determine whether this version of Docker supports memory+swap limits.
453 ($exited, $stdout, $stderr, $tempfail) = srun_sync(
454 ["srun", "--nodes=1"],
455 [$docker_bin, 'run', '--help'],
456 {label => "check --memory-swap feature"});
458 exit_retry_unlocked();
460 $docker_limitmem = ($stdout =~ /--memory-swap/);
462 # Find a non-root Docker user to use.
463 # Tries the default user for the container, then 'crunch', then 'nobody',
464 # testing for whether the actual user id is non-zero. This defends against
465 # mistakes but not malice, but we intend to harden the security in the future
466 # so we don't want anyone getting used to their jobs running as root in their
468 my @tryusers = ("", "crunch", "nobody");
469 foreach my $try_user (@tryusers) {
472 if ($try_user eq "") {
473 $label = "check whether default user is UID 0";
476 $label = "check whether user '$try_user' is UID 0";
477 $try_user_arg = "--user=$try_user";
479 my ($exited, $stdout, $stderr, $tempfail) = srun_sync(
480 ["srun", "--nodes=1"],
482 "$docker_bin run $docker_run_args $try_user_arg $docker_hash id --user"],
485 if ($exited == 0 && $stdout =~ /^\d+$/ && $stdout > 0) {
486 $dockeruserarg = $try_user_arg;
487 if ($try_user eq "") {
488 Log(undef, "Container will run with default user");
490 Log(undef, "Container will run with $dockeruserarg");
493 } elsif ($tempfail) {
494 exit_retry_unlocked();
498 if (!defined $dockeruserarg) {
499 croak("Could not find a user in container that is not UID 0 (tried default user, @tryusers) or there was a problem running 'id' in the container.");
502 if ($Job->{arvados_sdk_version}) {
503 # The job also specifies an Arvados SDK version. Add the SDKs to the
504 # tar file for the build script to install.
505 Log(undef, sprintf("Packing Arvados SDK version %s for installation",
506 $Job->{arvados_sdk_version}));
507 add_git_archive("git", "--git-dir=$git_dir", "archive",
508 "--prefix=.arvados.sdk/",
509 $Job->{arvados_sdk_version}, "sdk");
513 if (!defined $git_dir && $Job->{'script_version'} =~ m{^/}) {
514 # If script_version looks like an absolute path, *and* the --git-dir
515 # argument was not given -- which implies we were not invoked by
516 # crunch-dispatch -- we will use the given path as a working
517 # directory instead of resolving script_version to a git commit (or
518 # doing anything else with git).
519 $ENV{"CRUNCH_SRC_COMMIT"} = $Job->{'script_version'};
520 $ENV{"CRUNCH_SRC"} = $Job->{'script_version'};
523 # Resolve the given script_version to a git commit sha1. Also, if
524 # the repository is remote, clone it into our local filesystem: this
525 # ensures "git archive" will work, and is necessary to reliably
526 # resolve a symbolic script_version like "master^".
527 $ENV{"CRUNCH_SRC"} = "$ENV{CRUNCH_TMP}/src";
529 Log (undef, "Looking for version ".$Job->{script_version}." from repository ".$Job->{repository});
531 $ENV{"CRUNCH_SRC_COMMIT"} = $Job->{script_version};
533 # If we're running under crunch-dispatch, it will have already
534 # pulled the appropriate source tree into its own repository, and
535 # given us that repo's path as $git_dir.
537 # If we're running a "local" job, we might have to fetch content
538 # from a remote repository.
540 # (Currently crunch-dispatch gives a local path with --git-dir, but
541 # we might as well accept URLs there too in case it changes its
543 my $repo = $git_dir || $Job->{'repository'};
545 # Repository can be remote or local. If remote, we'll need to fetch it
546 # to a local dir before doing `git log` et al.
549 if ($repo =~ m{://|^[^/]*:}) {
550 # $repo is a git url we can clone, like git:// or https:// or
551 # file:/// or [user@]host:repo.git. Note "user/name@host:foo" is
552 # not recognized here because distinguishing that from a local
553 # path is too fragile. If you really need something strange here,
554 # use the ssh:// form.
555 $repo_location = 'remote';
556 } elsif ($repo =~ m{^\.*/}) {
557 # $repo is a local path to a git index. We'll also resolve ../foo
558 # to ../foo/.git if the latter is a directory. To help
559 # disambiguate local paths from named hosted repositories, this
560 # form must be given as ./ or ../ if it's a relative path.
561 if (-d "$repo/.git") {
562 $repo = "$repo/.git";
564 $repo_location = 'local';
566 # $repo is none of the above. It must be the name of a hosted
568 my $arv_repo_list = api_call("repositories/list",
569 'filters' => [['name','=',$repo]]);
570 my @repos_found = @{$arv_repo_list->{'items'}};
571 my $n_found = $arv_repo_list->{'serverResponse'}->{'items_available'};
573 Log(undef, "Repository '$repo' -> "
574 . join(", ", map { $_->{'uuid'} } @repos_found));
577 croak("Error: Found $n_found repositories with name '$repo'.");
579 $repo = $repos_found[0]->{'fetch_url'};
580 $repo_location = 'remote';
582 Log(undef, "Using $repo_location repository '$repo'");
583 $ENV{"CRUNCH_SRC_URL"} = $repo;
585 # Resolve given script_version (we'll call that $treeish here) to a
586 # commit sha1 ($commit).
587 my $treeish = $Job->{'script_version'};
589 if ($repo_location eq 'remote') {
590 # We minimize excess object-fetching by re-using the same bare
591 # repository in CRUNCH_TMP/.git for multiple crunch-jobs -- we
592 # just keep adding remotes to it as needed.
593 my $local_repo = $ENV{'CRUNCH_TMP'}."/.git";
594 my $gitcmd = "git --git-dir=\Q$local_repo\E";
596 # Set up our local repo for caching remote objects, making
598 if (!-d $local_repo) {
599 make_path($local_repo) or croak("Error: could not create $local_repo");
601 # This works (exits 0 and doesn't delete fetched objects) even
602 # if $local_repo is already initialized:
603 `$gitcmd init --bare`;
605 croak("Error: $gitcmd init --bare exited ".exit_status_s($?));
608 # If $treeish looks like a hash (or abbrev hash) we look it up in
609 # our local cache first, since that's cheaper. (We don't want to
610 # do that with tags/branches though -- those change over time, so
611 # they should always be resolved by the remote repo.)
612 if ($treeish =~ /^[0-9a-f]{7,40}$/s) {
613 # Hide stderr because it's normal for this to fail:
614 my $sha1 = `$gitcmd rev-list -n1 ''\Q$treeish\E 2>/dev/null`;
616 # Careful not to resolve a branch named abcdeff to commit 1234567:
617 $sha1 =~ /^$treeish/ &&
618 $sha1 =~ /^([0-9a-f]{40})$/s) {
620 Log(undef, "Commit $commit already present in $local_repo");
624 if (!defined $commit) {
625 # If $treeish isn't just a hash or abbrev hash, or isn't here
626 # yet, we need to fetch the remote to resolve it correctly.
628 # First, remove all local heads. This prevents a name that does
629 # not exist on the remote from resolving to (or colliding with)
630 # a previously fetched branch or tag (possibly from a different
632 remove_tree("$local_repo/refs/heads", {keep_root => 1});
634 Log(undef, "Fetching objects from $repo to $local_repo");
635 `$gitcmd fetch --no-progress --tags ''\Q$repo\E \Q+refs/heads/*:refs/heads/*\E`;
637 croak("Error: `$gitcmd fetch` exited ".exit_status_s($?));
641 # Now that the data is all here, we will use our local repo for
642 # the rest of our git activities.
646 my $gitcmd = "git --git-dir=\Q$repo\E";
647 my $sha1 = `$gitcmd rev-list -n1 ''\Q$treeish\E`;
648 unless ($? == 0 && $sha1 =~ /^([0-9a-f]{40})$/) {
649 croak("`$gitcmd rev-list` exited "
651 .", '$treeish' not found, giving up");
654 Log(undef, "Version $treeish is commit $commit");
656 if ($commit ne $Job->{'script_version'}) {
657 # Record the real commit id in the database, frozentokey, logs,
658 # etc. -- instead of an abbreviation or a branch name which can
659 # become ambiguous or point to a different commit in the future.
660 if (!$Job->update_attributes('script_version' => $commit)) {
661 croak("Error: failed to update job's script_version attribute");
665 $ENV{"CRUNCH_SRC_COMMIT"} = $commit;
666 add_git_archive("$gitcmd archive ''\Q$commit\E");
669 my $git_archive = combined_git_archive();
670 if (!defined $git_archive) {
671 Log(undef, "Skip install phase (no git archive)");
673 Log(undef, "Warning: This probably means workers have no source tree!");
678 my $install_script_tries_left = 3;
679 for (my $attempts = 0; $attempts < 3; $attempts++) {
680 my @srunargs = ("srun",
681 "--nodelist=$nodelist",
682 "-D", $ENV{'TMPDIR'}, "--job-name=$job_id");
683 my @execargs = ("sh", "-c",
684 "mkdir -p $ENV{CRUNCH_INSTALL} && cd $ENV{CRUNCH_TMP} && perl -");
686 $ENV{"CRUNCH_GIT_ARCHIVE_HASH"} = md5_hex($git_archive);
687 my ($stdout, $stderr, $tempfail);
688 ($exited, $stdout, $stderr, $tempfail) = srun_sync(
689 \@srunargs, \@execargs,
690 {label => "run install script on all workers"},
691 $build_script . $git_archive);
693 exit_retry_unlocked();
696 my $stderr_anything_from_script = 0;
697 for my $line (split(/\n/, $stderr)) {
698 if ($line !~ /^(srun: error: |starting: \[)/) {
699 $stderr_anything_from_script = 1;
703 last if $exited == 0 || $main::please_freeze;
705 # If the install script fails but doesn't print an error message,
706 # the next thing anyone is likely to do is just run it again in
707 # case it was a transient problem like "slurm communication fails
708 # because the network isn't reliable enough". So we'll just do
709 # that ourselves (up to 3 attempts in total). OTOH, if there is an
710 # error message, the problem is more likely to have a real fix and
711 # we should fail the job so the fixing process can start, instead
712 # of doing 2 more attempts.
713 last if $stderr_anything_from_script;
716 foreach my $tar_filename (map { tar_filename_n($_); } (1..$git_tar_count)) {
717 unlink($tar_filename);
725 foreach (qw (script script_version script_parameters runtime_constraints))
729 (ref($Job->{$_}) ? JSON::encode_json($Job->{$_}) : $Job->{$_}));
731 foreach (split (/\n/, $Job->{knobs}))
733 Log (undef, "knob " . $_);
737 'filters' => [['hostname', 'in', \@node]],
738 'order' => 'hostname',
739 'limit' => scalar(@node),
741 for my $n (@{$resp->{items}}) {
742 Log(undef, "$n->{hostname} $n->{uuid} ".JSON::encode_json($n->{properties}));
747 $main::success = undef;
753 my $thisround_succeeded = 0;
754 my $thisround_failed = 0;
755 my $thisround_failed_multiple = 0;
756 my $working_slot_count = scalar(@slot);
758 @jobstep_todo = sort { $jobstep[$a]->{level} <=> $jobstep[$b]->{level}
759 or $a <=> $b } @jobstep_todo;
760 my $level = $jobstep[$jobstep_todo[0]]->{level};
762 my $initial_tasks_this_level = 0;
763 foreach my $id (@jobstep_todo) {
764 $initial_tasks_this_level++ if ($jobstep[$id]->{level} == $level);
767 # If the number of tasks scheduled at this level #T is smaller than the number
768 # of slots available #S, only use the first #T slots, or the first slot on
769 # each node, whichever number is greater.
771 # When we dispatch tasks later, we'll allocate whole-node resources like RAM
772 # based on these numbers. Using fewer slots makes more resources available
773 # to each individual task, which should normally be a better strategy when
774 # there are fewer of them running with less parallelism.
776 # Note that this calculation is not redone if the initial tasks at
777 # this level queue more tasks at the same level. This may harm
778 # overall task throughput for that level.
780 if ($initial_tasks_this_level < @node) {
781 @freeslot = (0..$#node);
782 } elsif ($initial_tasks_this_level < @slot) {
783 @freeslot = (0..$initial_tasks_this_level - 1);
785 @freeslot = (0..$#slot);
787 my $round_num_freeslots = scalar(@freeslot);
788 print STDERR "crunch-job have ${round_num_freeslots} free slots for ${initial_tasks_this_level} initial tasks at this level, ".scalar(@node)." nodes, and ".scalar(@slot)." slots\n";
790 my %round_max_slots = ();
791 for (my $ii = $#freeslot; $ii >= 0; $ii--) {
792 my $this_slot = $slot[$freeslot[$ii]];
793 my $node_name = $this_slot->{node}->{name};
794 $round_max_slots{$node_name} ||= $this_slot->{cpu};
795 last if (scalar(keys(%round_max_slots)) >= @node);
798 Log(undef, "start level $level with $round_num_freeslots slots");
801 my $progress_is_dirty = 1;
802 my $progress_stats_updated = 0;
804 update_progress_stats();
808 for (my $todo_ptr = 0; $todo_ptr <= $#jobstep_todo; $todo_ptr ++)
810 # Don't create new tasks if we already know the job's final result.
811 last if defined($main::success);
813 my $id = $jobstep_todo[$todo_ptr];
814 my $Jobstep = $jobstep[$id];
815 if ($Jobstep->{level} != $level)
820 pipe $reader{$id}, "writer" or croak("pipe() failed: $!");
821 set_nonblocking($reader{$id});
823 my $childslot = $freeslot[0];
824 my $childnode = $slot[$childslot]->{node};
825 my $childslotname = join (".",
826 $slot[$childslot]->{node}->{name},
827 $slot[$childslot]->{cpu});
829 my $childpid = fork();
832 $SIG{'INT'} = 'DEFAULT';
833 $SIG{'QUIT'} = 'DEFAULT';
834 $SIG{'TERM'} = 'DEFAULT';
836 foreach (values (%reader))
840 fcntl ("writer", F_SETFL, 0) or croak ($!); # no close-on-exec
841 open(STDOUT,">&writer") or croak ($!);
842 open(STDERR,">&writer") or croak ($!);
847 delete $ENV{"GNUPGHOME"};
848 $ENV{"TASK_UUID"} = $Jobstep->{'arvados_task'}->{'uuid'};
849 $ENV{"TASK_QSEQUENCE"} = $id;
850 $ENV{"TASK_SEQUENCE"} = $level;
851 $ENV{"JOB_SCRIPT"} = $Job->{script};
852 while (my ($param, $value) = each %{$Job->{script_parameters}}) {
853 $param =~ tr/a-z/A-Z/;
854 $ENV{"JOB_PARAMETER_$param"} = $value;
856 $ENV{"TASK_SLOT_NODE"} = $slot[$childslot]->{node}->{name};
857 $ENV{"TASK_SLOT_NUMBER"} = $slot[$childslot]->{cpu};
858 $ENV{"TASK_WORK"} = $ENV{"CRUNCH_TMP"}."/task/$childslotname";
859 $ENV{"HOME"} = $ENV{"TASK_WORK"};
860 $ENV{"TASK_TMPDIR"} = $ENV{"TASK_WORK"}; # deprecated
861 $ENV{"CRUNCH_NODE_SLOTS"} = $round_max_slots{$ENV{TASK_SLOT_NODE}};
862 $ENV{"PATH"} = $ENV{"CRUNCH_INSTALL"} . "/bin:" . $ENV{"PATH"};
864 my $keep_mnt = $ENV{"TASK_WORK"}.".keep";
870 "--nodelist=".$childnode->{name},
871 qw(-n1 -c1 -N1 -D), $ENV{'TMPDIR'},
872 "--job-name=$job_id.$id.$$",
875 my $stdbuf = " stdbuf --output=0 --error=0 ";
877 my $arv_file_cache = "";
878 if (defined($Job->{'runtime_constraints'}->{'keep_cache_mb_per_task'})) {
879 $arv_file_cache = "--file-cache=" . ($Job->{'runtime_constraints'}->{'keep_cache_mb_per_task'} * 1024 * 1024);
883 "if [ -e \Q$ENV{TASK_WORK}\E ]; then rm -rf \Q$ENV{TASK_WORK}\E; fi; "
884 ."mkdir -p \Q$ENV{CRUNCH_TMP}\E \Q$ENV{JOB_WORK}\E \Q$ENV{TASK_WORK}\E \Q$keep_mnt\E "
885 ."&& cd \Q$ENV{CRUNCH_TMP}\E "
886 # These environment variables get used explicitly later in
887 # $command. No tool is expected to read these values directly.
888 .q{&& MEM=$(awk '($1 == "MemTotal:"){print $2}' </proc/meminfo) }
889 .q{&& SWAP=$(awk '($1 == "SwapTotal:"){print $2}' </proc/meminfo) }
890 ."&& MEMLIMIT=\$(( (\$MEM * 95) / ($ENV{CRUNCH_NODE_SLOTS} * 100) )) "
891 ."&& let SWAPLIMIT=\$MEMLIMIT+\$SWAP "
892 .q{&& declare -a VOLUMES=() }
893 .q{&& if which crunchrunner >/dev/null ; then VOLUMES+=("--volume=$(which crunchrunner):/usr/local/bin/crunchrunner:ro") ; fi }
894 .q{&& if test -f /etc/ssl/certs/ca-certificates.crt ; then VOLUMES+=("--volume=/etc/ssl/certs/ca-certificates.crt:/etc/arvados/ca-certificates.crt:ro") ; }
895 .q{elif test -f /etc/pki/tls/certs/ca-bundle.crt ; then VOLUMES+=("--volume=/etc/pki/tls/certs/ca-bundle.crt:/etc/arvados/ca-certificates.crt:ro") ; fi };
897 $command .= "&& exec arv-mount --read-write --mount-by-pdh=by_pdh --mount-tmp=tmp --crunchstat-interval=10 --allow-other $arv_file_cache \Q$keep_mnt\E --exec ";
898 $ENV{TASK_KEEPMOUNT} = "$keep_mnt/by_pdh";
899 $ENV{TASK_KEEPMOUNT_TMP} = "$keep_mnt/tmp";
903 my $containername = "$Jobstep->{arvados_task}->{uuid}-$Jobstep->{failures}";
904 my $cidfile = "$ENV{CRUNCH_TMP}/$containername.cid";
905 $command .= "crunchstat -cgroup-root=\Q$cgroup_root\E -cgroup-parent=docker -cgroup-cid=$cidfile -poll=10000 ";
906 $command .= "$docker_bin run $docker_run_args --name=$containername --attach=stdout --attach=stderr --attach=stdin -i \Q$dockeruserarg\E --cidfile=$cidfile --sig-proxy ";
907 # We only set memory limits if Docker lets us limit both memory and swap.
908 # Memory limits alone have been supported longer, but subprocesses tend
909 # to get SIGKILL if they exceed that without any swap limit set.
910 # See #5642 for additional background.
911 if ($docker_limitmem) {
912 $command .= "--memory=\${MEMLIMIT}k --memory-swap=\${SWAPLIMIT}k ";
915 # The source tree and $destdir directory (which we have
916 # installed on the worker host) are available in the container,
917 # under the same path.
918 $command .= "--volume=\Q$ENV{CRUNCH_SRC}:$ENV{CRUNCH_SRC}:ro\E ";
919 $command .= "--volume=\Q$ENV{CRUNCH_INSTALL}:$ENV{CRUNCH_INSTALL}:ro\E ";
921 # Currently, we make the "by_pdh" directory in arv-mount's mount
922 # point appear at /keep inside the container (instead of using
923 # the same path as the host like we do with CRUNCH_SRC and
924 # CRUNCH_INSTALL). However, crunch scripts and utilities must
925 # not rely on this. They must use $TASK_KEEPMOUNT.
926 $command .= "--volume=\Q$ENV{TASK_KEEPMOUNT}:/keep:ro\E ";
927 $ENV{TASK_KEEPMOUNT} = "/keep";
929 # Ditto TASK_KEEPMOUNT_TMP, as /keep_tmp.
930 $command .= "--volume=\Q$ENV{TASK_KEEPMOUNT_TMP}:/keep_tmp\E ";
931 $ENV{TASK_KEEPMOUNT_TMP} = "/keep_tmp";
933 # TASK_WORK is almost exactly like a docker data volume: it
934 # starts out empty, is writable, and persists until no
935 # containers use it any more. We don't use --volumes-from to
936 # share it with other containers: it is only accessible to this
937 # task, and it goes away when this task stops.
939 # However, a docker data volume is writable only by root unless
940 # the mount point already happens to exist in the container with
941 # different permissions. Therefore, we [1] assume /tmp already
942 # exists in the image and is writable by the crunch user; [2]
943 # avoid putting TASK_WORK inside CRUNCH_TMP (which won't be
944 # writable if they are created by docker while setting up the
945 # other --volumes); and [3] create $TASK_WORK inside the
946 # container using $build_script.
947 $command .= "--volume=/tmp ";
948 $ENV{"TASK_WORK"} = "/tmp/crunch-job-task-work/$childslotname";
949 $ENV{"HOME"} = $ENV{"TASK_WORK"};
950 $ENV{"TASK_TMPDIR"} = $ENV{"TASK_WORK"}; # deprecated
952 # TODO: Share a single JOB_WORK volume across all task
953 # containers on a given worker node, and delete it when the job
954 # ends (and, in case that doesn't work, when the next job
957 # For now, use the same approach as TASK_WORK above.
958 $ENV{"JOB_WORK"} = "/tmp/crunch-job-work";
960 # Bind mount the crunchrunner binary and host TLS certificates file into
962 $command .= '"${VOLUMES[@]}" ';
964 while (my ($env_key, $env_val) = each %ENV)
966 if ($env_key =~ /^(ARVADOS|CRUNCH|JOB|TASK)_/) {
967 $command .= "--env=\Q$env_key=$env_val\E ";
970 $command .= "--env=\QHOME=$ENV{HOME}\E ";
971 $command .= "\Q$docker_hash\E ";
973 if ($Job->{arvados_sdk_version}) {
975 $command .= "perl - \Q$ENV{CRUNCH_SRC}/crunch_scripts/$Job->{script}\E";
977 $command .= "/bin/sh -c \'python -c " .
978 '"from pkg_resources import get_distribution as get; print \"Using Arvados SDK version\", get(\"arvados-python-client\").version"' .
979 ">&2 2>/dev/null; " .
980 "mkdir -p \"$ENV{JOB_WORK}\" \"$ENV{TASK_WORK}\" && " .
981 "if which stdbuf >/dev/null ; then " .
982 " exec $stdbuf \Q$ENV{CRUNCH_SRC}/crunch_scripts/$Job->{script}\E ;" .
984 " exec \Q$ENV{CRUNCH_SRC}/crunch_scripts/$Job->{script}\E ;" .
989 $command .= "crunchstat -cgroup-root=\Q$cgroup_root\E -poll=10000 ";
991 $command .= "perl - $ENV{CRUNCH_SRC}/crunch_scripts/" . $Job->{"script"};
994 my @execargs = ('bash', '-c', $command);
995 srun (\@srunargs, \@execargs, undef, $build_script);
996 # exec() failed, we assume nothing happened.
997 die "srun() failed on build script\n";
1000 if (!defined $childpid)
1003 delete $reader{$id};
1007 $proc{$childpid} = {
1011 jobstepname => "$job_id.$id.$childpid",
1013 croak ("assert failed: \$slot[$childslot]->{'pid'} exists") if exists $slot[$childslot]->{pid};
1014 $slot[$childslot]->{pid} = $childpid;
1016 Log ($id, "job_task ".$Jobstep->{'arvados_task'}->{'uuid'});
1017 Log ($id, "child $childpid started on $childslotname");
1018 $Jobstep->{starttime} = time;
1019 $Jobstep->{node} = $childnode->{name};
1020 $Jobstep->{slotindex} = $childslot;
1021 delete $Jobstep->{stderr};
1022 delete $Jobstep->{finishtime};
1023 delete $Jobstep->{tempfail};
1025 $Jobstep->{'arvados_task'}->{started_at} = strftime "%Y-%m-%dT%H:%M:%SZ", gmtime($Jobstep->{starttime});
1026 retry_op(sub { $Jobstep->{'arvados_task'}->save; }, "job_tasks.update API");
1028 splice @jobstep_todo, $todo_ptr, 1;
1031 $progress_is_dirty = 1;
1035 ($round_num_freeslots > @freeslot && $todo_ptr+1 > $#jobstep_todo))
1037 last THISROUND if $main::please_freeze;
1038 if ($main::please_info)
1040 $main::please_info = 0;
1042 create_output_collection();
1044 update_progress_stats();
1049 if (!$gotsome || ($latest_refresh + 2 < scalar time))
1051 check_refresh_wanted();
1053 update_progress_stats();
1055 elsif (time - $progress_stats_updated >= 30 || $progress_is_dirty)
1057 update_progress_stats();
1060 select (undef, undef, undef, 0.1);
1062 $working_slot_count = scalar(grep { $_->{node}->{fail_count} == 0 &&
1063 $_->{node}->{hold_count} < 4 } @slot);
1064 if (($thisround_failed_multiple >= 8 && $thisround_succeeded == 0) ||
1065 ($thisround_failed_multiple >= 16 && $thisround_failed_multiple > $thisround_succeeded))
1067 my $message = "Repeated failure rate too high ($thisround_failed_multiple/"
1068 .($thisround_failed+$thisround_succeeded)
1069 .") -- giving up on this round";
1070 Log (undef, $message);
1074 # move slots from freeslot to holdslot (or back to freeslot) if necessary
1075 for (my $i=$#freeslot; $i>=0; $i--) {
1076 if ($slot[$freeslot[$i]]->{node}->{hold_until} > scalar time) {
1077 push @holdslot, (splice @freeslot, $i, 1);
1080 for (my $i=$#holdslot; $i>=0; $i--) {
1081 if ($slot[$holdslot[$i]]->{node}->{hold_until} <= scalar time) {
1082 push @freeslot, (splice @holdslot, $i, 1);
1086 # give up if no nodes are succeeding
1087 if ($working_slot_count < 1) {
1088 Log(undef, "Every node has failed -- giving up");
1095 push @freeslot, splice @holdslot;
1096 map { $slot[$freeslot[$_]]->{node}->{losing_streak} = 0 } (0..$#freeslot);
1099 Log (undef, "wait for last ".(scalar keys %proc)." children to finish");
1102 if ($main::please_continue) {
1103 $main::please_continue = 0;
1106 $main::please_info = 0, freeze(), create_output_collection(), save_meta(1) if $main::please_info;
1108 if (!reapchildren())
1110 check_refresh_wanted();
1112 update_progress_stats();
1113 select (undef, undef, undef, 0.1);
1114 killem (keys %proc) if $main::please_freeze;
1118 update_progress_stats();
1119 freeze_if_want_freeze();
1122 if (!defined $main::success)
1124 if (!@jobstep_todo) {
1126 } elsif ($working_slot_count < 1) {
1127 save_output_collection();
1129 exit_retry_unlocked();
1130 } elsif ($thisround_succeeded == 0 &&
1131 ($thisround_failed == 0 || $thisround_failed > 4)) {
1132 my $message = "stop because $thisround_failed tasks failed and none succeeded";
1133 Log (undef, $message);
1138 goto ONELEVEL if !defined $main::success;
1141 release_allocation();
1143 my $collated_output = save_output_collection();
1144 Log (undef, "finish");
1146 my $final_log = save_meta();
1149 if ($collated_output && $final_log && $main::success) {
1150 $final_state = 'Complete';
1152 $final_state = 'Failed';
1154 $Job->update_attributes('state' => $final_state);
1156 exit (($final_state eq 'Complete') ? 0 : 1);
1160 sub update_progress_stats
1162 $progress_stats_updated = time;
1163 return if !$progress_is_dirty;
1164 my ($todo, $done, $running) = (scalar @jobstep_todo,
1165 scalar @jobstep_done,
1166 scalar keys(%proc));
1167 $Job->{'tasks_summary'} ||= {};
1168 $Job->{'tasks_summary'}->{'todo'} = $todo;
1169 $Job->{'tasks_summary'}->{'done'} = $done;
1170 $Job->{'tasks_summary'}->{'running'} = $running;
1171 $Job->update_attributes('tasks_summary' => $Job->{'tasks_summary'});
1172 Log (undef, "status: $done done, $running running, $todo todo");
1173 $progress_is_dirty = 0;
1180 my $children_reaped = 0;
1181 my @successful_task_uuids = ();
1183 while((my $pid = waitpid (-1, WNOHANG)) > 0)
1185 my $childstatus = $?;
1187 my $whatslot = ($slot[$proc{$pid}->{slot}]->{node}->{name}
1189 . $slot[$proc{$pid}->{slot}]->{cpu});
1190 my $jobstepidx = $proc{$pid}->{jobstepidx};
1192 readfrompipes_after_exit ($jobstepidx);
1195 my $elapsed = time - $proc{$pid}->{time};
1196 my $Jobstep = $jobstep[$jobstepidx];
1198 my $exitvalue = $childstatus >> 8;
1199 my $exitinfo = "exit ".exit_status_s($childstatus);
1200 $Jobstep->{'arvados_task'}->reload;
1201 my $task_success = $Jobstep->{'arvados_task'}->{success};
1203 Log ($jobstepidx, "child $pid on $whatslot $exitinfo success=$task_success");
1205 if (!defined $task_success) {
1206 # task did not indicate one way or the other --> fail
1207 Log($jobstepidx, sprintf(
1208 "ERROR: Task process exited %s, but never updated its task record to indicate success and record its output.",
1209 exit_status_s($childstatus)));
1210 $Jobstep->{'arvados_task'}->{success} = 0;
1211 retry_op(sub { $Jobstep->{'arvados_task'}->save; }, "job_tasks.update API");
1218 $temporary_fail ||= $Jobstep->{tempfail};
1219 $temporary_fail ||= ($exitvalue == TASK_TEMPFAIL);
1221 ++$thisround_failed;
1222 ++$thisround_failed_multiple if $Jobstep->{'failures'} >= 1;
1224 # Check for signs of a failed or misconfigured node
1225 if (++$slot[$proc{$pid}->{slot}]->{node}->{losing_streak} >=
1226 2+$slot[$proc{$pid}->{slot}]->{node}->{ncpus}) {
1227 # Don't count this against jobstep failure thresholds if this
1228 # node is already suspected faulty and srun exited quickly
1229 if ($slot[$proc{$pid}->{slot}]->{node}->{hold_until} &&
1231 Log ($jobstepidx, "blaming failure on suspect node " .
1232 $slot[$proc{$pid}->{slot}]->{node}->{name});
1233 $temporary_fail ||= 1;
1235 ban_node_by_slot($proc{$pid}->{slot});
1238 Log ($jobstepidx, sprintf('failure (#%d, %s) after %d seconds',
1239 ++$Jobstep->{'failures'},
1240 $temporary_fail ? 'temporary' : 'permanent',
1243 if (!$temporary_fail || $Jobstep->{'failures'} >= 3) {
1244 # Give up on this task, and the whole job
1247 # Put this task back on the todo queue
1248 push @jobstep_todo, $jobstepidx;
1249 $Job->{'tasks_summary'}->{'failed'}++;
1253 push @successful_task_uuids, $Jobstep->{'arvados_task'}->{uuid};
1254 ++$thisround_succeeded;
1255 $slot[$proc{$pid}->{slot}]->{node}->{losing_streak} = 0;
1256 $slot[$proc{$pid}->{slot}]->{node}->{hold_until} = 0;
1257 $slot[$proc{$pid}->{slot}]->{node}->{fail_count} = 0;
1258 push @jobstep_done, $jobstepidx;
1259 Log ($jobstepidx, "success in $elapsed seconds");
1261 $Jobstep->{exitcode} = $childstatus;
1262 $Jobstep->{finishtime} = time;
1263 $Jobstep->{'arvados_task'}->{finished_at} = strftime "%Y-%m-%dT%H:%M:%SZ", gmtime($Jobstep->{finishtime});
1264 retry_op(sub { $Jobstep->{'arvados_task'}->save; }, "job_tasks.update API");
1265 Log ($jobstepidx, sprintf("task output (%d bytes): %s",
1266 length($Jobstep->{'arvados_task'}->{output}),
1267 $Jobstep->{'arvados_task'}->{output}));
1269 close $reader{$jobstepidx};
1270 delete $reader{$jobstepidx};
1271 delete $slot[$proc{$pid}->{slot}]->{pid};
1272 push @freeslot, $proc{$pid}->{slot};
1275 $progress_is_dirty = 1;
1278 if (scalar(@successful_task_uuids) > 0)
1280 Log (undef, sprintf("%d tasks exited (%d succeeded), checking for new tasks from API server.", $children_reaped, scalar(@successful_task_uuids)));
1282 my $newtask_list = [];
1283 my $newtask_results;
1285 $newtask_results = api_call(
1287 'filters' => [["created_by_job_task_uuid","in",\@successful_task_uuids]],
1288 'order' => 'qsequence',
1289 'offset' => scalar(@$newtask_list),
1291 push(@$newtask_list, @{$newtask_results->{items}});
1292 } while (@{$newtask_results->{items}});
1293 Log (undef, sprintf("Got %d new tasks from API server.", scalar(@$newtask_list)));
1294 foreach my $arvados_task (@$newtask_list) {
1296 'level' => $arvados_task->{'sequence'},
1298 'arvados_task' => $arvados_task
1300 push @jobstep, $jobstep;
1301 push @jobstep_todo, $#jobstep;
1305 return $children_reaped;
1308 sub check_refresh_wanted
1310 my @stat = stat $ENV{"CRUNCH_REFRESH_TRIGGER"};
1312 $stat[9] > $latest_refresh &&
1313 # ...and we have actually locked the job record...
1314 $job_id eq $Job->{'uuid'}) {
1315 $latest_refresh = scalar time;
1316 my $Job2 = api_call("jobs/get", uuid => $jobspec);
1317 for my $attr ('cancelled_at',
1318 'cancelled_by_user_uuid',
1319 'cancelled_by_client_uuid',
1321 $Job->{$attr} = $Job2->{$attr};
1323 if ($Job->{'state'} ne "Running") {
1324 if ($Job->{'state'} eq "Cancelled") {
1325 Log (undef, "Job cancelled at " . $Job->{'cancelled_at'} . " by user " . $Job->{'cancelled_by_user_uuid'});
1327 Log (undef, "Job state unexpectedly changed to " . $Job->{'state'});
1330 $main::please_freeze = 1;
1337 my $last_squeue_check = $squeue_checked;
1339 # Do not call `squeue` or check the kill list more than once every
1341 return if $last_squeue_check > time - 15;
1342 $squeue_checked = time;
1344 # Look for children from which we haven't received stderr data since
1345 # the last squeue check. If no such children exist, all procs are
1346 # alive and there's no need to even look at squeue.
1348 # As long as the crunchstat poll interval (10s) is shorter than the
1349 # squeue check interval (15s) this should make the squeue check an
1351 my $silent_procs = 0;
1352 for my $js (map {$jobstep[$_->{jobstepidx}]} values %proc)
1354 if (!exists($js->{stderr_at}))
1356 $js->{stderr_at} = 0;
1358 if ($js->{stderr_at} < $last_squeue_check)
1363 return if $silent_procs == 0;
1365 # use killem() on procs whose killtime is reached
1366 while (my ($pid, $procinfo) = each %proc)
1368 my $js = $jobstep[$procinfo->{jobstepidx}];
1369 if (exists $procinfo->{killtime}
1370 && $procinfo->{killtime} <= time
1371 && $js->{stderr_at} < $last_squeue_check)
1374 if ($js->{stderr_at}) {
1375 $sincewhen = " in last " . (time - $js->{stderr_at}) . "s";
1377 Log($procinfo->{jobstepidx}, "killing orphaned srun process $pid (task not in slurm queue, no stderr received$sincewhen)");
1384 # here is an opportunity to check for mysterious problems with local procs
1388 # Get a list of steps still running. Note: squeue(1) says --steps
1389 # selects a format (which we override anyway) and allows us to
1390 # specify which steps we're interested in (which we don't).
1391 # Importantly, it also changes the meaning of %j from "job name" to
1392 # "step name" and (although this isn't mentioned explicitly in the
1393 # docs) switches from "one line per job" mode to "one line per step"
1394 # mode. Without it, we'd just get a list of one job, instead of a
1396 my @squeue = `squeue --jobs=\Q$ENV{SLURM_JOB_ID}\E --steps --format='%j' --noheader`;
1399 Log(undef, "warning: squeue exit status $? ($!)");
1404 # which of my jobsteps are running, according to squeue?
1406 for my $jobstepname (@squeue)
1408 $ok{$jobstepname} = 1;
1411 # Check for child procs >60s old and not mentioned by squeue.
1412 while (my ($pid, $procinfo) = each %proc)
1414 if ($procinfo->{time} < time - 60
1415 && $procinfo->{jobstepname}
1416 && !exists $ok{$procinfo->{jobstepname}}
1417 && !exists $procinfo->{killtime})
1419 # According to slurm, this task has ended (successfully or not)
1420 # -- but our srun child hasn't exited. First we must wait (30
1421 # seconds) in case this is just a race between communication
1422 # channels. Then, if our srun child process still hasn't
1423 # terminated, we'll conclude some slurm communication
1424 # error/delay has caused the task to die without notifying srun,
1425 # and we'll kill srun ourselves.
1426 $procinfo->{killtime} = time + 30;
1427 Log($procinfo->{jobstepidx}, "notice: task is not in slurm queue but srun process $pid has not exited");
1434 # If a node fails in a multi-node "srun" call during job setup, the call
1435 # may hang instead of exiting with a nonzero code. This function checks
1436 # "sinfo" for the health of the nodes that were allocated and ensures that
1437 # they are all still in the "alloc" state. If a node that is allocated to
1438 # this job is not in "alloc" state, then set please_freeze.
1440 # This is only called from srun_sync() for node configuration. If a
1441 # node fails doing actual work, there are other recovery mechanisms.
1443 # Do not call `sinfo` more than once every 15 seconds.
1444 return if $sinfo_checked > time - 15;
1445 $sinfo_checked = time;
1447 # The output format "%t" means output node states.
1448 my @sinfo = `sinfo --nodes=\Q$ENV{SLURM_NODELIST}\E --noheader -o "%t"`;
1451 Log(undef, "warning: sinfo exit status $? ($!)");
1458 if ($_ != "alloc" && $_ != "alloc*") {
1459 $main::please_freeze = 1;
1464 sub release_allocation
1468 Log (undef, "release job allocation");
1469 system "scancel $ENV{SLURM_JOB_ID}";
1478 my $sel = IO::Select->new();
1479 foreach my $jobstepidx (keys %reader)
1481 my $fd = $reader{$jobstepidx};
1483 $fd_job{$fd} = $jobstepidx;
1485 if (my $stdout_fd = $jobstep[$jobstepidx]->{stdout_r}) {
1486 $sel->add($stdout_fd);
1487 $fd_job{$stdout_fd} = $jobstepidx;
1490 # select on all reader fds with 0.1s timeout
1491 my @ready_fds = $sel->can_read(0.1);
1492 foreach my $fd (@ready_fds)
1495 if (0 < sysread ($fd, $buf, 65536))
1498 print STDERR $buf if $ENV{CRUNCH_DEBUG};
1500 my $jobstepidx = $fd_job{$fd};
1501 if ($jobstep[$jobstepidx]->{stdout_r} == $fd) {
1502 $jobstep[$jobstepidx]->{stdout_captured} .= $buf;
1506 $jobstep[$jobstepidx]->{stderr_at} = time;
1507 $jobstep[$jobstepidx]->{stderr} .= $buf;
1509 # Consume everything up to the last \n
1510 preprocess_stderr ($jobstepidx);
1512 if (length ($jobstep[$jobstepidx]->{stderr}) > 16384)
1514 # If we get a lot of stderr without a newline, chop off the
1515 # front to avoid letting our buffer grow indefinitely.
1516 substr ($jobstep[$jobstepidx]->{stderr},
1517 0, length($jobstep[$jobstepidx]->{stderr}) - 8192) = "";
1525 # Consume all full lines of stderr for a jobstep. Everything after the
1526 # last newline will remain in $jobstep[$jobstepidx]->{stderr} after
1528 sub preprocess_stderr
1530 my $jobstepidx = shift;
1531 # slotindex is only defined for children running Arvados job tasks.
1532 # Be prepared to handle the undef case (for setup srun calls, etc.).
1533 my $job_slot_index = $jobstep[$jobstepidx]->{slotindex};
1535 while ($jobstep[$jobstepidx]->{stderr} =~ /^(.*?)\n/) {
1537 substr $jobstep[$jobstepidx]->{stderr}, 0, 1+length($line), "";
1538 Log ($jobstepidx, "stderr $line");
1539 if ($line =~ /srun: error: (SLURM job $ENV{SLURM_JOB_ID} has expired|Unable to confirm allocation for job $ENV{SLURM_JOB_ID})/i) {
1540 # If the allocation is revoked, we can't possibly continue, so mark all
1541 # nodes as failed. This will cause the overall exit code to be
1542 # EX_RETRY_UNLOCKED instead of failure so that crunch_dispatch can re-run
1544 $main::please_freeze = 1;
1545 foreach my $st (@slot) {
1546 $st->{node}->{fail_count}++;
1549 elsif ($line =~ /srun: error: .*?\b(Node failure on|Aborting, .*?\bio error\b|cannot communicate with node .* aborting job)/i) {
1550 $jobstep[$jobstepidx]->{tempfail} = 1;
1551 if (defined($job_slot_index)) {
1552 $slot[$job_slot_index]->{node}->{fail_count}++;
1553 ban_node_by_slot($job_slot_index);
1556 elsif ($line =~ /srun: error: (Unable to create job step|.*?: Communication connection failure)/i) {
1557 $jobstep[$jobstepidx]->{tempfail} = 1;
1558 ban_node_by_slot($job_slot_index) if (defined($job_slot_index));
1560 elsif ($line =~ /\bKeep(Read|Write|Request)Error:/) {
1561 $jobstep[$jobstepidx]->{tempfail} = 1;
1567 # Read whatever is still available on its stderr+stdout pipes after
1568 # the given child process has exited.
1569 sub readfrompipes_after_exit
1571 my $jobstepidx = shift;
1573 # The fact that the child has exited allows some convenient
1574 # simplifications: (1) all data must have already been written, so
1575 # there's no need to wait for more once sysread returns 0; (2) the
1576 # total amount of data available is bounded by the pipe buffer size,
1577 # so it's safe to read everything into one string.
1579 while (0 < sysread ($reader{$jobstepidx}, $buf, 65536)) {
1580 $jobstep[$jobstepidx]->{stderr_at} = time;
1581 $jobstep[$jobstepidx]->{stderr} .= $buf;
1583 if ($jobstep[$jobstepidx]->{stdout_r}) {
1584 while (0 < sysread ($jobstep[$jobstepidx]->{stdout_r}, $buf, 65536)) {
1585 $jobstep[$jobstepidx]->{stdout_captured} .= $buf;
1588 preprocess_stderr ($jobstepidx);
1591 Log ($jobstepidx, "stderr $_");
1592 } split ("\n", $jobstep[$jobstepidx]->{stderr});
1593 $jobstep[$jobstepidx]->{stderr} = '';
1600 if (!open($keep, "-|", "arv-get", "--retries", retry_count(), $hash)) {
1601 Log(undef, "fetch_block run error from arv-get $hash: $!");
1604 my $output_block = "";
1607 my $bytes = sysread($keep, $buf, 1024 * 1024);
1608 if (!defined $bytes) {
1609 Log(undef, "fetch_block read error from arv-get: $!");
1610 $output_block = undef;
1612 } elsif ($bytes == 0) {
1613 # sysread returns 0 at the end of the pipe.
1616 # some bytes were read into buf.
1617 $output_block .= $buf;
1622 Log(undef, "fetch_block arv-get exited " . exit_status_s($?));
1623 $output_block = undef;
1625 return $output_block;
1628 # Create a collection by concatenating the output of all tasks (each
1629 # task's output is either a manifest fragment, a locator for a
1630 # manifest fragment stored in Keep, or nothing at all). Return the
1631 # portable_data_hash of the new collection.
1632 sub create_output_collection
1634 Log (undef, "collate");
1636 my ($child_out, $child_in);
1637 my $pid = open2($child_out, $child_in, 'python', '-c', q{
1640 print (arvados.api("v1").collections().
1641 create(body={"manifest_text": sys.stdin.read(),
1642 "owner_uuid": sys.argv[2]}).
1643 execute(num_retries=int(sys.argv[1]))["portable_data_hash"])
1644 }, retry_count(), $Job->{owner_uuid});
1647 my $manifest_size = 0;
1651 my $output = $_->{'arvados_task'}->{output};
1652 next if (!defined($output));
1654 if ($output =~ /^[0-9a-f]{32}(\+\S+)*$/) {
1655 $next_write = fetch_block($output);
1657 $next_write = $output;
1659 if (defined($next_write)) {
1660 if (!defined(syswrite($child_in, $next_write))) {
1661 # There's been an error writing. Stop the loop.
1662 # We'll log details about the exit code later.
1665 $manifest_size += length($next_write);
1668 my $uuid = $_->{'arvados_task'}->{'uuid'};
1669 Log (undef, "Error retrieving '$output' output by task $task_idx ($uuid)");
1674 Log(undef, "collated output manifest text to send to API server is $manifest_size bytes with access tokens");
1677 my $s = IO::Select->new($child_out);
1678 if ($s->can_read(120)) {
1679 sysread($child_out, $joboutput, 1024 * 1024);
1682 Log(undef, "output collection creation exited " . exit_status_s($?));
1688 Log (undef, "timed out while creating output collection");
1689 foreach my $signal (2, 2, 2, 15, 15, 9) {
1690 kill($signal, $pid);
1691 last if waitpid($pid, WNOHANG) == -1;
1700 # Calls create_output_collection, logs the result, and returns it.
1701 # If that was successful, save that as the output in the job record.
1702 sub save_output_collection {
1703 my $collated_output = create_output_collection();
1705 if (!$collated_output) {
1706 Log(undef, "Failed to write output collection");
1709 Log(undef, "job output $collated_output");
1710 $Job->update_attributes('output' => $collated_output);
1712 return $collated_output;
1719 my $sig = 2; # SIGINT first
1720 if (exists $proc{$_}->{"sent_$sig"} &&
1721 time - $proc{$_}->{"sent_$sig"} > 4)
1723 $sig = 15; # SIGTERM if SIGINT doesn't work
1725 if (exists $proc{$_}->{"sent_$sig"} &&
1726 time - $proc{$_}->{"sent_$sig"} > 4)
1728 $sig = 9; # SIGKILL if SIGTERM doesn't work
1730 if (!exists $proc{$_}->{"sent_$sig"})
1732 Log ($proc{$_}->{jobstepidx}, "sending 2x signal $sig to pid $_");
1734 select (undef, undef, undef, 0.1);
1737 kill $sig, $_; # srun wants two SIGINT to really interrupt
1739 $proc{$_}->{"sent_$sig"} = time;
1740 $proc{$_}->{"killedafter"} = time - $proc{$_}->{"time"};
1750 vec($bits,fileno($_),1) = 1;
1756 # Send log output to Keep via arv-put.
1758 # $log_pipe_in and $log_pipe_out are the input and output filehandles to the arv-put pipe.
1759 # $log_pipe_out_buf is a string containing all output read from arv-put so far.
1760 # $log_pipe_out_select is an IO::Select object around $log_pipe_out.
1761 # $log_pipe_pid is the pid of the arv-put subprocess.
1763 # The only functions that should access these variables directly are:
1765 # log_writer_start($logfilename)
1766 # Starts an arv-put pipe, reading data on stdin and writing it to
1767 # a $logfilename file in an output collection.
1769 # log_writer_read_output([$timeout])
1770 # Read output from $log_pipe_out and append it to $log_pipe_out_buf.
1771 # Passes $timeout to the select() call, with a default of 0.01.
1772 # Returns the result of the last read() call on $log_pipe_out, or
1773 # -1 if read() wasn't called because select() timed out.
1774 # Only other log_writer_* functions should need to call this.
1776 # log_writer_send($txt)
1777 # Writes $txt to the output log collection.
1779 # log_writer_finish()
1780 # Closes the arv-put pipe and returns the output that it produces.
1782 # log_writer_is_active()
1783 # Returns a true value if there is currently a live arv-put
1784 # process, false otherwise.
1786 my ($log_pipe_in, $log_pipe_out, $log_pipe_out_buf, $log_pipe_out_select,
1789 sub log_writer_start($)
1791 my $logfilename = shift;
1792 $log_pipe_pid = open2($log_pipe_out, $log_pipe_in,
1796 '--filename', $logfilename,
1798 $log_pipe_out_buf = "";
1799 $log_pipe_out_select = IO::Select->new($log_pipe_out);
1802 sub log_writer_read_output {
1803 my $timeout = shift || 0.01;
1805 while ($read && $log_pipe_out_select->can_read($timeout)) {
1806 $read = read($log_pipe_out, $log_pipe_out_buf, 65536,
1807 length($log_pipe_out_buf));
1809 if (!defined($read)) {
1810 Log(undef, "error reading log manifest from arv-put: $!");
1815 sub log_writer_send($)
1818 print $log_pipe_in $txt;
1819 log_writer_read_output();
1822 sub log_writer_finish()
1824 return unless $log_pipe_pid;
1826 close($log_pipe_in);
1828 my $logger_failed = 0;
1829 my $read_result = log_writer_read_output(600);
1830 if ($read_result == -1) {
1831 $logger_failed = -1;
1832 Log (undef, "timed out reading from 'arv-put'");
1833 } elsif ($read_result != 0) {
1834 $logger_failed = -2;
1835 Log(undef, "failed to read arv-put log manifest to EOF");
1838 waitpid($log_pipe_pid, 0);
1840 $logger_failed ||= $?;
1841 Log(undef, "log_writer_finish: arv-put exited " . exit_status_s($?))
1844 close($log_pipe_out);
1845 my $arv_put_output = $logger_failed ? undef : $log_pipe_out_buf;
1846 $log_pipe_pid = $log_pipe_in = $log_pipe_out = $log_pipe_out_buf =
1847 $log_pipe_out_select = undef;
1849 return $arv_put_output;
1852 sub log_writer_is_active() {
1853 return $log_pipe_pid;
1856 sub Log # ($jobstepidx, $logmessage)
1858 my ($jobstepidx, $logmessage) = @_;
1859 if ($logmessage =~ /\n/) {
1860 for my $line (split (/\n/, $_[1])) {
1861 Log ($jobstepidx, $line);
1865 my $fh = select STDERR; $|=1; select $fh;
1867 if (defined($jobstepidx) && exists($jobstep[$jobstepidx]->{arvados_task})) {
1868 $task_qseq = $jobstepidx;
1870 my $message = sprintf ("%s %d %s %s", $job_id, $$, $task_qseq, $logmessage);
1871 $message =~ s{([^ -\176])}{"\\" . sprintf ("%03o", ord($1))}ge;
1874 if (log_writer_is_active() || -t STDERR) {
1875 my @gmtime = gmtime;
1876 $datetime = sprintf ("%04d-%02d-%02d_%02d:%02d:%02d",
1877 $gmtime[5]+1900, $gmtime[4]+1, @gmtime[3,2,1,0]);
1879 print STDERR ((-t STDERR) ? ($datetime." ".$message) : $message);
1881 if (log_writer_is_active()) {
1882 log_writer_send($datetime . " " . $message);
1889 my ($package, $file, $line) = caller;
1890 my $message = "@_ at $file line $line\n";
1891 Log (undef, $message);
1892 release_allocation();
1893 freeze() if @jobstep_todo;
1894 create_output_collection() if @jobstep_todo;
1904 if ($Job->{'state'} eq 'Cancelled') {
1905 $Job->update_attributes('finished_at' => scalar gmtime);
1907 $Job->update_attributes('state' => 'Failed');
1914 my $justcheckpoint = shift; # false if this will be the last meta saved
1915 return if $justcheckpoint; # checkpointing is not relevant post-Warehouse.pm
1916 return unless log_writer_is_active();
1917 my $log_manifest = log_writer_finish();
1918 return unless defined($log_manifest);
1921 my $prev_log_coll = api_call("collections/get", uuid => $Job->{log});
1922 $log_manifest = $prev_log_coll->{manifest_text} . $log_manifest;
1925 my $log_coll = api_call(
1926 "collections/create", ensure_unique_name => 1, collection => {
1927 manifest_text => $log_manifest,
1928 owner_uuid => $Job->{owner_uuid},
1929 name => sprintf("Log from %s job %s", $Job->{script}, $Job->{uuid}),
1931 Log(undef, "log collection is " . $log_coll->{portable_data_hash});
1932 $Job->update_attributes('log' => $log_coll->{portable_data_hash});
1934 return $log_coll->{portable_data_hash};
1938 sub freeze_if_want_freeze
1940 if ($main::please_freeze)
1942 release_allocation();
1945 # kill some srun procs before freeze+stop
1946 map { $proc{$_} = {} } @_;
1949 killem (keys %proc);
1950 select (undef, undef, undef, 0.1);
1952 while (($died = waitpid (-1, WNOHANG)) > 0)
1954 delete $proc{$died};
1959 create_output_collection();
1969 Log (undef, "Freeze not implemented");
1976 croak ("Thaw not implemented");
1992 $s =~ s{\\(.)}{$1 eq "n" ? "\n" : $1}ge;
1998 my $srunargs = shift;
1999 my $execargs = shift;
2000 my $opts = shift || {};
2003 my $label = exists $opts->{label} ? $opts->{label} : "@$execargs";
2004 Log (undef, "$label: start");
2006 my ($stderr_r, $stderr_w);
2007 pipe $stderr_r, $stderr_w or croak("pipe() failed: $!");
2009 my ($stdout_r, $stdout_w);
2010 pipe $stdout_r, $stdout_w or croak("pipe() failed: $!");
2012 my $started_srun = scalar time;
2014 my $srunpid = fork();
2019 fcntl($stderr_w, F_SETFL, 0) or croak($!); # no close-on-exec
2020 fcntl($stdout_w, F_SETFL, 0) or croak($!);
2021 open(STDERR, ">&", $stderr_w) or croak ($!);
2022 open(STDOUT, ">&", $stdout_w) or croak ($!);
2023 srun ($srunargs, $execargs, $opts, $stdin);
2029 set_nonblocking($stderr_r);
2030 set_nonblocking($stdout_r);
2032 # Add entries to @jobstep and %proc so check_squeue() and
2033 # freeze_if_want_freeze() can treat it like a job task process.
2037 stderr_captured => '',
2038 stdout_r => $stdout_r,
2039 stdout_captured => '',
2041 my $jobstepidx = $#jobstep;
2043 jobstepidx => $jobstepidx,
2045 $reader{$jobstepidx} = $stderr_r;
2047 while ($srunpid != waitpid ($srunpid, WNOHANG)) {
2048 my $busy = readfrompipes();
2049 if (!$busy || ($latest_refresh + 2 < scalar time)) {
2050 check_refresh_wanted();
2055 select(undef, undef, undef, 0.1);
2057 if (($started_srun + $srun_sync_timeout) < scalar time) {
2058 # Exceeded general timeout for "srun_sync" operations, likely
2059 # means something got stuck on the remote node.
2060 Log(undef, "srun_sync exceeded timeout, will fail.");
2061 $main::please_freeze = 1;
2063 killem(keys %proc) if $main::please_freeze;
2067 readfrompipes_after_exit ($jobstepidx);
2069 Log (undef, "$label: exit ".exit_status_s($exited));
2073 delete $proc{$srunpid};
2074 delete $reader{$jobstepidx};
2076 my $j = pop @jobstep;
2077 # If the srun showed signs of tempfail, ensure the caller treats that as a
2079 if ($main::please_freeze || $j->{tempfail}) {
2082 return ($exited, $j->{stdout_captured}, $j->{stderr_captured}, $j->{tempfail});
2088 my $srunargs = shift;
2089 my $execargs = shift;
2090 my $opts = shift || {};
2092 my $args = $have_slurm ? [@$srunargs, @$execargs] : $execargs;
2094 $Data::Dumper::Terse = 1;
2095 $Data::Dumper::Indent = 0;
2096 my $show_cmd = Dumper($args);
2097 $show_cmd =~ s/(TOKEN\\*=)[^\s\']+/${1}[...]/g;
2098 $show_cmd =~ s/\n/ /g;
2099 if ($opts->{fork}) {
2100 Log(undef, "starting: $show_cmd");
2102 # This is a child process: parent is in charge of reading our
2103 # stderr and copying it to Log() if needed.
2104 warn "starting: $show_cmd\n";
2107 if (defined $stdin) {
2108 my $child = open STDIN, "-|";
2109 defined $child or die "no fork: $!";
2111 print $stdin or die $!;
2112 close STDOUT or die $!;
2117 return system (@$args) if $opts->{fork};
2120 warn "ENV size is ".length(join(" ",%ENV));
2121 die "exec failed: $!: @$args";
2125 sub ban_node_by_slot {
2126 # Don't start any new jobsteps on this node for 60 seconds
2128 $slot[$slotid]->{node}->{hold_until} = 60 + scalar time;
2129 $slot[$slotid]->{node}->{hold_count}++;
2130 Log (undef, "backing off node " . $slot[$slotid]->{node}->{name} . " for 60 seconds");
2135 my ($lockfile, $error_message) = @_;
2136 open L, ">", $lockfile or croak("$lockfile: $!");
2137 if (!flock L, LOCK_EX|LOCK_NB) {
2138 croak("Can't lock $lockfile: $error_message\n");
2142 sub find_docker_image {
2143 # Given a Keep locator, check to see if it contains a Docker image.
2144 # If so, return its stream name and Docker hash.
2145 # If not, return undef for both values.
2146 my $locator = shift;
2147 my ($streamname, $filename);
2148 my $image = api_call("collections/get", uuid => $locator);
2150 foreach my $line (split(/\n/, $image->{manifest_text})) {
2151 my @tokens = split(/\s+/, $line);
2153 $streamname = shift(@tokens);
2154 foreach my $filedata (grep(/^\d+:\d+:/, @tokens)) {
2155 if (defined($filename)) {
2156 return (undef, undef); # More than one file in the Collection.
2158 $filename = (split(/:/, $filedata, 3))[2];
2163 if (defined($filename) and ($filename =~ /^((?:sha256:)?[0-9A-Fa-f]{64})\.tar$/)) {
2164 return ($streamname, $1);
2166 return (undef, undef);
2170 sub exit_retry_unlocked {
2171 Log(undef, "Transient failure with lock acquired; asking for re-dispatch by exiting ".EX_RETRY_UNLOCKED);
2172 exit(EX_RETRY_UNLOCKED);
2176 # Calculate the number of times an operation should be retried,
2177 # assuming exponential backoff, and that we're willing to retry as
2178 # long as tasks have been running. Enforce a minimum of 3 retries.
2179 my ($starttime, $endtime, $timediff, $retries);
2181 $starttime = $jobstep[0]->{starttime};
2182 $endtime = $jobstep[-1]->{finishtime};
2184 if (!defined($starttime)) {
2186 } elsif (!defined($endtime)) {
2187 $timediff = time - $starttime;
2189 $timediff = ($endtime - $starttime) - (time - $endtime);
2191 if ($timediff > 0) {
2192 $retries = int(log($timediff) / log(2));
2194 $retries = 1; # Use the minimum.
2196 return ($retries > 3) ? $retries : 3;
2200 # Pass in two function references.
2201 # This method will be called with the remaining arguments.
2202 # If it dies, retry it with exponential backoff until it succeeds,
2203 # or until the current retry_count is exhausted. After each failure
2204 # that can be retried, the second function will be called with
2205 # the current try count (0-based), next try time, and error message.
2206 my $operation = shift;
2207 my $op_text = shift;
2208 my $retries = retry_count();
2209 my $retry_callback = sub {
2210 my ($try_count, $next_try_at, $errmsg) = @_;
2211 $errmsg =~ s/\s*\bat \Q$0\E line \d+\.?\s*//;
2212 $errmsg =~ s/\s/ /g;
2213 $errmsg =~ s/\s+$//;
2215 if ($next_try_at < time) {
2216 $retry_msg = "Retrying.";
2218 my $next_try_fmt = strftime "%Y-%m-%dT%H:%M:%SZ", gmtime($next_try_at);
2219 $retry_msg = "Retrying at $next_try_fmt.";
2221 Log(undef, "$op_text failed: $errmsg. $retry_msg");
2223 foreach my $try_count (0..$retries) {
2224 my $next_try = time + (2 ** $try_count);
2225 my $result = eval { $operation->(@_); };
2228 } elsif ($try_count < $retries) {
2229 $retry_callback->($try_count, $next_try, $@);
2230 my $sleep_time = $next_try - time;
2231 sleep($sleep_time) if ($sleep_time > 0);
2234 # Ensure the error message ends in a newline, so Perl doesn't add
2235 # retry_op's line number to it.
2241 # Pass in a /-separated API method name, and arguments for it.
2242 # This function will call that method, retrying as needed until
2243 # the current retry_count is exhausted, with a log on the first failure.
2244 my $method_name = shift;
2246 foreach my $key (split(/\//, $method_name)) {
2247 $method = $method->{$key};
2249 return retry_op(sub { $method->execute(@_); }, "API method $method_name", @_);
2253 # Given a $?, return a human-readable exit code string like "0" or
2254 # "1" or "0 with signal 1" or "1 with signal 11".
2255 my $exitcode = shift;
2256 my $s = $exitcode >> 8;
2257 if ($exitcode & 0x7f) {
2258 $s .= " with signal " . ($exitcode & 0x7f);
2260 if ($exitcode & 0x80) {
2261 $s .= " with core dump";
2266 sub handle_readall {
2267 # Pass in a glob reference to a file handle.
2268 # Read all its contents and return them as a string.
2269 my $fh_glob_ref = shift;
2271 return <$fh_glob_ref>;
2274 sub tar_filename_n {
2276 return sprintf("%s/git.%s.%d.tar", $ENV{CRUNCH_TMP}, $job_id, $n);
2279 sub add_git_archive {
2280 # Pass in a git archive command as a string or list, a la system().
2281 # This method will save its output to be included in the archive sent to the
2285 if (!open(GIT_ARCHIVE, ">", tar_filename_n($git_tar_count))) {
2286 croak("Failed to save git archive: $!");
2288 my $git_pid = open2(">&GIT_ARCHIVE", $git_input, @_);
2290 waitpid($git_pid, 0);
2293 croak("Failed to save git archive: git exited " . exit_status_s($?));
2297 sub combined_git_archive {
2298 # Combine all saved tar archives into a single archive, then return its
2299 # contents in a string. Return undef if no archives have been saved.
2300 if ($git_tar_count < 1) {
2303 my $base_tar_name = tar_filename_n(1);
2304 foreach my $tar_to_append (map { tar_filename_n($_); } (2..$git_tar_count)) {
2305 my $tar_exit = system("tar", "-Af", $base_tar_name, $tar_to_append);
2306 if ($tar_exit != 0) {
2307 croak("Error preparing build archive: tar -A exited " .
2308 exit_status_s($tar_exit));
2311 if (!open(GIT_TAR, "<", $base_tar_name)) {
2312 croak("Could not open build archive: $!");
2314 my $tar_contents = handle_readall(\*GIT_TAR);
2316 return $tar_contents;
2319 sub set_nonblocking {
2321 my $flags = fcntl ($fh, F_GETFL, 0) or croak ($!);
2322 fcntl ($fh, F_SETFL, $flags | O_NONBLOCK) or croak ($!);
2328 # This is crunch-job's internal dispatch script. crunch-job running on the API
2329 # server invokes this script on individual compute nodes, or localhost if we're
2330 # running a job locally. It gets called in two modes:
2332 # * No arguments: Installation mode. Read a tar archive from the DATA
2333 # file handle; it includes the Crunch script's source code, and
2334 # maybe SDKs as well. Those should be installed in the proper
2335 # locations. This runs outside of any Docker container, so don't try to
2336 # introspect Crunch's runtime environment.
2338 # * With arguments: Crunch script run mode. This script should set up the
2339 # environment, then run the command specified in the arguments. This runs
2340 # inside any Docker container.
2343 use File::Path qw( make_path remove_tree );
2344 use POSIX qw(getcwd);
2346 use constant TASK_TEMPFAIL => 111;
2348 # Map SDK subdirectories to the path environments they belong to.
2349 my %SDK_ENVVARS = ("perl/lib" => "PERLLIB", "ruby/lib" => "RUBYLIB");
2351 my $destdir = $ENV{"CRUNCH_SRC"};
2352 my $archive_hash = $ENV{"CRUNCH_GIT_ARCHIVE_HASH"};
2353 my $repo = $ENV{"CRUNCH_SRC_URL"};
2354 my $install_dir = $ENV{"CRUNCH_INSTALL"} || (getcwd() . "/opt");
2355 my $job_work = $ENV{"JOB_WORK"};
2356 my $task_work = $ENV{"TASK_WORK"};
2358 open(STDOUT_ORIG, ">&", STDOUT);
2359 open(STDERR_ORIG, ">&", STDERR);
2361 for my $dir ($destdir, $job_work, $task_work) {
2364 -e $dir or die "Failed to create temporary directory ($dir): $!";
2369 remove_tree($task_work, {keep_root => 1});
2372 ### Crunch script run mode
2374 # We want to do routine logging during task 0 only. This gives the user
2375 # the information they need, but avoids repeating the information for every
2378 if ($ENV{TASK_SEQUENCE} eq "0") {
2381 printf STDERR_ORIG "[Crunch] $msg\n", @_;
2387 my $python_src = "$install_dir/python";
2388 my $venv_dir = "$job_work/.arvados.venv";
2389 my $venv_built = -e "$venv_dir/bin/activate";
2390 if ((!$venv_built) and (-d $python_src) and can_run("virtualenv")) {
2391 shell_or_die(undef, "virtualenv", "--quiet", "--system-site-packages",
2392 "--python=python2.7", $venv_dir);
2393 shell_or_die(TASK_TEMPFAIL, "$venv_dir/bin/pip", "--quiet", "install", "-I", $python_src);
2395 $Log->("Built Python SDK virtualenv");
2398 my @pysdk_version_cmd = ("python", "-c",
2399 "from pkg_resources import get_distribution as get; print get('arvados-python-client').version");
2401 $Log->("Running in Python SDK virtualenv");
2402 @pysdk_version_cmd = ();
2403 my $orig_argv = join(" ", map { quotemeta($_); } @ARGV);
2404 @ARGV = ("/bin/sh", "-ec",
2405 ". \Q$venv_dir/bin/activate\E; exec $orig_argv");
2406 } elsif (-d $python_src) {
2407 $Log->("Warning: virtualenv not found inside Docker container default " .
2408 "\$PATH. Can't install Python SDK.");
2411 if (@pysdk_version_cmd) {
2412 open(my $pysdk_version_pipe, "-|", @pysdk_version_cmd);
2413 my $pysdk_version = <$pysdk_version_pipe>;
2414 close($pysdk_version_pipe);
2416 chomp($pysdk_version);
2417 $Log->("Using Arvados SDK version $pysdk_version");
2419 # A lot could've gone wrong here, but pretty much all of it means that
2420 # Python won't be able to load the Arvados SDK.
2421 $Log->("Warning: Arvados SDK not found");
2425 while (my ($sdk_dir, $sdk_envkey) = each(%SDK_ENVVARS)) {
2426 my $sdk_path = "$install_dir/$sdk_dir";
2428 if ($ENV{$sdk_envkey}) {
2429 $ENV{$sdk_envkey} = "$sdk_path:" . $ENV{$sdk_envkey};
2431 $ENV{$sdk_envkey} = $sdk_path;
2433 $Log->("Arvados SDK added to %s", $sdk_envkey);
2438 die "Cannot exec `@ARGV`: $!";
2441 ### Installation mode
2442 open L, ">", "$destdir.lock" or die "$destdir.lock: $!";
2444 if (readlink ("$destdir.archive_hash") eq $archive_hash && -d $destdir) {
2445 # This exact git archive (source + arvados sdk) is already installed
2446 # here, so there's no need to reinstall it.
2448 # We must consume our DATA section, though: otherwise the process
2449 # feeding it to us will get SIGPIPE.
2451 while (read(DATA, $buf, 65536)) { }
2456 unlink "$destdir.archive_hash";
2460 # Ignore SIGPIPE: we check retval of close() instead. See perlipc(1).
2461 local $SIG{PIPE} = "IGNORE";
2462 warn "Extracting archive: $archive_hash\n";
2463 # --ignore-zeros is necessary sometimes: depending on how much NUL
2464 # padding tar -A put on our combined archive (which in turn depends
2465 # on the length of the component archives) tar without
2466 # --ignore-zeros will exit before consuming stdin and cause close()
2467 # to fail on the resulting SIGPIPE.
2468 if (!open(TARX, "|-", "tar", "--ignore-zeros", "-xC", $destdir)) {
2469 die "Error launching 'tar -xC $destdir': $!";
2471 # If we send too much data to tar in one write (> 4-5 MiB), it stops, and we
2472 # get SIGPIPE. We must feed it data incrementally.
2474 while (read(DATA, $tar_input, 65536)) {
2475 print TARX $tar_input;
2478 die "'tar -xC $destdir' exited $?: $!";
2484 my $sdk_root = "$destdir/.arvados.sdk/sdk";
2486 foreach my $sdk_lang (("python",
2487 map { (split /\//, $_, 2)[0]; } keys(%SDK_ENVVARS))) {
2488 if (-d "$sdk_root/$sdk_lang") {
2489 if (!rename("$sdk_root/$sdk_lang", "$install_dir/$sdk_lang")) {
2490 die "Failed to install $sdk_lang SDK: $!";
2496 my $python_dir = "$install_dir/python";
2497 if ((-d $python_dir) and can_run("python2.7")) {
2498 open(my $egg_info_pipe, "-|",
2499 "python2.7 \Q$python_dir/setup.py\E egg_info 2>&1 >/dev/null");
2500 my @egg_info_errors = <$egg_info_pipe>;
2501 close($egg_info_pipe);
2504 if (@egg_info_errors and (($egg_info_errors[-1] =~ /\bgit\b/) or ($egg_info_errors[-1] =~ /\[Errno 2\]/))) {
2505 # egg_info apparently failed because it couldn't ask git for a build tag.
2506 # Specify no build tag.
2507 open(my $pysdk_cfg, ">>", "$python_dir/setup.cfg");
2508 print $pysdk_cfg "\n[egg_info]\ntag_build =\n";
2511 my $egg_info_exit = $? >> 8;
2512 foreach my $errline (@egg_info_errors) {
2515 warn "python setup.py egg_info failed: exit $egg_info_exit";
2516 exit ($egg_info_exit || 1);
2521 # Hide messages from the install script (unless it fails: shell_or_die
2522 # will show $destdir.log in that case).
2523 open(STDOUT, ">>", "$destdir.log") or die ($!);
2524 open(STDERR, ">&", STDOUT) or die ($!);
2526 if (-e "$destdir/crunch_scripts/install") {
2527 shell_or_die (undef, "$destdir/crunch_scripts/install", $install_dir);
2528 } elsif (!-e "./install.sh" && -e "./tests/autotests.sh") {
2530 shell_or_die (undef, "./tests/autotests.sh", $install_dir);
2531 } elsif (-e "./install.sh") {
2532 shell_or_die (undef, "./install.sh", $install_dir);
2535 if ($archive_hash) {
2536 unlink "$destdir.archive_hash.new";
2537 symlink ($archive_hash, "$destdir.archive_hash.new") or die "$destdir.archive_hash.new: $!";
2538 rename ("$destdir.archive_hash.new", "$destdir.archive_hash") or die "$destdir.archive_hash: $!";
2544 my $command_name = shift;
2545 open(my $which, "-|", "which", $command_name) or die ($!);
2546 while (<$which>) { }
2553 my $exitcode = shift;
2555 if ($ENV{"DEBUG"}) {
2556 print STDERR "@_\n";
2558 if (system (@_) != 0) {
2561 my $exitstatus = sprintf("exit %d signal %d", $code >> 8, $code & 0x7f);
2562 open STDERR, ">&STDERR_ORIG";
2563 system ("cat $destdir.log >&2");
2564 warn "@_ failed ($err): $exitstatus";
2565 if (defined($exitcode)) {
2569 exit (($code >> 8) || 1);