Copilot commented on code in PR #17167:
URL: https://github.com/apache/pinot/pull/17167#discussion_r3008485204
##########
pinot-tools/src/main/java/org/apache/pinot/tools/admin/command/OperateClusterConfigCommand.java:
##########
@@ -144,36 +87,34 @@ public String run()
if (StringUtils.isEmpty(_config) && !_operation.equalsIgnoreCase("GET")) {
throw new UnsupportedOperationException("Empty config: " + _config);
}
- String clusterConfigUrl =
- _controllerProtocol + "://" + _controllerHost + ":" + _controllerPort
+ "/cluster/configs";
- List<Header> headers = AuthProviderUtils.makeAuthHeaders(
- AuthProviderUtils.makeAuthProvider(_authProvider, _authTokenUrl,
_authToken, _user,
- _password));
- switch (_operation.toUpperCase()) {
- case "ADD":
- case "UPDATE":
- String[] splits = _config.split("=");
- if (splits.length != 2) {
- throw new UnsupportedOperationException(
- "Bad config: " + _config + ". Please follow the pattern of
[Config Key]=[Config Value]");
- }
- String request =
JsonUtils.objectToString(Collections.singletonMap(splits[0], splits[1]));
- return sendRequest("POST", clusterConfigUrl, request, headers);
- case "GET":
- String response = sendRequest("GET", clusterConfigUrl, null, headers);
- JsonNode jsonNode = JsonUtils.stringToJsonNode(response);
- Iterator<String> fieldNamesIterator = jsonNode.fieldNames();
- String results = "";
- while (fieldNamesIterator.hasNext()) {
- String key = fieldNamesIterator.next();
- String value = jsonNode.get(key).textValue();
- results += String.format("%s=%s\n", key, value);
- }
- return results;
- case "DELETE":
- return sendRequest("DELETE", String.format("%s/%s", clusterConfigUrl,
_config), null, headers);
- default:
- throw new UnsupportedOperationException("Unsupported operation: " +
_operation);
+ String normalizedOperation = _operation.toUpperCase(Locale.ROOT);
+ try (PinotAdminClient adminClient = getPinotAdminClient()) {
+ switch (normalizedOperation) {
+ case "ADD":
+ case "UPDATE":
+ String[] splits = _config.split("=");
+ if (splits.length != 2) {
+ throw new UnsupportedOperationException(
+ "Bad config: " + _config + ". Please follow the pattern of
[Config Key]=[Config Value]");
+ }
+ String request =
JsonUtils.objectToString(java.util.Collections.singletonMap(splits[0],
splits[1]));
+ return adminClient.getClusterClient().updateClusterConfig(request);
+ case "GET":
+ String response = adminClient.getClusterClient().getClusterConfigs();
+ JsonNode jsonNode = JsonUtils.stringToJsonNode(response);
+ StringBuilder results = new StringBuilder();
+ jsonNode.fieldNames().forEachRemaining(key -> {
+ String value = jsonNode.get(key).textValue();
+ results.append(String.format("%s=%s%n", key, value));
+ });
+ return results.toString();
+ case "DELETE":
+ return adminClient.getClusterClient().deleteClusterConfig(_config);
+ default:
+ throw new UnsupportedOperationException("Unsupported operation: " +
_operation);
+ }
+ } catch (PinotAdminException e) {
+ throw new RuntimeException("Failed to operate on cluster config", e);
Review Comment:
Catching `PinotAdminException` and rethrowing a generic `RuntimeException`
loses useful context (e.g., HTTP status code / controller error payload) even
though `run()` already declares `throws Exception`. Consider either letting
`PinotAdminException` propagate, or wrapping it with operation-specific context
(include operation/config key) while preserving the original message/status to
aid debugging.
##########
pinot-connectors/pinot-flink-connector/src/main/java/org/apache/pinot/connector/flink/FlinkQuickStart.java:
##########
@@ -79,10 +77,8 @@ public static void main(String[] args)
execEnv.setParallelism(2);
DataStream<Row> srcDs =
execEnv.fromCollection(data).returns(TEST_TYPE_INFO).keyBy(r -> r.getField(0));
- HttpClient httpClient = HttpClient.getInstance();
- ControllerRequestClient client = new ControllerRequestClient(
- ControllerRequestURLBuilder.baseUrl(DEFAULT_CONTROLLER_URL),
httpClient);
- Schema schema = PinotConnectionUtils.getSchema(client, "starbucksStores");
+ PinotAdminClient client = new PinotAdminClient("localhost:9000");
+ Schema schema = client.getSchemaClient().getSchema("starbucksStores");
TableConfig tableConfig = PinotConnectionUtils.getTableConfig(client,
"starbucksStores", "OFFLINE");
srcDs.addSink(new PinotSinkFunction<>(new
FlinkRowGenericRowConverter(TEST_TYPE_INFO), tableConfig, schema));
execEnv.execute();
Review Comment:
`PinotAdminClient` is created in `main()` but never closed. Since it owns an
underlying HTTP client/transport, this can leak threads/resources. Wrap it in
try-with-resources (or explicitly call `close()`) around the schema/config
fetches.
```suggestion
try (PinotAdminClient client = new PinotAdminClient("localhost:9000")) {
Schema schema = client.getSchemaClient().getSchema("starbucksStores");
TableConfig tableConfig = PinotConnectionUtils.getTableConfig(client,
"starbucksStores", "OFFLINE");
srcDs.addSink(new PinotSinkFunction<>(new
FlinkRowGenericRowConverter(TEST_TYPE_INFO), tableConfig, schema));
execEnv.execute();
}
```
##########
pinot-tools/src/main/java/org/apache/pinot/tools/admin/command/AddTenantCommand.java:
##########
@@ -110,36 +77,36 @@ public AddTenantCommand setRealtime(int realtime) {
return this;
}
- public AddTenantCommand setUser(String user) {
- _user = user;
- return this;
- }
-
- public AddTenantCommand setPassword(String password) {
- _password = password;
+ public AddTenantCommand setControllerUrl(String url) {
+ URI uri = URI.create(url);
+ if (uri.getScheme() != null) {
+ _controllerProtocol = uri.getScheme();
+ }
+ if (uri.getHost() != null) {
+ _controllerHost = uri.getHost();
+ }
+ if (uri.getPort() > 0) {
+ _controllerPort = Integer.toString(uri.getPort());
+ }
return this;
Review Comment:
`setControllerUrl(String url)` uses `URI.create(url)` and then reads
scheme/host/port. For common inputs like `"localhost:9000"` (no scheme),
`URI.create` treats `localhost` as the scheme and leaves host/port unset, which
will misconfigure `_controllerProtocol/_controllerHost/_controllerPort`.
Consider handling scheme-less inputs (e.g., prefix with `http://`), or preserve
the previous behavior of storing a full controller address/URI when no scheme
is present.
##########
pinot-tools/src/main/java/org/apache/pinot/tools/admin/command/OperateClusterConfigCommand.java:
##########
@@ -144,36 +87,34 @@ public String run()
if (StringUtils.isEmpty(_config) && !_operation.equalsIgnoreCase("GET")) {
throw new UnsupportedOperationException("Empty config: " + _config);
}
- String clusterConfigUrl =
- _controllerProtocol + "://" + _controllerHost + ":" + _controllerPort
+ "/cluster/configs";
- List<Header> headers = AuthProviderUtils.makeAuthHeaders(
- AuthProviderUtils.makeAuthProvider(_authProvider, _authTokenUrl,
_authToken, _user,
- _password));
- switch (_operation.toUpperCase()) {
- case "ADD":
- case "UPDATE":
- String[] splits = _config.split("=");
- if (splits.length != 2) {
- throw new UnsupportedOperationException(
- "Bad config: " + _config + ". Please follow the pattern of
[Config Key]=[Config Value]");
- }
- String request =
JsonUtils.objectToString(Collections.singletonMap(splits[0], splits[1]));
- return sendRequest("POST", clusterConfigUrl, request, headers);
- case "GET":
- String response = sendRequest("GET", clusterConfigUrl, null, headers);
- JsonNode jsonNode = JsonUtils.stringToJsonNode(response);
- Iterator<String> fieldNamesIterator = jsonNode.fieldNames();
- String results = "";
- while (fieldNamesIterator.hasNext()) {
- String key = fieldNamesIterator.next();
- String value = jsonNode.get(key).textValue();
- results += String.format("%s=%s\n", key, value);
- }
- return results;
- case "DELETE":
- return sendRequest("DELETE", String.format("%s/%s", clusterConfigUrl,
_config), null, headers);
- default:
- throw new UnsupportedOperationException("Unsupported operation: " +
_operation);
+ String normalizedOperation = _operation.toUpperCase(Locale.ROOT);
+ try (PinotAdminClient adminClient = getPinotAdminClient()) {
+ switch (normalizedOperation) {
+ case "ADD":
+ case "UPDATE":
+ String[] splits = _config.split("=");
+ if (splits.length != 2) {
+ throw new UnsupportedOperationException(
+ "Bad config: " + _config + ". Please follow the pattern of
[Config Key]=[Config Value]");
+ }
+ String request =
JsonUtils.objectToString(java.util.Collections.singletonMap(splits[0],
splits[1]));
+ return adminClient.getClusterClient().updateClusterConfig(request);
Review Comment:
Minor: `java.util.Collections.singletonMap(...)` is used via a
fully-qualified reference here. Prefer adding an import for
`java.util.Collections` (consistent with the rest of the file) to improve
readability and keep imports explicit.
##########
pinot-clients/pinot-java-client/src/main/java/org/apache/pinot/client/admin/PinotLogicalTableAdminClient.java:
##########
@@ -0,0 +1,72 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.pinot.client.admin;
+
+import com.fasterxml.jackson.databind.JsonNode;
+
+/**
+ * Client for logical table administration operations.
+ */
+public class PinotLogicalTableAdminClient {
+ private final PinotAdminTransport _transport;
+ private final String _controllerAddress;
+ private final java.util.Map<String, String> _headers;
+
+ public PinotLogicalTableAdminClient(PinotAdminTransport transport, String
controllerAddress,
+ java.util.Map<String, String> headers) {
+ _transport = transport;
+ _controllerAddress = controllerAddress;
+ _headers = headers;
+ }
+
+ public String createLogicalTable(String logicalTableConfigJson)
+ throws PinotAdminException {
+ JsonNode response = _transport.executePost(_controllerAddress,
"/logicalTables", logicalTableConfigJson, null,
+ _headers);
+ System.out.println("createLogicalTable response: " + response.toString());
Review Comment:
`createLogicalTable()` prints the response via `System.out.println`, which
is unexpected for a library client and can pollute caller output. Please remove
the stdout write, or replace it with SLF4J logging at an appropriate level (or
just return the response and let callers log).
```suggestion
```
##########
pinot-tools/src/main/java/org/apache/pinot/tools/admin/command/ImportDataCommand.java:
##########
@@ -413,4 +399,24 @@ private String getRecordReaderClass(FileFormat format) {
throw new IllegalArgumentException("Unsupported file format - " +
format);
}
}
+
+ private String resolveControllerURI() {
+ if (_controllerUriProvidedExplicitly) {
+ return _controllerURI;
+ }
+ boolean controllerFieldsCustomized =
+ _controllerHost != null ||
!_controllerPort.equals(DEFAULT_CONTROLLER_PORT)
+ || !_controllerProtocol.equals(CommonConstants.HTTP_PROTOCOL);
+ if (!controllerFieldsCustomized) {
+ return _controllerURI;
+ }
+ try {
+ if (_controllerHost == null) {
+ _controllerHost = NetUtils.getHostAddress();
+ }
+ } catch (SocketException | UnknownHostException e) {
+ throw new IllegalStateException("Failed to determine controller host",
e);
+ }
+ return _controllerProtocol + "://" + _controllerHost + ":" +
_controllerPort;
+ }
Review Comment:
`_controllerUriProvidedExplicitly` is only set in `setControllerURI()`, but
picocli will set `_controllerURI` directly when `-controllerURI` is passed on
the command line. As a result, `resolveControllerURI()` can incorrectly treat a
user-supplied `-controllerURI` as “not explicit” and override it when
`-controllerHost/-controllerPort/-controllerProtocol` are also set. Consider
detecting explicit CLI usage via picocli parse results/spec, or alternatively
fail fast when both `-controllerURI` and any controller host/port/protocol
options are provided to avoid ambiguous precedence.
--
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
To unsubscribe, e-mail: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]