Skip to content

Commit ee95e0c

Browse files
authored
Merge pull request #23522 from mmaslankaprv/polaris-catalog
Polaris catalog in ducktape
2 parents 68c9aca + db63a9a commit ee95e0c

File tree

8 files changed

+397
-1
lines changed

8 files changed

+397
-1
lines changed

tests/docker/Dockerfile

+7
Original file line numberDiff line numberDiff line change
@@ -236,6 +236,12 @@ RUN /ocsf-server && rm /ocsf-server
236236

237237
#################################
238238

239+
FROM base AS polaris
240+
COPY --chown=0:0 --chmod=0755 tests/docker/ducktape-deps/polaris /
241+
RUN /polaris && rm /polaris
242+
243+
#################################
244+
239245
FROM librdkafka AS final
240246

241247
COPY --chown=0:0 --chmod=0755 tests/docker/ducktape-deps/python-deps /
@@ -325,6 +331,7 @@ COPY --from=keycloak /opt/keycloak/ /opt/keycloak/
325331
COPY --from=wasi-transforms /opt/transforms/ /opt/transforms/
326332
COPY --from=ocsf /opt/ocsf-schema/ /opt/ocsf-schema/
327333
COPY --from=ocsf /opt/ocsf-server/ /opt/ocsf-server/
334+
COPY --from=polaris /opt/polaris/ /opt/polaris/
328335
COPY --from=flink /opt/flink/ /opt/flink/
329336

330337
RUN ldconfig

tests/docker/ducktape-deps/java-dev-tools

+16
Original file line numberDiff line numberDiff line change
@@ -4,6 +4,7 @@ apt update
44
apt install -y \
55
build-essential \
66
openjdk-17-jdk \
7+
openjdk-21-jdk \
78
git \
89
maven \
910
cmake \
@@ -14,3 +15,18 @@ SCRIPTPATH="$(
1415
pwd -P
1516
)"
1617
$SCRIPTPATH/protobuf
18+
19+
update-java-alternatives -s java-1.17.0-openjdk-amd64
20+
mkdir /opt/java
21+
22+
cat <<EOF >/opt/java/java-21
23+
JAVA_HOME="/usr/lib/jvm/java-21-openjdk-amd64"
24+
/usr/lib/jvm/java-21-openjdk-amd64/bin/java "\$@"
25+
EOF
26+
chmod +x /opt/java/java-21
27+
28+
cat <<EOF >/opt/java/java-17
29+
JAVA_HOME="/usr/lib/jvm/java-17-openjdk-amd64"
30+
/usr/lib/jvm/java-17-openjdk-amd64/bin/java "\$@"
31+
EOF
32+
chmod +x /opt/java/java-17

tests/docker/ducktape-deps/polaris

+8
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,8 @@
1+
#!/usr/bin/env bash
2+
set -e
3+
update-java-alternatives -s java-1.21.0-openjdk-amd64
4+
git -C /opt clone https://github.com/apache/polaris.git
5+
cd /opt/polaris
6+
git reset --hard 1a6b3eb3963355f78c5ca916cc1d66ecd1493092
7+
./gradlew --no-daemon --info shadowJar
8+
update-java-alternatives -s java-1.17.0-openjdk-amd64
+165
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,165 @@
1+
# Copyright 2020 Vectorized, Inc.
2+
#
3+
# Use of this software is governed by the Business Source License
4+
# included in the file licenses/BSL.md
5+
#
6+
# As of the Change Date specified in that file, in accordance with
7+
# the Business Source License, use of this software will be governed
8+
# by the Apache License, Version 2.0
9+
10+
import os
11+
import json
12+
import collections
13+
import re
14+
from typing import Optional, Any
15+
16+
from ducktape.services.service import Service
17+
from ducktape.utils.util import wait_until
18+
from ducktape.cluster.cluster import ClusterNode
19+
20+
from polaris.management.api_client import ApiClient
21+
from polaris.management.configuration import Configuration
22+
from polaris.management.api.polaris_default_api import PolarisDefaultApi
23+
24+
25+
class PolarisCatalog(Service):
26+
"""Polaris Catalog service
27+
28+
The polaris catalog service maintain lifecycle of catalog process on the nodes.
29+
The service deploys polaris in a test mode with in-memory storage which is intended
30+
to be used for dev/test purposes.
31+
"""
32+
PERSISTENT_ROOT = "/var/lib/polaris"
33+
INSTALL_PATH = "/opt/polaris"
34+
JAR = "polaris-service-1.0.0-all.jar"
35+
JAR_PATH = os.path.join(INSTALL_PATH, "polaris-service/build/libs", JAR)
36+
LOG_FILE = os.path.join(PERSISTENT_ROOT, "polaris.log")
37+
POLARIS_CONFIG = os.path.join(PERSISTENT_ROOT, "polaris-server.yml")
38+
logs = {
39+
# Includes charts/ and results/ directories along with benchmark.log
40+
"polaris_logs": {
41+
"path": LOG_FILE,
42+
"collect_default": True
43+
},
44+
}
45+
# the only way to access polaris credentials running with the in-memory
46+
# storage is to parse them from standard output
47+
credentials_pattern = re.compile(
48+
"realm: default-realm root principal credentials: (?P<client_id>.+):(?P<password>.+)"
49+
)
50+
51+
nodes: list[ClusterNode]
52+
53+
def _cmd(self, node):
54+
java = "/opt/java/java-21"
55+
return f"{java} -jar {PolarisCatalog.JAR_PATH} server {PolarisCatalog.POLARIS_CONFIG} \
56+
1>> {PolarisCatalog.LOG_FILE} 2>> {PolarisCatalog.LOG_FILE} &"
57+
58+
def __init__(self, ctx, node: ClusterNode | None = None):
59+
super(PolarisCatalog, self).__init__(ctx, num_nodes=0 if node else 1)
60+
61+
if node:
62+
self.nodes = [node]
63+
self._ctx = ctx
64+
# catalog API url
65+
self.catalog_url = None
66+
# polaris management api url
67+
self.management_url = None
68+
self.client_id = None
69+
self.password = None
70+
71+
def _parse_credentials(self, node):
72+
line = node.account.ssh_output(
73+
f"grep 'root principal credentials' {PolarisCatalog.LOG_FILE}"
74+
).decode('utf-8')
75+
m = PolarisCatalog.credentials_pattern.match(line)
76+
if m is None:
77+
raise Exception(f"Unable to find credentials in line: {line}")
78+
self.client_id = m['client_id']
79+
self.password = m['password']
80+
81+
def start_node(self, node, timeout_sec=60, **kwargs):
82+
node.account.ssh("mkdir -p %s" % PolarisCatalog.PERSISTENT_ROOT,
83+
allow_fail=False)
84+
# polaris server settings
85+
cfg_yaml = self.render("polaris-server.yml")
86+
node.account.create_file(PolarisCatalog.POLARIS_CONFIG, cfg_yaml)
87+
cmd = self._cmd(node)
88+
self.logger.info(
89+
f"Starting polaris catalog service on {node.name} with command {cmd}"
90+
)
91+
node.account.ssh(cmd, allow_fail=False)
92+
93+
# wait for the healthcheck to return 200
94+
def _polaris_ready():
95+
out = node.account.ssh_output(
96+
"curl -s -o /dev/null -w '%{http_code}' http://localhost:8182/healthcheck"
97+
)
98+
status_code = int(out.decode('utf-8'))
99+
self.logger.info(f"health check result status code: {status_code}")
100+
return status_code == 200
101+
102+
wait_until(_polaris_ready,
103+
timeout_sec=timeout_sec,
104+
backoff_sec=0.4,
105+
err_msg="Error waiting for polaris catalog to start",
106+
retry_on_exc=True)
107+
108+
# setup urls and credentials
109+
self.catalog_url = f"http://{node.account.hostname}:8181/api/catalog/v1"
110+
self.management_url = f'http://{node.account.hostname}:8181/api/management/v1'
111+
self._parse_credentials(node)
112+
self.logger.info(
113+
f"Polaris catalog ready, credentials - client_id: {self.client_id}, password: {self.password}"
114+
)
115+
116+
def _get_token(self) -> str:
117+
client = ApiClient(configuration=Configuration(host=self.catalog_url))
118+
response = client.call_api('POST',
119+
f'{self.catalog_url}/oauth/tokens',
120+
header_params={
121+
'Content-Type':
122+
'application/x-www-form-urlencoded'
123+
},
124+
post_params={
125+
'grant_type': 'client_credentials',
126+
'client_id': self.client_id,
127+
'client_secret': self.password,
128+
'scope': 'PRINCIPAL_ROLE:ALL'
129+
}).response.data
130+
131+
if 'access_token' not in json.loads(response):
132+
raise Exception('Failed to get access token')
133+
return json.loads(response)['access_token']
134+
135+
def management_client(self) -> ApiClient:
136+
token = self._get_token()
137+
return ApiClient(configuration=Configuration(host=self.management_url,
138+
access_token=token))
139+
140+
def catalog_client(self) -> ApiClient:
141+
token = self._get_token()
142+
return ApiClient(configuration=Configuration(host=self.catalog_url,
143+
access_token=token))
144+
145+
def wait_node(self, node, timeout_sec=None):
146+
## unused as there is nothing to wait for here
147+
return False
148+
149+
def stop_node(self, node, allow_fail=False, **_):
150+
151+
node.account.kill_java_processes(PolarisCatalog.JAR,
152+
allow_fail=allow_fail)
153+
154+
def _stopped():
155+
out = node.account.ssh_output("jcmd").decode('utf-8')
156+
return PolarisCatalog.JAR not in out
157+
158+
wait_until(_stopped,
159+
timeout_sec=10,
160+
backoff_sec=1,
161+
err_msg="Error stopping Polaris")
162+
163+
def clean_node(self, node, **_):
164+
self.stop_node(node, allow_fail=True)
165+
node.account.remove(PolarisCatalog.PERSISTENT_ROOT, allow_fail=True)
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,121 @@
1+
#
2+
# Copyright (c) 2024 Snowflake Computing Inc.
3+
#
4+
# Licensed under the Apache License, Version 2.0 (the "License");
5+
# you may not use this file except in compliance with the License.
6+
# You may obtain a copy of the License at
7+
#
8+
# http://www.apache.org/licenses/LICENSE-2.0
9+
#
10+
# Unless required by applicable law or agreed to in writing, software
11+
# distributed under the License is distributed on an "AS IS" BASIS,
12+
# WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
13+
# See the License for the specific language governing permissions and
14+
# limitations under the License.
15+
#
16+
server:
17+
# Maximum number of threads.
18+
maxThreads: 200
19+
20+
# Minimum number of thread to keep alive.
21+
minThreads: 10
22+
applicationConnectors:
23+
# HTTP-specific options.
24+
- type: http
25+
26+
# The port on which the HTTP server listens for service requests.
27+
port: 8181
28+
29+
adminConnectors:
30+
- type: http
31+
port: 8182
32+
33+
# The hostname of the interface to which the HTTP server socket wil be found. If omitted, the
34+
# socket will listen on all interfaces.
35+
#bindHost: localhost
36+
37+
# ssl:
38+
# keyStore: ./example.keystore
39+
# keyStorePassword: example
40+
#
41+
# keyStoreType: JKS # (optional, JKS is default)
42+
43+
# HTTP request log settings
44+
requestLog:
45+
appenders:
46+
- type: console
47+
# Either 'jdbc' or 'polaris'; specifies the underlying delegate catalog
48+
baseCatalogType: "polaris"
49+
50+
featureConfiguration:
51+
ENFORCE_PRINCIPAL_CREDENTIAL_ROTATION_REQUIRED_CHECKING: false
52+
DISABLE_TOKEN_GENERATION_FOR_USER_PRINCIPALS: true
53+
SUPPORTED_CATALOG_STORAGE_TYPES:
54+
- S3
55+
- GCS
56+
- AZURE
57+
- FILE
58+
59+
# Whether we want to enable Snowflake OAuth locally. Setting this to true requires
60+
# that you go through the setup outlined in the `README.md` file, specifically the
61+
# `OAuth + Snowflake: Local Testing And Then Some` section
62+
callContextResolver:
63+
type: default
64+
65+
realmContextResolver:
66+
type: default
67+
68+
defaultRealms:
69+
- default-realm
70+
71+
metaStoreManager:
72+
type: in-memory
73+
74+
# TODO - avoid duplicating token broker config
75+
oauth2:
76+
type: test
77+
# type: default # - uncomment to support Auth0 JWT tokens
78+
# tokenBroker:
79+
# type: symmetric-key
80+
# secret: polaris
81+
82+
authenticator:
83+
class: io.polaris.service.auth.TestInlineBearerTokenPolarisAuthenticator
84+
# class: io.polaris.service.auth.DefaultPolarisAuthenticator # - uncomment to support Auth0 JWT tokens
85+
# tokenBroker:
86+
# type: symmetric-key
87+
# secret: polaris
88+
89+
cors:
90+
allowed-origins:
91+
- http://localhost:8080
92+
allowed-timing-origins:
93+
- http://localhost:8080
94+
allowed-methods:
95+
- PATCH
96+
- POST
97+
- DELETE
98+
- GET
99+
- PUT
100+
allowed-headers:
101+
- "*"
102+
exposed-headers:
103+
- "*"
104+
preflight-max-age: 600
105+
allowed-credentials: true
106+
107+
# Logging settings.
108+
109+
logging:
110+
# The default level of all loggers. Can be OFF, ERROR, WARN, INFO, DEBUG, TRACE, or ALL.
111+
level: INFO
112+
113+
# Logger-specific levels.
114+
loggers:
115+
org.apache.iceberg.rest: DEBUG
116+
io.polaris: DEBUG
117+
118+
appenders:
119+
- type: console
120+
threshold: ALL
121+
logFormat: "%-5p [%d{ISO8601} - %-6r] [%t] [%X{aid}%X{sid}%X{tid}%X{wid}%X{oid}%X{srv}%X{job}%X{rid}] %c{30}: %m %kvp%n%ex"
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,53 @@
1+
# Copyright 2020 Redpanda Data, Inc.
2+
#
3+
# Use of this software is governed by the Business Source License
4+
# included in the file licenses/BSL.md
5+
#
6+
# As of the Change Date specified in that file, in accordance with
7+
# the Business Source License, use of this software will be governed
8+
# by the Apache License, Version 2.0
9+
10+
from rptest.services.cluster import cluster
11+
12+
from rptest.services.polaris_catalog import PolarisCatalog
13+
from rptest.tests.polaris_catalog_test import PolarisCatalogTest
14+
import polaris.catalog
15+
16+
from polaris.management.api.polaris_default_api import PolarisDefaultApi
17+
from polaris.management.models.create_catalog_request import CreateCatalogRequest
18+
from polaris.management.models.catalog_properties import CatalogProperties
19+
from polaris.management.models.catalog import Catalog
20+
from polaris.management.models.storage_config_info import StorageConfigInfo
21+
22+
23+
class PolarisCatalogSmokeTest(PolarisCatalogTest):
24+
def __init__(self, test_ctx, *args, **kwargs):
25+
super(PolarisCatalogSmokeTest, self).__init__(test_ctx,
26+
num_brokers=1,
27+
*args,
28+
extra_rp_conf={},
29+
**kwargs)
30+
31+
"""
32+
Validates if the polaris catalog is accessible from ducktape tests harness
33+
"""
34+
35+
@cluster(num_nodes=2)
36+
def test_creating_catalog(self):
37+
"""The very basic test checking interaction with polaris catalog
38+
"""
39+
polaris_api = PolarisDefaultApi(self.polaris.management_client())
40+
catalog = Catalog(
41+
type="INTERNAL",
42+
name="test-catalog",
43+
properties=CatalogProperties(
44+
default_base_location=
45+
f"file://{PolarisCatalog.PERSISTENT_ROOT}/catalog_data",
46+
additional_properties={}),
47+
storageConfigInfo=StorageConfigInfo(storageType="FILE"))
48+
49+
polaris_api.create_catalog(CreateCatalogRequest(catalog=catalog))
50+
resp = polaris_api.list_catalogs()
51+
52+
assert len(resp.catalogs) == 1
53+
assert resp.catalogs[0].name == "test-catalog"

0 commit comments

Comments
 (0)