Skip to content

Commit 76ec66f

Browse files
committed
Redact CREATE CATALOG
This commit introduces redacting of security-sensitive information in the following statements: * CREATE CATALOG * EXPLAIN CREATE CATALOG * PREPARE CREATE CATALOG The current approach is as follows: * For syntactically valid statements, only properties containing sensitive information are masked. * If a query is syntactically valid but retrieving security-sensitive properties fails for any reason (e.g., the query references a nonexistent connector or catalog property evaluation fails), all properties are masked. * If a query fails before or during parsing, nothing is masked. The redacted form is created right before initialization of the QueryStateMachine and is propagated to all places that create QueryInfo and BasicQueryInfo (e.g., REST endpoints, query events, and the system.runtime.queries table). Before this change, QueryInfo/BasicQueryInfo stored the raw query text received from the end user. From now on, the text will be altered for the cases listed above.
1 parent 152bd6a commit 76ec66f

34 files changed

+874
-60
lines changed

core/trino-main/src/main/java/io/trino/FeaturesConfig.java

Lines changed: 14 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -122,6 +122,8 @@ public class FeaturesConfig
122122

123123
private boolean faultTolerantExecutionExchangeEncryptionEnabled = true;
124124

125+
private boolean statementRedactingEnabled = true;
126+
125127
public enum DataIntegrityVerification
126128
{
127129
NONE,
@@ -514,6 +516,18 @@ public FeaturesConfig setFaultTolerantExecutionExchangeEncryptionEnabled(boolean
514516
return this;
515517
}
516518

519+
public boolean isStatementRedactingEnabled()
520+
{
521+
return statementRedactingEnabled;
522+
}
523+
524+
@Config("deprecated.statement-redacting-enabled")
525+
public FeaturesConfig setStatementRedactingEnabled(boolean statementRedactingEnabled)
526+
{
527+
this.statementRedactingEnabled = statementRedactingEnabled;
528+
return this;
529+
}
530+
517531
public void applyFaultTolerantExecutionDefaults()
518532
{
519533
exchangeCompressionCodec = LZ4;

core/trino-main/src/main/java/io/trino/connector/CatalogFactory.java

Lines changed: 4 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -20,6 +20,8 @@
2020
import io.trino.spi.connector.ConnectorFactory;
2121
import io.trino.spi.connector.ConnectorName;
2222

23+
import java.util.Set;
24+
2325
@ThreadSafe
2426
public interface CatalogFactory
2527
{
@@ -28,4 +30,6 @@ public interface CatalogFactory
2830
CatalogConnector createCatalog(CatalogProperties catalogProperties);
2931

3032
CatalogConnector createCatalog(CatalogHandle catalogHandle, ConnectorName connectorName, Connector connector);
33+
34+
Set<String> getSecuritySensitivePropertyNames(CatalogProperties catalogProperties);
3135
}

core/trino-main/src/main/java/io/trino/connector/DefaultCatalogFactory.java

Lines changed: 31 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -13,6 +13,7 @@
1313
*/
1414
package io.trino.connector;
1515

16+
import com.google.common.collect.ImmutableSet;
1617
import com.google.errorprone.annotations.ThreadSafe;
1718
import com.google.inject.Inject;
1819
import io.airlift.configuration.secrets.SecretsResolver;
@@ -45,6 +46,7 @@
4546

4647
import java.util.Map;
4748
import java.util.Optional;
49+
import java.util.Set;
4850
import java.util.concurrent.ConcurrentHashMap;
4951
import java.util.concurrent.ConcurrentMap;
5052

@@ -144,6 +146,25 @@ public CatalogConnector createCatalog(CatalogHandle catalogHandle, ConnectorName
144146
return createCatalog(catalogHandle, connectorName, connector, Optional.empty());
145147
}
146148

149+
@Override
150+
public Set<String> getSecuritySensitivePropertyNames(CatalogProperties catalogProperties)
151+
{
152+
ConnectorFactory connectorFactory = connectorFactories.get(catalogProperties.connectorName());
153+
if (connectorFactory == null) {
154+
// If someone tries to use a non-existent connector, we assume they
155+
// misspelled the name and, for safety, we redact all the properties.
156+
return ImmutableSet.copyOf(catalogProperties.properties().keySet());
157+
}
158+
159+
ConnectorContext context = createConnectorContext(catalogProperties.catalogHandle());
160+
String catalogName = catalogProperties.catalogHandle().getCatalogName().toString();
161+
Map<String, String> config = secretsResolver.getResolvedConfiguration(catalogProperties.properties());
162+
163+
try (ThreadContextClassLoader _ = new ThreadContextClassLoader(connectorFactory.getClass().getClassLoader())) {
164+
return connectorFactory.getSecuritySensitivePropertyNames(catalogName, config, context);
165+
}
166+
}
167+
147168
private CatalogConnector createCatalog(CatalogHandle catalogHandle, ConnectorName connectorName, Connector connector, Optional<CatalogProperties> catalogProperties)
148169
{
149170
Tracer tracer = createTracer(catalogHandle);
@@ -196,7 +217,16 @@ private Connector createConnector(
196217
ConnectorFactory connectorFactory,
197218
Map<String, String> properties)
198219
{
199-
ConnectorContext context = new ConnectorContextInstance(
220+
ConnectorContext context = createConnectorContext(catalogHandle);
221+
222+
try (ThreadContextClassLoader _ = new ThreadContextClassLoader(connectorFactory.getClass().getClassLoader())) {
223+
return connectorFactory.create(catalogName, properties, context);
224+
}
225+
}
226+
227+
private ConnectorContext createConnectorContext(CatalogHandle catalogHandle)
228+
{
229+
return new ConnectorContextInstance(
200230
catalogHandle,
201231
openTelemetry,
202232
createTracer(catalogHandle),
@@ -206,10 +236,6 @@ private Connector createConnector(
206236
new InternalMetadataProvider(metadata, typeManager),
207237
pageSorter,
208238
pageIndexerFactory);
209-
210-
try (ThreadContextClassLoader _ = new ThreadContextClassLoader(connectorFactory.getClass().getClassLoader())) {
211-
return connectorFactory.create(catalogName, properties, context);
212-
}
213239
}
214240

215241
private Tracer createTracer(CatalogHandle catalogHandle)

core/trino-main/src/main/java/io/trino/connector/LazyCatalogFactory.java

Lines changed: 7 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -19,6 +19,7 @@
1919
import io.trino.spi.connector.ConnectorFactory;
2020
import io.trino.spi.connector.ConnectorName;
2121

22+
import java.util.Set;
2223
import java.util.concurrent.atomic.AtomicReference;
2324

2425
import static com.google.common.base.Preconditions.checkState;
@@ -51,6 +52,12 @@ public CatalogConnector createCatalog(CatalogHandle catalogHandle, ConnectorName
5152
return getDelegate().createCatalog(catalogHandle, connectorName, connector);
5253
}
5354

55+
@Override
56+
public Set<String> getSecuritySensitivePropertyNames(CatalogProperties catalogProperties)
57+
{
58+
return getDelegate().getSecuritySensitivePropertyNames(catalogProperties);
59+
}
60+
5461
private CatalogFactory getDelegate()
5562
{
5663
CatalogFactory catalogFactory = delegate.get();

core/trino-main/src/main/java/io/trino/dispatcher/DispatchManager.java

Lines changed: 12 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -13,6 +13,7 @@
1313
*/
1414
package io.trino.dispatcher;
1515

16+
import com.google.common.base.Function;
1617
import com.google.common.util.concurrent.AbstractFuture;
1718
import com.google.common.util.concurrent.Futures;
1819
import com.google.common.util.concurrent.ListenableFuture;
@@ -44,6 +45,8 @@
4445
import io.trino.spi.TrinoException;
4546
import io.trino.spi.resourcegroups.SelectionContext;
4647
import io.trino.spi.resourcegroups.SelectionCriteria;
48+
import io.trino.sql.RedactedQuery;
49+
import io.trino.sql.SensitiveStatementRedactor;
4750
import jakarta.annotation.PostConstruct;
4851
import jakarta.annotation.PreDestroy;
4952
import org.weakref.jmx.Flatten;
@@ -84,6 +87,7 @@ public class DispatchManager
8487
private final SessionPropertyDefaults sessionPropertyDefaults;
8588
private final SessionPropertyManager sessionPropertyManager;
8689
private final Tracer tracer;
90+
private final SensitiveStatementRedactor sensitiveStatementRedactor;
8791

8892
private final int maxQueryLength;
8993

@@ -107,6 +111,7 @@ public DispatchManager(
107111
SessionPropertyDefaults sessionPropertyDefaults,
108112
SessionPropertyManager sessionPropertyManager,
109113
Tracer tracer,
114+
SensitiveStatementRedactor sensitiveStatementRedactor,
110115
QueryManagerConfig queryManagerConfig,
111116
DispatchExecutor dispatchExecutor,
112117
QueryMonitor queryMonitor)
@@ -121,6 +126,7 @@ public DispatchManager(
121126
this.sessionPropertyDefaults = requireNonNull(sessionPropertyDefaults, "sessionPropertyDefaults is null");
122127
this.sessionPropertyManager = sessionPropertyManager;
123128
this.tracer = requireNonNull(tracer, "tracer is null");
129+
this.sensitiveStatementRedactor = requireNonNull(sensitiveStatementRedactor, "sensitiveStatementRedactor is null");
124130

125131
this.maxQueryLength = queryManagerConfig.getMaxQueryLength();
126132

@@ -240,7 +246,7 @@ private <C> void createQueryInternal(QueryId queryId, Span querySpan, Slug slug,
240246
DispatchQuery dispatchQuery = dispatchQueryFactory.createDispatchQuery(
241247
session,
242248
sessionContext.getTransactionId(),
243-
query,
249+
getRedactedQueryProvider(preparedQuery, query),
244250
preparedQuery,
245251
slug,
246252
selectionContext.getResourceGroupId());
@@ -280,6 +286,11 @@ private <C> void createQueryInternal(QueryId queryId, Span querySpan, Slug slug,
280286
}
281287
}
282288

289+
private Function<Session, RedactedQuery> getRedactedQueryProvider(PreparedQuery preparedQuery, String query)
290+
{
291+
return session -> sensitiveStatementRedactor.redact(query, preparedQuery, session);
292+
}
293+
283294
private boolean queryCreated(DispatchQuery dispatchQuery)
284295
{
285296
boolean queryAdded = queryTracker.addQuery(dispatchQuery);

core/trino-main/src/main/java/io/trino/dispatcher/DispatchQueryFactory.java

Lines changed: 3 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -17,16 +17,18 @@
1717
import io.trino.execution.QueryPreparer.PreparedQuery;
1818
import io.trino.server.protocol.Slug;
1919
import io.trino.spi.resourcegroups.ResourceGroupId;
20+
import io.trino.sql.RedactedQuery;
2021
import io.trino.transaction.TransactionId;
2122

2223
import java.util.Optional;
24+
import java.util.function.Function;
2325

2426
public interface DispatchQueryFactory
2527
{
2628
DispatchQuery createDispatchQuery(
2729
Session session,
2830
Optional<TransactionId> transactionId,
29-
String query,
31+
Function<Session, RedactedQuery> queryProvider,
3032
PreparedQuery preparedQuery,
3133
Slug slug,
3234
ResourceGroupId resourceGroup);

core/trino-main/src/main/java/io/trino/dispatcher/LocalDispatchQueryFactory.java

Lines changed: 4 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -38,12 +38,14 @@
3838
import io.trino.server.protocol.Slug;
3939
import io.trino.spi.TrinoException;
4040
import io.trino.spi.resourcegroups.ResourceGroupId;
41+
import io.trino.sql.RedactedQuery;
4142
import io.trino.sql.tree.Statement;
4243
import io.trino.transaction.TransactionId;
4344
import io.trino.transaction.TransactionManager;
4445

4546
import java.util.Map;
4647
import java.util.Optional;
48+
import java.util.function.Function;
4749

4850
import static io.trino.spi.StandardErrorCode.NOT_SUPPORTED;
4951
import static io.trino.util.StatementUtils.getQueryType;
@@ -108,7 +110,7 @@ public LocalDispatchQueryFactory(
108110
public DispatchQuery createDispatchQuery(
109111
Session session,
110112
Optional<TransactionId> existingTransactionId,
111-
String query,
113+
Function<Session, RedactedQuery> queryProvider,
112114
PreparedQuery preparedQuery,
113115
Slug slug,
114116
ResourceGroupId resourceGroup)
@@ -117,8 +119,7 @@ public DispatchQuery createDispatchQuery(
117119
PlanOptimizersStatsCollector planOptimizersStatsCollector = new PlanOptimizersStatsCollector(queryReportedRuleStatsLimit);
118120
QueryStateMachine stateMachine = QueryStateMachine.begin(
119121
existingTransactionId,
120-
query,
121-
preparedQuery.getPrepareSql(),
122+
queryProvider,
122123
session,
123124
locationFactory.createQueryLocation(session.getQueryId()),
124125
resourceGroup,

core/trino-main/src/main/java/io/trino/execution/CreateCatalogTask.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -86,7 +86,7 @@ public ListenableFuture<Void> execute(
8686
return immediateVoidFuture();
8787
}
8888

89-
private static Map<String, String> evaluateProperties(
89+
public static Map<String, String> evaluateProperties(
9090
CreateCatalog statement,
9191
Session session,
9292
PlannerContext plannerContext,

core/trino-main/src/main/java/io/trino/execution/QueryPreparer.java

Lines changed: 9 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -86,7 +86,7 @@ else if (wrappedStatement instanceof ExecuteImmediate executeImmediateStatement)
8686
}
8787
validateParameters(statement, parameters);
8888

89-
return new PreparedQuery(statement, parameters, prepareSql);
89+
return new PreparedQuery(wrappedStatement, statement, parameters, prepareSql);
9090
}
9191

9292
private static void validateParameters(Statement node, List<Expression> parameterValues)
@@ -102,17 +102,24 @@ private static void validateParameters(Statement node, List<Expression> paramete
102102

103103
public static class PreparedQuery
104104
{
105+
private final Statement wrappedStatement;
105106
private final Statement statement;
106107
private final List<Expression> parameters;
107108
private final Optional<String> prepareSql;
108109

109-
public PreparedQuery(Statement statement, List<Expression> parameters, Optional<String> prepareSql)
110+
public PreparedQuery(Statement wrappedStatement, Statement statement, List<Expression> parameters, Optional<String> prepareSql)
110111
{
112+
this.wrappedStatement = requireNonNull(wrappedStatement, "wrappedStatement is null");
111113
this.statement = requireNonNull(statement, "statement is null");
112114
this.parameters = ImmutableList.copyOf(requireNonNull(parameters, "parameters is null"));
113115
this.prepareSql = requireNonNull(prepareSql, "prepareSql is null");
114116
}
115117

118+
public Statement getWrappedStatement()
119+
{
120+
return wrappedStatement;
121+
}
122+
116123
public Statement getStatement()
117124
{
118125
return statement;

core/trino-main/src/main/java/io/trino/execution/QueryStateMachine.java

Lines changed: 9 additions & 8 deletions
Original file line numberDiff line numberDiff line change
@@ -53,6 +53,7 @@
5353
import io.trino.spi.resourcegroups.ResourceGroupId;
5454
import io.trino.spi.security.SelectedRole;
5555
import io.trino.spi.type.Type;
56+
import io.trino.sql.RedactedQuery;
5657
import io.trino.sql.analyzer.Output;
5758
import io.trino.sql.planner.PlanFragment;
5859
import io.trino.tracing.TrinoAttributes;
@@ -80,6 +81,7 @@
8081
import java.util.concurrent.atomic.AtomicLong;
8182
import java.util.concurrent.atomic.AtomicReference;
8283
import java.util.function.Consumer;
84+
import java.util.function.Function;
8385
import java.util.function.Supplier;
8486

8587
import static com.google.common.base.Preconditions.checkArgument;
@@ -231,8 +233,7 @@ private QueryStateMachine(
231233
*/
232234
public static QueryStateMachine begin(
233235
Optional<TransactionId> existingTransactionId,
234-
String query,
235-
Optional<String> preparedQuery,
236+
Function<Session, RedactedQuery> queryProvider,
236237
Session session,
237238
URI self,
238239
ResourceGroupId resourceGroup,
@@ -249,8 +250,7 @@ public static QueryStateMachine begin(
249250
{
250251
return beginWithTicker(
251252
existingTransactionId,
252-
query,
253-
preparedQuery,
253+
queryProvider,
254254
session,
255255
self,
256256
resourceGroup,
@@ -269,8 +269,7 @@ public static QueryStateMachine begin(
269269

270270
static QueryStateMachine beginWithTicker(
271271
Optional<TransactionId> existingTransactionId,
272-
String query,
273-
Optional<String> preparedQuery,
272+
Function<Session, RedactedQuery> queryProvider,
274273
Session session,
275274
URI self,
276275
ResourceGroupId resourceGroup,
@@ -318,9 +317,11 @@ static QueryStateMachine beginWithTicker(
318317

319318
querySpan.setAttribute(TrinoAttributes.QUERY_TYPE, queryType.map(Enum::name).orElse("UNKNOWN"));
320319

320+
RedactedQuery redactedQuery = queryProvider.apply(session);
321+
321322
QueryStateMachine queryStateMachine = new QueryStateMachine(
322-
query,
323-
preparedQuery,
323+
redactedQuery.query(),
324+
redactedQuery.preparedQuery(),
324325
session,
325326
self,
326327
resourceGroup,

0 commit comments

Comments
 (0)