Skip to content

Commit

Permalink
Add more options to S3FileSystemConfig
Browse files Browse the repository at this point in the history
  • Loading branch information
pettyjamesm committed Feb 2, 2024
1 parent ebff693 commit 8a3bef5
Show file tree
Hide file tree
Showing 3 changed files with 97 additions and 1 deletion.
Original file line number Diff line number Diff line change
Expand Up @@ -18,11 +18,14 @@
import io.airlift.configuration.ConfigDescription;
import io.airlift.configuration.ConfigSecuritySensitive;
import io.airlift.units.DataSize;
import io.airlift.units.Duration;
import io.airlift.units.MaxDataSize;
import io.airlift.units.MinDataSize;
import jakarta.validation.constraints.Min;
import jakarta.validation.constraints.NotNull;

import java.util.Optional;

import static io.airlift.units.DataSize.Unit.MEGABYTE;

public class S3FileSystemConfig
Expand All @@ -47,6 +50,11 @@ public enum S3SseType
private DataSize streamingPartSize = DataSize.of(16, MEGABYTE);
private boolean requesterPays;
private Integer maxConnections;
private Duration connectionTtl;
private Duration connectionMaxIdleTime;
private Duration socketConnectTimeout;
private Duration socketReadTimeout;
private boolean tcpKeepAlive;
private HostAndPort httpProxy;
private boolean httpProxySecure;

Expand Down Expand Up @@ -243,6 +251,71 @@ public S3FileSystemConfig setMaxConnections(Integer maxConnections)
return this;
}

public Optional<Duration> getConnectionTtl()
{
return Optional.ofNullable(connectionTtl);
}

@Config("s3.connection-ttl")
@ConfigDescription("Maximum time allowed for connections to be reused before being replaced in the connection pool")
public S3FileSystemConfig setConnectionTtl(Duration connectionTtl)
{
this.connectionTtl = connectionTtl;
return this;
}

public Optional<Duration> getConnectionMaxIdleTime()
{
return Optional.ofNullable(connectionMaxIdleTime);
}

@Config("s3.connection-max-idle-time")
@ConfigDescription("Maximum time allowed for connections to remain idle in the connection pool before being closed")
public S3FileSystemConfig setConnectionMaxIdleTime(Duration connectionMaxIdleTime)
{
this.connectionMaxIdleTime = connectionMaxIdleTime;
return this;
}

public Optional<Duration> getSocketConnectTimeout()
{
return Optional.ofNullable(socketConnectTimeout);
}

@Config("s3.socket-connect-timeout")
@ConfigDescription("Maximum time allowed for socket connect to complete before timing out")
public S3FileSystemConfig setSocketConnectTimeout(Duration socketConnectTimeout)
{
this.socketConnectTimeout = socketConnectTimeout;
return this;
}

public Optional<Duration> getSocketReadTimeout()
{
return Optional.ofNullable(socketReadTimeout);
}

@Config("s3.socket-read-timeout")
@ConfigDescription("Maximum time allowed for socket reads before timing out")
public S3FileSystemConfig setSocketReadTimeout(Duration socketReadTimeout)
{
this.socketReadTimeout = socketReadTimeout;
return this;
}

public boolean getTcpKeepAlive()
{
return tcpKeepAlive;
}

@Config("s3.tcp-keep-alive")
@ConfigDescription("Enable TCP keep alive on created connections")
public S3FileSystemConfig setTcpKeepAlive(boolean tcpKeepAlive)
{
this.tcpKeepAlive = tcpKeepAlive;
return this;
}

public HostAndPort getHttpProxy()
{
return httpProxy;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -81,7 +81,13 @@ public S3FileSystemFactory(OpenTelemetry openTelemetry, S3FileSystemConfig confi
}

ApacheHttpClient.Builder httpClient = ApacheHttpClient.builder()
.maxConnections(config.getMaxConnections());
.maxConnections(config.getMaxConnections())
.tcpKeepAlive(config.getTcpKeepAlive());

config.getConnectionTtl().ifPresent(connectionTtl -> httpClient.connectionTimeToLive(connectionTtl.toJavaTime()));
config.getConnectionMaxIdleTime().ifPresent(connectionMaxIdleTime -> httpClient.connectionMaxIdleTime(connectionMaxIdleTime.toJavaTime()));
config.getSocketConnectTimeout().ifPresent(socketConnectTimeout -> httpClient.connectionTimeout(socketConnectTimeout.toJavaTime()));
config.getSocketReadTimeout().ifPresent(socketReadTimeout -> httpClient.socketTimeout(socketReadTimeout.toJavaTime()));

if (config.getHttpProxy() != null) {
URI endpoint = URI.create("%s://%s".formatted(
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -16,6 +16,7 @@
import com.google.common.collect.ImmutableMap;
import com.google.common.net.HostAndPort;
import io.airlift.units.DataSize;
import io.airlift.units.Duration;
import io.trino.filesystem.s3.S3FileSystemConfig.S3SseType;
import org.junit.jupiter.api.Test;

Expand All @@ -25,6 +26,7 @@
import static io.airlift.configuration.testing.ConfigAssertions.assertRecordedDefaults;
import static io.airlift.configuration.testing.ConfigAssertions.recordDefaults;
import static io.airlift.units.DataSize.Unit.MEGABYTE;
import static java.util.concurrent.TimeUnit.MINUTES;

public class TestS3FileSystemConfig
{
Expand All @@ -47,6 +49,11 @@ public void testDefaults()
.setStreamingPartSize(DataSize.of(16, MEGABYTE))
.setRequesterPays(false)
.setMaxConnections(null)
.setConnectionTtl(null)
.setConnectionMaxIdleTime(null)
.setSocketConnectTimeout(null)
.setSocketReadTimeout(null)
.setTcpKeepAlive(false)
.setHttpProxy(null)
.setHttpProxySecure(false));
}
Expand All @@ -70,6 +77,11 @@ public void testExplicitPropertyMappings()
.put("s3.streaming.part-size", "42MB")
.put("s3.requester-pays", "true")
.put("s3.max-connections", "42")
.put("s3.connection-ttl", "1m")
.put("s3.connection-max-idle-time", "2m")
.put("s3.socket-connect-timeout", "3m")
.put("s3.socket-read-timeout", "4m")
.put("s3.tcp-keep-alive", "true")
.put("s3.http-proxy", "localhost:8888")
.put("s3.http-proxy.secure", "true")
.buildOrThrow();
Expand All @@ -90,6 +102,11 @@ public void testExplicitPropertyMappings()
.setSseKmsKeyId("mykey")
.setRequesterPays(true)
.setMaxConnections(42)
.setConnectionTtl(new Duration(1, MINUTES))
.setConnectionMaxIdleTime(new Duration(2, MINUTES))
.setSocketConnectTimeout(new Duration(3, MINUTES))
.setSocketReadTimeout(new Duration(4, MINUTES))
.setTcpKeepAlive(true)
.setHttpProxy(HostAndPort.fromParts("localhost", 8888))
.setHttpProxySecure(true);

Expand Down

0 comments on commit 8a3bef5

Please sign in to comment.