From 5b885e3379769844db6a9fac4da563e35f147eb4 Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Fri, 24 May 2024 17:58:54 +0800 Subject: [PATCH 01/15] Use sigterm --- dask_jobqueue/slurm.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/dask_jobqueue/slurm.py b/dask_jobqueue/slurm.py index 8e6c1a07..687bca48 100644 --- a/dask_jobqueue/slurm.py +++ b/dask_jobqueue/slurm.py @@ -12,7 +12,7 @@ class SLURMJob(Job): # Override class variables submit_command = "sbatch" - cancel_command = "scancel" + cancel_command = "scancel --signal=SIGTERM" config_name = "slurm" def __init__( From a6cb957290d450ff67c7159dd8696d5166f5be35 Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Sat, 25 May 2024 09:35:28 +0800 Subject: [PATCH 02/15] Add extra submit/cancel args to core --- dask_jobqueue/core.py | 29 +++++++++++++++++++++++++++++ 1 file changed, 29 insertions(+) diff --git a/dask_jobqueue/core.py b/dask_jobqueue/core.py index 62af14aa..129e7c16 100644 --- a/dask_jobqueue/core.py +++ b/dask_jobqueue/core.py @@ -53,6 +53,10 @@ Additional arguments to pass to `dask-worker` env_extra : list Deprecated: use ``job_script_prologue`` instead. This parameter will be removed in a future version. + submit_command_extra : list + Extra arguments to pass to the job scheduler submit command + cancel_command_extra : list + Extra arguments to pass to the job scheduler cancel command job_script_prologue : list Other commands to add to script before launching worker. header_skip : list @@ -172,6 +176,8 @@ def __init__( job_extra=None, job_extra_directives=None, env_extra=None, + submit_command_extra=None, + cancel_command_extra=None, job_script_prologue=None, header_skip=None, job_directives_skip=None, @@ -270,6 +276,29 @@ def __init__( if env_extra is None: env_extra = dask.config.get("jobqueue.%s.env-extra" % self.config_name) + + if self.submit_command_extra is None: + self.submit_command_extra = dask.config.get( + "jobqueue.%s.submit-command-extra" % self.config_name, [] + ) + + self.submit_command = ( + Job.submit_command + + " " + + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) + ) + + if self.cancel_command_extra is None: + self.cancel_command_extra = dask.config.get( + "jobqueue.%s.cancel-command-extra" % self.config_name, [] + ) + + self.cancel_command = ( + Job.cancel_command + + " " + + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) + ) + if job_script_prologue is None: job_script_prologue = dask.config.get( "jobqueue.%s.job-script-prologue" % self.config_name From 320dfc076293ce856bea4baf4a7a820aef58fb02 Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Sat, 25 May 2024 09:36:02 +0800 Subject: [PATCH 03/15] Rework condor to match core --- dask_jobqueue/htcondor.py | 20 +++++++------------- 1 file changed, 7 insertions(+), 13 deletions(-) diff --git a/dask_jobqueue/htcondor.py b/dask_jobqueue/htcondor.py index 4ab5c07a..e8eff970 100644 --- a/dask_jobqueue/htcondor.py +++ b/dask_jobqueue/htcondor.py @@ -38,8 +38,6 @@ def __init__( name=None, disk=None, config_name=None, - submit_command_extra=None, - cancel_command_extra=None, **base_class_kwargs ): super().__init__( @@ -95,26 +93,26 @@ def __init__( if self.job_extra_directives: self.job_header_dict.update(self.job_extra_directives) - if submit_command_extra is None: - submit_command_extra = dask.config.get( + if self.submit_command_extra is None: + self.submit_command_extra = dask.config.get( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) self.submit_command = ( - HTCondorJob.submit_command + HTCondorJob.submit_commsand + " " - + " ".join(shlex.quote(arg) for arg in submit_command_extra) + + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) ) - if cancel_command_extra is None: - cancel_command_extra = dask.config.get( + if self.cancel_command_extra is None: + self.cancel_command_extra = dask.config.get( "jobqueue.%s.cancel-command-extra" % self.config_name, [] ) self.cancel_command = ( HTCondorJob.cancel_command + " " - + " ".join(shlex.quote(arg) for arg in cancel_command_extra) + + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) ) def job_script(self): @@ -227,10 +225,6 @@ class HTCondorCluster(JobQueueCluster): job_extra_directives : dict Extra submit file attributes for the job as key-value pairs. They will be inserted as ``key = value``. - submit_command_extra : list of str - Extra arguments to pass to condor_submit - cancel_command_extra : list of str - Extra arguments to pass to condor_rm {job} {cluster} From 8752fb4e8b41f6674dccc5a5519d1b2667906653 Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Sat, 25 May 2024 09:36:29 +0800 Subject: [PATCH 04/15] Add slurm default --- dask_jobqueue/jobqueue.yaml | 2 ++ dask_jobqueue/slurm.py | 26 +++++++++++++++++++++++++- 2 files changed, 27 insertions(+), 1 deletion(-) diff --git a/dask_jobqueue/jobqueue.yaml b/dask_jobqueue/jobqueue.yaml index f9424158..06fb6e9d 100644 --- a/dask_jobqueue/jobqueue.yaml +++ b/dask_jobqueue/jobqueue.yaml @@ -122,6 +122,8 @@ jobqueue: account: null walltime: '00:30:00' env-extra: null + submit-command-extra: [] # Extra sbatch arguments + cancel-command-extra: ["--signal=SIGTERM"] # Extra scancel arguments job-script-prologue: [] job-cpu: null job-mem: null diff --git a/dask_jobqueue/slurm.py b/dask_jobqueue/slurm.py index 687bca48..a78fc072 100644 --- a/dask_jobqueue/slurm.py +++ b/dask_jobqueue/slurm.py @@ -1,5 +1,6 @@ import logging import math +import shlex import warnings import dask @@ -12,7 +13,7 @@ class SLURMJob(Job): # Override class variables submit_command = "sbatch" - cancel_command = "scancel --signal=SIGTERM" + cancel_command = "scancel" config_name = "slurm" def __init__( @@ -26,6 +27,7 @@ def __init__( job_cpu=None, job_mem=None, config_name=None, + cancel_command_extra=["--signal=SIGTERM"], **base_class_kwargs ): super().__init__( @@ -57,6 +59,28 @@ def __init__( if job_mem is None: job_mem = dask.config.get("jobqueue.%s.job-mem" % self.config_name) + if self.submit_command_extra is None: + self.submit_command_extra = dask.config.get( + "jobqueue.%s.submit-command-extra" % self.config_name, [] + ) + + self.submit_command = ( + SLURMJob.submit_command + + " " + + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) + ) + + if cancel_command_extra is None: + cancel_command_extra = dask.config.get( + "jobqueue.%s.cancel-command-extra" % self.config_name, [] + ) + + self.cancel_command = ( + SLURMJob.cancel_command + + " " + + " ".join(shlex.quote(arg) for arg in cancel_command_extra) + ) + header_lines = [] # SLURM header build if self.job_name is not None: From 012f121774f0b71eb3a818eb227ff012b363fabc Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Sat, 25 May 2024 09:37:00 +0800 Subject: [PATCH 05/15] Add boilerplate to other clusters --- dask_jobqueue/lsf.py | 23 +++++++++++++++++++++++ dask_jobqueue/oar.py | 22 ++++++++++++++++++++++ dask_jobqueue/pbs.py | 23 +++++++++++++++++++++++ dask_jobqueue/sge.py | 23 +++++++++++++++++++++++ 4 files changed, 91 insertions(+) diff --git a/dask_jobqueue/lsf.py b/dask_jobqueue/lsf.py index d165371d..2537e7f7 100644 --- a/dask_jobqueue/lsf.py +++ b/dask_jobqueue/lsf.py @@ -4,6 +4,7 @@ import math import os import re +import shlex import subprocess import toolz @@ -54,6 +55,28 @@ def __init__( use_stdin = dask.config.get("jobqueue.%s.use-stdin" % self.config_name) self.use_stdin = use_stdin + if self.submit_command_extra is None: + self.submit_command_extra = dask.config.get( + "jobqueue.%s.submit-command-extra" % self.config_name, [] + ) + + self.submit_command = ( + LSFJob.submit_command + + " " + + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) + ) + + if self.cancel_command_extra is None: + self.cancel_command_extra = dask.config.get( + "jobqueue.%s.cancel-command-extra" % self.config_name, [] + ) + + self.cancel_command = ( + LSFJob.cancel_command + + " " + + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) + ) + header_lines = [] # LSF header build if self.name is not None: diff --git a/dask_jobqueue/oar.py b/dask_jobqueue/oar.py index 0a23f4b0..6f90c0b9 100644 --- a/dask_jobqueue/oar.py +++ b/dask_jobqueue/oar.py @@ -44,6 +44,28 @@ def __init__( if walltime is None: walltime = dask.config.get("jobqueue.%s.walltime" % self.config_name) + if self.submit_command_extra is None: + self.submit_command_extra = dask.config.get( + "jobqueue.%s.submit-command-extra" % self.config_name, [] + ) + + self.submit_command = ( + OARJob.submit_command + + " " + + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) + ) + + if self.cancel_command_extra is None: + self.cancel_command_extra = dask.config.get( + "jobqueue.%s.cancel-command-extra" % self.config_name, [] + ) + + self.cancel_command = ( + OARJob.cancel_command + + " " + + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) + ) + header_lines = [] if self.job_name is not None: header_lines.append("#OAR -n %s" % self.job_name) diff --git a/dask_jobqueue/pbs.py b/dask_jobqueue/pbs.py index 719fd9ce..cee59883 100644 --- a/dask_jobqueue/pbs.py +++ b/dask_jobqueue/pbs.py @@ -1,6 +1,7 @@ import logging import math import os +import shlex import warnings import dask @@ -83,6 +84,28 @@ def __init__( if not account: account = project + if self.submit_command_extra is None: + self.submit_command_extra = dask.config.get( + "jobqueue.%s.submit-command-extra" % self.config_name, [] + ) + + self.submit_command = ( + PBSJob.submit_command + + " " + + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) + ) + + if self.cancel_command_extra is None: + self.cancel_command_extra = dask.config.get( + "jobqueue.%s.cancel-command-extra" % self.config_name, [] + ) + + self.cancel_command = ( + PBSJob.cancel_command + + " " + + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) + ) + header_lines = [] # PBS header build if self.job_name is not None: diff --git a/dask_jobqueue/sge.py b/dask_jobqueue/sge.py index 64bb1b4d..ebafa8d0 100644 --- a/dask_jobqueue/sge.py +++ b/dask_jobqueue/sge.py @@ -1,4 +1,5 @@ import logging +import shlex import dask @@ -38,6 +39,28 @@ def __init__( if walltime is None: walltime = dask.config.get("jobqueue.%s.walltime" % self.config_name) + if self.submit_command_extra is None: + self.submit_command_extra = dask.config.get( + "jobqueue.%s.submit-command-extra" % self.config_name, [] + ) + + self.submit_command = ( + SGEJob.submit_command + + " " + + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) + ) + + if self.cancel_command_extra is None: + self.cancel_command_extra = dask.config.get( + "jobqueue.%s.cancel-command-extra" % self.config_name, [] + ) + + self.cancel_command = ( + SGEJob.cancel_command + + " " + + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) + ) + header_lines = [] if self.job_name is not None: header_lines.append("#$ -N %s" % self.job_name) From 6407ea6490c4d5a288de4a5afc7535e8e59c420a Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Mon, 17 Jun 2024 16:36:47 +0800 Subject: [PATCH 06/15] Not self yet --- dask_jobqueue/core.py | 6 +++--- 1 file changed, 3 insertions(+), 3 deletions(-) diff --git a/dask_jobqueue/core.py b/dask_jobqueue/core.py index 129e7c16..d57f83e5 100644 --- a/dask_jobqueue/core.py +++ b/dask_jobqueue/core.py @@ -277,15 +277,15 @@ def __init__( if env_extra is None: env_extra = dask.config.get("jobqueue.%s.env-extra" % self.config_name) - if self.submit_command_extra is None: - self.submit_command_extra = dask.config.get( + if submit_command_extra is None: + submit_command_extra = dask.config.get( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) self.submit_command = ( Job.submit_command + " " - + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) + + " ".join(shlex.quote(arg) for arg in submit_command_extra) ) if self.cancel_command_extra is None: From 38ed4a93ff7d22ea2ccae22ac4f4bd06e40dd9e5 Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Mon, 17 Jun 2024 16:57:43 +0800 Subject: [PATCH 07/15] Fix args --- dask_jobqueue/core.py | 5 ++++- dask_jobqueue/htcondor.py | 4 ++-- dask_jobqueue/lsf.py | 4 ++-- dask_jobqueue/oar.py | 4 ++-- dask_jobqueue/pbs.py | 4 ++-- dask_jobqueue/sge.py | 4 ++-- dask_jobqueue/slurm.py | 6 +++--- 7 files changed, 17 insertions(+), 14 deletions(-) diff --git a/dask_jobqueue/core.py b/dask_jobqueue/core.py index d57f83e5..afaeb745 100644 --- a/dask_jobqueue/core.py +++ b/dask_jobqueue/core.py @@ -281,17 +281,20 @@ def __init__( submit_command_extra = dask.config.get( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) + logger.critical(f"{submit_command_extra=}") + self.submit_command_extra = submit_command_extra self.submit_command = ( Job.submit_command + " " - + " ".join(shlex.quote(arg) for arg in submit_command_extra) + + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) ) if self.cancel_command_extra is None: self.cancel_command_extra = dask.config.get( "jobqueue.%s.cancel-command-extra" % self.config_name, [] ) + self.cancel_command_extra = cancel_command_extra self.cancel_command = ( Job.cancel_command diff --git a/dask_jobqueue/htcondor.py b/dask_jobqueue/htcondor.py index e8eff970..fdbcba57 100644 --- a/dask_jobqueue/htcondor.py +++ b/dask_jobqueue/htcondor.py @@ -93,7 +93,7 @@ def __init__( if self.job_extra_directives: self.job_header_dict.update(self.job_extra_directives) - if self.submit_command_extra is None: + if self.submit_command_extra is None or self.submit_command_extra == []: self.submit_command_extra = dask.config.get( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) @@ -104,7 +104,7 @@ def __init__( + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) ) - if self.cancel_command_extra is None: + if self.cancel_command_extra is None or self.cancel_command_extra == []: self.cancel_command_extra = dask.config.get( "jobqueue.%s.cancel-command-extra" % self.config_name, [] ) diff --git a/dask_jobqueue/lsf.py b/dask_jobqueue/lsf.py index 2537e7f7..56e63b34 100644 --- a/dask_jobqueue/lsf.py +++ b/dask_jobqueue/lsf.py @@ -55,7 +55,7 @@ def __init__( use_stdin = dask.config.get("jobqueue.%s.use-stdin" % self.config_name) self.use_stdin = use_stdin - if self.submit_command_extra is None: + if self.submit_command_extra is None or self.submit_command_extra == []: self.submit_command_extra = dask.config.get( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) @@ -66,7 +66,7 @@ def __init__( + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) ) - if self.cancel_command_extra is None: + if self.cancel_command_extra is None or self.cancel_command_extra == []: self.cancel_command_extra = dask.config.get( "jobqueue.%s.cancel-command-extra" % self.config_name, [] ) diff --git a/dask_jobqueue/oar.py b/dask_jobqueue/oar.py index 6f90c0b9..15c8ef8e 100644 --- a/dask_jobqueue/oar.py +++ b/dask_jobqueue/oar.py @@ -44,7 +44,7 @@ def __init__( if walltime is None: walltime = dask.config.get("jobqueue.%s.walltime" % self.config_name) - if self.submit_command_extra is None: + if self.submit_command_extra is None or self.submit_command_extra == []: self.submit_command_extra = dask.config.get( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) @@ -55,7 +55,7 @@ def __init__( + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) ) - if self.cancel_command_extra is None: + if self.cancel_command_extra is None or self.cancel_command_extra == []: self.cancel_command_extra = dask.config.get( "jobqueue.%s.cancel-command-extra" % self.config_name, [] ) diff --git a/dask_jobqueue/pbs.py b/dask_jobqueue/pbs.py index cee59883..2aec3979 100644 --- a/dask_jobqueue/pbs.py +++ b/dask_jobqueue/pbs.py @@ -84,7 +84,7 @@ def __init__( if not account: account = project - if self.submit_command_extra is None: + if self.submit_command_extra is None or self.submit_command_extra == []: self.submit_command_extra = dask.config.get( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) @@ -95,7 +95,7 @@ def __init__( + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) ) - if self.cancel_command_extra is None: + if self.cancel_command_extra is None or self.cancel_command_extra == []: self.cancel_command_extra = dask.config.get( "jobqueue.%s.cancel-command-extra" % self.config_name, [] ) diff --git a/dask_jobqueue/sge.py b/dask_jobqueue/sge.py index ebafa8d0..db211e3d 100644 --- a/dask_jobqueue/sge.py +++ b/dask_jobqueue/sge.py @@ -39,7 +39,7 @@ def __init__( if walltime is None: walltime = dask.config.get("jobqueue.%s.walltime" % self.config_name) - if self.submit_command_extra is None: + if self.submit_command_extra is None or self.submit_command_extra == []: self.submit_command_extra = dask.config.get( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) @@ -50,7 +50,7 @@ def __init__( + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) ) - if self.cancel_command_extra is None: + if self.cancel_command_extra is None or self.cancel_command_extra == []: self.cancel_command_extra = dask.config.get( "jobqueue.%s.cancel-command-extra" % self.config_name, [] ) diff --git a/dask_jobqueue/slurm.py b/dask_jobqueue/slurm.py index a78fc072..8049b211 100644 --- a/dask_jobqueue/slurm.py +++ b/dask_jobqueue/slurm.py @@ -15,6 +15,7 @@ class SLURMJob(Job): submit_command = "sbatch" cancel_command = "scancel" config_name = "slurm" + cancel_command_extra = ["--signal=SIGTERM"] def __init__( self, @@ -27,7 +28,6 @@ def __init__( job_cpu=None, job_mem=None, config_name=None, - cancel_command_extra=["--signal=SIGTERM"], **base_class_kwargs ): super().__init__( @@ -59,7 +59,7 @@ def __init__( if job_mem is None: job_mem = dask.config.get("jobqueue.%s.job-mem" % self.config_name) - if self.submit_command_extra is None: + if self.submit_command_extra is None or self.submit_command_extra == []: self.submit_command_extra = dask.config.get( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) @@ -70,7 +70,7 @@ def __init__( + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) ) - if cancel_command_extra is None: + if self.cancel_command_extra is None or self.cancel_command_extra == []: cancel_command_extra = dask.config.get( "jobqueue.%s.cancel-command-extra" % self.config_name, [] ) From 7adc55ea418ac3372408164f3749337c41fef093 Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Mon, 17 Jun 2024 17:03:10 +0800 Subject: [PATCH 08/15] Fix core --- dask_jobqueue/core.py | 27 ++++++++++++++------------- 1 file changed, 14 insertions(+), 13 deletions(-) diff --git a/dask_jobqueue/core.py b/dask_jobqueue/core.py index afaeb745..0ea680a7 100644 --- a/dask_jobqueue/core.py +++ b/dask_jobqueue/core.py @@ -281,26 +281,27 @@ def __init__( submit_command_extra = dask.config.get( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) - logger.critical(f"{submit_command_extra=}") self.submit_command_extra = submit_command_extra - self.submit_command = ( - Job.submit_command - + " " - + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) - ) + if self.submit_command is not None: + self.submit_command = ( + Job.submit_command + + " " + + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) + ) - if self.cancel_command_extra is None: - self.cancel_command_extra = dask.config.get( + if cancel_command_extra is None: + cancel_command_extra = dask.config.get( "jobqueue.%s.cancel-command-extra" % self.config_name, [] ) self.cancel_command_extra = cancel_command_extra - self.cancel_command = ( - Job.cancel_command - + " " - + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) - ) + if self.cancel_command is not None: + self.cancel_command = ( + Job.cancel_command + + " " + + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) + ) if job_script_prologue is None: job_script_prologue = dask.config.get( From 5d4fdc5952076884dfb9aaaed084b6bbcc1eb800 Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Wed, 7 Aug 2024 12:03:43 +0800 Subject: [PATCH 09/15] Type fixes --- dask_jobqueue/core.py | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/dask_jobqueue/core.py b/dask_jobqueue/core.py index 0ea680a7..680844b7 100644 --- a/dask_jobqueue/core.py +++ b/dask_jobqueue/core.py @@ -285,7 +285,7 @@ def __init__( if self.submit_command is not None: self.submit_command = ( - Job.submit_command + self.submit_command + " " + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) ) @@ -298,7 +298,7 @@ def __init__( if self.cancel_command is not None: self.cancel_command = ( - Job.cancel_command + self.cancel_command + " " + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) ) From 7ad8ebc3dede75afa15674a95e92915a06436642 Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Wed, 7 Aug 2024 12:12:12 +0800 Subject: [PATCH 10/15] Initialise variables --- dask_jobqueue/core.py | 12 ++++++------ 1 file changed, 6 insertions(+), 6 deletions(-) diff --git a/dask_jobqueue/core.py b/dask_jobqueue/core.py index 680844b7..d253b0d4 100644 --- a/dask_jobqueue/core.py +++ b/dask_jobqueue/core.py @@ -283,10 +283,10 @@ def __init__( ) self.submit_command_extra = submit_command_extra + self.submit_command = Job.submit_command if self.submit_command is not None: - self.submit_command = ( - self.submit_command - + " " + self.submit_command += ( + " " + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) ) @@ -296,10 +296,10 @@ def __init__( ) self.cancel_command_extra = cancel_command_extra + self.cancel_command = Job.cancel_command if self.cancel_command is not None: - self.cancel_command = ( - self.cancel_command - + " " + self.cancel_command += ( + " " + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) ) From 6074a3c28c34489ecc42fefa99ab920467d7c6f6 Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Wed, 7 Aug 2024 12:12:27 +0800 Subject: [PATCH 11/15] typo --- dask_jobqueue/htcondor.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/dask_jobqueue/htcondor.py b/dask_jobqueue/htcondor.py index fdbcba57..f34bde1f 100644 --- a/dask_jobqueue/htcondor.py +++ b/dask_jobqueue/htcondor.py @@ -99,7 +99,7 @@ def __init__( ) self.submit_command = ( - HTCondorJob.submit_commsand + HTCondorJob.submit_command + " " + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) ) From a7cdfb58092f8ee8da2c3343f4eb6968d6793144 Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Wed, 7 Aug 2024 13:04:59 +0800 Subject: [PATCH 12/15] Use base class --- dask_jobqueue/core.py | 2 -- dask_jobqueue/htcondor.py | 22 --------------------- dask_jobqueue/lsf.py | 22 --------------------- dask_jobqueue/oar.py | 22 --------------------- dask_jobqueue/pbs.py | 22 --------------------- dask_jobqueue/sge.py | 22 --------------------- dask_jobqueue/slurm.py | 40 +++++++++++++++++++++------------------ 7 files changed, 22 insertions(+), 130 deletions(-) diff --git a/dask_jobqueue/core.py b/dask_jobqueue/core.py index d253b0d4..35e0b8f4 100644 --- a/dask_jobqueue/core.py +++ b/dask_jobqueue/core.py @@ -283,7 +283,6 @@ def __init__( ) self.submit_command_extra = submit_command_extra - self.submit_command = Job.submit_command if self.submit_command is not None: self.submit_command += ( " " @@ -296,7 +295,6 @@ def __init__( ) self.cancel_command_extra = cancel_command_extra - self.cancel_command = Job.cancel_command if self.cancel_command is not None: self.cancel_command += ( " " diff --git a/dask_jobqueue/htcondor.py b/dask_jobqueue/htcondor.py index f34bde1f..35645a90 100644 --- a/dask_jobqueue/htcondor.py +++ b/dask_jobqueue/htcondor.py @@ -93,28 +93,6 @@ def __init__( if self.job_extra_directives: self.job_header_dict.update(self.job_extra_directives) - if self.submit_command_extra is None or self.submit_command_extra == []: - self.submit_command_extra = dask.config.get( - "jobqueue.%s.submit-command-extra" % self.config_name, [] - ) - - self.submit_command = ( - HTCondorJob.submit_command - + " " - + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) - ) - - if self.cancel_command_extra is None or self.cancel_command_extra == []: - self.cancel_command_extra = dask.config.get( - "jobqueue.%s.cancel-command-extra" % self.config_name, [] - ) - - self.cancel_command = ( - HTCondorJob.cancel_command - + " " - + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) - ) - def job_script(self): """Construct a job submission script""" quoted_arguments = quote_arguments(["-c", self._command_template]) diff --git a/dask_jobqueue/lsf.py b/dask_jobqueue/lsf.py index 56e63b34..0b4e4bd6 100644 --- a/dask_jobqueue/lsf.py +++ b/dask_jobqueue/lsf.py @@ -55,28 +55,6 @@ def __init__( use_stdin = dask.config.get("jobqueue.%s.use-stdin" % self.config_name) self.use_stdin = use_stdin - if self.submit_command_extra is None or self.submit_command_extra == []: - self.submit_command_extra = dask.config.get( - "jobqueue.%s.submit-command-extra" % self.config_name, [] - ) - - self.submit_command = ( - LSFJob.submit_command - + " " - + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) - ) - - if self.cancel_command_extra is None or self.cancel_command_extra == []: - self.cancel_command_extra = dask.config.get( - "jobqueue.%s.cancel-command-extra" % self.config_name, [] - ) - - self.cancel_command = ( - LSFJob.cancel_command - + " " - + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) - ) - header_lines = [] # LSF header build if self.name is not None: diff --git a/dask_jobqueue/oar.py b/dask_jobqueue/oar.py index 463e3152..d0302dbc 100644 --- a/dask_jobqueue/oar.py +++ b/dask_jobqueue/oar.py @@ -44,28 +44,6 @@ def __init__( if walltime is None: walltime = dask.config.get("jobqueue.%s.walltime" % self.config_name) - if self.submit_command_extra is None or self.submit_command_extra == []: - self.submit_command_extra = dask.config.get( - "jobqueue.%s.submit-command-extra" % self.config_name, [] - ) - - self.submit_command = ( - OARJob.submit_command - + " " - + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) - ) - - if self.cancel_command_extra is None or self.cancel_command_extra == []: - self.cancel_command_extra = dask.config.get( - "jobqueue.%s.cancel-command-extra" % self.config_name, [] - ) - - self.cancel_command = ( - OARJob.cancel_command - + " " - + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) - ) - header_lines = [] if self.job_name is not None: header_lines.append("#OAR -n %s" % self.job_name) diff --git a/dask_jobqueue/pbs.py b/dask_jobqueue/pbs.py index 2aec3979..585a7645 100644 --- a/dask_jobqueue/pbs.py +++ b/dask_jobqueue/pbs.py @@ -84,28 +84,6 @@ def __init__( if not account: account = project - if self.submit_command_extra is None or self.submit_command_extra == []: - self.submit_command_extra = dask.config.get( - "jobqueue.%s.submit-command-extra" % self.config_name, [] - ) - - self.submit_command = ( - PBSJob.submit_command - + " " - + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) - ) - - if self.cancel_command_extra is None or self.cancel_command_extra == []: - self.cancel_command_extra = dask.config.get( - "jobqueue.%s.cancel-command-extra" % self.config_name, [] - ) - - self.cancel_command = ( - PBSJob.cancel_command - + " " - + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) - ) - header_lines = [] # PBS header build if self.job_name is not None: diff --git a/dask_jobqueue/sge.py b/dask_jobqueue/sge.py index db211e3d..219fb011 100644 --- a/dask_jobqueue/sge.py +++ b/dask_jobqueue/sge.py @@ -39,28 +39,6 @@ def __init__( if walltime is None: walltime = dask.config.get("jobqueue.%s.walltime" % self.config_name) - if self.submit_command_extra is None or self.submit_command_extra == []: - self.submit_command_extra = dask.config.get( - "jobqueue.%s.submit-command-extra" % self.config_name, [] - ) - - self.submit_command = ( - SGEJob.submit_command - + " " - + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) - ) - - if self.cancel_command_extra is None or self.cancel_command_extra == []: - self.cancel_command_extra = dask.config.get( - "jobqueue.%s.cancel-command-extra" % self.config_name, [] - ) - - self.cancel_command = ( - SGEJob.cancel_command - + " " - + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) - ) - header_lines = [] if self.job_name is not None: header_lines.append("#$ -N %s" % self.job_name) diff --git a/dask_jobqueue/slurm.py b/dask_jobqueue/slurm.py index 8049b211..5a7d4e00 100644 --- a/dask_jobqueue/slurm.py +++ b/dask_jobqueue/slurm.py @@ -15,7 +15,6 @@ class SLURMJob(Job): submit_command = "sbatch" cancel_command = "scancel" config_name = "slurm" - cancel_command_extra = ["--signal=SIGTERM"] def __init__( self, @@ -28,10 +27,15 @@ def __init__( job_cpu=None, job_mem=None, config_name=None, + cancel_command_extra=["--signal=SIGTERM"], **base_class_kwargs ): super().__init__( - scheduler=scheduler, name=name, config_name=config_name, **base_class_kwargs + scheduler=scheduler, + name=name, + config_name=config_name, + cancel_command_extra=cancel_command_extra, + **base_class_kwargs ) if queue is None: @@ -64,22 +68,22 @@ def __init__( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) - self.submit_command = ( - SLURMJob.submit_command - + " " - + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) - ) - - if self.cancel_command_extra is None or self.cancel_command_extra == []: - cancel_command_extra = dask.config.get( - "jobqueue.%s.cancel-command-extra" % self.config_name, [] - ) - - self.cancel_command = ( - SLURMJob.cancel_command - + " " - + " ".join(shlex.quote(arg) for arg in cancel_command_extra) - ) + # self.submit_command = ( + # SLURMJob.submit_command + # + " " + # + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) + # ) + + # if self.cancel_command_extra is None or self.cancel_command_extra == []: + # cancel_command_extra = dask.config.get( + # "jobqueue.%s.cancel-command-extra" % self.config_name, [] + # ) + + # self.cancel_command = ( + # SLURMJob.cancel_command + # + " " + # + " ".join(shlex.quote(arg) for arg in cancel_command_extra) + # ) header_lines = [] # SLURM header build From 5c99b7eff4b2ec6c86bb32a99b34bd4dfffa099b Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Wed, 7 Aug 2024 13:09:42 +0800 Subject: [PATCH 13/15] Cleanup --- dask_jobqueue/slurm.py | 17 ----------------- 1 file changed, 17 deletions(-) diff --git a/dask_jobqueue/slurm.py b/dask_jobqueue/slurm.py index 5a7d4e00..8abad649 100644 --- a/dask_jobqueue/slurm.py +++ b/dask_jobqueue/slurm.py @@ -68,23 +68,6 @@ def __init__( "jobqueue.%s.submit-command-extra" % self.config_name, [] ) - # self.submit_command = ( - # SLURMJob.submit_command - # + " " - # + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) - # ) - - # if self.cancel_command_extra is None or self.cancel_command_extra == []: - # cancel_command_extra = dask.config.get( - # "jobqueue.%s.cancel-command-extra" % self.config_name, [] - # ) - - # self.cancel_command = ( - # SLURMJob.cancel_command - # + " " - # + " ".join(shlex.quote(arg) for arg in cancel_command_extra) - # ) - header_lines = [] # SLURM header build if self.job_name is not None: From 4f03167e2ced5b58df4c5e93e632798c22561a0a Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Wed, 7 Aug 2024 13:10:40 +0800 Subject: [PATCH 14/15] Cleanup --- dask_jobqueue/slurm.py | 5 ----- 1 file changed, 5 deletions(-) diff --git a/dask_jobqueue/slurm.py b/dask_jobqueue/slurm.py index 8abad649..27a87551 100644 --- a/dask_jobqueue/slurm.py +++ b/dask_jobqueue/slurm.py @@ -63,11 +63,6 @@ def __init__( if job_mem is None: job_mem = dask.config.get("jobqueue.%s.job-mem" % self.config_name) - if self.submit_command_extra is None or self.submit_command_extra == []: - self.submit_command_extra = dask.config.get( - "jobqueue.%s.submit-command-extra" % self.config_name, [] - ) - header_lines = [] # SLURM header build if self.job_name is not None: From 64f47af3b3ccacfdf88b0bce4d5f09237c5d9de7 Mon Sep 17 00:00:00 2001 From: "Alec Thomson (S&A, Kensington WA)" Date: Wed, 7 Aug 2024 18:07:39 +0800 Subject: [PATCH 15/15] lint --- dask_jobqueue/core.py | 12 +++++------- dask_jobqueue/htcondor.py | 1 - dask_jobqueue/pbs.py | 1 - dask_jobqueue/sge.py | 1 - dask_jobqueue/slurm.py | 9 ++++----- 5 files changed, 9 insertions(+), 15 deletions(-) diff --git a/dask_jobqueue/core.py b/dask_jobqueue/core.py index 35e0b8f4..080c17b3 100644 --- a/dask_jobqueue/core.py +++ b/dask_jobqueue/core.py @@ -276,7 +276,7 @@ def __init__( if env_extra is None: env_extra = dask.config.get("jobqueue.%s.env-extra" % self.config_name) - + if submit_command_extra is None: submit_command_extra = dask.config.get( "jobqueue.%s.submit-command-extra" % self.config_name, [] @@ -284,9 +284,8 @@ def __init__( self.submit_command_extra = submit_command_extra if self.submit_command is not None: - self.submit_command += ( - " " - + " ".join(shlex.quote(arg) for arg in self.submit_command_extra) + self.submit_command += " " + " ".join( + shlex.quote(arg) for arg in self.submit_command_extra ) if cancel_command_extra is None: @@ -296,9 +295,8 @@ def __init__( self.cancel_command_extra = cancel_command_extra if self.cancel_command is not None: - self.cancel_command += ( - " " - + " ".join(shlex.quote(arg) for arg in self.cancel_command_extra) + self.cancel_command += " " + " ".join( + shlex.quote(arg) for arg in self.cancel_command_extra ) if job_script_prologue is None: diff --git a/dask_jobqueue/htcondor.py b/dask_jobqueue/htcondor.py index 35645a90..196ffb3b 100644 --- a/dask_jobqueue/htcondor.py +++ b/dask_jobqueue/htcondor.py @@ -1,6 +1,5 @@ import logging import re -import shlex import dask from dask.utils import parse_bytes diff --git a/dask_jobqueue/pbs.py b/dask_jobqueue/pbs.py index 585a7645..719fd9ce 100644 --- a/dask_jobqueue/pbs.py +++ b/dask_jobqueue/pbs.py @@ -1,7 +1,6 @@ import logging import math import os -import shlex import warnings import dask diff --git a/dask_jobqueue/sge.py b/dask_jobqueue/sge.py index 219fb011..64bb1b4d 100644 --- a/dask_jobqueue/sge.py +++ b/dask_jobqueue/sge.py @@ -1,5 +1,4 @@ import logging -import shlex import dask diff --git a/dask_jobqueue/slurm.py b/dask_jobqueue/slurm.py index 27a87551..8b8942b4 100644 --- a/dask_jobqueue/slurm.py +++ b/dask_jobqueue/slurm.py @@ -1,6 +1,5 @@ import logging import math -import shlex import warnings import dask @@ -31,10 +30,10 @@ def __init__( **base_class_kwargs ): super().__init__( - scheduler=scheduler, - name=name, - config_name=config_name, - cancel_command_extra=cancel_command_extra, + scheduler=scheduler, + name=name, + config_name=config_name, + cancel_command_extra=cancel_command_extra, **base_class_kwargs )