Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -39,6 +39,7 @@
import java.util.concurrent.Executor;

import static com.google.common.util.concurrent.Futures.immediateVoidFuture;
import static io.airlift.units.Duration.succinctDuration;
import static io.trino.server.DynamicFilterService.DynamicFiltersStats;
import static io.trino.util.Failures.toFailure;
import static java.util.Objects.requireNonNull;
Expand Down Expand Up @@ -195,7 +196,7 @@ public Optional<Instant> getEndTime()
@Override
public Duration getTotalCpuTime()
{
return new Duration(0, MILLISECONDS);
return succinctDuration(0, MILLISECONDS);
}

@Override
Expand Down Expand Up @@ -270,16 +271,16 @@ private static QueryStats immediateFailureQueryStats()
now,
now,
now,
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
0,
0,
0,
Expand All @@ -304,19 +305,19 @@ private static QueryStats immediateFailureQueryStats()
false,
OptionalDouble.empty(),
OptionalDouble.empty(),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
false,
ImmutableSet.of(),
DataSize.ofBytes(0),
DataSize.ofBytes(0),
0,
0,
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
DataSize.ofBytes(0),
DataSize.ofBytes(0),
0,
Expand All @@ -325,14 +326,14 @@ private static QueryStats immediateFailureQueryStats()
DataSize.ofBytes(0),
0,
0,
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
DataSize.ofBytes(0),
DataSize.ofBytes(0),
0,
0,
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
DataSize.ofBytes(0),
DataSize.ofBytes(0),
ImmutableList.of(),
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -42,6 +42,7 @@
import static io.airlift.concurrent.MoreFutures.addExceptionCallback;
import static io.airlift.concurrent.MoreFutures.addSuccessCallback;
import static io.airlift.concurrent.MoreFutures.tryGetFutureValue;
import static io.airlift.units.Duration.succinctDuration;
import static io.trino.SystemSessionProperties.getRequiredWorkers;
import static io.trino.SystemSessionProperties.getRequiredWorkersMaxWait;
import static io.trino.spi.StandardErrorCode.GENERIC_INTERNAL_ERROR;
Expand Down Expand Up @@ -241,7 +242,7 @@ public Duration getTotalCpuTime()
{
return tryGetQueryExecution()
.map(QueryExecution::getTotalCpuTime)
.orElseGet(() -> new Duration(0, MILLISECONDS));
.orElseGet(() -> succinctDuration(0, MILLISECONDS));
}

@Override
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -85,6 +85,7 @@
import static com.google.common.util.concurrent.MoreExecutors.directExecutor;
import static io.airlift.concurrent.Threads.daemonThreadsNamed;
import static io.airlift.jaxrs.AsyncResponseHandler.bindAsyncResponse;
import static io.airlift.units.Duration.succinctDuration;
import static io.trino.client.ProtocolHeaders.TRINO_HEADERS;
import static io.trino.dispatcher.QueuedStatementResource.SubmissionState.ABANDONED;
import static io.trino.dispatcher.QueuedStatementResource.SubmissionState.NOT_SUBMITTED;
Expand All @@ -111,7 +112,7 @@ public class QueuedStatementResource
{
private static final Logger log = Logger.get(QueuedStatementResource.class);
private static final Duration MAX_WAIT_TIME = new Duration(1, SECONDS);
private static final Duration NO_DURATION = new Duration(0, MILLISECONDS);
private static final Duration NO_DURATION = succinctDuration(0, MILLISECONDS);

private final HttpRequestSessionContextFactory sessionContextFactory;
private final DispatchManager dispatchManager;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -23,6 +23,7 @@
import java.util.Set;

import static io.airlift.units.DataSize.succinctBytes;
import static io.airlift.units.Duration.succinctDuration;
import static java.lang.Math.min;
import static java.util.Objects.requireNonNull;
import static java.util.concurrent.TimeUnit.MILLISECONDS;
Expand All @@ -42,7 +43,7 @@ public class BasicStageStats

DataSize.ofBytes(0),
0,
new Duration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
DataSize.ofBytes(0),

DataSize.ofBytes(0),
Expand All @@ -57,10 +58,10 @@ public class BasicStageStats
DataSize.ofBytes(0),
DataSize.ofBytes(0),

new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
new Duration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),

false,
ImmutableSet.of(),
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -42,8 +42,9 @@
import static com.google.common.base.Preconditions.checkArgument;
import static com.google.common.base.Throwables.throwIfInstanceOf;
import static com.google.common.util.concurrent.MoreExecutors.directExecutor;
import static io.airlift.units.Duration.succinctDuration;
import static java.util.Objects.requireNonNull;
import static java.util.concurrent.TimeUnit.NANOSECONDS;
import static java.util.concurrent.TimeUnit.MILLISECONDS;

public class DataDefinitionExecution<T extends Statement>
implements QueryExecution
Expand Down Expand Up @@ -128,7 +129,7 @@ public Optional<Instant> getEndTime()
@Override
public Duration getTotalCpuTime()
{
return new Duration(0, NANOSECONDS);
return succinctDuration(0, MILLISECONDS);
}

@Override
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -45,8 +45,8 @@ public static List<Parameter> extractParameters(Statement statement)
return parameterExtractingVisitor.getParameters().stream()
.sorted(Comparator.comparing(
parameter -> parameter.getLocation().get(),
Comparator.comparing(NodeLocation::getLineNumber)
.thenComparing(NodeLocation::getColumnNumber)))
Comparator.comparingInt(NodeLocation::getLineNumber)
.thenComparingInt(NodeLocation::getColumnNumber)))
.collect(toImmutableList());
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -22,6 +22,7 @@
import java.util.Optional;
import java.util.concurrent.atomic.AtomicReference;

import static io.airlift.units.Duration.succinctDuration;
import static io.airlift.units.Duration.succinctNanos;
import static java.lang.Math.max;
import static java.util.Objects.requireNonNull;
Expand Down Expand Up @@ -306,7 +307,7 @@ private Duration getDuration(AtomicReference<Duration> finalDuration, AtomicRefe
if (startNanos != null) {
return nanosSince(startNanos, tickerNanos());
}
return new Duration(0, MILLISECONDS);
return succinctDuration(0, MILLISECONDS);
}

private Optional<Instant> toInstant(AtomicReference<Long> instantNanos)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -93,6 +93,7 @@
import static com.google.common.base.Throwables.throwIfInstanceOf;
import static com.google.common.util.concurrent.MoreExecutors.directExecutor;
import static io.airlift.units.DataSize.succinctBytes;
import static io.airlift.units.Duration.succinctDuration;
import static io.trino.SystemSessionProperties.getRetryPolicy;
import static io.trino.SystemSessionProperties.isEnableDynamicFiltering;
import static io.trino.execution.ParameterExtractor.bindParameters;
Expand All @@ -104,7 +105,7 @@
import static io.trino.tracing.ScopedSpan.scopedSpan;
import static java.lang.Thread.currentThread;
import static java.util.Objects.requireNonNull;
import static java.util.concurrent.TimeUnit.SECONDS;
import static java.util.concurrent.TimeUnit.MILLISECONDS;

@ThreadSafe
public class SqlQueryExecution
Expand Down Expand Up @@ -375,7 +376,7 @@ public Duration getTotalCpuTime()
return finalQueryInfo.get().getQueryStats().getTotalCpuTime();
}
if (scheduler == null) {
return new Duration(0, SECONDS);
return succinctDuration(0, MILLISECONDS);
}
return scheduler.getTotalCpuTime();
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -398,15 +398,15 @@ public synchronized void stateChanged(TaskStatus taskStatus)

public interface LocalExchangeBucketCountProvider
{
Optional<Integer> getBucketCount(Session session, PartitioningHandle partitioning);
OptionalInt getBucketCount(Session session, PartitioningHandle partitioning);
}

private static final class LocalExchangePartitionRewriter
extends SimplePlanRewriter<Void>
{
private final Function<PartitioningHandle, Optional<Integer>> bucketCountProvider;
private final Function<PartitioningHandle, OptionalInt> bucketCountProvider;

public LocalExchangePartitionRewriter(Function<PartitioningHandle, Optional<Integer>> bucketCountProvider)
public LocalExchangePartitionRewriter(Function<PartitioningHandle, OptionalInt> bucketCountProvider)
{
this.bucketCountProvider = requireNonNull(bucketCountProvider, "bucketCountProvider is null");
}
Expand Down
6 changes: 4 additions & 2 deletions core/trino-main/src/main/java/io/trino/execution/SqlTask.java
Original file line number Diff line number Diff line change
Expand Up @@ -54,6 +54,7 @@
import java.util.List;
import java.util.Map;
import java.util.Optional;
import java.util.OptionalInt;
import java.util.Set;
import java.util.UUID;
import java.util.concurrent.Executor;
Expand All @@ -69,6 +70,7 @@
import static com.google.common.util.concurrent.Futures.immediateFuture;
import static com.google.common.util.concurrent.MoreExecutors.directExecutor;
import static io.airlift.units.DataSize.succinctBytes;
import static io.airlift.units.Duration.succinctDuration;
import static io.trino.execution.DynamicFiltersCollector.INITIAL_DYNAMIC_FILTERS_VERSION;
import static io.trino.execution.DynamicFiltersCollector.INITIAL_DYNAMIC_FILTER_DOMAINS;
import static io.trino.execution.TaskState.FAILED;
Expand Down Expand Up @@ -344,12 +346,12 @@ private TaskStatus createTaskStatus(TaskHolder taskHolder)
DataSize outputDataSize = DataSize.ofBytes(0);
DataSize writerInputDataSize = DataSize.ofBytes(0);
DataSize physicalWrittenDataSize = DataSize.ofBytes(0);
Optional<Integer> writerCount = Optional.empty();
OptionalInt writerCount = OptionalInt.empty();
DataSize userMemoryReservation = DataSize.ofBytes(0);
DataSize peakUserMemoryReservation = DataSize.ofBytes(0);
DataSize revocableMemoryReservation = DataSize.ofBytes(0);
long fullGcCount = 0;
Duration fullGcTime = new Duration(0, MILLISECONDS);
Duration fullGcTime = succinctDuration(0, MILLISECONDS);
long dynamicFiltersVersion = INITIAL_DYNAMIC_FILTERS_VERSION;
if (taskHolder.getFinalTaskInfo() != null) {
TaskInfo taskInfo = taskHolder.getFinalTaskInfo();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -37,10 +37,11 @@

import static com.google.common.base.Preconditions.checkArgument;
import static io.airlift.units.DataSize.Unit.BYTE;
import static io.airlift.units.Duration.succinctDuration;
import static io.trino.execution.StageState.RUNNING;
import static java.lang.Math.min;
import static java.util.Objects.requireNonNull;
import static java.util.concurrent.TimeUnit.SECONDS;
import static java.util.concurrent.TimeUnit.MILLISECONDS;

@Immutable
public class StageStats
Expand Down Expand Up @@ -660,7 +661,7 @@ public BasicStageStats toBasicStageStats(StageState stageState)
public static StageStats createInitial()
{
DataSize zeroBytes = DataSize.of(0, BYTE);
Duration zeroSeconds = new Duration(0, SECONDS);
Duration zeroSeconds = succinctDuration(0, MILLISECONDS);
return new StageStats(
null,
ImmutableMap.of(),
Expand Down
13 changes: 7 additions & 6 deletions core/trino-main/src/main/java/io/trino/execution/TaskStatus.java
Original file line number Diff line number Diff line change
Expand Up @@ -22,11 +22,12 @@

import java.net.URI;
import java.util.List;
import java.util.Optional;
import java.util.OptionalInt;

import static com.google.common.base.MoreObjects.toStringHelper;
import static com.google.common.base.Preconditions.checkArgument;
import static com.google.common.base.Preconditions.checkState;
import static io.airlift.units.Duration.succinctDuration;
import static io.trino.execution.DynamicFiltersCollector.INITIAL_DYNAMIC_FILTERS_VERSION;
import static io.trino.execution.TaskState.PLANNED;
import static java.util.Objects.requireNonNull;
Expand Down Expand Up @@ -62,7 +63,7 @@ public class TaskStatus
private final DataSize writerInputDataSize;
private final DataSize outputDataSize;
private final DataSize physicalWrittenDataSize;
private final Optional<Integer> maxWriterCount;
private final OptionalInt maxWriterCount;
private final DataSize memoryReservation;
private final DataSize peakMemoryReservation;
private final DataSize revocableMemoryReservation;
Expand Down Expand Up @@ -90,7 +91,7 @@ public TaskStatus(
@JsonProperty("outputDataSize") DataSize outputDataSize,
@JsonProperty("writerInputDataSize") DataSize writerInputDataSize,
@JsonProperty("physicalWrittenDataSize") DataSize physicalWrittenDataSize,
@JsonProperty("writerCount") Optional<Integer> maxWriterCount,
@JsonProperty("writerCount") OptionalInt maxWriterCount,
@JsonProperty("memoryReservation") DataSize memoryReservation,
@JsonProperty("peakMemoryReservation") DataSize peakMemoryReservation,
@JsonProperty("revocableMemoryReservation") DataSize revocableMemoryReservation,
Expand Down Expand Up @@ -212,7 +213,7 @@ public DataSize getPhysicalWrittenDataSize()
}

@JsonProperty
public Optional<Integer> getMaxWriterCount()
public OptionalInt getMaxWriterCount()
{
return maxWriterCount;
}
Expand Down Expand Up @@ -303,12 +304,12 @@ public static TaskStatus initialTaskStatus(TaskId taskId, URI location, String n
DataSize.ofBytes(0),
DataSize.ofBytes(0),
DataSize.ofBytes(0),
Optional.empty(),
OptionalInt.empty(),
DataSize.ofBytes(0),
DataSize.ofBytes(0),
DataSize.ofBytes(0),
0,
new Duration(0, MILLISECONDS),
succinctDuration(0, MILLISECONDS),
INITIAL_DYNAMIC_FILTERS_VERSION,
0L,
0L);
Expand Down
Loading