Compare commits

...

59 Commits

Author SHA1 Message Date
YannC
cd7b64b83e chore: upgrade to version v0.20.8 2024-12-24 16:28:13 +01:00
Miloš Paunović
a211947228 chore(ui): automatically add namespace filter where needed (#6296) 2024-12-23 14:21:05 +01:00
Loïc Mathieu
7b68b1b3b3 fix(core): existing ns based on KV and not only flows
Lookup if there are existing KV to know if a ns exist from the kv function and not only if flows exist in the KV.
2024-12-20 09:59:44 +01:00
Loïc Mathieu
8e5a84b0b3 fix(jdbc): read the disabled flag from the DB 2024-12-20 09:59:37 +01:00
YannC
1151976e9f fix(core): save flowable's output when flowable is child of another flowable (#6500)
close #6494
2024-12-19 08:40:38 +01:00
Miloš Paunović
2bd0232e18 fix(ui): if no trigger state filter is selected, show them all (#6522) 2024-12-19 08:37:56 +01:00
Barthélémy Ledoux
d998ac4c7d fix: add missing mutation when loading plugin doc form cache (#6502) 2024-12-18 09:15:01 +01:00
brian.mulier
52af8f44cd chore(deps): version 0.20.7 2024-12-17 11:24:51 +01:00
brian.mulier
88aa2be61a fix: bump package-lock.json versions 2024-12-17 11:24:51 +01:00
Florian Hussonnois
ca007695ed fix(core): properly check next scheduled date for backfill execution (#6413)
Changes:
When a trigger is evaluated for in a back-fill context, we have to make sure
that current-date is strictly after the next execution date for an execution to be eligible.

fix: #6413
2024-12-17 10:33:21 +01:00
Barthélémy Ledoux
122b409bd3 fix: avoid redirect loops when axios calls an unauthorized API (#6450)
* fix: avoid redirect loops when axios calls an unauthorized API

* use the proper structure for axios

* protect against empty request data
2024-12-13 12:34:55 +01:00
Ludovic DEHON
d2500bb4de chore(core): refactor SecretService 2024-12-13 00:22:16 +01:00
Bart Ledoux
30e6bc4364 npm audit fix 2024-12-12 16:37:55 +01:00
brian.mulier
7640bcb16d fix(ui): avoid unsaved changes pop-up upon clicking on plugin property type definition anchors
closes #6297
2024-12-11 18:36:00 +01:00
brian.mulier
a55c6c1c52 fix(ui): total is not needed in FlowCreate.vue 2024-12-11 17:28:55 +01:00
brian.mulier
cfb2ba526e fix(ui): Flow create was no longer generating graph 2024-12-11 17:28:53 +01:00
GitHub Action
cde81c2868 chore(translations): auto generate values for languages other than english 2024-12-11 15:05:03 +01:00
Piyush Bhaskar
28e9697d00 feat(ui): Add new filters to Administration -> Triggers page (#6328)
Co-authored-by: Piyush-r-bhaskar <piyush.bhaskar@gmail.com>
Co-authored-by: Miloš Paunović <paun992@hotmail.com>
2024-12-11 15:04:54 +01:00
michascant
2a3b14f451 feat(UI): added new filters to Flows -> Metrics tab (#6305)
Co-authored-by: Miloš Paunović <paun992@hotmail.com>
2024-12-11 15:04:32 +01:00
GitHub Action
335f33b5bb chore(translations): auto generate values for languages other than english 2024-12-11 15:04:26 +01:00
Miloš Paunović
d613034263 feat(ui): add missing filter options for metrics (#6409) 2024-12-11 15:04:17 +01:00
Loïc Mathieu
854d43cdba chore(deps): version 0.20.6 2024-12-10 15:50:07 +01:00
Loïc Mathieu
70dd343ddc feat(core,jdbc): small trigger / scheduler improvements 2024-12-10 15:49:35 +01:00
Loïc Mathieu
364c74d033 chore(deps): version 0.20.5 2024-12-10 15:01:55 +01:00
Miloš Paunović
35a180b32a fix(ui): pass flow revision on execution overview (#6380) 2024-12-10 10:48:14 +01:00
Yerin Lee
70ea1f6e64 chore(ui): add scrolling to totals chart legend if more than 4 items present (#5971)
Co-authored-by: MilosPaunovic <paun992@hotmail.com>
2024-12-10 08:41:48 +01:00
Miloš Paunović
0ee401819a feat(ui): add triggers sorting by next execution date (#6318) 2024-12-10 08:15:14 +01:00
MilosPaunovic
34e4dfa152 chore(docs): switch link to good first issues in readme file 2024-12-10 08:14:50 +01:00
Miloš Paunović
224e750009 chore(ui): prevent text wrap inside trigger id column (#6336) 2024-12-10 08:14:13 +01:00
Miloš Paunović
b90a5ae9d6 chore(ui): respect date format form setting inside filter label (#6335) 2024-12-10 08:14:04 +01:00
Barthélémy Ledoux
e0b89dc425 feat(ui): add flow validation to FlowCreate component (#6370) 2024-12-09 14:57:28 +01:00
Shivam
716b8dbdfe fix(ui): properly handle filename with multiple dots in editor sidebar (#6362)
Co-authored-by: Miloš Paunović <paun992@hotmail.com>
2024-12-09 10:02:59 +01:00
Ludovic DEHON
1d82840a51 chore(version): version 0.20.4 2024-12-08 00:17:22 +01:00
Ludovic DEHON
cc1d813e20 chore(core): add unit test for if nested in parallel 2024-12-08 00:15:34 +01:00
Loïc Mathieu
44a8c7d63a chore(version): version 0.20.3 2024-12-06 14:13:14 +01:00
Florian Hussonnois
56afa318cd fix(core): fix cannot create Metric from null in Worker class
fix: kestra-io/kestra-ee#2417
2024-12-06 13:29:50 +01:00
Loïc Mathieu
620f894a4d fix(core): catch errors on task run
Fixes https://github.com/kestra-io/kestra-ee/issues/2416
2024-12-06 11:42:15 +01:00
YannC
37287d5e4c fix(ui): axios missing content type 2024-12-06 10:41:20 +01:00
brian.mulier
c653a1adc3 fix(jdbc): topology was built across all tenants 2024-12-06 09:53:17 +01:00
Piyush Bhaskar
1abfa5e23e chore(ui): improve bulk actions design in the executions listing (#6240)
Co-authored-by: Piyush-r-bhaskar <piyush.bhaskar@gmail.com>
Co-authored-by: Miloš Paunović <paun992@hotmail.com>
2024-12-05 14:51:13 +01:00
Manoj Balaraj
03d8855309 fix(ui): properly handle pebble expression if it contains dash character (#6062)
Co-authored-by: manu2931 <manojb912@gmai.com>
Co-authored-by: Miloš Paunović <paun992@hotmail.com>
2024-12-05 14:11:40 +01:00
Loïc Mathieu
0ab7b0e57a chore(version): upgrade to 0.20.2 2024-12-05 11:04:52 +01:00
Loïc Mathieu
5f8de5106b Revert "feat(core): Add displayName to flow level outputs(backend) (#5605)"
This reverts commit a5741aa424.

This reverts commit 42f721fdec.

This reverts commit 0de24c4448.
2024-12-05 10:30:51 +01:00
Miloš Paunović
c749301944 fix(ui): filter out system labels from executing using prefill (#6311) 2024-12-05 09:21:24 +01:00
Piyush Bhaskar
0831e9d356 chore(ui): remove default editor outline (#6303)
Co-authored-by: Piyush-r-bhaskar <piyush.bhaskar@gmail.com>
2024-12-05 08:37:23 +01:00
Miloš Paunović
9e4d36e70d fix(ui): only apply editor padding on main editor (#6310) 2024-12-05 08:34:12 +01:00
Ludovic DEHON
2bbb7a83b8 chore(version): update to version 'v0.20.1'. 2024-12-04 22:36:36 +01:00
Piyush Bhaskar
bad60b4663 chore(ui): Improvement in Welcome Page. (#6077)
* chore(ui): Improvement in Welcome Page.

* Update Welcome.vue | scoped the styling

* fix bad merge

* remove special behavior of navbar on welcome

* finish the welcome page (thank you)

* fix: better adaptive layout

* use container queries and flex for better responsive design

* chore(translations): auto generate values for languages other than english

---------

Co-authored-by: Barthélémy Ledoux <ledouxb@me.com>
Co-authored-by: Bart Ledoux <bledoux@kestra.io>
Co-authored-by: Miloš Paunović <paun992@hotmail.com>
Co-authored-by: GitHub Action <actions@github.com>
2024-12-04 14:38:28 +01:00
Abhishek Pawar
4b1c700b5e fix(ui): handle logs selector overflow in a good manner (#6224)
Co-authored-by: MilosPaunovic <paun992@hotmail.com>
2024-12-04 14:18:23 +01:00
Ian Cheng
1323c95785 feat(ui): add right click menu on file tree view in editor (#5936)
Co-authored-by: Miloš Paunović <paun992@hotmail.com>
2024-12-04 13:21:04 +01:00
Loïc Mathieu
3e726b5848 fix(core, webserver): properly close the queue on Flux.onFinally
Two fixes:
- close the queue onFinally and not onComplete and onCancel to take into accunt errors.
- close the queue onFinally in the execution creation as now it is only done on the success path and not even via a Flux lifecycle method

This may fix or improve some incosistent behavior reported by users on the webserver.
2024-12-04 12:18:05 +01:00
Loïc Mathieu
97ad281566 fix(core): Correctly parse Content-Disposition in the Download task
Fixes #6270
2024-12-04 12:16:46 +01:00
Nitin Bisht
31f6e3fe25 chore(ui): amend spacing on plugins page (#6223)
Co-authored-by: MilosPaunovic <paun992@hotmail.com>
2024-12-04 08:55:20 +01:00
Joe Celaster
97f16e989b chore(ui): remove search field background on single plugin page (#6220)
Co-authored-by: MilosPaunovic <paun992@hotmail.com>
2024-12-04 08:49:34 +01:00
Manoj Balaraj
b72fb29377 fix(ui): improve debug outputs expression on initial load (#6094)
Co-authored-by: manu2931 <manojb912@gmai.com>
Co-authored-by: Miloš Paunović <paun992@hotmail.com>
2024-12-04 08:45:34 +01:00
Piyush Bhaskar
e45bbdb9e7 chore(ui): add top and left padding to editor component (#6191)
Co-authored-by: Piyush-r-bhaskar <piyush.bhaskar@gmail.com>
Co-authored-by: MilosPaunovic <paun992@hotmail.com>
2024-12-04 08:39:41 +01:00
Ines Qian
178ee0e7df chore(ui): properly highlight selected options in all of the filter dropdowns (#6173)
Co-authored-by: Miloš Paunović <paun992@hotmail.com>
2024-12-04 08:36:28 +01:00
Florian Hussonnois
aa1ba59983 chore(version): update to version 'v0.20.0'. 2024-12-03 12:20:53 +01:00
Loïc Mathieu
2e9a0d132a fix(core): possible NPE when the Executor didn't have the flow 2024-12-03 12:19:50 +01:00
78 changed files with 1722 additions and 866 deletions

View File

@@ -178,7 +178,7 @@ Stay connected and get support:
We welcome contributions of all kinds!
- **Report Issues:** Found a bug or have a feature request? Open an [issue on GitHub](https://github.com/kestra-io/kestra/issues).
- **Contribute Code:** Check out our [Contributor Guide](https://kestra.io/docs/getting-started/contributing) for initial guidelines, and explore our [good first issues](https://go.kestra.io/contribute) for beginner-friendly tasks to tackle first.
- **Contribute Code:** Check out our [Contributor Guide](https://kestra.io/docs/getting-started/contributing) for initial guidelines, and explore our [good first issues](https://go.kestra.io/contributing) for beginner-friendly tasks to tackle first.
- **Develop Plugins:** Build and share plugins using our [Plugin Developer Guide](https://kestra.io/docs/plugin-developer-guide/).
- **Contribute to our Docs:** Contribute edits or updates to keep our [documentation](https://github.com/kestra-io/docs) top-notch.

View File

@@ -54,7 +54,11 @@ public abstract class AbstractWorkerCallable implements Callable<State.Type> {
try {
return doCall();
} catch (Exception e) {
} catch (Throwable e) {
// Catching Throwable is usually a bad idea.
// However, here, we want to be sure that the task fails whatever happens,
// and some plugins may throw errors, for example, for dependency issues or worst,
// bad behavior that throws errors and not exceptions.
return this.exceptionHandler(e);
} finally {
shutdownLatch.countDown();

View File

@@ -323,9 +323,7 @@ public class ExecutorService {
);
if (!nexts.isEmpty()) {
return nexts.stream()
.map(throwFunction(NextTaskRun::getTaskRun))
.toList();
return saveFlowableOutput(nexts, executor);
}
} catch (Exception e) {
log.warn("Unable to resolve the next tasks to run", e);
@@ -379,7 +377,9 @@ public class ExecutorService {
if (flow.getOutputs() != null) {
RunContext runContext = runContextFactory.of(executor.getFlow(), executor.getExecution());
try {
Map<String, Object> outputs = flowInputOutput.flowOutputsToMap(flow.getOutputs());
Map<String, Object> outputs = flow.getOutputs()
.stream()
.collect(HashMap::new, (map, entry) -> map.put(entry.getId(), entry.getValue()), Map::putAll);
outputs = runContext.render(outputs);
outputs = flowInputOutput.typedOutputs(flow, executor.getExecution(), outputs);
newExecution = newExecution.withOutputs(outputs);
@@ -1046,7 +1046,7 @@ public class ExecutorService {
* WARNING: ATM, only the first violation will update the execution.
*/
public Executor handleExecutionChangedSLA(Executor executor) throws QueueException {
if (ListUtils.isEmpty(executor.getFlow().getSla()) || executor.getExecution().getState().isTerminated()) {
if (executor.getFlow() == null || ListUtils.isEmpty(executor.getFlow().getSla()) || executor.getExecution().getState().isTerminated()) {
return executor;
}

View File

@@ -1,7 +1,5 @@
package io.kestra.core.runners;
import com.fasterxml.jackson.core.JsonProcessingException;
import com.fasterxml.jackson.core.type.TypeReference;
import com.fasterxml.jackson.databind.ObjectMapper;
import com.google.common.annotations.VisibleForTesting;
import com.google.common.collect.ImmutableMap;
@@ -9,7 +7,12 @@ import io.kestra.core.encryption.EncryptionService;
import io.kestra.core.exceptions.IllegalVariableEvaluationException;
import io.kestra.core.exceptions.KestraRuntimeException;
import io.kestra.core.models.executions.Execution;
import io.kestra.core.models.flows.*;
import io.kestra.core.models.flows.Data;
import io.kestra.core.models.flows.DependsOn;
import io.kestra.core.models.flows.Flow;
import io.kestra.core.models.flows.Input;
import io.kestra.core.models.flows.RenderableInput;
import io.kestra.core.models.flows.Type;
import io.kestra.core.models.flows.input.FileInput;
import io.kestra.core.models.flows.input.InputAndValue;
import io.kestra.core.models.flows.input.ItemTypeInterface;
@@ -28,7 +31,6 @@ import jakarta.inject.Inject;
import jakarta.inject.Singleton;
import jakarta.validation.ConstraintViolationException;
import jakarta.validation.constraints.NotNull;
import org.reactivestreams.Publisher;
import reactor.core.publisher.Flux;
import reactor.core.publisher.Mono;
@@ -43,8 +45,6 @@ import java.time.Instant;
import java.time.LocalDate;
import java.time.LocalTime;
import java.util.AbstractMap;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collection;
import java.util.Collections;
import java.util.HashMap;
@@ -85,21 +85,6 @@ public class FlowInputOutput {
this.secretKey = Optional.ofNullable(secretKey);
}
/**
* Transform a list of flow outputs to a Map of output id -> output value map.
* An Output value map is a map with value and displayName.
*/
public Map<String, Object> flowOutputsToMap(List<Output> flowOutputs) {
return ListUtils.emptyOnNull(flowOutputs)
.stream()
.collect(HashMap::new, (map, entry) -> {
final HashMap<String, Object> entryInfo = new HashMap<>();
entryInfo.put("value", entry.getValue());
entryInfo.put("displayName", Optional.ofNullable(entry.getDisplayName()).orElse(entry.getId()));
map.put(entry.getId(), entryInfo);
}, Map::putAll);
}
/**
* Validate all the inputs of a given execution of a flow.
*
@@ -370,21 +355,9 @@ public class FlowInputOutput {
.getOutputs()
.stream()
.map(output -> {
final HashMap<String, Object> current;
final Object currentValue;
Object current = in == null ? null : in.get(output.getId());
try {
current = in == null ? null : JSON_MAPPER.readValue(
JSON_MAPPER.writeValueAsString(in.get(output.getId())), new TypeReference<>() {});
} catch (JsonProcessingException e) {
throw new RuntimeException(e);
}
if (current == null) {
currentValue = null;
} else {
currentValue = current.get("value");
}
try {
return parseData(execution, output, currentValue)
return parseData(execution, output, current)
.map(entry -> {
if (output.getType().equals(Type.SECRET)) {
return new AbstractMap.SimpleEntry<>(
@@ -395,26 +368,12 @@ public class FlowInputOutput {
return entry;
});
} catch (Exception e) {
throw output.toConstraintViolationException(e.getMessage(), currentValue);
throw output.toConstraintViolationException(e.getMessage(), current);
}
})
.filter(Optional::isPresent)
.map(Optional::get)
.collect(HashMap::new,
(map, entry) -> {
map.compute(entry.getKey(), (key, existingValue) -> {
if (existingValue == null) {
return entry.getValue();
}
if (existingValue instanceof List) {
((List<Object>) existingValue).add(entry.getValue());
return existingValue;
}
return new ArrayList<>(Arrays.asList(existingValue, entry.getValue()));
});
},
Map::putAll
);
.collect(HashMap::new, (map, entry) -> map.put(entry.getKey(), entry.getValue()), Map::putAll);
// Ensure outputs are compliant with tasks outputs.
return JacksonMapper.toMap(results);
@@ -432,7 +391,7 @@ public class FlowInputOutput {
final Type elementType = data instanceof ItemTypeInterface itemTypeInterface ? itemTypeInterface.getItemType() : null;
return Optional.of(new AbstractMap.SimpleEntry<>(
Optional.ofNullable(data.getDisplayName()).orElse(data.getId()),
data.getId(),
parseType(execution, data.getType(), data.getId(), elementType, current)
));
}

View File

@@ -51,6 +51,8 @@ import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicBoolean;
import java.util.concurrent.atomic.AtomicInteger;
import java.util.concurrent.atomic.AtomicReference;
import java.util.stream.Collectors;
import java.util.stream.Stream;
import static io.kestra.core.models.flows.State.Type.*;
import static io.kestra.core.server.Service.ServiceState.TERMINATED_FORCED;
@@ -180,11 +182,16 @@ public class Worker implements Service, Runnable, AutoCloseable {
return Collections.emptySet();
}
return Set.of(
Metric.of(this.metricRegistry.findGauge(MetricRegistry.METRIC_WORKER_JOB_THREAD_COUNT)),
Metric.of(this.metricRegistry.findGauge(MetricRegistry.METRIC_WORKER_JOB_PENDING_COUNT)),
Metric.of(this.metricRegistry.findGauge(MetricRegistry.METRIC_WORKER_JOB_RUNNING_COUNT))
Stream<String> metrics = Stream.of(
MetricRegistry.METRIC_WORKER_JOB_THREAD_COUNT,
MetricRegistry.METRIC_WORKER_JOB_PENDING_COUNT,
MetricRegistry.METRIC_WORKER_JOB_RUNNING_COUNT
);
return metrics
.flatMap(metric -> Optional.ofNullable(metricRegistry.findGauge(metric)).stream())
.map(Metric::of)
.collect(Collectors.toSet());
}
@Override

View File

@@ -469,11 +469,6 @@ public abstract class AbstractScheduler implements Scheduler, Service {
)
.build()
)
.peek(f -> {
if (f.getTriggerContext().getEvaluateRunningDate() != null || !isExecutionNotRunning(f)) {
this.triggerState.unlock(f.getTriggerContext());
}
})
.filter(f -> f.getTriggerContext().getEvaluateRunningDate() == null)
.filter(this::isExecutionNotRunning)
.map(FlowWithWorkerTriggerNextDate::of)

View File

@@ -1,4 +1,14 @@
package io.kestra.core.schedulers;
import java.util.function.Consumer;
/**
* This context is used by the Scheduler to allow evaluating and updating triggers in a transaction from the main evaluation loop.
* See AbstractScheduler.handle().
*/
public interface ScheduleContextInterface {
/**
* Do trigger retrieval and updating in a single transaction.
*/
void doInTransaction(Consumer<ScheduleContextInterface> consumer);
}

View File

@@ -25,15 +25,14 @@ public interface SchedulerTriggerStateInterface {
Trigger update(Flow flow, AbstractTrigger abstractTrigger, ConditionContext conditionContext) throws Exception;
/**
* Used by the JDBC implementation: find triggers in all tenants.
*/
List<Trigger> findByNextExecutionDateReadyForAllTenants(ZonedDateTime now, ScheduleContextInterface scheduleContext);
/**
* Required for Kafka
* Used by the Kafka implementation: find triggers in the scheduler assigned flow (as in Kafka partition assignment).
*/
List<Trigger> findByNextExecutionDateReadyForGivenFlows(List<FlowWithSource> flows, ZonedDateTime now, ScheduleContextInterface scheduleContext);
/**
* Required for Kafka
*/
void unlock(Trigger trigger);
}

View File

@@ -17,6 +17,10 @@ public class SecretService {
@PostConstruct
private void postConstruct() {
this.decode();
}
public void decode() {
decodedSecrets = System.getenv().entrySet().stream()
.filter(entry -> entry.getKey().startsWith(SECRET_PREFIX))
.<Map.Entry<String, String>>mapMulti((entry, consumer) -> {

View File

@@ -61,12 +61,7 @@ public class ExecutionLogService {
}
}));
}, FluxSink.OverflowStrategy.BUFFER)
.doOnCancel(() -> {
if (disposable.get() != null) {
disposable.get().run();
}
})
.doOnComplete(() -> {
.doFinally(ignored -> {
if (disposable.get() != null) {
disposable.get().run();
}

View File

@@ -8,6 +8,8 @@ import jakarta.annotation.Nullable;
import jakarta.inject.Inject;
import jakarta.inject.Singleton;
import java.io.IOException;
@Singleton
public class KVStoreService {
@@ -29,19 +31,7 @@ public class KVStoreService {
* @return The {@link KVStore}.
*/
public KVStore get(String tenant, String namespace, @Nullable String fromNamespace) {
// Only check namespace existence if not a descendant
boolean checkIfNamespaceExists = fromNamespace == null || isNotParentNamespace(namespace, fromNamespace);
if (checkIfNamespaceExists && !namespaceService.isNamespaceExists(tenant, namespace)) {
throw new KVStoreException(String.format(
"Cannot access the KV store. The namespace '%s' does not exist.",
namespace
));
}
boolean isNotSameNamespace = fromNamespace != null && !namespace.equals(fromNamespace);
if (isNotSameNamespace && isNotParentNamespace(namespace, fromNamespace)) {
try {
flowService.checkAllowedNamespace(tenant, namespace, tenant, fromNamespace);
@@ -52,6 +42,24 @@ public class KVStoreService {
}
}
// Only check namespace existence if not a descendant
boolean checkIfNamespaceExists = fromNamespace == null || isNotParentNamespace(namespace, fromNamespace);
if (checkIfNamespaceExists && !namespaceService.isNamespaceExists(tenant, namespace)) {
// if it didn't exist, we still check if there are KV as you can add KV without creating a namespace in DB or having flows in it
KVStore kvStore = new InternalKVStore(tenant, namespace, storageInterface);
try {
if (kvStore.list().isEmpty()) {
throw new KVStoreException(String.format(
"Cannot access the KV store. The namespace '%s' does not exist.",
namespace
));
}
} catch (IOException e) {
throw new KVStoreException(e);
}
return kvStore;
}
return new InternalKVStore(tenant, namespace, storageInterface);
}

View File

@@ -8,7 +8,6 @@ import io.kestra.core.models.annotations.Example;
import io.kestra.core.models.annotations.Plugin;
import io.kestra.core.models.annotations.PluginProperty;
import io.kestra.core.models.executions.Execution;
import io.kestra.core.models.flows.Flow;
import io.kestra.core.models.property.Property;
import io.kestra.core.models.executions.TaskRun;
import io.kestra.core.models.executions.TaskRunAttempt;
@@ -37,13 +36,16 @@ import lombok.ToString;
import jakarta.validation.constraints.NotEmpty;
import jakarta.validation.constraints.NotNull;
import org.apache.commons.lang3.stream.Streams;
import java.time.ZonedDateTime;
import java.util.ArrayList;
import java.util.Collections;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
import java.util.Optional;
import java.util.stream.Collectors;
@SuperBuilder
@ToString
@@ -160,7 +162,7 @@ public class Subflow extends Task implements ExecutableTask<Subflow.Output>, Chi
@Override
public List<SubflowExecution<?>> createSubflowExecutions(RunContext runContext,
FlowExecutorInterface flowExecutorInterface,
Flow currentFlow,
io.kestra.core.models.flows.Flow currentFlow,
Execution currentExecution,
TaskRun currentTaskRun) throws InternalException {
Map<String, Object> inputs = new HashMap<>();
@@ -186,7 +188,7 @@ public class Subflow extends Task implements ExecutableTask<Subflow.Output>, Chi
public Optional<SubflowExecutionResult> createSubflowExecutionResult(
RunContext runContext,
TaskRun taskRun,
Flow flow,
io.kestra.core.models.flows.Flow flow,
Execution execution
) {
// we only create a worker task result when the execution is terminated
@@ -202,16 +204,25 @@ public class Subflow extends Task implements ExecutableTask<Subflow.Output>, Chi
.executionId(execution.getId())
.state(execution.getState().getCurrent());
FlowInputOutput flowInputOutput = ((DefaultRunContext)runContext).getApplicationContext().getBean(FlowInputOutput.class); // this is hacking
final Map<String, Object> subflowOutputs = Optional
.ofNullable(flow.getOutputs())
.map(outputs -> flowInputOutput.flowOutputsToMap(flow.getOutputs()))
.map(outputs -> flowInputOutput.typedOutputs(flow, execution, outputs))
.map(outputs -> outputs
.stream()
.collect(Collectors.toMap(
io.kestra.core.models.flows.Output::getId,
io.kestra.core.models.flows.Output::getValue)
)
)
.orElseGet(() -> isOutputsAllowed ? this.getOutputs() : null);
if (subflowOutputs != null) {
try {
builder.outputs(runContext.render(subflowOutputs));
Map<String, Object> outputs = runContext.render(subflowOutputs);
FlowInputOutput flowInputOutput = ((DefaultRunContext)runContext).getApplicationContext().getBean(FlowInputOutput.class); // this is hacking
if (flow.getOutputs() != null && flowInputOutput != null) {
outputs = flowInputOutput.typedOutputs(flow, execution, outputs);
}
builder.outputs(outputs);
} catch (Exception e) {
runContext.logger().warn("Failed to extract outputs with the error: '{}'", e.getLocalizedMessage(), e);
var state = this.isAllowFailure() ? this.isAllowWarning() ? State.Type.SUCCESS : State.Type.WARNING : State.Type.FAILED;

View File

@@ -23,6 +23,8 @@ import java.io.FileOutputStream;
import java.net.URI;
import java.net.URLEncoder;
import java.nio.charset.StandardCharsets;
import java.nio.file.Path;
import java.nio.file.Paths;
import java.util.List;
import java.util.Map;
@@ -162,21 +164,31 @@ public class Download extends AbstractHttp implements RunnableTask<Download.Outp
}
}
// Note: this is a naive basic implementation that may bot cover all possible use cases.
// Note: this is a basic implementation that should cover all possible use cases.
// If this is not enough, we should find some helper method somewhere to cover all possible rules of the Content-Disposition header.
private String filenameFromHeader(RunContext runContext, String contentDisposition) {
try {
String[] parts = contentDisposition.split(" ");
// Content-Disposition parts are separated by ';'
String[] parts = contentDisposition.split(";");
String filename = null;
for (String part : parts) {
if (part.startsWith("filename")) {
filename = part.substring(part.lastIndexOf('=') + 1);
String stripped = part.strip();
if (stripped.startsWith("filename")) {
filename = stripped.substring(stripped.lastIndexOf('=') + 1);
}
if (part.startsWith("filename*")) {
if (stripped.startsWith("filename*")) {
// following https://datatracker.ietf.org/doc/html/rfc5987 the filename* should be <ENCODING>'(lang)'<filename>
filename = part.substring(part.lastIndexOf('\'') + 2, part.length() - 1);
filename = stripped.substring(stripped.lastIndexOf('\'') + 2, stripped.length() - 1);
}
}
// filename may be in double-quotes
if (filename != null && filename.charAt(0) == '"') {
filename = filename.substring(1, filename.length() - 1);
}
// if filename contains a path: use only the last part to avoid security issues due to host file overwriting
if (filename != null && filename.contains(File.separator)) {
filename = filename.substring(filename.lastIndexOf(File.separator) + 1);
}
return filename;
} catch (Exception e) {
// if we cannot parse the Content-Disposition header, we return null

View File

@@ -337,7 +337,8 @@ public class Schedule extends AbstractTrigger implements Schedulable, TriggerOut
RunContext runContext = conditionContext.getRunContext();
ExecutionTime executionTime = this.executionTime();
ZonedDateTime currentDateTimeExecution = convertDateTime(triggerContext.getDate());
Backfill backfill = triggerContext.getBackfill();
final Backfill backfill = triggerContext.getBackfill();
if (backfill != null) {
if (backfill.getPaused()) {
@@ -352,7 +353,14 @@ public class Schedule extends AbstractTrigger implements Schedulable, TriggerOut
return Optional.empty();
}
ZonedDateTime next = scheduleDates.getDate();
final ZonedDateTime next = scheduleDates.getDate();
// If the trigger is evaluated for 'back-fill', we have to make sure
// that 'current-date' is strictly after the next execution date for an execution to be eligible.
if (backfill != null && currentDateTimeExecution.isBefore(next)) {
// Otherwise, skip the execution.
return Optional.empty();
}
// we are in the future don't allow
// No use case, just here for prevention but it should never happen

View File

@@ -2,7 +2,9 @@ package io.kestra.core.runners;
import io.kestra.core.junit.annotations.KestraTest;
import io.kestra.core.models.executions.Execution;
import io.kestra.core.models.flows.*;
import io.kestra.core.models.flows.DependsOn;
import io.kestra.core.models.flows.Input;
import io.kestra.core.models.flows.Type;
import io.kestra.core.models.flows.input.FileInput;
import io.kestra.core.models.flows.input.InputAndValue;
import io.kestra.core.models.flows.input.IntInput;
@@ -28,11 +30,6 @@ import java.util.List;
import java.util.Map;
import java.util.Optional;
import static org.hamcrest.MatcherAssert.assertThat;
import static org.hamcrest.Matchers.aMapWithSize;
import static org.hamcrest.Matchers.notNullValue;
import static org.hamcrest.Matchers.is;
@KestraTest
class FlowInputOutputTest {
@@ -252,17 +249,6 @@ class FlowInputOutputTest {
);
}
@Test
@SuppressWarnings("unchecked")
void flowOutputsToMap() {
Flow flow = Flow.builder().id("flow").outputs(List.of(Output.builder().id("output").value("something").build())).build();
Map<String, Object> stringObjectMap = flowInputOutput.flowOutputsToMap(flow.getOutputs());
assertThat(stringObjectMap, aMapWithSize(1));
assertThat(stringObjectMap.get("output"), notNullValue());
assertThat(((Map<String, Object>) stringObjectMap.get("output")).get("value"), is("something"));
}
private static final class MemoryCompletedFileUpload implements CompletedFileUpload {
private final String name;

View File

@@ -2,12 +2,15 @@ package io.kestra.core.services;
import io.kestra.core.junit.annotations.KestraTest;
import io.kestra.core.repositories.FlowRepositoryInterface;
import io.kestra.core.storages.kv.KVStoreException;
import io.kestra.core.storages.StorageInterface;
import io.kestra.core.storages.kv.*;
import io.micronaut.test.annotation.MockBean;
import jakarta.inject.Inject;
import org.junit.jupiter.api.Assertions;
import org.junit.jupiter.api.Test;
import java.io.IOException;
import java.time.Duration;
import java.util.Optional;
@KestraTest
@@ -19,7 +22,7 @@ class KVStoreServiceTest {
KVStoreService storeService;
@Inject
FlowRepositoryInterface flowRepository;
StorageInterface storageInterface;
@Test
void shouldGetKVStoreForExistingNamespaceGivenFromNull() {
@@ -37,6 +40,13 @@ class KVStoreServiceTest {
Assertions.assertNotNull(storeService.get(null, "io.kestra", TEST_EXISTING_NAMESPACE));
}
@Test
void shouldGetKVStoreFromNonExistingNamespaceWithAKV() throws IOException {
KVStore kvStore = new InternalKVStore(null, "system", storageInterface);
kvStore.put("key", new KVValueAndMetadata(new KVMetadata(Duration.ofHours(1)), "value"));
Assertions.assertNotNull(storeService.get(null, "system", null));
}
@MockBean(NamespaceService.class)
public static class MockNamespaceService extends NamespaceService {

View File

@@ -39,13 +39,4 @@ class FlowOutputTest extends AbstractMemoryRunnerTest {
assertThat(execution.getOutputs(), nullValue());
assertThat(execution.getState().getCurrent(), is(State.Type.FAILED));
}
@Test
void shouldGetSuccessExecutionForFlowWithOutputsDisplayName() throws TimeoutException, QueueException {
Execution execution = runnerUtils.runOne(null, NAMESPACE, "flow-with-outputs-display-name", null, null);
assertThat(execution.getOutputs(), aMapWithSize(1));
assertThat(execution.getOutputs().get("key"), nullValue());
assertThat(execution.getOutputs().get("Sample Output"), is("{\"value\":\"flow-with-outputs-display-name\"}"));
assertThat(execution.getState().getCurrent(), is(State.Type.SUCCESS));
}
}

View File

@@ -4,6 +4,7 @@ import io.kestra.core.models.executions.Execution;
import io.kestra.core.models.flows.State;
import io.kestra.core.queues.QueueException;
import io.kestra.core.runners.AbstractMemoryRunnerTest;
import org.junit.jupiter.api.RepeatedTest;
import org.junit.jupiter.api.Test;
import java.time.Duration;
@@ -14,6 +15,15 @@ import static org.hamcrest.MatcherAssert.assertThat;
import static org.hamcrest.Matchers.*;
class IfTest extends AbstractMemoryRunnerTest {
@Test
void multipleIf() throws TimeoutException, QueueException {
Execution execution = runnerUtils.runOne(null, "io.kestra.tests", "if", null,
(f, e) -> Map.of("if1", true, "if2", false, "if3", true));
assertThat(execution.getTaskRunList(), hasSize(12));
assertThat(execution.getState().getCurrent(), is(State.Type.SUCCESS));
}
@Test
void ifTruthy() throws TimeoutException, QueueException {
Execution execution = runnerUtils.runOne(null, "io.kestra.tests", "if-condition", null,

View File

@@ -1,22 +1,25 @@
package io.kestra.plugin.core.flow;
import io.kestra.core.junit.annotations.KestraTest;
import io.kestra.core.exceptions.IllegalVariableEvaluationException;
import io.kestra.core.models.executions.Execution;
import io.kestra.core.models.executions.TaskRun;
import io.kestra.core.models.flows.Flow;
import io.kestra.core.models.flows.Output;
import io.kestra.core.models.flows.State;
import io.kestra.core.models.flows.Type;
import io.kestra.core.runners.DefaultRunContext;
import io.kestra.core.runners.RunContext;
import io.kestra.core.runners.RunContextFactory;
import io.kestra.core.runners.SubflowExecutionResult;
import io.micronaut.context.annotation.Property;
import jakarta.inject.Inject;
import io.micronaut.context.ApplicationContext;
import org.hamcrest.Matchers;
import org.junit.jupiter.api.BeforeEach;
import org.junit.jupiter.api.Test;
import org.junit.jupiter.api.extension.ExtendWith;
import org.mockito.Mock;
import org.mockito.Mockito;
import org.mockito.junit.jupiter.MockitoExtension;
import org.mockito.junit.jupiter.MockitoSettings;
import org.mockito.quality.Strictness;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import java.time.Instant;
import java.util.Collections;
@@ -29,14 +32,26 @@ import static org.hamcrest.Matchers.hasProperty;
import static org.hamcrest.Matchers.is;
import static org.junit.jupiter.api.Assertions.assertTrue;
@KestraTest
@ExtendWith(MockitoExtension.class)
@MockitoSettings(strictness = Strictness.LENIENT)
class SubflowTest {
private static final Logger LOG = LoggerFactory.getLogger(SubflowTest.class);
private static final State DEFAULT_SUCCESS_STATE = State.of(State.Type.SUCCESS, List.of(new State.History(State.Type.CREATED, Instant.now()), new State.History(State.Type.RUNNING, Instant.now()), new State.History(State.Type.SUCCESS, Instant.now())));
public static final String EXECUTION_ID = "executionId";
@Inject
private RunContextFactory runContextFactory;
@Mock
private DefaultRunContext runContext;
@Mock
private ApplicationContext applicationContext;
@BeforeEach
void beforeEach() {
Mockito.when(runContext.logger()).thenReturn(LOG);
Mockito.when(runContext.getApplicationContext()).thenReturn(applicationContext);
}
@Test
void shouldNotReturnResultForExecutionNotTerminated() {
@@ -44,7 +59,6 @@ class SubflowTest {
.builder()
.state(State.of(State.Type.CREATED, Collections.emptyList()))
.build();
RunContext runContext = runContextFactory.of();
Optional<SubflowExecutionResult> result = new Subflow().createSubflowExecutionResult(
runContext,
@@ -59,14 +73,14 @@ class SubflowTest {
@SuppressWarnings("deprecation")
@Test
void shouldNotReturnOutputsForSubflowOutputsDisabled() {
// Given
Mockito.when(applicationContext.getProperty(Subflow.PLUGIN_FLOW_OUTPUTS_ENABLED, Boolean.class))
.thenReturn(Optional.of(false));
Map<String, Object> outputs = Map.of("key", "value");
Subflow subflow = Subflow.builder()
.outputs(outputs)
.build();
DefaultRunContext defaultRunContext = (DefaultRunContext) runContextFactory.of();
DefaultRunContext runContext = Mockito.mock(DefaultRunContext.class);
Mockito.when(runContext.pluginConfiguration(Subflow.PLUGIN_FLOW_OUTPUTS_ENABLED)).thenReturn(Optional.of(false));
Mockito.when(runContext.getApplicationContext()).thenReturn(defaultRunContext.getApplicationContext());
// When
Optional<SubflowExecutionResult> result = subflow.createSubflowExecutionResult(
@@ -81,6 +95,7 @@ class SubflowTest {
Map<String, Object> expected = Subflow.Output.builder()
.executionId(EXECUTION_ID)
.state(DEFAULT_SUCCESS_STATE.getCurrent())
.outputs(Collections.emptyMap())
.build()
.toMap();
assertThat(result.get().getParentTaskRun().getOutputs(), is(expected));
@@ -94,10 +109,14 @@ class SubflowTest {
@SuppressWarnings("deprecation")
@Test
void shouldReturnOutputsForSubflowOutputsEnabled() {
void shouldReturnOutputsForSubflowOutputsEnabled() throws IllegalVariableEvaluationException {
// Given
Mockito.when(applicationContext.getProperty(Subflow.PLUGIN_FLOW_OUTPUTS_ENABLED, Boolean.class))
.thenReturn(Optional.of(true));
Map<String, Object> outputs = Map.of("key", "value");
RunContext runContext = runContextFactory.of(outputs);
Mockito.when(runContext.render(Mockito.anyMap())).thenReturn(outputs);
Subflow subflow = Subflow.builder()
.outputs(outputs)
@@ -129,10 +148,13 @@ class SubflowTest {
}
@Test
void shouldOnlyReturnOutputsFromFlowOutputs() {
void shouldOnlyReturnOutputsFromFlowOutputs() throws IllegalVariableEvaluationException {
// Given
Output output = Output.builder().id("key").value("value").type(Type.STRING).build();
RunContext runContext = runContextFactory.of(Map.of(output.getId(), output.getValue()));
Mockito.when(applicationContext.getProperty(Subflow.PLUGIN_FLOW_OUTPUTS_ENABLED, Boolean.class))
.thenReturn(Optional.of(true));
Output output = Output.builder().id("key").value("value").build();
Mockito.when(runContext.render(Mockito.anyMap())).thenReturn(Map.of(output.getId(), output.getValue()));
Flow flow = Flow.builder()
.outputs(List.of(output))
.build();

View File

@@ -134,7 +134,26 @@ class DownloadTest {
Download.Output output = task.run(runContext);
assertThat(output.getUri().toString(), containsString("filename.jpg"));
assertThat(output.getUri().toString(), endsWith("filename.jpg"));
}
@Test
void contentDispositionWithPath() throws Exception {
EmbeddedServer embeddedServer = applicationContext.getBean(EmbeddedServer.class);
embeddedServer.start();
Download task = Download.builder()
.id(DownloadTest.class.getSimpleName())
.type(DownloadTest.class.getName())
.uri(embeddedServer.getURI() + "/content-disposition")
.build();
RunContext runContext = TestsUtils.mockRunContext(this.runContextFactory, task, ImmutableMap.of());
Download.Output output = task.run(runContext);
assertThat(output.getUri().toString(), not(containsString("/secure-path/")));
assertThat(output.getUri().toString(), endsWith("filename.jpg"));
}
@Test
@@ -177,5 +196,11 @@ class DownloadTest {
return HttpResponse.ok("Hello World".getBytes())
.header(HttpHeaders.CONTENT_DISPOSITION, "attachment; filename=\"filename.jpg\"");
}
@Get("content-disposition-path")
public HttpResponse<byte[]> contentDispositionWithPath() {
return HttpResponse.ok("Hello World".getBytes())
.header(HttpHeaders.CONTENT_DISPOSITION, "attachment; filename=\"/secure-path/filename.jpg\"");
}
}
}

View File

@@ -130,7 +130,7 @@ public class SetTest {
.type(Set.class.getName())
.key("{{ inputs.key }}")
.value("{{ inputs.value }}")
.namespace("???")
.namespace("not-found")
.build();
// When - Then

View File

@@ -2,6 +2,7 @@ package io.kestra.plugin.core.trigger;
import io.kestra.core.models.Label;
import io.kestra.core.models.conditions.ConditionContext;
import io.kestra.core.models.triggers.Backfill;
import io.kestra.core.runners.DefaultRunContext;
import io.kestra.core.runners.RunContextInitializer;
import io.kestra.plugin.core.condition.DateTimeBetween;
@@ -21,6 +22,8 @@ import org.junit.jupiter.api.Test;
import java.time.DayOfWeek;
import java.time.Duration;
import java.time.Instant;
import java.time.LocalTime;
import java.time.ZoneId;
import java.time.ZonedDateTime;
import java.time.format.DateTimeFormatter;
@@ -36,6 +39,8 @@ import static org.hamcrest.Matchers.*;
@KestraTest
class ScheduleTest {
private static final String TEST_CRON_EVERYDAY_AT_8 = "0 8 * * *";
@Inject
RunContextFactory runContextFactory;
@@ -214,6 +219,45 @@ class ScheduleTest {
assertThat(dateFromVars(vars.get("previous"), date), is(date.minus(Duration.ofSeconds(1))));
}
@Test
void shouldNotReturnExecutionForBackFillWhenCurrentDateIsBeforeScheduleDate() throws Exception {
// Given
Schedule trigger = Schedule.builder().id("schedule").cron(TEST_CRON_EVERYDAY_AT_8).build();
ZonedDateTime now = ZonedDateTime.now();
TriggerContext triggerContext = triggerContext(now, trigger).toBuilder()
.backfill(Backfill
.builder()
.currentDate(ZonedDateTime.now().with(LocalTime.MIN))
.end(ZonedDateTime.now().with(LocalTime.MAX))
.build()
).build();
// When
Optional<Execution> result = trigger.evaluate(conditionContext(trigger), triggerContext);
// Then
assertThat(result.isEmpty(), is(true));
}
@Test
void shouldReturnExecutionForBackFillWhenCurrentDateIsAfterScheduleDate() throws Exception {
// Given
Schedule trigger = Schedule.builder().id("schedule").cron(TEST_CRON_EVERYDAY_AT_8).build();
ZonedDateTime now = ZonedDateTime.now();
TriggerContext triggerContext = triggerContext(now, trigger).toBuilder()
.backfill(Backfill
.builder()
.currentDate(ZonedDateTime.now().with(LocalTime.MIN).plus(Duration.ofHours(8)))
.end(ZonedDateTime.now().with(LocalTime.MAX))
.build()
)
.build();
// When
Optional<Execution> result = trigger.evaluate(conditionContext(trigger), triggerContext);
// Then
assertThat(result.isPresent(), is(true));
}
@Test
void noBackfillNextDate() throws Exception {
Schedule trigger = Schedule.builder().id("schedule").cron("0 0 * * *").build();

View File

@@ -1,13 +0,0 @@
id: flow-with-outputs-display-name
namespace: io.kestra.tests
tasks:
- id: return
type: io.kestra.plugin.core.debug.Return
format: "{{ flow.id }}"
outputs:
- id: "key"
value: "{{ outputs.return }}"
type: STRING
displayName: Sample Output

View File

@@ -0,0 +1,57 @@
id: if
namespace: io.kestra.tests
inputs:
- id: if1
type: BOOLEAN
- id: if2
type: BOOLEAN
- id: if3
type: BOOLEAN
tasks:
- id: parallel
type: io.kestra.plugin.core.flow.Parallel
concurrent: 4
tasks:
- id: if-1
type: io.kestra.plugin.core.flow.If
condition: "{{ inputs.if1 }}"
then:
- id: if-1-log
type: io.kestra.plugin.core.log.Log
message: "Hello World!"
- id: if-2
type: io.kestra.plugin.core.flow.If
condition: "{{ inputs.if2 }}"
then:
- id: if-2-log
type: io.kestra.plugin.core.log.Log
message: "Hello World!"
- id: if-3
type: io.kestra.plugin.core.flow.If
condition: "{{ inputs.if3 }}"
then:
- id: if-3-log
type: io.kestra.plugin.core.log.Log
message: "Hello World!"
- id: log-parallel-1
type: io.kestra.plugin.core.log.Log
message: "Hello World!"
- id: log-parallel-2
type: io.kestra.plugin.core.log.Log
message: "Hello World!"
- id: log-parallel-3
type: io.kestra.plugin.core.log.Log
message: "Hello World!"
- id: log-parallel-4
type: io.kestra.plugin.core.log.Log
message: "Hello World!"
- id: log-parallel-5
type: io.kestra.plugin.core.log.Log
message: "Hello World!"
- id: log-parallel-6
type: io.kestra.plugin.core.log.Log
message: "Hello World!"

View File

@@ -1,11 +1,6 @@
id: subflow-grand-child
namespace: io.kestra.tests
outputs:
- id: myResult
type: STRING
value: something
tasks:
- id: firstLevel
type: io.kestra.plugin.core.log.Log

View File

@@ -1,4 +1,4 @@
version=0.20.0-SNAPSHOT
version=0.20.8
org.gradle.parallel=true
org.gradle.caching=true

View File

@@ -8,8 +8,8 @@ import io.kestra.core.exceptions.InternalException;
import io.kestra.core.metrics.MetricRegistry;
import io.kestra.core.models.executions.*;
import io.kestra.core.models.executions.statistics.ExecutionCount;
import io.kestra.core.models.flows.*;
import io.kestra.core.models.flows.Flow;
import io.kestra.core.models.flows.*;
import io.kestra.core.models.flows.sla.*;
import io.kestra.core.models.tasks.ExecutableTask;
import io.kestra.core.models.tasks.Task;
@@ -47,9 +47,11 @@ import org.jooq.Configuration;
import org.slf4j.event.Level;
import java.io.IOException;
import java.time.*;
import java.time.Duration;
import java.time.Instant;
import java.util.ArrayList;
import java.util.List;
import java.util.Objects;
import java.util.Optional;
import java.util.concurrent.*;
import java.util.concurrent.atomic.AtomicBoolean;
@@ -294,7 +296,7 @@ public class JdbcExecutor implements ExecutorInterface, Service {
flowTopologyService
.topology(
flow,
this.allFlows
this.allFlows.stream().filter(f -> Objects.equals(f.getTenantId(), flow.getTenantId())).toList()
)
)
.distinct()

View File

@@ -97,7 +97,7 @@ public class JdbcScheduler extends AbstractScheduler {
public void handleNext(List<FlowWithSource> flows, ZonedDateTime now, BiConsumer<List<Trigger>, ScheduleContextInterface> consumer) {
JdbcSchedulerContext schedulerContext = new JdbcSchedulerContext(this.dslContextWrapper);
schedulerContext.startTransaction(scheduleContextInterface -> {
schedulerContext.doInTransaction(scheduleContextInterface -> {
List<Trigger> triggers = this.triggerState.findByNextExecutionDateReadyForAllTenants(now, scheduleContextInterface);
consumer.accept(triggers, scheduleContextInterface);

View File

@@ -18,17 +18,14 @@ public class JdbcSchedulerContext implements ScheduleContextInterface {
this.dslContextWrapper = dslContextWrapper;
}
public void startTransaction(Consumer<ScheduleContextInterface> consumer) {
@Override
public void doInTransaction(Consumer<ScheduleContextInterface> consumer) {
this.dslContextWrapper.transaction(configuration -> {
this.context = DSL.using(configuration);
consumer.accept(this);
this.commit();
this.context.commit();
});
}
public void commit() {
this.context.commit();
}
}

View File

@@ -63,8 +63,15 @@ public class JdbcSchedulerTriggerState implements SchedulerTriggerStateInterface
@Override
public Trigger update(Trigger trigger) {
// here we save a trigger after evaluation, but as during its evaluation it can have been disabled in DB,
// we need to load it form DB and copy the disabled flag if set
Optional<Trigger> existing = findLast(trigger);
Trigger updated = trigger;
if (existing.isPresent() && existing.get().getDisabled()) {
updated = trigger.toBuilder().disabled(true).build();
}
return this.triggerRepository.update(trigger);
return this.triggerRepository.update(updated);
}
public Trigger updateExecution(Trigger trigger) {
@@ -85,7 +92,4 @@ public class JdbcSchedulerTriggerState implements SchedulerTriggerStateInterface
public List<Trigger> findByNextExecutionDateReadyForGivenFlows(List<FlowWithSource> flows, ZonedDateTime now, ScheduleContextInterface scheduleContext) {
throw new NotImplementedException();
}
@Override
public void unlock(Trigger trigger) {}
}

53
ui/package-lock.json generated
View File

@@ -1,15 +1,15 @@
{
"name": "kestra",
"version": "0.19.11",
"version": "0.0.0",
"lockfileVersion": 3,
"requires": true,
"packages": {
"": {
"name": "kestra",
"version": "0.19.11",
"version": "0.0.0",
"dependencies": {
"@js-joda/core": "^5.6.3",
"@kestra-io/ui-libs": "^0.0.69",
"@kestra-io/ui-libs": "^0.0.72",
"@vue-flow/background": "^1.3.2",
"@vue-flow/controls": "^1.1.2",
"@vue-flow/core": "^1.41.5",
@@ -742,13 +742,13 @@
}
},
"node_modules/@intlify/core-base": {
"version": "10.0.4",
"resolved": "https://registry.npmjs.org/@intlify/core-base/-/core-base-10.0.4.tgz",
"integrity": "sha512-GG428DkrrWCMhxRMRQZjuS7zmSUzarYcaHJqG9VB8dXAxw4iQDoKVQ7ChJRB6ZtsCsX3Jse1PEUlHrJiyQrOTg==",
"version": "10.0.5",
"resolved": "https://registry.npmjs.org/@intlify/core-base/-/core-base-10.0.5.tgz",
"integrity": "sha512-F3snDTQs0MdvnnyzTDTVkOYVAZOE/MHwRvF7mn7Jw1yuih4NrFYLNYIymGlLmq4HU2iIdzYsZ7f47bOcwY73XQ==",
"license": "MIT",
"dependencies": {
"@intlify/message-compiler": "10.0.4",
"@intlify/shared": "10.0.4"
"@intlify/message-compiler": "10.0.5",
"@intlify/shared": "10.0.5"
},
"engines": {
"node": ">= 16"
@@ -758,12 +758,12 @@
}
},
"node_modules/@intlify/message-compiler": {
"version": "10.0.4",
"resolved": "https://registry.npmjs.org/@intlify/message-compiler/-/message-compiler-10.0.4.tgz",
"integrity": "sha512-AFbhEo10DP095/45EauinQJ5hJ3rJUmuuqltGguvc3WsvezZN+g8qNHLGWKu60FHQVizMrQY7VJ+zVlBXlQQkQ==",
"version": "10.0.5",
"resolved": "https://registry.npmjs.org/@intlify/message-compiler/-/message-compiler-10.0.5.tgz",
"integrity": "sha512-6GT1BJ852gZ0gItNZN2krX5QAmea+cmdjMvsWohArAZ3GmHdnNANEcF9JjPXAMRtQ6Ux5E269ymamg/+WU6tQA==",
"license": "MIT",
"dependencies": {
"@intlify/shared": "10.0.4",
"@intlify/shared": "10.0.5",
"source-map-js": "^1.0.2"
},
"engines": {
@@ -774,9 +774,9 @@
}
},
"node_modules/@intlify/shared": {
"version": "10.0.4",
"resolved": "https://registry.npmjs.org/@intlify/shared/-/shared-10.0.4.tgz",
"integrity": "sha512-ukFn0I01HsSgr3VYhYcvkTCLS7rGa0gw4A4AMpcy/A9xx/zRJy7PS2BElMXLwUazVFMAr5zuiTk3MQeoeGXaJg==",
"version": "10.0.5",
"resolved": "https://registry.npmjs.org/@intlify/shared/-/shared-10.0.5.tgz",
"integrity": "sha512-bmsP4L2HqBF6i6uaMqJMcFBONVjKt+siGluRq4Ca4C0q7W2eMaVZr8iCgF9dKbcVXutftkC7D6z2SaSMmLiDyA==",
"license": "MIT",
"engines": {
"node": ">= 16"
@@ -878,9 +878,9 @@
"integrity": "sha512-T1rRxzdqkEXcou0ZprN1q9yDRlvzCPLqmlNt5IIsGBzoEVgLCCYrKEwc84+TvsXuAc95VAZwtWD2zVsKPY4bcA=="
},
"node_modules/@kestra-io/ui-libs": {
"version": "0.0.69",
"resolved": "https://registry.npmjs.org/@kestra-io/ui-libs/-/ui-libs-0.0.69.tgz",
"integrity": "sha512-kHFJ09fKZWTlzkpAYYK6ELo+9tUAgJE+LRrlFK5O8DqMr/NSaL5tzuAiPE5alsPOAT7LOhGs2RXj+rMtJ/SVqQ==",
"version": "0.0.72",
"resolved": "https://registry.npmjs.org/@kestra-io/ui-libs/-/ui-libs-0.0.72.tgz",
"integrity": "sha512-dPJ6XqMjVZcU0KoaxLK4nwoiPHXT6kenrxQhklPzzea7wFVrGxthbeaMZwmMg5J042Ys8Z2pWVaXKMs3qORCLg==",
"dependencies": {
"@nuxtjs/mdc": "^0.9.0",
"@popperjs/core": "^2.11.8",
@@ -7873,15 +7873,16 @@
"peer": true
},
"node_modules/nanoid": {
"version": "3.3.7",
"resolved": "https://registry.npmjs.org/nanoid/-/nanoid-3.3.7.tgz",
"integrity": "sha512-eSRppjcPIatRIMC1U6UngP8XFcz8MQWGQdt1MTBQ7NaAmvXDfvNxbvWV3x2y6CdEUciCSsDHDQZbhYaB8QEo2g==",
"version": "3.3.8",
"resolved": "https://registry.npmjs.org/nanoid/-/nanoid-3.3.8.tgz",
"integrity": "sha512-WNLf5Sd8oZxOm+TzppcYk8gVOgP+l58xNy58D0nbUnOxOWRWvlcCV4kUF7ltmI6PsrLl/BgKEyS4mqsGChFN0w==",
"funding": [
{
"type": "github",
"url": "https://github.com/sponsors/ai"
}
],
"license": "MIT",
"bin": {
"nanoid": "bin/nanoid.cjs"
},
@@ -11230,13 +11231,13 @@
}
},
"node_modules/vue-i18n": {
"version": "10.0.4",
"resolved": "https://registry.npmjs.org/vue-i18n/-/vue-i18n-10.0.4.tgz",
"integrity": "sha512-1xkzVxqBLk2ZFOmeI+B5r1J7aD/WtNJ4j9k2mcFcQo5BnOmHBmD7z4/oZohh96AAaRZ4Q7mNQvxc9h+aT+Md3w==",
"version": "10.0.5",
"resolved": "https://registry.npmjs.org/vue-i18n/-/vue-i18n-10.0.5.tgz",
"integrity": "sha512-9/gmDlCblz3i8ypu/afiIc/SUIfTTE1mr0mZhb9pk70xo2csHAM9mp2gdQ3KD2O0AM3Hz/5ypb+FycTj/lHlPQ==",
"license": "MIT",
"dependencies": {
"@intlify/core-base": "10.0.4",
"@intlify/shared": "10.0.4",
"@intlify/core-base": "10.0.5",
"@intlify/shared": "10.0.5",
"@vue/devtools-api": "^6.5.0"
},
"engines": {

View File

@@ -1,6 +1,6 @@
{
"name": "kestra",
"version": "0.19.11",
"version": "0.0.0",
"private": true,
"type": "module",
"packageManager": "npm@9.9.3",

View File

@@ -0,0 +1,120 @@
<svg width="164" height="116" viewBox="0 0 164 116" fill="none" xmlns="http://www.w3.org/2000/svg">
<ellipse cx="82.0001" cy="57.058" rx="81.8075" ry="50.9725" fill="#5C29DC" fill-opacity="0.05"/>
<ellipse cx="82.0002" cy="57.0577" rx="71.8178" ry="44.7482" fill="#5C29DC" fill-opacity="0.22"/>
<ellipse cx="82.0002" cy="57.0581" rx="60.1184" ry="37.4585" fill="#5C29DC"/>
<g filter="url(#filter0_d_4459_37617)">
<path d="M84.8423 99.1473L146.085 64.5658C150.538 62.0508 150.595 55.6558 146.186 53.0626L84.9445 17.0409C82.8563 15.8126 80.2648 15.8194 78.183 17.0585L17.6657 53.0806C13.2967 55.6812 13.3531 62.0268 17.7676 64.5494L78.2843 99.1305C80.3147 100.291 82.8059 100.297 84.8423 99.1473Z" fill="#1F232C"/>
</g>
<g filter="url(#filter1_d_4459_37617)">
<path d="M84.8423 92.5061L146.085 57.9247C150.538 55.4097 150.595 49.0147 146.186 46.4215L84.9445 10.3998C82.8563 9.17152 80.2648 9.17828 78.183 10.4174L17.6657 46.4395C13.2967 49.0401 13.3531 55.3857 17.7676 57.9083L78.2843 92.4894C80.3147 93.6497 82.8059 93.656 84.8423 92.5061Z" fill="#1F232C"/>
</g>
<path d="M84.8423 95.8269L146.085 61.2455C150.538 58.7305 150.595 52.3355 146.186 49.7423L84.9445 13.7206C82.8563 12.4923 80.2648 12.4991 78.183 13.7382L17.6657 49.7603C13.2967 52.3609 13.3531 58.7065 17.7676 61.2291L78.2843 95.8102C80.3147 96.9705 82.8059 96.9768 84.8423 95.8269Z" fill="#1F232C"/>
<g filter="url(#filter2_d_4459_37617)">
<path d="M84.8423 90.1834L146.085 55.6019C150.538 53.087 150.595 46.6919 146.186 44.0987L84.9445 8.07703C82.8563 6.84877 80.2648 6.85553 78.183 8.09467L17.6657 44.1167C13.2967 46.7173 13.3531 53.0629 17.7676 55.5855L78.2843 90.1666C80.3147 91.3269 82.8059 91.3333 84.8423 90.1834Z" fill="#11192B"/>
<path d="M84.6791 89.8943L145.921 55.3129C150.153 52.9237 150.206 46.8484 146.018 44.3849L84.7762 8.36315C82.7924 7.1963 80.3305 7.20272 78.3528 8.37991L17.8355 44.402C13.6849 46.8725 13.7385 52.9008 17.9323 55.2973L78.449 89.8784C80.3779 90.9807 82.7445 90.9867 84.6791 89.8943Z" stroke="url(#paint0_linear_4459_37617)" stroke-width="0.663881"/>
</g>
<g clip-path="url(#clip0_4459_37617)">
<mask id="mask0_4459_37617" style="mask-type:alpha" maskUnits="userSpaceOnUse" x="14" y="7" width="136" height="85">
<path d="M84.6791 89.8943L145.921 55.3129C150.153 52.9237 150.206 46.8484 146.018 44.3849L84.7762 8.36315C82.7924 7.1963 80.3305 7.20272 78.3528 8.37991L17.8355 44.402C13.6849 46.8725 13.7385 52.9008 17.9323 55.2973L78.449 89.8784C80.3779 90.9807 82.7445 90.9867 84.6791 89.8943Z" fill="#11192B" stroke="url(#paint1_linear_4459_37617)" stroke-width="0.663881"/>
</mask>
<g mask="url(#mask0_4459_37617)">
<g filter="url(#filter3_f_4459_37617)">
<rect x="55.8655" y="56.9795" width="43.8063" height="15.8446" transform="rotate(180 55.8655 56.9795)" fill="#8874B2"/>
</g>
<g filter="url(#filter4_f_4459_37617)">
<rect x="139.261" y="82.3198" width="43.8063" height="32.7457" transform="rotate(180 139.261 82.3198)" fill="#8874B2"/>
</g>
</g>
</g>
<g filter="url(#filter5_di_4459_37617)">
<path d="M87.578 47.3682C87.5701 46.458 86.311 45.7311 84.7658 45.7446L78.4142 45.8C76.869 45.8135 75.6228 46.5623 75.6308 47.4725L75.6634 51.2139C75.6714 52.1241 76.9304 52.851 78.4756 52.8375L84.8273 52.7821C86.3724 52.7686 87.6186 52.0198 87.6107 51.1096L87.578 47.3682Z" fill="#A950FF"/>
<path d="M72.9007 38.8646C72.8929 37.9707 71.6564 37.2568 70.1388 37.27L63.6871 37.3263C62.1696 37.3396 60.9457 38.0749 60.9535 38.9688L60.9866 42.7692C60.9944 43.6631 62.231 44.377 63.7485 44.3638L70.2003 44.3075C71.7178 44.2942 72.9417 43.5589 72.9339 42.665L72.9007 38.8646Z" fill="#A950FF"/>
<path d="M102.105 38.61C102.097 37.7161 100.861 37.0022 99.343 37.0154L92.8912 37.0717C91.3737 37.085 90.1498 37.8203 90.1576 38.7142L90.1908 42.5146C90.1986 43.4085 91.4351 44.1224 92.9526 44.1092L99.4044 44.0529C100.922 44.0396 102.146 43.3043 102.138 42.4104L102.105 38.61Z" fill="#E9C1FF"/>
<path d="M93.0156 45.6723C91.4704 45.6857 90.2242 46.4345 90.2322 47.3447L90.2648 51.0861C90.2728 51.9963 91.5318 52.7232 93.077 52.7097L99.4287 52.6543C100.974 52.6408 102.22 51.892 102.212 50.9818L102.179 47.2404C102.171 46.3303 100.912 45.6033 99.3672 45.6168L93.0156 45.6723Z" fill="#CD88FF"/>
<path d="M93.0413 54.274C91.5238 54.2872 90.2999 55.0226 90.3077 55.9165L90.3409 59.7169C90.3487 60.6108 91.5852 61.3247 93.1028 61.3115L99.5545 61.2552C101.072 61.2419 102.296 60.5065 102.288 59.6126L102.255 55.8123C102.247 54.9184 101.011 54.2045 99.4931 54.2177L93.0413 54.274Z" fill="#A950FF"/>
<path d="M78.3389 37.1986C76.7937 37.2121 75.5475 37.9609 75.5554 38.8711L75.5881 42.6125C75.596 43.5227 76.8551 44.2496 78.4003 44.2361L84.7519 44.1807C86.2971 44.1672 87.5433 43.4184 87.5353 42.5082L87.5027 38.7668C87.4947 37.8566 86.2357 37.1297 84.6905 37.1432L78.3389 37.1986Z" fill="#CD88FF"/>
<path d="M66.3843 54.7446C62.9323 54.7747 60.1482 56.4475 60.1659 58.481C60.1837 60.5144 62.9965 62.1384 66.4486 62.1083C69.9007 62.0781 72.6847 60.4053 72.667 58.3719C72.6493 56.3385 69.8364 54.7145 66.3843 54.7446Z" fill="#F62E76"/>
</g>
<g filter="url(#filter6_d_4459_37617)">
<path d="M119.151 55.6691L128.653 74.5943C128.776 74.8393 128.959 75.0494 129.184 75.2054C129.41 75.3614 129.671 75.4583 129.944 75.4873C130.216 75.5162 130.492 75.4763 130.745 75.3711C130.999 75.2659 131.221 75.0989 131.393 74.8853L133.178 65.0719L142.018 61.6387C142.187 61.4278 142.301 61.1783 142.35 60.9125C142.399 60.6467 142.382 60.3729 142.299 60.1155C142.217 59.8581 142.072 59.6252 141.877 59.4375C141.683 59.2498 141.445 59.1133 141.184 59.04L121.061 53.3653C120.756 53.2792 120.433 53.2835 120.13 53.3777C119.828 53.4719 119.559 53.6519 119.357 53.8958C119.154 54.1398 119.027 54.4372 118.991 54.752C118.955 55.0668 119.01 55.3854 119.151 55.6691Z" fill="white"/>
</g>
<defs>
<filter id="filter0_d_4459_37617" x="6.45604" y="15.4606" width="150.97" height="99.814" filterUnits="userSpaceOnUse" color-interpolation-filters="sRGB">
<feFlood flood-opacity="0" result="BackgroundImageFix"/>
<feColorMatrix in="SourceAlpha" type="matrix" values="0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 127 0" result="hardAlpha"/>
<feOffset dy="7.30269"/>
<feGaussianBlur stdDeviation="3.98328"/>
<feComposite in2="hardAlpha" operator="out"/>
<feColorMatrix type="matrix" values="0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0.25 0"/>
<feBlend mode="normal" in2="BackgroundImageFix" result="effect1_dropShadow_4459_37617"/>
<feBlend mode="normal" in="SourceGraphic" in2="effect1_dropShadow_4459_37617" result="shape"/>
</filter>
<filter id="filter1_d_4459_37617" x="4.4644" y="0.852951" width="154.953" height="103.797" filterUnits="userSpaceOnUse" color-interpolation-filters="sRGB">
<feFlood flood-opacity="0" result="BackgroundImageFix"/>
<feColorMatrix in="SourceAlpha" type="matrix" values="0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 127 0" result="hardAlpha"/>
<feOffset dy="1.32776"/>
<feGaussianBlur stdDeviation="4.9791"/>
<feComposite in2="hardAlpha" operator="out"/>
<feColorMatrix type="matrix" values="0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0.25 0"/>
<feBlend mode="normal" in2="BackgroundImageFix" result="effect1_dropShadow_4459_37617"/>
<feBlend mode="normal" in="SourceGraphic" in2="effect1_dropShadow_4459_37617" result="shape"/>
</filter>
<filter id="filter2_d_4459_37617" x="11.4351" y="5.50094" width="141.011" height="89.8558" filterUnits="userSpaceOnUse" color-interpolation-filters="sRGB">
<feFlood flood-opacity="0" result="BackgroundImageFix"/>
<feColorMatrix in="SourceAlpha" type="matrix" values="0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 127 0" result="hardAlpha"/>
<feOffset dy="1.32776"/>
<feGaussianBlur stdDeviation="1.49373"/>
<feComposite in2="hardAlpha" operator="out"/>
<feColorMatrix type="matrix" values="0 0 0 0 0.253904 0 0 0 0 0.093523 0 0 0 0 0.968327 0 0 0 0.25 0"/>
<feBlend mode="normal" in2="BackgroundImageFix" result="effect1_dropShadow_4459_37617"/>
<feBlend mode="normal" in="SourceGraphic" in2="effect1_dropShadow_4459_37617" result="shape"/>
</filter>
<filter id="filter3_f_4459_37617" x="-31.6608" y="-2.5851" width="131.246" height="103.284" filterUnits="userSpaceOnUse" color-interpolation-filters="sRGB">
<feFlood flood-opacity="0" result="BackgroundImageFix"/>
<feBlend mode="normal" in="SourceGraphic" in2="BackgroundImageFix" result="shape"/>
<feGaussianBlur stdDeviation="21.8599" result="effect1_foregroundBlur_4459_37617"/>
</filter>
<filter id="filter4_f_4459_37617" x="51.7347" y="5.85435" width="131.246" height="120.185" filterUnits="userSpaceOnUse" color-interpolation-filters="sRGB">
<feFlood flood-opacity="0" result="BackgroundImageFix"/>
<feBlend mode="normal" in="SourceGraphic" in2="BackgroundImageFix" result="shape"/>
<feGaussianBlur stdDeviation="21.8599" result="effect1_foregroundBlur_4459_37617"/>
</filter>
<filter id="filter5_di_4459_37617" x="58.838" y="37.0151" width="44.7778" height="27.7493" filterUnits="userSpaceOnUse" color-interpolation-filters="sRGB">
<feFlood flood-opacity="0" result="BackgroundImageFix"/>
<feColorMatrix in="SourceAlpha" type="matrix" values="0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 127 0" result="hardAlpha"/>
<feOffset dy="1.32776"/>
<feGaussianBlur stdDeviation="0.663881"/>
<feComposite in2="hardAlpha" operator="out"/>
<feColorMatrix type="matrix" values="0 0 0 0 0.222242 0 0 0 0 0.247729 0 0 0 0 0.604557 0 0 0 0.6 0"/>
<feBlend mode="normal" in2="BackgroundImageFix" result="effect1_dropShadow_4459_37617"/>
<feBlend mode="normal" in="SourceGraphic" in2="effect1_dropShadow_4459_37617" result="shape"/>
<feColorMatrix in="SourceAlpha" type="matrix" values="0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 127 0" result="hardAlpha"/>
<feOffset dy="0.995821"/>
<feGaussianBlur stdDeviation="1.32776"/>
<feComposite in2="hardAlpha" operator="arithmetic" k2="-1" k3="1"/>
<feColorMatrix type="matrix" values="0 0 0 0 1 0 0 0 0 0.923763 0 0 0 0 0.923763 0 0 0 0.45 0"/>
<feBlend mode="normal" in2="shape" result="effect2_innerShadow_4459_37617"/>
</filter>
<filter id="filter6_d_4459_37617" x="110.407" y="48.1595" width="40.5442" height="39.3401" filterUnits="userSpaceOnUse" color-interpolation-filters="sRGB">
<feFlood flood-opacity="0" result="BackgroundImageFix"/>
<feColorMatrix in="SourceAlpha" type="matrix" values="0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 127 0" result="hardAlpha"/>
<feOffset dy="3.42944"/>
<feGaussianBlur stdDeviation="4.2868"/>
<feComposite in2="hardAlpha" operator="out"/>
<feColorMatrix type="matrix" values="0 0 0 0 0.0969792 0 0 0 0 0.127894 0 0 0 0 0.2375 0 0 0 1 0"/>
<feBlend mode="normal" in2="BackgroundImageFix" result="effect1_dropShadow_4459_37617"/>
<feBlend mode="normal" in="SourceGraphic" in2="effect1_dropShadow_4459_37617" result="shape"/>
</filter>
<linearGradient id="paint0_linear_4459_37617" x1="109.337" y1="36.0149" x2="77.2234" y2="84.8818" gradientUnits="userSpaceOnUse">
<stop stop-color="#F9C1FF"/>
<stop offset="1" stop-color="#A950FF"/>
</linearGradient>
<linearGradient id="paint1_linear_4459_37617" x1="109.337" y1="36.0149" x2="77.2234" y2="84.8818" gradientUnits="userSpaceOnUse">
<stop stop-color="#F9C1FF"/>
<stop offset="1" stop-color="#A950FF"/>
</linearGradient>
<clipPath id="clip0_4459_37617">
<rect width="148.229" height="85.9522" fill="white" transform="matrix(-1 0 0 1 156.114 6.08545)"/>
</clipPath>
</defs>
</svg>

After

Width:  |  Height:  |  Size: 11 KiB

View File

@@ -1,3 +1,3 @@
* Explore our [documentation](https://kestra.io/docs)
* Learn the [concepts](https://kestra.io/docs/concepts)
* Browse Kestra [integrations](https://kestra.io/plugins)
* [Video Tutorials](https://kestra.io/tutorial-videos/all)
* [Documentation](https://kestra.io/docs)
* [Blueprints](https://kestra.io/blueprints)

View File

@@ -1,3 +1 @@
Our community of data engineers and developers are here to help.
[Join our Slack](https://kestra.io/slack)
Ask any question in our Slack community. If you are stuck, we are help to help you. ✋

View File

@@ -1,3 +1 @@
Follow each step one by one with this advanced tutorial.
[Follow the tutorial](/ui/flows/new?reset=true)
Chhose your use case and follow a step-by-step guide to learn Kestra 's features and capabilities. ❤️

View File

@@ -8,29 +8,11 @@
:total="total"
>
<template #navbar>
<el-form-item>
<search-field />
</el-form-item>
<el-form-item>
<namespace-select
data-type="flow"
:value="$route.query.namespace"
@update:model-value="onDataTableValue('namespace', $event)"
/>
</el-form-item>
<el-form-item>
<el-select v-model="state" clearable :placeholder="$t('triggers_state.state')">
<el-option
v-for="(s, index) in states"
:key="index"
:label="s.label"
:value="s.value"
/>
</el-select>
</el-form-item>
<el-form-item>
<refresh-button @refresh="load(onDataLoaded)" />
</el-form-item>
<KestraFilter
prefix="triggers"
:include="['namespace', 'trigger_state']"
:refresh="{shown: true, callback: load}"
/>
</template>
<template #table>
<select-table
@@ -84,7 +66,13 @@
sortable="custom"
:sort-orders="['ascending', 'descending']"
:label="$t('id')"
/>
>
<template #default="scope">
<div class="text-nowrap">
{{ scope.row.id }}
</div>
</template>
</el-table-column>
<el-table-column
v-if="visibleColumns.flowId"
prop="flowId"
@@ -145,7 +133,13 @@
<date-ago :inverted="true" :date="scope.row.updatedDate" />
</template>
</el-table-column>
<el-table-column v-if="visibleColumns.nextExecutionDate" :label="$t('next execution date')">
<el-table-column
v-if="visibleColumns.nextExecutionDate"
prop="nextExecutionDate"
sortable="custom"
:sort-orders="['ascending', 'descending']"
:label="$t('next execution date')"
>
<template #default="scope">
<date-ago :inverted="true" :date="scope.row.nextExecutionDate" />
</template>
@@ -266,29 +260,25 @@
import TriggerAvatar from "../flows/TriggerAvatar.vue"
</script>
<script>
import NamespaceSelect from "../namespace/NamespaceSelect.vue";
import RouteContext from "../../mixins/routeContext";
import RestoreUrl from "../../mixins/restoreUrl";
import SearchField from "../layout/SearchField.vue";
import DataTable from "../layout/DataTable.vue";
import DataTableActions from "../../mixins/dataTableActions";
import MarkdownTooltip from "../layout/MarkdownTooltip.vue";
import RefreshButton from "../layout/RefreshButton.vue";
import DateAgo from "../layout/DateAgo.vue";
import Id from "../Id.vue";
import {mapState} from "vuex";
import SelectTableActions from "../../mixins/selectTableActions";
import _merge from "lodash/merge";
import LogsWrapper from "../logs/LogsWrapper.vue";
import KestraFilter from "../filter/KestraFilter.vue"
export default {
mixins: [RouteContext, RestoreUrl, DataTableActions, SelectTableActions],
components: {
RefreshButton,
KestraFilter,
MarkdownTooltip,
DataTable,
SearchField,
NamespaceSelect,
DateAgo,
Id,
LogsWrapper
@@ -475,9 +465,9 @@
}
})
if (!this.state) return all;
if(!this.$route.query.trigger_state?.length) return all;
const disabled = this.state === "DISABLED" ? true : false;
const disabled = this.$route.query?.trigger_state?.[0] === "disabled" ? true : false;
return all.filter(trigger => trigger.disabled === disabled);
},
visibleColumns() {

View File

@@ -648,4 +648,20 @@ $spacing: 20px;
}
}
}
:deep(.legend) {
&::-webkit-scrollbar {
height: 5px;
width: 5px;
}
&::-webkit-scrollbar-track {
background: var(--card-bg);
}
&::-webkit-scrollbar-thumb {
background: var(--bs-primary);
border-radius: 0px;
}
}
</style>

View File

@@ -12,14 +12,14 @@ const getOrCreateLegendList = (chart, id, direction = "row") => {
if (!listContainer) {
listContainer = document.createElement("ul");
listContainer.classList.add("fw-light", "small");
listContainer.classList.add("w-100", "fw-light", "small", "legend");
listContainer.style.display = "flex";
listContainer.style.flexDirection = direction;
listContainer.style.margin = 0;
listContainer.style.padding = 0;
listContainer.style.maxHeight = "200px";
listContainer.style.flexWrap = "wrap";
listContainer.style.maxHeight = "196px"; // 4 visible items
listContainer.style.overflow = "auto";
legendContainer?.appendChild(listContainer);
}

View File

@@ -186,5 +186,4 @@ code {
}
}
}
</style>

View File

@@ -48,6 +48,7 @@
"execution/loadFlowForExecutionByExecutionId",
{
id: execution.id,
revision: this.$route.query.revision
}
);
}

View File

@@ -73,40 +73,48 @@
@update:select-all="toggleAllSelection"
@unselect="toggleAllUnselected"
>
<!-- Always visible buttons -->
<el-button v-if="canUpdate" :icon="StateMachine" @click="changeStatusDialogVisible = !changeStatusDialogVisible">
{{ $t("change state") }}
</el-button>
<el-button v-if="canUpdate" :icon="Restart" @click="restartExecutions()">
{{ $t("restart") }}
</el-button>
<el-button v-if="canCreate" :icon="PlayBoxMultiple" @click="replayExecutions()">
{{ $t("replay") }}
</el-button>
<el-button v-if="canUpdate" :icon="StateMachine" @click="changeStatusDialogVisible = !changeStatusDialogVisible">
{{ $t("change state") }}
</el-button>
<el-button v-if="canUpdate" :icon="StopCircleOutline" @click="killExecutions()">
{{ $t("kill") }}
</el-button>
<el-button v-if="canDelete" :icon="Delete" @click="deleteExecutions()">
{{ $t("delete") }}
</el-button>
<el-button
v-if="canUpdate"
:icon="LabelMultiple"
@click="isOpenLabelsModal = !isOpenLabelsModal"
>
{{ $t("Set labels") }}
</el-button>
<el-button v-if="canUpdate" :icon="PlayBox" @click="resumeExecutions()">
{{ $t("resume") }}
</el-button>
<el-button v-if="canUpdate" :icon="PauseBox" @click="pauseExecutions()">
{{ $t("pause") }}
</el-button>
<el-button v-if="canUpdate" :icon="QueueFirstInLastOut" @click="unqueueExecutions()">
{{ $t("unqueue") }}
</el-button>
<el-button v-if="canUpdate" :icon="RunFast" @click="forceRunExecutions()">
{{ $t("force run") }}
</el-button>
<!-- Dropdown with additional actions -->
<el-dropdown>
<el-button>
<DotsVertical />
</el-button>
<template #dropdown>
<el-dropdown-menu>
<el-dropdown-item v-if="canUpdate" :icon="LabelMultiple" @click=" isOpenLabelsModal = !isOpenLabelsModal">
{{ $t("Set labels") }}
</el-dropdown-item>
<el-dropdown-item v-if="canUpdate" :icon="PlayBox" @click="resumeExecutions()">
{{ $t("resume") }}
</el-dropdown-item>
<el-dropdown-item v-if="canUpdate" :icon="PauseBox" @click="pauseExecutions()">
{{ $t("pause") }}
</el-dropdown-item>
<el-dropdown-item v-if="canUpdate" :icon="QueueFirstInLastOut" @click="unqueueExecutions()">
{{ $t("unqueue") }}
</el-dropdown-item>
<el-dropdown-item v-if="canUpdate" :icon="RunFast" @click="forceRunExecutions()">
{{ $t("force run") }}
</el-dropdown-item>
</el-dropdown-menu>
</template>
</el-dropdown>
</bulk-select>
<el-dialog
v-if="isOpenLabelsModal"
@@ -285,7 +293,7 @@
<el-table-column column-key="action" class-name="row-action">
<template #default="scope">
<router-link
:to="{name: 'executions/update', params: {namespace: scope.row.namespace, flowId: scope.row.flowId, id: scope.row.id}}"
:to="{name: 'executions/update', params: {namespace: scope.row.namespace, flowId: scope.row.flowId, id: scope.row.id}, query: {revision: scope.row.flowRevision}}"
>
<kicon :tooltip="$t('details')" placement="left">
<TextSearch />
@@ -344,6 +352,7 @@
import SelectTable from "../layout/SelectTable.vue";
import PlayBox from "vue-material-design-icons/PlayBox.vue";
import PlayBoxMultiple from "vue-material-design-icons/PlayBoxMultiple.vue";
import DotsVertical from "vue-material-design-icons/DotsVertical.vue";
import Restart from "vue-material-design-icons/Restart.vue";
import Delete from "vue-material-design-icons/Delete.vue";
import StopCircleOutline from "vue-material-design-icons/StopCircleOutline.vue";

View File

@@ -27,6 +27,7 @@
@previous="previousLogForLevel(logLevel)"
@next="nextLogForLevel(logLevel)"
@close="logCursor = undefined"
class="w-100"
/>
</el-form-item>
<el-form-item>

View File

@@ -127,13 +127,28 @@
const debugEditor = ref(null);
const debugExpression = ref("");
const computedDebugValue = computed(() => {
const task = selectedTask()?.taskId;
if(!task) return "";
const formatTask = (task) => {
if (!task) return "";
return task.includes("-") ? `["${task}"]` : `.${task}`;
};
const path = expandedValue.value;
if(!path) return `{{ outputs.${task} }}`
const formatPath = (path) => {
if (!path.includes("-")) return `.${path}`;
return `{{ outputs.${path} }}`
const bracketIndex = path.indexOf("[");
const task = path.substring(0, bracketIndex);
const rest = path.substring(bracketIndex);
return `["${task}"]${rest}`;
}
let task = selectedTask()?.taskId;
if (!task) return "";
let path = expandedValue.value;
if (!path) return `{{ outputs${formatTask(task)} }}`;
return `{{ outputs${formatPath(path)} }}`;
});
const debugError = ref("");
const debugStackTrace = ref("");
@@ -225,11 +240,18 @@
if (!task) return;
selected.value = [task.value];
expandedValue.value = task.value;
const child = task.children?.[1];
if (child) {
selected.value.push(child.value);
expandedValue.value = child.path
expandedValue.value = child.path;
const grandChild = child.children?.[1];
if (grandChild) {
selected.value.push(grandChild.value);
expandedValue.value = grandChild.path;
}
}
debugCollapse.value = "debug";

View File

@@ -53,6 +53,11 @@
:key="comparator.value"
:value="comparator"
:label="comparator.label"
:class="{
selected: current.some(
(c) => c.comparator === comparator,
),
}"
@click="() => comparatorCallback(comparator)"
/>
</template>
@@ -62,6 +67,11 @@
:key="filter.value"
:value="filter"
:label="filter.label"
:class="{
selected: current.some((c) =>
c.value.includes(filter.value),
),
}"
@click="() => valueCallback(filter)"
/>
</template>
@@ -105,13 +115,13 @@
import Magnify from "vue-material-design-icons/Magnify.vue";
import State from "../../utils/state.js";
import DateRange from "../layout/DateRange.vue";
const emits = defineEmits(["dashboard"]);
const props = defineProps({
prefix: {type: String, required: true},
include: {type: Array, required: true},
values: {type: Object, required: false, default: undefined},
refresh: {
type: Object,
default: () => ({shown: false, callback: () => {}}),
@@ -140,6 +150,9 @@
} = useFilters(props.prefix);
const select = ref<InstanceType<typeof ElSelect> | null>(null);
const updateHoveringIndex = (index) => {
select.value.states.hoveringIndex = index >= 0 ? index : 0;
};
const emptyLabel = ref(t("filters.empty"));
const INITIAL_DROPDOWNS = {
first: {shown: true, value: {}},
@@ -208,7 +221,9 @@
dropdowns.value.second = {shown: false, index: -1};
dropdowns.value.third = {shown: true, index: current.value.length - 1};
select.value.states.hoveringIndex = 0;
// Set hover index to the selected comparator for highlighting
const index = valueOptions.value.findIndex((o) => o.value === value.value);
updateHoveringIndex(index);
};
const dropdownClosedCallback = (visible) => {
if (!visible) {
@@ -216,6 +231,12 @@
// If last filter item selection was not completed, remove it from array
if (current.value?.at(-1)?.value?.length === 0) current.value.pop();
} else {
// Highlight all selected items by setting hoveringIndex to match the first selected item
const index = valueOptions.value.findIndex((o) => {
return current.value.some((c) => c.value.includes(o.value));
});
updateHoveringIndex(index);
}
};
const valueCallback = (filter, isDate = false) => {
@@ -225,6 +246,12 @@
if (index === -1) values.push(filter.value);
else values.splice(index, 1);
// Update the hover index for better UX
const hoverIndex = valueOptions.value.findIndex(
(o) => o.value === filter.value,
);
updateHoveringIndex(hoverIndex);
} else {
const match = current.value.find((v) => v.label === "absolute_date");
if (match) match.value = [filter];
@@ -272,51 +299,8 @@
// Load all namespaces only if that filter is included
if (props.include.includes("namespace")) loadNamespaces();
const scopeOptions = [
{
label: t("scope_filter.user", {label: props.prefix}),
value: "USER",
},
{
label: t("scope_filter.system", {label: props.prefix}),
value: "SYSTEM",
},
];
const childOptions = [
{
label: t("trigger filter.options.ALL"),
value: "ALL",
},
{
label: t("trigger filter.options.CHILD"),
value: "CHILD",
},
{
label: t("trigger filter.options.MAIN"),
value: "MAIN",
},
];
const levelOptions = [
{label: "TRACE", value: "TRACE"},
{label: "DEBUG", value: "DEBUG"},
{label: "INFO", value: "INFO"},
{label: "WARN", value: "WARN"},
{label: "ERROR", value: "ERROR"},
];
const relativeDateOptions = [
{label: t("datepicker.last5minutes"), value: "PT5M"},
{label: t("datepicker.last15minutes"), value: "PT15M"},
{label: t("datepicker.last1hour"), value: "PT1H"},
{label: t("datepicker.last12hours"), value: "PT12H"},
{label: t("datepicker.last24hours"), value: "PT24H"},
{label: t("datepicker.last48hours"), value: "PT48H"},
{label: t("datepicker.last7days"), value: "PT168H"},
{label: t("datepicker.last30days"), value: "PT720H"},
{label: t("datepicker.last365days"), value: "PT8760H"},
];
import {useValues} from "./useValues.js";
const {VALUES} = useValues(props.prefix);
const isDatePickerShown = computed(() => {
const c = current?.value?.at(-1);
@@ -330,23 +314,32 @@
case "namespace":
return namespaceOptions.value;
case "scope":
return scopeOptions;
case "state":
return State.arrayAllStates().map((s) => ({
label: s.name,
value: s.name,
}));
return VALUES.EXECUTION_STATE;
case "trigger_state":
return VALUES.TRIGGER_STATE;
case "scope":
return VALUES.SCOPE;
case "child":
return childOptions;
return VALUES.CHILD;
case "level":
return levelOptions;
return VALUES.LEVEL;
case "relative_date":
return relativeDateOptions;
return VALUES.RELATIVE_DATE;
case "task":
return props.values?.task || [];
case "metric":
return props.values?.metric || [];
case "aggregation":
return VALUES.AGGREGATION;
case "absolute_date":
return [];
@@ -429,14 +422,20 @@
// Include paramters from URL directly to filter
current.value = decodeParams(route.query, props.include);
if (route.name === "flows/update" && route.params.namespace) {
const addNamespaceFilter = (namespace) => {
if (!namespace) return;
current.value.push({
label: "namespace",
value: [route.params.namespace],
value: [namespace],
comparator: COMPARATORS.STARTS_WITH,
persistent: true,
});
}
};
const {name, params} = route;
if (name === "flows/update") addNamespaceFilter(params?.namespace);
else if (name === "namespaces/update") addNamespaceFilter(params.id);
</script>
<style lang="scss">

View File

@@ -1,8 +1,9 @@
<template>
<span v-if="label">{{ $t("filters.options." + label) }}</span>
<span v-if="label" class="text-lowercase">
{{ $t(`filters.options.${label}`) }}
</span>
<span v-if="comparator" class="comparator">{{ comparator }}</span>
<!-- TODO: Amend line below after merging issue: https://github.com/kestra-io/kestra/issues/5955 -->
<span v-if="value">{{ !comparator ? ":" : "" }}{{ value }}</span>
<span v-if="value">{{ value }}</span>
</template>
<script setup lang="ts">
@@ -10,8 +11,10 @@
const props = defineProps({option: {type: Object, required: true}});
const DATE_FORMATS: Intl.DateTimeFormatOptions = {timeStyle: "short", dateStyle: "short"};
const formatter = new Intl.DateTimeFormat("en-US", DATE_FORMATS);
import moment from "moment";
const DATE_FORMAT = localStorage.getItem("dateFormat") || "llll";
const formatter = (date) => moment(date).format(DATE_FORMAT);
const label = computed(() => props.option?.label);
const comparator = computed(() => props.option?.comparator?.label);
@@ -25,15 +28,15 @@
}
const {startDate, endDate} = value[0];
return `${startDate ? formatter.format(new Date(startDate)) : "unknown"}:and:${endDate ? formatter.format(new Date(endDate)) : "unknown"}`;
return `${startDate ? formatter(new Date(startDate)) : "unknown"}:and:${endDate ? formatter(new Date(endDate)) : "unknown"}`;
});
</script>
<style lang="scss" scoped>
.comparator {
background: var(--bs-gray-500);
padding: 0.30rem 0.35rem;
margin: 0 0.5rem;
display: inline-block;
}
.comparator {
background: var(--bs-gray-500);
padding: 0.3rem 0.35rem;
margin: 0 0.5rem;
display: inline-block;
}
</style>

View File

@@ -1,7 +1,11 @@
import {useI18n} from "vue-i18n";
import DotsSquare from "vue-material-design-icons/DotsSquare.vue";
import TagOutline from "vue-material-design-icons/TagOutline.vue";
import MathLog from "vue-material-design-icons/MathLog.vue";
import Sigma from "vue-material-design-icons/Sigma.vue";
import TimelineTextOutline from "vue-material-design-icons/TimelineTextOutline.vue";
import ChartBar from "vue-material-design-icons/ChartBar.vue";
import CalendarRangeOutline from "vue-material-design-icons/CalendarRangeOutline.vue";
import CalendarEndOutline from "vue-material-design-icons/CalendarEndOutline.vue";
import FilterVariantMinus from "vue-material-design-icons/FilterVariantMinus.vue";
@@ -89,6 +93,13 @@ export function useFilters(prefix) {
value: {label: "state", comparator: undefined, value: []},
comparators: [COMPARATORS.IS_ONE_OF],
},
{
key: "trigger_state",
icon: StateMachine,
label: t("filters.options.state"),
value: {label: "trigger_state", comparator: undefined, value: []},
comparators: [COMPARATORS.IS],
},
{
key: "scope",
icon: FilterSettingsOutline,
@@ -96,13 +107,6 @@ export function useFilters(prefix) {
value: {label: "scope", comparator: undefined, value: []},
comparators: [COMPARATORS.IS_ONE_OF],
},
{
key: "labels",
icon: TagOutline,
label: t("filters.options.labels"),
value: {label: "labels", comparator: undefined, value: []},
comparators: [COMPARATORS.CONTAINS],
},
{
key: "childFilter",
icon: FilterVariantMinus,
@@ -117,6 +121,27 @@ export function useFilters(prefix) {
value: {label: "level", comparator: undefined, value: []},
comparators: [COMPARATORS.IS],
},
{
key: "task",
icon: TimelineTextOutline,
label: t("filters.options.task"),
value: {label: "task", comparator: undefined, value: []},
comparators: [COMPARATORS.IS],
},
{
key: "metric",
icon: ChartBar,
label: t("filters.options.metric"),
value: {label: "metric", comparator: undefined, value: []},
comparators: [COMPARATORS.IS],
},
{
key: "aggregation",
icon: Sigma,
label: t("filters.options.aggregation"),
value: {label: "aggregation", comparator: undefined, value: []},
comparators: [COMPARATORS.IS],
},
{
key: "timeRange",
icon: CalendarRangeOutline,
@@ -131,6 +156,13 @@ export function useFilters(prefix) {
value: {label: "absolute_date", comparator: undefined, value: []},
comparators: [COMPARATORS.BETWEEN],
},
{
key: "labels",
icon: TagOutline,
label: t("filters.options.labels"),
value: {label: "labels", comparator: undefined, value: []},
comparators: [COMPARATORS.CONTAINS],
},
];
const encodeParams = (filters) => {
const encode = (values, key) => {
@@ -196,6 +228,7 @@ export function useFilters(prefix) {
});
}
// TODO: Will need tweaking once we introduce multiple comparators for filters
return params.map((p) => {
const comparator = OPTIONS.find((o) => o.value.label === p.label);
return {...p, comparator: comparator?.comparators?.[0]};

View File

@@ -0,0 +1,50 @@
import {useI18n} from "vue-i18n";
import State from "../../utils/state.js";
export function useValues(label?: string) {
const {t} = useI18n({useScope: "global"});
const VALUES = {
SCOPE: [
{label: t("scope_filter.user", {label}), value: "USER"},
{label: t("scope_filter.system", {label}), value: "SYSTEM"},
],
CHILD: [
{label: t("trigger filter.options.ALL"), value: "ALL"},
{label: t("trigger filter.options.CHILD"), value: "CHILD"},
{label: t("trigger filter.options.MAIN"), value: "MAIN"},
],
LEVEL: ["TRACE", "DEBUG", "INFO", "WARN", "ERROR"].map((value) => ({
label: value,
value,
})),
RELATIVE_DATE: [
{label: t("datepicker.last5minutes"), value: "PT5M"},
{label: t("datepicker.last15minutes"), value: "PT15M"},
{label: t("datepicker.last1hour"), value: "PT1H"},
{label: t("datepicker.last12hours"), value: "PT12H"},
{label: t("datepicker.last24hours"), value: "PT24H"},
{label: t("datepicker.last48hours"), value: "PT48H"},
{label: t("datepicker.last7days"), value: "PT168H"},
{label: t("datepicker.last30days"), value: "PT720H"},
{label: t("datepicker.last365days"), value: "PT8760H"},
],
EXECUTION_STATE: State.arrayAllStates().map(
(state: { name: string }) => ({
label: state.name,
value: state.name,
}),
),
AGGREGATION: ["sum", "avg", "min", "max"].map((value) => ({
label: value,
value,
})),
TRIGGER_STATE: ["enabled", "disabled"].map((value) => ({
label: `${value.charAt(0).toUpperCase()}${value.slice(1)}`,
value,
})),
};
return {VALUES};
}

View File

@@ -6,6 +6,7 @@
:flow-id="flowParsed?.id"
:namespace="flowParsed?.namespace"
:is-creating="true"
:flow-validation="flowValidation"
:flow-graph="flowGraph"
:is-read-only="false"
:is-dirty="true"
@@ -74,7 +75,7 @@ tasks:
sourceWrapper() {
return {source: this.source};
},
...mapState("flow", ["flowGraph", "total"]),
...mapState("flow", ["flowGraph"]),
...mapState("auth", ["user"]),
...mapState("plugin", ["pluginSingleList", "pluginsDocumentation"]),
...mapGetters("core", ["guidedProperties"]),

View File

@@ -1,72 +1,25 @@
<template>
<nav>
<collapse>
<el-form-item>
<el-select
:model-value="$route.query.task"
filterable
:persistent="false"
:placeholder="$t('task')"
clearable
@update:model-value="updateQuery({'task': $event})"
>
<el-option
v-for="item in tasksWithMetrics"
:key="item"
:label="item"
:value="item"
>
{{ item }}
</el-option>
</el-select>
</el-form-item>
<el-form-item>
<el-select
:model-value="$route.query.metric"
filterable
:clearable="true"
:persistent="false"
:placeholder="$t('metric')"
@update:model-value="updateQuery({'metric': $event})"
>
<el-option
v-for="item in metrics"
:key="item"
:label="item"
:value="item"
>
{{ item }}
</el-option>
</el-select>
</el-form-item>
<el-form-item>
<el-select
:model-value="$route.query.aggregation"
filterable
:clearable="true"
:persistent="false"
:placeholder="$t('aggregation')"
@update:model-value="updateQuery({'aggregation': $event})"
>
<el-option
v-for="item in ['sum','avg','min','max']"
:key="item"
:label="$t(item)"
:value="item"
/>
</el-select>
</el-form-item>
<el-form-item>
<date-filter
@update:is-relative="onDateFilterTypeChange"
@update:filter-value="updateQuery"
/>
</el-form-item>
<el-form-item>
<refresh-button @refresh="load" :can-auto-refresh="canAutoRefresh" />
</el-form-item>
</collapse>
</nav>
<KestraFilter
prefix="flow_metrics"
:include="[
'task',
'metric',
'aggregation',
'relative_date',
'absolute_date',
]"
:values="{
task: tasksWithMetrics.map((value) => ({
label: value,
value,
})),
metric: metrics.map((value) => ({
label: value,
value,
})),
}"
:refresh="{shown: true, callback: load}"
/>
<div v-bind="$attrs" v-loading="isLoading">
<el-card>
@@ -76,13 +29,20 @@
:persistent="false"
:hide-after="0"
transition=""
:popper-class="tooltipContent === '' ? 'd-none' : 'tooltip-stats'"
:popper-class="
tooltipContent === '' ? 'd-none' : 'tooltip-stats'
"
v-if="aggregatedMetric"
>
<template #content>
<span v-html="tooltipContent" />
</template>
<Bar ref="chartRef" :data="chartData" :options="options" v-if="aggregatedMetric" />
<Bar
ref="chartRef"
:data="chartData"
:options="options"
v-if="aggregatedMetric"
/>
</el-tooltip>
<span v-else>
<el-alert type="info" :closable="false">
@@ -99,58 +59,79 @@
import moment from "moment";
import {defaultConfig, getFormat, tooltip} from "../../utils/charts";
import {cssVariable} from "@kestra-io/ui-libs/src/utils/global";
import Collapse from "../layout/Collapse.vue";
import DateFilter from "../executions/date-select/DateFilter.vue";
import RefreshButton from "../layout/RefreshButton.vue";
import KestraFilter from "../filter/KestraFilter.vue";
export default {
name: "FlowMetrics",
components: {
Collapse,
Bar,
DateFilter,
RefreshButton
KestraFilter,
},
created() {
this.loadMetrics();
},
computed: {
...mapState("flow", ["flow", "metrics", "aggregatedMetric","tasksWithMetrics"]),
...mapState("flow", [
"flow",
"metrics",
"aggregatedMetric",
"tasksWithMetrics",
]),
theme() {
return localStorage.getItem("theme") || "light";
},
xGrid() {
return this.theme === "light" ?
{}
return this.theme === "light"
? {}
: {
borderColor: "#404559",
color: "#404559"
}
color: "#404559",
};
},
yGrid() {
return this.theme === "light" ?
{}
return this.theme === "light"
? {}
: {
borderColor: "#404559",
color: "#404559"
}
color: "#404559",
};
},
chartData() {
return {
labels: this.aggregatedMetric.aggregations.map(e => moment(e.date).format(getFormat(this.aggregatedMetric.groupBy))),
labels: this.aggregatedMetric.aggregations.map((e) =>
moment(e.date).format(
getFormat(this.aggregatedMetric.groupBy),
),
),
datasets: [
!this.display ? [] : {
label: this.$t(this.$route.query.aggregation.toLowerCase()) + " " + this.$t("of") + " " + this.$route.query.metric,
backgroundColor: cssVariable("--el-color-success"),
borderRadius: 4,
data: this.aggregatedMetric.aggregations.map(e => e.value ? e.value : 0)
}
]
!this.display
? []
: {
label:
this.$t(this.$route.query.aggregation) +
" " +
this.$t("of") +
" " +
this.$route.query.metric,
backgroundColor:
cssVariable("--el-color-success"),
borderRadius: 4,
data: this.aggregatedMetric.aggregations.map(
(e) => (e.value ? e.value : 0),
),
},
],
};
},
options() {
const darken = this.theme === "light" ? cssVariable("--bs-gray-700") : cssVariable("--bs-gray-800");
const lighten = this.theme === "light" ? cssVariable("--bs-gray-200") : cssVariable("--bs-gray-400");
const darken =
this.theme === "light"
? cssVariable("--bs-gray-700")
: cssVariable("--bs-gray-800");
const lighten =
this.theme === "light"
? cssVariable("--bs-gray-200")
: cssVariable("--bs-gray-400");
return defaultConfig({
plugins: {
@@ -158,7 +139,7 @@
external: (context) => {
this.tooltipContent = tooltip(context.tooltip);
},
}
},
},
scales: {
x: {
@@ -166,59 +147,38 @@
grid: {
borderColor: lighten,
color: lighten,
drawTicks: false
drawTicks: false,
},
ticks: {
color: darken,
autoSkip: true,
minRotation: 0,
maxRotation: 0,
}
},
},
y: {
display: true,
grid: {
borderColor: lighten,
color: lighten,
drawTicks: false
drawTicks: false,
},
ticks: {
color: darken
}
}
}
})
color: darken,
},
},
},
});
},
display() {
return this.$route.query.metric && this.$route.query.aggregation;
},
endDate() {
if (this.$route.query.endDate) {
return this.$route.query.endDate;
}
return undefined;
},
startDate() {
// This allow to force refresh this computed property especially when using timeRange
this.refreshDates;
if (this.$route.query.startDate) {
return this.$route.query.startDate;
}
if (this.$route.query.timeRange) {
return this.$moment().subtract(this.$moment.duration(this.$route.query.timeRange).as("milliseconds")).toISOString(true);
}
// the default is PT30D
return this.$moment().subtract(30, "days").toISOString(true);
}
},
data() {
return {
tooltipContent: undefined,
isLoading: false,
canAutoRefresh: false,
refreshDates: false
}
};
},
methods: {
onDateFilterTypeChange(event) {
@@ -228,23 +188,35 @@
return {
...base,
startDate: this.startDate,
endDate: this.endDate
}
endDate: this.endDate,
};
},
loadMetrics() {
this.$store.dispatch("flow/loadTasksWithMetrics",{...this.$route.params})
this.$store.dispatch("flow/loadTasksWithMetrics", {
...this.$route.params,
});
this.$store
.dispatch(this.$route.query.task ? "flow/loadTaskMetrics" : "flow/loadFlowMetrics", this.loadQuery({
...this.$route.params,
taskId: this.$route.query.task,
}))
.dispatch(
this.$route.query.task
? "flow/loadTaskMetrics"
: "flow/loadFlowMetrics",
this.loadQuery({
...this.$route.params,
taskId: this.$route.query.task,
}),
)
.then(() => {
if (this.metrics.length > 0) {
if (this.$route.query.metric && !this.metrics.includes(this.$route.query.metric)) {
if (
this.$route.query.metric &&
!this.metrics.includes(this.$route.query.metric)
) {
let query = {...this.$route.query};
delete query.metric;
this.$router.push({query: query}).then(_ => this.loadAggregatedMetrics());
this.$router
.push({query: query})
.then((_) => this.loadAggregatedMetrics());
} else {
this.loadAggregatedMetrics();
}
@@ -255,15 +227,20 @@
this.isLoading = true;
if (this.display) {
this.$store.dispatch(this.$route.query.task ? "flow/loadTaskAggregatedMetrics" : "flow/loadFlowAggregatedMetrics", this.loadQuery({
...this.$route.params,
...this.$route.query,
metric: this.$route.query.metric,
aggregate: this.$route.query.aggregation,
taskId: this.$route.query.task
}))
this.$store.dispatch(
this.$route.query.task
? "flow/loadTaskAggregatedMetrics"
: "flow/loadFlowAggregatedMetrics",
this.loadQuery({
...this.$route.params,
...this.$route.query,
metric: this.$route.query.metric,
aggregate: this.$route.query.aggregation,
taskId: this.$route.query.task,
}),
);
} else {
this.$store.commit("flow/setAggregatedMetric", undefined)
this.$store.commit("flow/setAggregatedMetric", undefined);
}
this.isLoading = false;
},
@@ -271,7 +248,7 @@
let query = {...this.$route.query};
for (const [key, value] of Object.entries(queryParam)) {
if (value === undefined || value === "" || value === null) {
delete query[key]
delete query[key];
} else {
query[key] = value;
}
@@ -283,17 +260,27 @@
if (!this.$route.query.metric) {
this.loadMetrics();
} else {
this.refreshDates = !this.refreshDates;
this.loadAggregatedMetrics();
}
}
}
}
},
},
watch: {
"$route.query": {
handler(query) {
if (!query.metric) {
this.loadMetrics();
} else {
this.loadAggregatedMetrics();
}
},
},
},
};
</script>
<style>
.navbar-flow-metrics {
display: flex;
width: 100%;
}
</style>
.navbar-flow-metrics {
display: flex;
width: 100%;
}
</style>

View File

@@ -132,7 +132,7 @@
},
fillInputsFromExecution(){
// Add all labels except the one from flow to prevent duplicates
this.executionLabels = this.getExecutionLabels();
this.executionLabels = this.getExecutionLabels().filter(item => !item.key.startsWith("system."));
if (!this.flow.inputs) {
return;

View File

@@ -524,6 +524,7 @@
.custom-dark-vs-theme {
.monaco-editor, .monaco-editor-background {
outline: none;
background-color: $input-bg;
--vscode-editor-background: $input-bg;
--vscode-breadcrumb-background: $input-bg;

View File

@@ -1,5 +1,10 @@
<template>
<div v-show="explorerVisible" class="p-3 sidebar" @click="$refs.tree.setCurrentKey(undefined)">
<div
v-show="explorerVisible"
class="p-3 sidebar"
@click="$refs.tree.setCurrentKey(undefined)"
@contextmenu.prevent="onTabContextMenu"
>
<div class="d-flex flex-row">
<el-select
v-model="filter"
@@ -29,10 +34,7 @@
:persistent="false"
popper-class="text-base"
>
<el-button
class="px-2"
@click="toggleDialog(true, 'file')"
>
<el-button class="px-2" @click="toggleDialog(true, 'file')">
<FilePlus />
</el-button>
</el-tooltip>
@@ -94,10 +96,7 @@
:persistent="false"
popper-class="text-base"
>
<el-button
class="px-2"
@click="exportFiles()"
>
<el-button class="px-2" @click="exportFiles()">
<FolderDownloadOutline />
</el-button>
</el-tooltip>
@@ -110,7 +109,9 @@
:load="loadNodes"
:data="items"
highlight-current
:allow-drop="(_, drop, dropType) => !drop.data?.leaf || dropType !== 'inner'"
:allow-drop="
(_, drop, dropType) => !drop.data?.leaf || dropType !== 'inner'
"
draggable
node-key="id"
v-loading="items === undefined"
@@ -122,12 +123,17 @@
? changeOpenedTabs({
action: 'open',
name: data.fileName,
extension: data.fileName.split('.')[1],
extension: data.fileName.split('.').pop(),
path: getPath(node),
})
: undefined
"
@node-drag-start="nodeBeforeDrag = {parent: $event.parent.data.id, path: getPath($event.data.id)}"
@node-drag-start="
nodeBeforeDrag = {
parent: $event.parent.data.id,
path: getPath($event.data.id),
}
"
@node-drop="nodeMoved"
@keydown.delete.prevent="deleteKeystroke"
>
@@ -141,13 +147,19 @@
<template #default="{data, node}">
<el-dropdown
:ref="`dropdown__${data.id}`"
@contextmenu.prevent.stop="toggleDropdown(`dropdown__${data.id}`)"
@contextmenu.prevent.stop="
toggleDropdown(`dropdown__${data.id}`)
"
trigger="contextmenu"
class="w-100"
>
<el-row justify="space-between" class="w-100">
<el-col class="w-100">
<TypeIcon :name="data.fileName" :folder="!data.leaf" class="me-2" />
<TypeIcon
:name="data.fileName"
:folder="!data.leaf"
class="me-2"
/>
<span class="filename"> {{ data.fileName }}</span>
</el-col>
</el-row>
@@ -174,7 +186,7 @@
true,
!data.leaf ? 'folder' : 'file',
data.fileName,
node
node,
)
"
>
@@ -182,18 +194,16 @@
$t(
`namespace files.rename.${
!data.leaf ? "folder" : "file"
}`
}`,
)
}}
</el-dropdown-item>
<el-dropdown-item
@click="confirmRemove(node)"
>
<el-dropdown-item @click="confirmRemove(node)">
{{
$t(
`namespace files.delete.${
!data.leaf ? "folder" : "file"
}`
}`,
)
}}
</el-dropdown-item>
@@ -308,7 +318,7 @@
{{
Array.isArray(confirmation.node?.data?.children)
? $t(
"namespace files.dialog.folder_deletion_description"
"namespace files.dialog.folder_deletion_description",
)
: $t("namespace files.dialog.file_deletion_description")
}}
@@ -324,6 +334,22 @@
</div>
</template>
</el-dialog>
<el-menu
v-if="tabContextMenu.visible"
:style="{
left: `${tabContextMenu.x}px`,
top: `${tabContextMenu.y}px`,
}"
class="tabs-context"
>
<el-menu-item @click="toggleDialog(true, 'file')">
{{ $t("namespace files.create.file") }}
</el-menu-item>
<el-menu-item @click="toggleDialog(true, 'folder')">
{{ $t("namespace files.create.folder") }}
</el-menu-item>
</el-menu>
</div>
</template>
@@ -332,7 +358,7 @@
import Utils from "../../utils/utils";
import FileExplorerEmpty from "../../assets/icons/file_explorer_empty.svg"
import FileExplorerEmpty from "../../assets/icons/file_explorer_empty.svg";
import Magnify from "vue-material-design-icons/Magnify.vue";
import FilePlus from "vue-material-design-icons/FilePlus.vue";
@@ -340,7 +366,7 @@
import PlusBox from "vue-material-design-icons/PlusBox.vue";
import FolderDownloadOutline from "vue-material-design-icons/FolderDownloadOutline.vue";
import TypeIcon from "../utils/icons/Type.vue"
import TypeIcon from "../utils/icons/Type.vue";
const DIALOG_DEFAULTS = {
visible: false,
@@ -361,8 +387,8 @@
props: {
currentNS: {
type: String,
default: null
}
default: null,
},
},
components: {
Magnify,
@@ -370,7 +396,7 @@
FolderPlus,
PlusBox,
FolderDownloadOutline,
TypeIcon
TypeIcon,
},
data() {
return {
@@ -385,7 +411,8 @@
confirmation: {visible: false, data: {}},
items: undefined,
nodeBeforeDrag: undefined,
searchResults: []
searchResults: [],
tabContextMenu: {visible: false, x: 0, y: 0},
};
},
computed: {
@@ -401,7 +428,12 @@
if (item.type === "Directory") {
const folderPath = `${basePath}${item.fileName}`;
paths.push(folderPath);
paths.push(...extractPaths(`${folderPath}/`, item.children ?? []));
paths.push(
...extractPaths(
`${folderPath}/`,
item.children ?? [],
),
);
}
});
return paths;
@@ -411,7 +443,10 @@
},
},
methods: {
...mapMutations("editor", ["toggleExplorerVisibility", "changeOpenedTabs"]),
...mapMutations("editor", [
"toggleExplorerVisibility",
"changeOpenedTabs",
]),
...mapActions("namespace", [
"createDirectory",
"readDirectory",
@@ -425,14 +460,23 @@
]),
sorted(items) {
return items.sort((a, b) => {
if (a.type === "Directory" && b.type !== "Directory")
return -1;
if (a.type === "Directory" && b.type !== "Directory") return -1;
else if (a.type !== "Directory" && b.type === "Directory")
return 1;
return a.fileName.localeCompare(b.fileName);
});
},
getFileNameWithExtension(fileNameWithExtension) {
const lastDotIdx = fileNameWithExtension.lastIndexOf(".");
return lastDotIdx !== -1
? [
fileNameWithExtension.slice(0, lastDotIdx),
fileNameWithExtension.slice(lastDotIdx + 1),
]
: [fileNameWithExtension, ""];
},
renderNodes(items) {
if (this.items === undefined) {
this.items = [];
@@ -443,7 +487,9 @@
if (type === "Directory") {
this.addFolder({fileName});
} else if (type === "File") {
const [fileName, extension] = items[i].fileName.split(".");
const [fileName, extension] = this.getFileNameWithExtension(
items[i].fileName,
);
const file = {fileName, extension, leaf: true};
this.addFile({file});
}
@@ -451,11 +497,13 @@
},
async loadNodes(node, resolve) {
if (node.level === 0) {
const payload = {namespace: this.currentNS ?? this.$route.params.namespace};
const payload = {
namespace: this.currentNS ?? this.$route.params.namespace,
};
const items = await this.readDirectory(payload);
this.renderNodes(items);
this.items = this.sorted(this.items)
this.items = this.sorted(this.items);
}
if (node.level >= 1) {
@@ -470,7 +518,7 @@
...item,
id: Utils.uid(),
leaf: item.type === "File",
}))
})),
);
// eslint-disable-next-line no-inner-declarations
@@ -481,34 +529,39 @@
items[index].children = newChildren;
} else if (Array.isArray(item.children)) {
// Recursively search in children array
updateChildren(
item.children,
path,
newChildren
);
updateChildren(item.children, path, newChildren);
}
});
}
};
updateChildren(this.items, this.getPath(node.data.id), children);
updateChildren(
this.items,
this.getPath(node.data.id),
children,
);
resolve(children);
}
},
async searchFilesList(value) {
if(!value) return;
if (!value) return;
const results = await this.searchFiles({namespace: this.currentNS ?? this.$route.params.namespace, query: value});
this.searchResults = results.map(result => result.replace(/^\/*/, ""));
const results = await this.searchFiles({
namespace: this.currentNS ?? this.$route.params.namespace,
query: value,
});
this.searchResults = results.map((result) =>
result.replace(/^\/*/, ""),
);
return this.searchResults;
},
chooseSearchResults(item){
chooseSearchResults(item) {
this.changeOpenedTabs({
action: "open",
name: item.split("/").pop(),
extension: item.split(".")[1],
extension: item.split(".").pop(),
path: item,
})
});
this.filter = "";
},
@@ -586,7 +639,10 @@
});
} catch (e) {
this.$refs.tree.remove(draggedNode.data.id);
this.$refs.tree.append(draggedNode.data, this.nodeBeforeDrag.parent);
this.$refs.tree.append(
draggedNode.data,
this.nodeBeforeDrag.parent,
);
}
},
focusCreationInput() {
@@ -628,7 +684,7 @@
const folderIndex = currentFolder.findIndex(
(item) =>
typeof item === "object" &&
item.fileName === folderName
item.fileName === folderName,
);
if (folderIndex === -1) {
// If the folder doesn't exist, create it
@@ -636,7 +692,7 @@
id: Utils.uid(),
fileName: folderName,
children: [],
type: "Directory"
type: "Directory",
};
currentFolder.push(newFolder);
this.sorted(currentFolder);
@@ -650,13 +706,15 @@
// Extract file details
const fileName = pathParts[pathParts.length - 1];
const [name, extension] = fileName.split(".");
const [name, extension] =
this.getFileNameWithExtension(fileName);
// Read file content
const content = await this.readFile(file);
this.importFileDirectory({
namespace: this.currentNS ?? this.$route.params.namespace,
namespace:
this.currentNS ?? this.$route.params.namespace,
content,
path: `${folderPath}/${fileName}`,
});
@@ -668,15 +726,18 @@
extension ? `.${extension}` : ""
}`,
extension,
type: "File"
type: "File",
});
} else {
// Process files at root level (not in any folder)
const content = await this.readFile(file);
const [name, extension] = file.name.split(".");
const [name, extension] = this.getFileNameWithExtension(
file.name,
);
this.importFileDirectory({
namespace: this.currentNS ?? this.$route.params.namespace,
namespace:
this.currentNS ?? this.$route.params.namespace,
content,
path: file.name,
});
@@ -688,13 +749,13 @@
}`,
extension,
leaf: !!extension,
type: "File"
type: "File",
});
}
}
this.$toast().success(
this.$t("namespace files.import.success")
this.$t("namespace files.import.success"),
);
} catch (error) {
this.$toast().error(this.$t("namespace files.import.error"));
@@ -705,20 +766,17 @@
}
},
exportFiles() {
this.exportFileDirectory({namespace: this.currentNS ?? this.$route.params.namespace});
this.exportFileDirectory({
namespace: this.currentNS ?? this.$route.params.namespace,
});
},
async addFile({file, creation, shouldReset = true}) {
let FILE;
if (creation) {
const separateString = (str) => {
const lastIndex = str.lastIndexOf(".");
return lastIndex !== -1
? [str.slice(0, lastIndex), str.slice(lastIndex + 1)]
: [str, ""];
};
const [fileName, extension] = separateString(this.dialog.name);
const [fileName, extension] = this.getFileNameWithExtension(
this.dialog.name,
);
FILE = {fileName, extension, content: "", leaf: true};
} else {
@@ -733,14 +791,15 @@
extension,
content,
leaf,
type: "File"
type: "File",
};
const path = `${this.dialog.folder ? `${this.dialog.folder}/` : ""}${NAME}`;
if (creation) {
if ((await this.searchFilesList(path)).includes(path)) {
this.$toast().error(this.$t("namespace files.create.already_exists"));
this.$toast().error(
this.$t("namespace files.create.already_exists"),
);
return;
}
await this.createFile({
@@ -755,7 +814,7 @@
action: "open",
name: NAME,
path,
extension: extension
extension: extension,
});
this.dialog.folder = path.substring(0, path.lastIndexOf("/"));
@@ -767,16 +826,28 @@
} else {
const SELF = this;
(function pushItemToFolder(basePath = "", array, pathParts) {
for (const item of array) {
const folderPath = `${basePath}${item.fileName}`;
if (folderPath === SELF.dialog.folder && Array.isArray(item.children)) {
item.children = SELF.sorted([...item.children, NEW]);
if (
folderPath === SELF.dialog.folder &&
Array.isArray(item.children)
) {
item.children = SELF.sorted([
...item.children,
NEW,
]);
return true; // Return true if the folder is found and item is pushed
}
if (Array.isArray(item.children) && pushItemToFolder(`${folderPath}/`, item.children, pathParts.slice(1))) {
if (
Array.isArray(item.children) &&
pushItemToFolder(
`${folderPath}/`,
item.children,
pathParts.slice(1),
)
) {
// Return true if the folder is found and item is pushed in recursive call
return true;
}
@@ -787,7 +858,9 @@
const folderPath = `${basePath}${pathParts[0]}`;
if (folderPath === SELF.dialog.folder) {
const newFolder = SELF.folderNode(pathParts[0], [NEW]);
const newFolder = SELF.folderNode(pathParts[0], [
NEW,
]);
array.push(newFolder);
array = SELF.sorted(array);
@@ -797,7 +870,11 @@
array.push(newFolder);
array = SELF.sorted(array);
return pushItemToFolder(`${basePath}${pathParts[0]}/`, newFolder.children, pathParts.slice(1));
return pushItemToFolder(
`${basePath}${pathParts[0]}/`,
newFolder.children,
pathParts.slice(1),
);
}
return false;
@@ -812,7 +889,10 @@
this.confirmation = {visible: true, node};
},
async removeItem() {
const {node, node: {data}} = this.confirmation;
const {
node,
node: {data},
} = this.confirmation;
await this.deleteFileDirectory({
namespace: this.currentNS ?? this.$route.params.namespace,
@@ -832,7 +912,11 @@
},
deleteKeystroke() {
if (this.$refs.tree.getCurrentNode()) {
this.confirmRemove(this.$refs.tree.getNode(this.$refs.tree.getCurrentNode().id));
this.confirmRemove(
this.$refs.tree.getNode(
this.$refs.tree.getCurrentNode().id,
),
);
}
},
async addFolder(folder, creation) {
@@ -842,7 +926,7 @@
fileName: this.dialog.name,
};
const NEW = this.folderNode(fileName, folder?.children ?? [])
const NEW = this.folderNode(fileName, folder?.children ?? []);
if (creation) {
const path = `${
@@ -873,7 +957,12 @@
item.children = SELF.sorted(item.children);
return true; // Return true if the folder is found and item is pushed
} else if (Array.isArray(item.children)) {
if (pushItemToFolder(`${folderPath}/`, item.children)) {
if (
pushItemToFolder(
`${folderPath}/`,
item.children,
)
) {
return true; // Return true if the folder is found and item is pushed in recursive call
}
}
@@ -890,8 +979,8 @@
fileName,
leaf: false,
children: children ?? [],
type: "Directory"
}
type: "Directory",
};
},
getPath(name) {
const nodes = this.$refs.tree.getNodePath(name);
@@ -906,7 +995,20 @@
} catch (_error) {
this.$toast().error(this.$t("namespace files.path.error"));
}
}
},
onTabContextMenu(event) {
this.tabContextMenu = {
visible: true,
x: event.clientX,
y: event.clientY,
};
document.addEventListener("click", this.hideTabContextMenu);
},
hideTabContextMenu() {
this.tabContextMenu.visible = false;
document.removeEventListener("click", this.hideTabContextMenu);
},
},
watch: {
flow: {
@@ -929,99 +1031,116 @@
</script>
<style lang="scss">
.filter .el-input__wrapper {
padding-right: 0px;
.filter .el-input__wrapper {
padding-right: 0px;
}
.el-tree {
height: calc(100% - 64px);
overflow: hidden auto;
.el-tree__empty-block {
height: auto;
}
.el-tree {
height: calc(100% - 64px);
overflow: hidden auto;
&::-webkit-scrollbar {
width: 2px;
}
.el-tree__empty-block {
height: auto;
}
&::-webkit-scrollbar-track {
background: var(--card-bg);
}
&::-webkit-scrollbar {
width: 2px;
}
&::-webkit-scrollbar-thumb {
background: var(--bs-primary);
border-radius: 0px;
}
&::-webkit-scrollbar-track {
background: var(--card-bg);
}
.node {
--el-tree-node-content-height: 36px;
--el-tree-node-hover-bg-color: transparent;
line-height: 36px;
&::-webkit-scrollbar-thumb {
background: var(--bs-primary);
border-radius: 0px;
}
.node {
--el-tree-node-content-height: 36px;
--el-tree-node-hover-bg-color: transparent;
line-height: 36px;
.el-tree-node__content {
width: 100%;
}
.el-tree-node__content {
width: 100%;
}
}
}
</style>
<style lang="scss" scoped>
@import "@kestra-io/ui-libs/src/scss/variables.scss";
@import "@kestra-io/ui-libs/src/scss/variables.scss";
.sidebar {
background: var(--card-bg);
border-right: 1px solid var(--bs-border-color);
.sidebar {
background: var(--card-bg);
border-right: 1px solid var(--bs-border-color);
.empty {
position: relative;
top: 100px;
text-align: center;
color: white;
.empty {
position: relative;
top: 100px;
text-align: center;
color: white;
html.light & {
color: $tertiary;
}
& img {
margin-bottom: 2rem;
}
& h3 {
font-size: var(--font-size-lg);
font-weight: 500;
margin-bottom: .5rem;
}
& p {
font-size: var(--font-size-sm);
}
html.light & {
color: $tertiary;
}
:deep(.el-button):not(.el-dialog .el-button) {
border: 0;
background: none;
outline: none;
opacity: 0.5;
padding-left: calc(var(--spacer) / 2);
padding-right: calc(var(--spacer) / 2);
&.el-button--primary {
opacity: 1;
}
& img {
margin-bottom: 2rem;
}
.hidden {
display: none;
& h3 {
font-size: var(--font-size-lg);
font-weight: 500;
margin-bottom: 0.5rem;
}
.filename {
& p {
font-size: var(--font-size-sm);
}
}
:deep(.el-button):not(.el-dialog .el-button) {
border: 0;
background: none;
outline: none;
opacity: 0.5;
padding-left: calc(var(--spacer) / 2);
padding-right: calc(var(--spacer) / 2);
&.el-button--primary {
opacity: 1;
}
}
.hidden {
display: none;
}
.filename {
font-size: var(--el-font-size-small);
color: var(--el-text-color-regular);
&:hover {
color: var(--el-text-color-primary);
}
}
ul.tabs-context {
position: fixed;
z-index: 9999;
border: 1px solid var(--bs-border-color);
& li {
height: 30px;
padding: 16px;
font-size: var(--el-font-size-small);
color: var(--el-text-color-regular);
color: var(--bs-gray-900);
&:hover {
color: var(--el-text-color-primary);
color: var(--bs-secondary);
}
}
}
</style>
}
</style>

View File

@@ -720,6 +720,10 @@
height: 100%;
outline: none;
}
.main-editor > #editorWrapper .monaco-editor {
padding: 1rem 0 0 1rem;
}
</style>
<style lang="scss">

View File

@@ -1,5 +1,5 @@
<template>
<nav data-component="FILENAME_PLACEHOLDER" class="d-flex w-100 gap-3 top-bar" v-if="displayNavBar">
<nav data-component="FILENAME_PLACEHOLDER" class="d-flex w-100 gap-3 top-bar">
<div class="d-flex flex-column flex-grow-1 flex-shrink-1 overflow-hidden top-title">
<el-breadcrumb v-if="breadcrumb">
<el-breadcrumb-item v-for="(item, x) in breadcrumb" :key="x">
@@ -74,9 +74,6 @@
...mapState("bookmarks", ["pages"]),
...mapGetters("core", ["guidedProperties"]),
...mapGetters("auth", ["user"]),
displayNavBar() {
return this.$route?.name !== "welcome";
},
tourEnabled(){
// Temporary solution to not showing the tour menu item for EE
return this.tutorialFlows?.length && !Object.keys(this.user).length

View File

@@ -1,23 +1,14 @@
<template>
<el-row justify="space-between" :gutter="20">
<el-col
<div class="onboarding-bottom">
<onboarding-card
v-for="card in cards"
:key="card.title"
:xs="24"
:sm="12"
:md="12"
:lg="6"
:xl="6"
class="pb-4"
>
<onboarding-card
:title="card.title"
:content="card.content"
:category="card.category"
:link="card.link"
/>
</el-col>
</el-row>
:title="card.title"
:content="card.content"
:category="card.category"
:link="card.link"
/>
</div>
</template>
<script>
import {mapGetters} from "vuex";
@@ -30,17 +21,13 @@
data() {
return {
cards: [
{
title: this.$t("welcome.started.title"),
category: "started",
},
{
title: this.$t("welcome.product-tour.title"),
category: "product",
},
{
title: this.$t("welcome.doc.title"),
title: this.$t("welcome.tutorial.title"),
category: "docs",
},
{
@@ -54,4 +41,15 @@
...mapGetters("core", ["guidedProperties"])
}
}
</script>
</script>
<style lang="scss" scoped>
.onboarding-bottom {
display: flex;
gap: 1rem;
margin-top: 1.5rem;
justify-items: center;
flex-wrap: wrap;
max-width: 1000px;
}
</style>

View File

@@ -1,22 +1,35 @@
<template>
<el-card>
<template #header>
<img :src="img" alt="">
</template>
<div class="content row">
<p class="fw-bold text-uppercase smaller-text">
{{ title }}
</p>
<markdown :source="mdContent" class="mt-4" />
<el-card class="box-card">
<div class="card-content">
<div class="card-header">
<el-link v-if="isOpenInNewCategory" :underline="false" :icon="OpenInNew" :href="getLink()" target="_blank" />
</div>
<div class="icon-title">
<el-icon size="25px">
<component :is="getIcon()" />
</el-icon>
<div class="card">
<h5 class="cat_title">
{{ title }}
</h5>
<div class="cat_description">
<markdown :source="mdContent" />
</div>
</div>
</div>
</div>
</el-card>
</template>
<script setup>
import OpenInNew from "vue-material-design-icons/OpenInNew.vue"
import Monitor from "vue-material-design-icons/Monitor.vue"
import Slack from "vue-material-design-icons/Slack.vue"
import PlayBox from "vue-material-design-icons/PlayBoxMultiple.vue"
</script>
<script>
import imageStarted from "../../assets/onboarding/onboarding-started-dark.svg"
import imageHelp from "../../assets/onboarding/onboarding-help-dark.svg"
import imageDoc from "../../assets/onboarding/onboarding-docs-dark.svg"
import imageProduct from "../../assets/onboarding/onboarding-product-dark.svg"
import Markdown from "../layout/Markdown.vue";
import Utils from "../../utils/utils.js";
@@ -47,6 +60,26 @@
.then((module) => {
this.markdownContent = module.default;
})
},
getIcon() {
switch (this.category) {
case "help":
return Slack;
case "docs":
return PlayBox;
case "product":
return Monitor;
default:
return Monitor;
}
},
getLink() {
// Define links for the specific categories
const links = {
help: "https://kestra.io/slack",
docs: "https://kestra.io/docs"
};
return links[this.category] || "#"; // Default to "#" if no link is found
}
},
computed: {
@@ -57,48 +90,67 @@
}
return ""
},
img() {
switch (this.category) {
case "started":
return imageStarted;
case "help":
return imageHelp;
case "docs":
return imageDoc;
case "product":
return imageProduct;
}
return imageStarted
},
mdContent() {
return this.markdownContent;
},
isOpenInNewCategory() {
// Define which categories should show the OpenInNew icon
return this.category === "help" || this.category === "docs";
}
}
}
</script>
<style scoped lang="scss">
a:hover {
text-decoration: none;
}
.el-card {
background-color: var(--card-bg);
border-color: var(--el-border-color);
box-shadow: var(--el-box-shadow);
position: relative;
min-width: 250px;
flex: 1;
cursor: pointer;
&:deep(.el-card__header) {
padding: 0;
}
position: relative;
height: 100%;
cursor: pointer;
}
.smaller-text {
font-size: 0.86em;
.box-card {
.card-header {
position: absolute;
top: 5px;
right: 5px;
}
.cat_title {
width: 100%;
margin: 3px 0 10px;
padding-left: 20px;
font-weight: 600;
font-size: var(--el-font-size-small);
}
.cat_description {
width: 100%;
margin: 0;
padding-left: 20px;
}
}
p {
margin-bottom: 0;
.icon-title {
display: inline-flex;
&.icon-title-left {
margin-right: 10px;
}
}
img {
width: 100%;
height: 100%;
.el-link {
font-size: 20px;
}
</style>

View File

@@ -1,90 +1,136 @@
<template>
<el-col class="main">
<el-row :gutter="20">
<el-col :xs="24" :sm="24" :md="24" :lg="14" :xl="14" class="mb-4">
<el-card class="px-3 pt-4">
<el-row justify="space-around" class="p-5">
<el-col :xs="24" :sm="24" :md="24" :lg="12" :xl="12" justify="space-between">
<el-row class="mb-5" justify="center">
<img class="img-fluid" :src="logo" alt="Kestra Logo">
</el-row>
<el-row justify="center">
<router-link :to="{name: 'flows/create'}">
<el-button size="large" type="primary">
<Plus />
{{ $t("welcome button create") }}
</el-button>
</router-link>
</el-row>
</el-col>
<el-col :xs="24" :sm="24" :md="24" :lg="12" :xl="12" justify="center" class="mt-4">
<img :src="codeImage" class="img-fluid" alt="code example">
</el-col>
</el-row>
</el-card>
</el-col>
<el-col :xs="24" :sm="24" :md="24" :lg="10" :xl="10" class="mb-4">
<iframe
width="100%"
height="100%"
src="https://www.youtube.com/embed/a2BZ7vOihjg?si=gHZuap7frp5c8HVx"
frameborder="0"
allow="accelerometer; autoplay; clipboard-write; encrypted-media; gyroscope; picture-in-picture; web-share"
allowfullscreen
/>
</el-col>
</el-row>
<onboarding-bottom />
</el-col>
<top-nav-bar v-if="topbar" :title="routeInfo.title">
<template #additional-right>
<ul>
<li>
<el-button v-if="canCreate" tag="router-link" :to="{name: 'flows/create', query: {namespace: $route.query.namespace}}" :icon="Plus" type="primary">
{{ $t('create_flow') }}
</el-button>
</li>
</ul>
</template>
</top-nav-bar>
<div class="main">
<div class="section-1">
<div class="section-1-main">
<div class="section-content">
<img
:src="logo"
alt="Kestra"
class="section-1-img img-fluid"
width="180px"
>
<h2 class="section-1-title">
{{ $t("homeDashboard.wel_text") }}
</h2>
<p class="section-1-desc">
{{ $t("homeDashboard.start") }}
</p>
<router-link :to="{name: 'flows/create'}">
<el-button
:icon="Plus"
size="large"
type="primary"
class="px-3 p-4 section-1-link product-link"
>
{{ $t("welcome button create") }}
</el-button>
</router-link>
<el-button
:icon="Play"
tag="a"
href="https://www.youtube.com/watch?v=a2BZ7vOihjg"
target="_blank"
class="p-3 px-4 mt-0 mb-lg-5 watch"
>
Watch Video
</el-button>
</div>
<div class="mid-bar mb-3">
<div class="title title--center-line">
{{ $t("homeDashboard.guide") }}
</div>
</div>
<onboarding-bottom />
</div>
</div>
</div>
</template>
<script>
import {mapGetters} from "vuex";
<script setup>
import Plus from "vue-material-design-icons/Plus.vue";
import Play from "vue-material-design-icons/Play.vue";
</script>
<script>
import {mapGetters, mapState} from "vuex";
import OnboardingBottom from "./OnboardingBottom.vue";
import onboardingImage from "../../assets/onboarding/onboarding-dark.svg"
import onboardingImageLight from "../../assets/onboarding/onboarding-light.svg"
import codeImageDark from "../../assets/onboarding/onboarding-code-dark.svg"
import codeImageLight from "../../assets/onboarding/onboarding-code-light.svg"
import kestraWelcome from "../../assets/onboarding/kestra_welcome.svg";
import TopNavBar from "../../components/layout/TopNavBar.vue";
import RouteContext from "../../mixins/routeContext";
import RestoreUrl from "../../mixins/restoreUrl";
import permission from "../../models/permission";
import action from "../../models/action";
export default {
name: "CreateFlow",
mixins: [RouteContext, RestoreUrl],
components: {
OnboardingBottom,
Plus
TopNavBar
},
data() {
return {
onboardingImage,
props: {
topbar: {
type: Boolean,
default: true
}
},
computed: {
...mapGetters("core", ["guidedProperties"]),
...mapState("auth", ["user"]),
logo() {
// get theme
return (localStorage.getItem("theme") || "light") === "light" ? onboardingImageLight : onboardingImage;
return (localStorage.getItem("theme") || "light") === "light" ? kestraWelcome : kestraWelcome;
},
codeImage() {
return (localStorage.getItem("theme") || "light") === "light" ? codeImageLight : codeImageDark;
routeInfo() {
return {
title: this.$t("homeDashboard.welcome")
};
},
canCreate() {
return this.user && this.user.hasAnyActionOnAnyNamespace(permission.FLOW, action.CREATE);
}
}
}
</script>
<style scoped lang="scss">
.main {
margin: 3rem 1rem 1rem;
padding: 3rem 1rem 1rem;
background: var(--el-text-color-primary);
background: radial-gradient(ellipse at top, rgba(102,51,255,0.6) 0%, rgba(253, 253, 253, 0) 20%);
background-size: 4000px;
background-position: center;
height: 100%;
width: auto;
display: flex;
flex-direction: column;
container-type: inline-size;
@media (min-width: 768px) {
margin: 3rem 2rem 1rem;
padding: 3rem 2rem 1rem;
}
@media (min-width: 992px) {
margin: 3rem 3rem 1rem;
padding: 3rem 3rem 1rem;
}
@media (min-width: 1920px) {
margin: 3rem 10rem 1rem;
padding: 3rem 10rem 1rem;
}
}
@@ -93,8 +139,114 @@
height: auto;
}
.el-button {
font-size: var(--font-size-lg);
margin-bottom: calc(var(--spacer) * 2);
.product-link, .watch {
background: var(--el-button-bg-color);
color: var(--el-button-text-color);
font-weight: 700;
border-radius: 5px;
border: 1px solid var(--el-button-border-color);
text-decoration: none;
font-size: var(--el-font-size-small);
width: 200px;
margin-bottom: calc(var(--spacer));
}
.watch {
font-weight: 500;
background-color: var(--el-bg-color);
color: var(--el-text-color-regular);
font-size: var(--el-font-size-small);
}
.main .section-1 {
display: flex;
flex-grow: 1;
justify-content: center;
align-items: center;
border-radius: var(--bs-border-radius);
}
.section-1-main {
.section-content {
width: 100%;
display: flex;
flex-direction: column;
align-items: center;
.section-1-title {
line-height: var(--el-font-line-height-primary);
text-align: center;
font-size: var(--el-font-size-extra-large);
font-weight: 600;
color: var(--el-text-color-regular);
}
.section-1-desc {
line-height: var(--el-font-line-height-primary);
font-weight: 500;
font-size: 1rem;
text-align: center;
color: var(--el-text-color-regular);
}
}
.mid-bar {
margin-top: 50px;
.title {
font-weight: 500;
color: var(--bs-gray-900-lighten-5);
display: flex;
align-items: center;
white-space: nowrap;
font-size: var(--el-font-size-extra-small);
&--center-line {
text-align: center;
padding: 0;
&::before,
&::after {
content: "";
background-color: var(--bs-gray-600-lighten-10);
height: 2px;
width: 50%;
}
&::before {
margin-right: 1rem;
}
&::after {
margin-left: 1rem;
}
}
}
}
}
@container (max-width: 20px) {
.main .section-1 .section-1-main {
width: 90%;
}
}
@container (max-width: 50px) {
.main .section-1 .section-1-main {
padding-top: 30px;
}
.section-1 .section-1-main .container {
width: 76%;
}
.title--center-line {
&::before,
&::after {
width: 50%;
}
}
}
</style>

View File

@@ -233,7 +233,7 @@
.plugin-card {
display: flex;
width: 232px;
width: 100%;
min-width: 130px;
padding: 8px 16px;
align-items: center;

View File

@@ -1,7 +1,7 @@
<template>
<div class="plugins-list">
<el-input
class="search p-2"
class="p-2 bg-transparent search"
:placeholder="$t('pluginPage.search', {count: countPlugin})"
v-model="searchInput"
clearable

View File

@@ -52,7 +52,7 @@
>
<div class="left">
<div>
<div class="title">
<div class="ps-0 title">
{{ blueprint.title }}
</div>
<div v-if="!system" class="tags text-uppercase">

View File

@@ -329,7 +329,8 @@ export default {
)
},
loadFlowForExecution({commit}, options) {
return this.$http.get(`${apiUrl(this)}/executions/flows/${options.namespace}/${options.flowId}`, {params: {revision: options.revision}})
const revision = options.revision ? `?revision=${options.revision}` : "";
return this.$http.get(`${apiUrl(this)}/executions/flows/${options.namespace}/${options.flowId}${revision}`)
.then(response => {
commit("setFlow", response.data)
return response.data;

View File

@@ -35,6 +35,7 @@ export default {
const cachedPluginDoc = state.pluginsDocumentation[options.cls];
if (!options.all && cachedPluginDoc) {
commit("setPlugin", cachedPluginDoc);
return Promise.resolve(cachedPluginDoc);
}

View File

@@ -360,7 +360,7 @@
},
"cancel": "Abbrechen",
"homeDashboard": {
"title": "Übersicht",
"title": "Dashboard",
"today": "Heute",
"yesterday": "Gestern",
"last28Days": "Letzte 28 Tage",
@@ -369,7 +369,11 @@
"namespacesErrorExecutions": "Ausführungsfehler pro Namespace",
"failedExecutions": "Fehlgeschlagene Ausführungen",
"errorLogs": "Fehler-Logs",
"no executions": "Bereit, Ihren Flow in Aktion zu sehen?"
"no executions": "Bereit, Ihren Flow in Aktion zu sehen?",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"welcome": "Welcome to Kestra"
},
"executions replayed": "<code>{executionCount}</code> Ausführung(en) erneut abgespielt",
"executions resumed": "<code>{executionCount}</code> Ausführung(en) fortgesetzt",
@@ -654,6 +658,9 @@
},
"need-help": {
"title": "Brauchen Sie Hilfe?"
},
"tutorial": {
"title": "Tutorial"
}
},
"pluginPage": {
@@ -819,7 +826,7 @@
},
"curl": {
"command": "cURL-Befehl",
"note": "Die Inputs vom Typ SECRET und FILE müssen immer bei der Ausführung angegeben werden."
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "Temporale Ansicht",
@@ -956,7 +963,11 @@
"child": "Kind",
"absolute_date": "Absolutes Datum",
"relative_date": "Relatives Datum",
"level": "Log-Ebene"
"level": "Log-Ebene",
"task": "Aufgabe",
"aggregation": "Aggregation",
"metric": "Metrik",
"trigger_state": "Zustand"
},
"empty": "Keine Daten.",
"label": "Filter auswählen",

View File

@@ -410,7 +410,11 @@
},
"cancel": "Cancel",
"homeDashboard": {
"title": "Overview",
"title": "Dashboard",
"welcome": "Welcome to Kestra",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"today": "Today",
"yesterday": "Yesterday",
"last28Days": "Last 28 days",
@@ -719,6 +723,9 @@
"doc": {
"title": "Docs"
},
"tutorial": {
"title": "Tutorial"
},
"need-help": {
"title": "Need help?"
}
@@ -888,7 +895,7 @@
},
"curl": {
"command": "cURL command",
"note": "For SECRET and FILE-type inputs, adjust the command to match the actual value."
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "Temporal view",
@@ -962,13 +969,17 @@
"options": {
"namespace": "Namespace",
"state": "State",
"trigger_state": "State",
"scope": "Scope",
"labels": "Labels",
"child": "Child",
"level": "Log level",
"task": "Task",
"metric": "Metric",
"aggregation": "Aggregation",
"relative_date": "Relative date",
"absolute_date": "Absolute date",
"text": "text"
"text": "text",
"labels": "Labels"
},
"comparators": {
"is": "is",

View File

@@ -360,7 +360,7 @@
},
"cancel": "Cancelar",
"homeDashboard": {
"title": "Resumen",
"title": "Dashboard",
"today": "Hoy",
"yesterday": "Ayer",
"last28Days": "Últimos 28 días",
@@ -369,7 +369,11 @@
"namespacesErrorExecutions": "Errores de ejecuciones por namespace",
"failedExecutions": "Ejecuciones FAILED",
"errorLogs": "Logs de errores",
"no executions": "¿Listo para ver tu flow en acción?"
"no executions": "¿Listo para ver tu flow en acción?",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"welcome": "Welcome to Kestra"
},
"executions replayed": "<code>{executionCount}</code> ejecución(es) reproducidas",
"executions resumed": "<code>{executionCount}</code> ejecución(es) reanudadas",
@@ -654,6 +658,9 @@
},
"need-help": {
"title": "¿Necesitas ayuda?"
},
"tutorial": {
"title": "Tutorial"
}
},
"pluginPage": {
@@ -819,7 +826,7 @@
},
"curl": {
"command": "Comando cURL",
"note": "Ten en cuenta que para el tipo de input SECRET y FILE, el comando debe adaptarse para coincidir con los valores reales."
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "Vista temporal",
@@ -956,7 +963,11 @@
"child": "Niño",
"absolute_date": "Fecha absoluta",
"relative_date": "Fecha relativa",
"level": "Nivel de log"
"level": "Nivel de log",
"task": "Tarea",
"aggregation": "Agregación",
"metric": "Métrica",
"trigger_state": "Estado"
},
"empty": "Sin datos.",
"label": "Elegir filtros",

View File

@@ -360,7 +360,7 @@
},
"cancel": "Annuler",
"homeDashboard": {
"title": "Vue d'ensemble",
"title": "Dashboard",
"today": "Aujourd'hui",
"yesterday": "Hier",
"last28Days": "Dernier 28 jours",
@@ -369,7 +369,11 @@
"namespacesErrorExecutions": "Exécutions en erreur par espace de nom",
"failedExecutions": "Exécutions en échec",
"errorLogs": "Journaux d'erreurs",
"no executions": "Prêt à voir votre flow Kestra en action ?"
"no executions": "Prêt à voir votre flow Kestra en action ?",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"welcome": "Welcome to Kestra"
},
"executions resumed": "<code>{executionCount}</code> exécution(s) reprise(s)",
"executions replayed": "<code>{executionCount}</code> exécution(s) rejouée(s)",
@@ -654,6 +658,9 @@
},
"need-help": {
"title": "Besoin d'aide ?"
},
"tutorial": {
"title": "Tutorial"
}
},
"pluginPage": {
@@ -819,7 +826,7 @@
},
"curl": {
"command": "Commande cURL",
"note": "Notez que pour les types d'entrée SECRET et FILE, la commande doit être adaptée pour correspondre aux valeurs réelles."
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "Vue temporelle",
@@ -956,7 +963,11 @@
"child": "Enfant",
"absolute_date": "Date absolue",
"relative_date": "Date relative",
"level": "Niveau de log"
"level": "Niveau de log",
"task": "Tâche",
"aggregation": "Agrégation",
"metric": "Métrique",
"trigger_state": "État"
},
"empty": "Aucune donnée.",
"label": "Choisir des filtres",

View File

@@ -360,7 +360,7 @@
},
"cancel": "रद्द करें",
"homeDashboard": {
"title": "सारांश",
"title": "Dashboard",
"today": "आज",
"yesterday": "कल",
"last28Days": "पिछले 28 दिन",
@@ -369,7 +369,11 @@
"namespacesErrorExecutions": "प्रति namespace निष्पादन त्रुटियाँ",
"failedExecutions": "असफल निष्पादन",
"errorLogs": "त्रुटि Logs",
"no executions": "क्या आप अपने flow को क्रियान्वित होते देखना चाहते हैं?"
"no executions": "क्या आप अपने flow को क्रियान्वित होते देखना चाहते हैं?",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"welcome": "Welcome to Kestra"
},
"executions replayed": "<code>{executionCount}</code> निष्पादन पुनः चलाए गए",
"executions resumed": "<code>{executionCount}</code> निष्पादन फिर से शुरू किए गए",
@@ -654,6 +658,9 @@
},
"need-help": {
"title": "मदद चाहिए?"
},
"tutorial": {
"title": "Tutorial"
}
},
"pluginPage": {
@@ -819,7 +826,7 @@
},
"curl": {
"command": "cURL कमांड",
"note": "ध्यान दें की SECRET और FILE इनपुट प्रकार के लिए, कमांड को वास्तविक मानों से मेल खाने के लिए समायोजित करना होगा।"
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "Temporal दृश्य",
@@ -956,7 +963,11 @@
"child": "बच्चा",
"absolute_date": "सटीक तिथि",
"relative_date": "सापेक्ष तिथि",
"level": "Log स्तर"
"level": "Log स्तर",
"task": "कार्य",
"aggregation": "एग्रीगेशन",
"metric": "मेट्रिक",
"trigger_state": "स्थिति"
},
"empty": "कोई डेटा नहीं।",
"label": "फिल्टर चुनें",

View File

@@ -360,7 +360,7 @@
},
"cancel": "Annulla",
"homeDashboard": {
"title": "Panoramica",
"title": "Dashboard",
"today": "Oggi",
"yesterday": "Ieri",
"last28Days": "Ultimi 28 giorni",
@@ -369,7 +369,11 @@
"namespacesErrorExecutions": "Errori di esecuzione per namespace",
"failedExecutions": "Esecuzioni FAILED",
"errorLogs": "Error logs",
"no executions": "Pronto a vedere il tuo flow in azione?"
"no executions": "Pronto a vedere il tuo flow in azione?",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"welcome": "Welcome to Kestra"
},
"executions replayed": "<code>{executionCount}</code> esecuzione/i ripetute",
"executions resumed": "<code>{executionCount}</code> esecuzione/i riprese",
@@ -654,6 +658,9 @@
},
"need-help": {
"title": "Hai bisogno di aiuto?"
},
"tutorial": {
"title": "Tutorial"
}
},
"pluginPage": {
@@ -819,7 +826,7 @@
},
"curl": {
"command": "Comando cURL",
"note": "Nota che per il tipo di input SECRET e FILE, il comando deve essere adattato per corrispondere ai valori reali."
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "Vista temporale",
@@ -956,7 +963,11 @@
"child": "Bambino",
"absolute_date": "Data assoluta",
"relative_date": "Data relativa",
"level": "Livello di Log"
"level": "Livello di Log",
"task": "Compito",
"aggregation": "Aggregazione",
"metric": "Metrica",
"trigger_state": "Stato"
},
"empty": "Nessun dato.",
"label": "Scegli filtri",

View File

@@ -360,7 +360,7 @@
},
"cancel": "キャンセル",
"homeDashboard": {
"title": "概要",
"title": "Dashboard",
"today": "今日",
"yesterday": "昨日",
"last28Days": "過去28日間",
@@ -369,7 +369,11 @@
"namespacesErrorExecutions": "namespaceごとの実行エラー",
"failedExecutions": "FAILED実行",
"errorLogs": "エラーログ",
"no executions": "flowを実行してみませんか"
"no executions": "flowを実行してみませんか",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"welcome": "Welcome to Kestra"
},
"executions replayed": "<code>{executionCount}</code>件の実行が再実行されました",
"executions resumed": "<code>{executionCount}</code>件の実行が再開されました",
@@ -654,6 +658,9 @@
},
"need-help": {
"title": "ヘルプが必要ですか?"
},
"tutorial": {
"title": "Tutorial"
}
},
"pluginPage": {
@@ -819,7 +826,7 @@
},
"curl": {
"command": "cURLコマンド",
"note": "SECRETおよびFILE入力タイプの場合、コマンドは実際の値に合わせて調整する必要があります。"
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "Temporalビュー",
@@ -956,7 +963,11 @@
"child": "子",
"absolute_date": "絶対日付",
"relative_date": "相対日付",
"level": "ログレベル"
"level": "ログレベル",
"task": "タスク",
"aggregation": "集計",
"metric": "メトリック",
"trigger_state": "状態"
},
"empty": "データがありません。",
"label": "フィルターを選択",

View File

@@ -360,7 +360,7 @@
},
"cancel": "취소",
"homeDashboard": {
"title": "개요",
"title": "Dashboard",
"today": "오늘",
"yesterday": "어제",
"last28Days": "지난 28일",
@@ -369,7 +369,11 @@
"namespacesErrorExecutions": "namespace별 실행 오류",
"failedExecutions": "실패한 실행",
"errorLogs": "오류 로그",
"no executions": "flow를 실행해 보시겠습니까?"
"no executions": "flow를 실행해 보시겠습니까?",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"welcome": "Welcome to Kestra"
},
"executions replayed": "<code>{executionCount}</code> 개의 실행이 재실행되었습니다",
"executions resumed": "<code>{executionCount}</code> 개의 실행이 재개되었습니다",
@@ -654,6 +658,9 @@
},
"need-help": {
"title": "도움이 필요하십니까?"
},
"tutorial": {
"title": "Tutorial"
}
},
"pluginPage": {
@@ -819,7 +826,7 @@
},
"curl": {
"command": "cURL 명령어",
"note": "SECRET FILE input 유형의 경우, 명령어는 실제 값에 맞게 조정되어야 합니다."
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "Temporal 보기",
@@ -956,7 +963,11 @@
"child": "자식",
"absolute_date": "절대 날짜",
"relative_date": "상대 날짜",
"level": "로그 레벨"
"level": "로그 레벨",
"task": "작업",
"aggregation": "집계",
"metric": "메트릭",
"trigger_state": "상태"
},
"empty": "데이터 없음.",
"label": "필터 선택",

View File

@@ -360,7 +360,7 @@
},
"cancel": "Anuluj",
"homeDashboard": {
"title": "Przegląd",
"title": "Dashboard",
"today": "Dzisiaj",
"yesterday": "Wczoraj",
"last28Days": "Ostatnie 28 dni",
@@ -369,7 +369,11 @@
"namespacesErrorExecutions": "Błędy wykonania na namespace",
"failedExecutions": "Nieudane wykonania",
"errorLogs": "Logi błędów",
"no executions": "Gotowy, aby zobaczyć swój flow w akcji?"
"no executions": "Gotowy, aby zobaczyć swój flow w akcji?",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"welcome": "Welcome to Kestra"
},
"executions replayed": "<code>{executionCount}</code> wykonanie(a) odtworzone",
"executions resumed": "<code>{executionCount}</code> wykonanie(a) wznowione",
@@ -654,6 +658,9 @@
},
"need-help": {
"title": "Potrzebujesz pomocy?"
},
"tutorial": {
"title": "Tutorial"
}
},
"pluginPage": {
@@ -819,7 +826,7 @@
},
"curl": {
"command": "Komenda cURL",
"note": "Zauważ, że dla typu input SECRET i FILE, komenda musi być dostosowana do rzeczywistych wartości."
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "Widok czasowy",
@@ -956,7 +963,11 @@
"child": "Dziecko",
"absolute_date": "Data bezwzględna",
"relative_date": "Data względna",
"level": "Poziom logowania"
"level": "Poziom logowania",
"task": "Zadanie",
"aggregation": "Agregacja",
"metric": "Metryka",
"trigger_state": "Stan"
},
"empty": "Brak danych.",
"label": "Wybierz filtry",

View File

@@ -360,7 +360,7 @@
},
"cancel": "Cancelar",
"homeDashboard": {
"title": "Visão Geral",
"title": "Dashboard",
"today": "Hoje",
"yesterday": "Ontem",
"last28Days": "Últimos 28 dias",
@@ -369,7 +369,11 @@
"namespacesErrorExecutions": "Erros de execuções por namespace",
"failedExecutions": "Execuções FAILED",
"errorLogs": "Logs de erros",
"no executions": "Pronto para ver seu flow em ação?"
"no executions": "Pronto para ver seu flow em ação?",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"welcome": "Welcome to Kestra"
},
"executions replayed": "<code>{executionCount}</code> execução(ões) reproduzida(s)",
"executions resumed": "<code>{executionCount}</code> execução(ões) retomada(s)",
@@ -654,6 +658,9 @@
},
"need-help": {
"title": "Precisa de ajuda?"
},
"tutorial": {
"title": "Tutorial"
}
},
"pluginPage": {
@@ -819,7 +826,7 @@
},
"curl": {
"command": "Comando cURL",
"note": "Note que para os tipos de input SECRET e FILE, o comando deve ser ajustado para corresponder aos valores reais."
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "Visualização temporal",
@@ -956,7 +963,11 @@
"child": "Filho",
"absolute_date": "Data absoluta",
"relative_date": "Data relativa",
"level": "Nível de Log"
"level": "Nível de Log",
"task": "Tarefa",
"aggregation": "Agregação",
"metric": "Métrica",
"trigger_state": "Estado"
},
"empty": "Sem dados.",
"label": "Escolher filtros",

View File

@@ -360,7 +360,7 @@
},
"cancel": "Отмена",
"homeDashboard": {
"title": "Обзор",
"title": "Dashboard",
"today": "Сегодня",
"yesterday": "Вчера",
"last28Days": "Последние 28 дней",
@@ -369,7 +369,11 @@
"namespacesErrorExecutions": "Ошибки выполнения по namespace",
"failedExecutions": "Неудачные выполнения",
"errorLogs": "Логи ошибок",
"no executions": "Готовы увидеть ваш flow в действии?"
"no executions": "Готовы увидеть ваш flow в действии?",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"welcome": "Welcome to Kestra"
},
"executions replayed": "<code>{executionCount}</code> выполнение(я) воспроизведено",
"executions resumed": "<code>{executionCount}</code> выполнение(я) возобновлено",
@@ -654,6 +658,9 @@
},
"need-help": {
"title": "Нужна помощь?"
},
"tutorial": {
"title": "Tutorial"
}
},
"pluginPage": {
@@ -819,7 +826,7 @@
},
"curl": {
"command": "Команда cURL",
"note": "Обратите внимание, что для типов input SECRET и FILE команда должна быть адаптирована для соответствия реальным значениям."
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "Временной вид",
@@ -956,7 +963,11 @@
"child": "Ребенок",
"absolute_date": "Абсолютная дата",
"relative_date": "Относительная дата",
"level": "Уровень Log"
"level": "Уровень Log",
"task": "Задача",
"aggregation": "Агрегация",
"metric": "Метрика",
"trigger_state": "Состояние"
},
"empty": "Нет данных.",
"label": "Выберите фильтры",

View File

@@ -360,7 +360,7 @@
},
"cancel": "取消",
"homeDashboard": {
"title": "概览",
"title": "Dashboard",
"today": "今天",
"yesterday": "昨天",
"last28Days": "最近28天",
@@ -369,7 +369,11 @@
"namespacesErrorExecutions": "每个命名空间的执行错误",
"failedExecutions": "失败的执行",
"errorLogs": "错误日志",
"no executions": "准备好看到你的流程在行动了吗?"
"no executions": "准备好看到你的流程在行动了吗?",
"wel_text": "Welcome!",
"start": "Let's get your first workflow up and running.",
"guide": "Need guidance to execute your first flow?",
"welcome": "Welcome to Kestra"
},
"executions replayed": "<code>{executionCount}</code> 个执行已重放",
"executions resumed": "<code>{executionCount}</code> 个执行已恢复",
@@ -654,6 +658,9 @@
},
"need-help": {
"title": "需要帮助?"
},
"tutorial": {
"title": "Tutorial"
}
},
"pluginPage": {
@@ -819,7 +826,7 @@
},
"curl": {
"command": "cURL命令",
"note": "请注意对于SECRET和FILE输入类型命令必须适应以匹配实际值。"
"note": "Note that for SECRET and FILE input type, the command must be accommodate to match the real values."
},
"logs_view": {
"raw": "时间视图",
@@ -956,7 +963,11 @@
"child": "子项",
"absolute_date": "绝对日期",
"relative_date": "相对日期",
"level": "Log级别"
"level": "Log级别",
"task": "任务",
"aggregation": "聚合",
"metric": "指标",
"trigger_state": "状态"
},
"empty": "无数据。",
"label": "选择过滤器",

View File

@@ -7,6 +7,8 @@ let requestsTotal = 0
let requestsCompleted = 0
let latencyThreshold = 0
const JWT_REFRESHED_QUERY = "__jwt_refreshed__";
const progressComplete = () => {
requestsTotal = 0
requestsCompleted = 0
@@ -115,17 +117,27 @@ export default (callback, store, router) => {
const originalRequest = errorResponse.config
if (!refreshing) {
const originalRequestData = JSON.parse(originalRequest.data ?? "{}");
// if we already tried refreshing the token,
// the user simply does not have access to this feature
if(originalRequestData[JWT_REFRESHED_QUERY] === 1) {
return Promise.reject(errorResponse)
}
refreshing = true;
try {
await instance.post("/oauth/access_token?grant_type=refresh_token");
await instance.post("/oauth/access_token?grant_type=refresh_token", null, {headers: {"Content-Type": "application/json"}});
toRefreshQueue.forEach(({config, resolve, reject}) => {
instance.request(config).then(response => { resolve(response) }).catch(error => { reject(error) })
})
toRefreshQueue = [];
refreshing = false;
originalRequestData[JWT_REFRESHED_QUERY] = 1;
originalRequest.data = JSON.stringify(originalRequestData);
return instance(originalRequest)
} catch (refreshError) {
} catch {
document.body.classList.add("login");
store.dispatch("core/isUnsaved", false);
store.commit("layout/setTopNavbar", undefined);
@@ -140,7 +152,7 @@ export default (callback, store, router) => {
}
} else {
toRefreshQueue.push(originalRequest);
return;
}
}

View File

@@ -8,8 +8,27 @@ export default (app, store, router) => {
}
});
router.beforeEach(async () => {
if (store.getters["core/unsavedChange"]) {
const routeEqualsExceptHash = (route1, route2) => {
const deleteTenantIfEmpty = route => {
if (route.params.tenant === "") {
delete route.params.tenant;
}
}
const filteredRouteForEquals = route => ({
path: route.path,
query: route.query,
params: route.params
})
deleteTenantIfEmpty(route1);
deleteTenantIfEmpty(route2);
return JSON.stringify(filteredRouteForEquals(route1)) === JSON.stringify(filteredRouteForEquals(route2))
}
router.beforeEach(async (to, from) => {
if (store.getters["core/unsavedChange"] && !routeEqualsExceptHash(from, to)) {
if (confirm(confirmationMessage)) {
store.commit("editor/changeOpenedTabs", {
action: "dirty",

View File

@@ -587,6 +587,7 @@ public class ExecutionController {
Flow flow = flowService.getFlowIfExecutableOrThrow(tenantService.resolveTenant(), namespace, id, revision);
List<Label> parsedLabels = parseLabels(labels);
Execution current = Execution.newExecution(flow, null, parsedLabels, scheduleDate);
final AtomicReference<Runnable> disposable = new AtomicReference<>();
Mono<CompletableFuture<ExecutionResponse>> handle = flowInputOutput.readExecutionInputs(flow, current, inputs)
.handle((executionInputs, sink) -> {
Execution executionWithInputs = current.withInputs(executionInputs);
@@ -598,7 +599,6 @@ public class ExecutionController {
if (!wait) {
future.complete(ExecutionResponse.fromExecution(executionWithInputs, executionUrl(executionWithInputs)));
} else {
final AtomicReference<Runnable> disposable = new AtomicReference<>();
disposable.set(this.executionQueue.receive(either -> {
if (either.isRight()) {
log.error("Unable to deserialize the execution: {}", either.getRight().getMessage());
@@ -607,7 +607,6 @@ public class ExecutionController {
Execution item = either.getLeft();
if (item.getId().equals(executionWithInputs.getId()) && this.isStopFollow(flow, item)) {
disposable.get().run();
future.complete(ExecutionResponse.fromExecution(item, executionUrl(item)));
}
}));
@@ -617,7 +616,11 @@ public class ExecutionController {
sink.error(e);
}
});
return handle.flatMap(Mono::fromFuture);
return handle.flatMap(Mono::fromFuture).doFinally(ignored -> {
if (disposable.get() != null) {
disposable.get().run();
}
});
}
private URI executionUrl(Execution execution) {
@@ -1511,12 +1514,7 @@ public class ExecutionController {
cancel.set(receive);
}, FluxSink.OverflowStrategy.BUFFER)
.doOnCancel(() -> {
if (cancel.get() != null) {
cancel.get().run();
}
})
.doOnComplete(() -> {
.doFinally(ignored -> {
if (cancel.get() != null) {
cancel.get().run();
}