From 5b26f1ced95d48bbedeff8231e51db4fc6e66b09 Mon Sep 17 00:00:00 2001 From: rernas35 Date: Mon, 28 Feb 2022 21:54:53 +0100 Subject: [PATCH 1/2] first version of r2dbc --- clickhouse-r2dbc/README.md | 26 + clickhouse-r2dbc/pom.xml | 286 ++++ .../com/clickhouse/r2dbc/ClickHouseBatch.java | 38 + .../r2dbc/ClickHouseColumnMetadata.java | 27 + .../clickhouse/r2dbc/ClickHouseResult.java | 118 ++ .../com/clickhouse/r2dbc/ClickHouseRow.java | 48 + .../r2dbc/ClickHouseRowMetadata.java | 35 + .../clickhouse/r2dbc/ClickHouseStatement.java | 164 +++ .../r2dbc/ClickHouseStatementBinding.java | 57 + .../connection/ClickHouseConnection.java | 198 +++ .../ClickHouseConnectionFactory.java | 28 + .../ClickHouseConnectionFactoryMetadata.java | 13 + .../ClickHouseConnectionFactoryProvider.java | 67 + .../ClickHouseConnectionMetadata.java | 47 + .../types/ClickHouseDataTypeWrapper.java | 28 + .../io.r2dbc.spi.ConnectionFactoryProvider | 1 + .../r2dbc/spi/test/R2DBCTestKitImplTest.java | 302 ++++ .../config.d/docker_related_config.xml | 12 + .../clickhouse-docker-mount/config.xml | 1294 +++++++++++++++++ .../docker_related_config.xml | 12 + .../clickhouse-docker-mount/users.xml | 123 ++ .../README.md | 11 + .../pom.xml | 50 + .../spring/webflux/sample/Application.java | 17 + .../webflux/sample/config/R2DBCConfig.java | 34 + .../sample/controller/ClickController.java | 46 + .../spring/webflux/sample/model/Click.java | 24 + .../webflux/sample/model/ClickStats.java | 53 + .../sample/repository/ClickRepository.java | 47 + .../src/main/resources/application.yaml | 7 + .../src/main/resources/init.sql | 11 + .../src/main/resources/log4j.properties | 4 + ...house-r2dbc-sample.postman_collection.json | 57 + .../config.d/docker_related_config.xml | 12 + .../docker/clickhouse-docker-mount/config.xml | 1294 +++++++++++++++++ .../docker_related_config.xml | 12 + .../docker/clickhouse-docker-mount/users.xml | 123 ++ .../misc/docker/compose.yaml | 10 + examples/clickhouse-r2dbc-samples/pom.xml | 15 + pom.xml | 10 + 40 files changed, 4761 insertions(+) create mode 100644 clickhouse-r2dbc/README.md create mode 100644 clickhouse-r2dbc/pom.xml create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseBatch.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseColumnMetadata.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseResult.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseRow.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseRowMetadata.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseStatement.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseStatementBinding.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnection.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactory.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryMetadata.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryProvider.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionMetadata.java create mode 100644 clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/types/ClickHouseDataTypeWrapper.java create mode 100644 clickhouse-r2dbc/src/main/resources/META-INF/services/io.r2dbc.spi.ConnectionFactoryProvider create mode 100644 clickhouse-r2dbc/src/test/java/com/clickhouse/r2dbc/spi/test/R2DBCTestKitImplTest.java create mode 100644 clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.d/docker_related_config.xml create mode 100644 clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.xml create mode 100644 clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/docker_related_config.xml create mode 100644 clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/users.xml create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/README.md create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/pom.xml create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/Application.java create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/config/R2DBCConfig.java create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/controller/ClickController.java create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/Click.java create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/ClickStats.java create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/repository/ClickRepository.java create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/application.yaml create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/init.sql create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/log4j.properties create mode 100644 examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/postman/clickhouse-r2dbc-sample.postman_collection.json create mode 100644 examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.d/docker_related_config.xml create mode 100644 examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.xml create mode 100644 examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/docker_related_config.xml create mode 100644 examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/users.xml create mode 100644 examples/clickhouse-r2dbc-samples/misc/docker/compose.yaml create mode 100644 examples/clickhouse-r2dbc-samples/pom.xml diff --git a/clickhouse-r2dbc/README.md b/clickhouse-r2dbc/README.md new file mode 100644 index 000000000..7556c459e --- /dev/null +++ b/clickhouse-r2dbc/README.md @@ -0,0 +1,26 @@ +# clickhouse-r2dbc + +This module provides r2dbc support to clickhouse-jdbc driver. + +r2dbc link : https://r2dbc.io/ + +Sample code: +```java +ConnectionFactory connectionFactory = ConnectionFactories + .get("r2dbc:clickhouse:http://{username}:{password}@{host}:{port}/{database}"); + + Mono.from(connectionFactory.create()) + .flatMapMany(connection -> connection + .createStatement("select domain, path, toDate(cdate) as d, count(1) as count from clickdb.clicks where domain = :domain group by domain, path, d") + .bind("domain", domain) + .execute()) + .flatMap(result -> result + .map((row, rowMetadata) -> String.format("%s%s[%s]:%d", row.get("domain", String.class), + row.get("path", String.class), + row.get("d", LocalDate.class), + row.get("count", Long.class)) )) + .doOnNext(System.out::println) + .subscribe(); +``` + +for full example please check clickhouse-jdbc/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample . \ No newline at end of file diff --git a/clickhouse-r2dbc/pom.xml b/clickhouse-r2dbc/pom.xml new file mode 100644 index 000000000..49285bbdd --- /dev/null +++ b/clickhouse-r2dbc/pom.xml @@ -0,0 +1,286 @@ + + + + clickhouse-java + com.clickhouse + ${revision} + + 4.0.0 + + clickhouse-r2dbc + + + 4.0.3 + 1.8 + 1.8 + 1.0.0.RELEASE + R2DBC + 1.0 + + + + + io.projectreactor + reactor-core + provided + + + io.r2dbc + r2dbc-spi + ${r2dbc-spi.version} + + + io.r2dbc + r2dbc-spi-test + ${r2dbc-spi.version} + test + + + com.clickhouse + clickhouse-client + ${project.version} + + + com.clickhouse + clickhouse-grpc-client + ${project.version} + + + com.clickhouse + clickhouse-http-client + ${project.version} + + + com.clickhouse + clickhouse-jdbc + ${project.version} + + + org.apache.commons + commons-lang3 + ${commons-lang3.version} + + + org.testcontainers + testcontainers + test + + + com.clickhouse + clickhouse-client + ${project.version} + test-jar + test + + + com.zaxxer + HikariCP + ${hikari-cp.version} + test + + + org.slf4j + slf4j-api + + + + + org.junit.jupiter + junit-jupiter-api + 5.8.2 + test + + + + + + + org.apache.maven.plugins + maven-surefire-plugin + ${surefire-plugin.version} + + ${excludedGroups} + ${skipUTs} + false + + + + org.apache.maven.plugins + maven-shade-plugin + + + shade + package + + shade + + + true + true + true + shaded + + + com.google.gson + ${shade.base}.gson + + + org.apache + ${shade.base}.apache + + + + + + + + + ${project.groupId}.r2dbc + ${spec.title} + ${spec.version} + + + + + + ${project.parent.groupId}:clickhouse-grpc-client + + ** + + + + *:* + + mozilla/** + **/darwin/** + **/linux/** + **/win32/** + **/module-info.class + META-INF/DEPENDENCIES + META-INF/MANIFEST.MF + META-INF/maven/** + META-INF/native-image/** + META-INF/*.xml + + + + + + + shade-all + package + + shade + + + true + true + true + all + + + + + + + ${project.groupId}.r2dbc + ${spec.title} + ${spec.version} + + + + + + *:* + + com/google/** + mozilla/** + org/** + **/module-info.class + META-INF/DEPENDENCIES + META-INF/MANIFEST.MF + META-INF/maven/** + META-INF/native-image/** + META-INF/*.xml + + + + + + + shade-http + package + + shade + + + true + true + true + http + + + + + + + ${project.groupId}.r2dbc + ${spec.title} + ${spec.version} + + + + + + ${project.parent.groupId}:clickhouse-cli-client + + ** + + + + ${project.parent.groupId}:clickhouse-grpc-client + + ** + + + + *:* + + com/google/** + mozilla/** + org/** + ru/** + **/darwin/** + **/linux/** + **/win32/** + **/module-info.class + META-INF/DEPENDENCIES + META-INF/MANIFEST.MF + META-INF/maven/** + META-INF/native-image/** + META-INF/*.xml + + + + + + + + + org.codehaus.mojo + flatten-maven-plugin + + + flatten + package + + flatten + + + + + + + \ No newline at end of file diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseBatch.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseBatch.java new file mode 100644 index 000000000..c8d1344d8 --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseBatch.java @@ -0,0 +1,38 @@ +package com.clickhouse.r2dbc; + +import com.clickhouse.client.ClickHouseFormat; +import com.clickhouse.client.ClickHouseRequest; +import io.r2dbc.spi.Batch; +import io.r2dbc.spi.Result; +import org.reactivestreams.Publisher; +import reactor.core.publisher.Flux; +import reactor.core.publisher.Mono; + +import java.util.ArrayList; +import java.util.List; + +public class ClickHouseBatch implements Batch { + + private static final ClickHouseFormat PREFERRED_FORMAT = ClickHouseFormat.TabSeparatedWithNamesAndTypes; + private ClickHouseRequest request; + List sqlList = new ArrayList<>(); + + public ClickHouseBatch(ClickHouseRequest request) { + this.request = request; + } + + @Override + public Batch add(String sql) { + sqlList.add(sql); + return this; + } + + @Override + public Publisher execute() { + return Flux.fromStream(sqlList.stream().map(sql -> { + request.query(sql).format(PREFERRED_FORMAT); + return Mono.fromFuture(request::execute); })) + .flatMap(Mono::flux) + .map(ClickHouseResult::new); + } +} diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseColumnMetadata.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseColumnMetadata.java new file mode 100644 index 000000000..ac40c18a0 --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseColumnMetadata.java @@ -0,0 +1,27 @@ +package com.clickhouse.r2dbc; + +import com.clickhouse.client.ClickHouseColumn; +import com.clickhouse.r2dbc.types.ClickHouseDataTypeWrapper; +import io.r2dbc.spi.ColumnMetadata; +import io.r2dbc.spi.Type; + +public class ClickHouseColumnMetadata implements ColumnMetadata { + + final Type type; + final String name; + + ClickHouseColumnMetadata(ClickHouseColumn col) { + this.name = col.getColumnName(); // TODO :check alias handling. + this.type = ClickHouseDataTypeWrapper.of(col.getDataType()); + } + + @Override + public Type getType() { + return type; + } + + @Override + public String getName() { + return name; + } +} diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseResult.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseResult.java new file mode 100644 index 000000000..0df8734c2 --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseResult.java @@ -0,0 +1,118 @@ +package com.clickhouse.r2dbc; + +import com.clickhouse.client.ClickHouseResponse; +import com.clickhouse.client.ClickHouseResponseSummary; +import com.clickhouse.client.logging.Logger; +import com.clickhouse.client.logging.LoggerFactory; +import io.r2dbc.spi.Result; +import io.r2dbc.spi.Row; +import io.r2dbc.spi.RowMetadata; +import org.apache.commons.lang3.tuple.Pair; +import org.reactivestreams.Publisher; +import reactor.core.publisher.Flux; +import reactor.core.publisher.Mono; + +import java.util.function.BiFunction; +import java.util.function.Function; +import java.util.function.Predicate; +import java.util.stream.StreamSupport; + +public class ClickHouseResult implements Result { + + private static final Logger log = LoggerFactory.getLogger(ClickHouseResult.class); + + private final Flux rowSegments; + private final Mono updatedCount; + private final Flux segments; + + ClickHouseResult(ClickHouseResponse response) { + this.rowSegments = Mono.just(response) + .flatMapMany(resp -> Flux + .fromStream(StreamSupport.stream(resp.records().spliterator(), false) + .map(rec -> Pair.of(resp.getColumns(), rec)))) + .map(pair -> new ClickHouseRow(pair.getRight(), pair.getLeft())) + .map(RowSegment::new); + this.updatedCount = Mono.just(response).map(ClickHouseResponse::getSummary) + .map(ClickHouseResponseSummary::getProgress) + .map(ClickHouseResponseSummary.Progress::getWrittenRows) + .map(UpdateCount::new); + this.segments = Flux.concat(this.updatedCount, this.rowSegments); + } + + ClickHouseResult(Flux rowSegments, Mono updatedCount) { + this.rowSegments = rowSegments; + this.updatedCount = updatedCount; + this.segments = Flux.concat(this.updatedCount, this.rowSegments); + } + + /** + * Returns updated count(written rows from summary of {@link ClickHouseResponse}).Important! if writtenRows is greater than MAX_INT then it will return MAX_INT. + * @return updated count + */ + @Override + public Mono getRowsUpdated() { + return updatedCount.map(val -> ((UpdateCount) val).value()); + } + + @Override + public Publisher map(BiFunction biFunction) { + return rowSegments.cast(RowSegment.class) + .map(RowSegment::row).handle((row, sink) -> { + try { + sink.next(biFunction.apply(row, row.getMetadata())); + } catch (Exception e) { + log.error("Provided function caused exception:", e); + } + }); + } + + @Override + public Result filter(Predicate predicate) { + return new ClickHouseResult(segments.filter(predicate), updatedCount.filter(predicate)); + } + + @Override + public Publisher flatMap(Function> function) { + return segments.flatMap(segment -> { + try { + Publisher retValue = function.apply(segment); + if (retValue == null) { + return Mono.error(new IllegalStateException("flatmap function returned null value")); + } + return retValue; + } catch (Exception e) { + log.error("Provided function caused exception:", e); + return Mono.error(e); + } + }); + } + + + class RowSegment implements Result.RowSegment { + + final ClickHouseRow row; + + RowSegment(ClickHouseRow row) { + this.row = row; + } + + @Override + public Row row() { + return row; + } + } + + class UpdateCount implements Result.UpdateCount { + + final long updateCount; + + UpdateCount(long updateCount) { + this.updateCount = updateCount; + } + + @Override + public long value() { + return updateCount; + } + } +} diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseRow.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseRow.java new file mode 100644 index 000000000..0a217894c --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseRow.java @@ -0,0 +1,48 @@ +package com.clickhouse.r2dbc; + +import com.clickhouse.client.ClickHouseColumn; +import com.clickhouse.client.ClickHouseRecord; +import io.r2dbc.spi.Row; +import io.r2dbc.spi.RowMetadata; + +import java.util.LinkedHashMap; +import java.util.List; +import java.util.NoSuchElementException; +import java.util.function.Function; +import java.util.stream.Collectors; + +public class ClickHouseRow implements Row { + + final ClickHouseRecord record; + final ClickHouseRowMetadata rowMetadata; + + ClickHouseRow(ClickHouseRecord record, List columnList) { + this.record = record; + this.rowMetadata = new ClickHouseRowMetadata(columnList.stream() + .map(ClickHouseColumnMetadata::new) + .collect(Collectors + .toMap(ClickHouseColumnMetadata::getName, + Function.identity(), + (v1,v2) -> v2, // since every key will be unique, won't need to merge so just overwrite with the latest one. + LinkedHashMap::new))); + } + + @Override + public RowMetadata getMetadata() { + return rowMetadata; + } + + @Override + public T get(int i, Class aClass) { + return aClass.cast(record.getValue(i).asObject(aClass)); + } + + @Override + public T get(String name, Class aClass) { + try { + return aClass.cast(record.getValue(name).asObject(aClass)); + } catch (IllegalArgumentException e) { + throw new NoSuchElementException(String.format("Unknown element with a name %s", name)); + } + } +} diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseRowMetadata.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseRowMetadata.java new file mode 100644 index 000000000..fdeb26d20 --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseRowMetadata.java @@ -0,0 +1,35 @@ +package com.clickhouse.r2dbc; + +import io.r2dbc.spi.ColumnMetadata; +import io.r2dbc.spi.RowMetadata; + +import java.util.ArrayList; +import java.util.Collections; +import java.util.LinkedHashMap; +import java.util.List; + +public class ClickHouseRowMetadata implements RowMetadata { + + LinkedHashMap columnNameMetadataMap; + + ClickHouseRowMetadata( LinkedHashMap columnNameMetadataMap) { + this.columnNameMetadataMap = columnNameMetadataMap; + } + + @Override + public ColumnMetadata getColumnMetadata(int i) { + if (i > columnNameMetadataMap.size()) + throw new IllegalArgumentException("Given index is greater than size column metadata array."); + return columnNameMetadataMap.entrySet().stream().skip(i-1).findFirst().get().getValue(); + } + + @Override + public ColumnMetadata getColumnMetadata(String columnName) { + return columnNameMetadataMap.get(columnName); + } + + @Override + public List getColumnMetadatas() { + return Collections.unmodifiableList(new ArrayList<>(columnNameMetadataMap.values())); + } +} diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseStatement.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseStatement.java new file mode 100644 index 000000000..07e8cb477 --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseStatement.java @@ -0,0 +1,164 @@ +package com.clickhouse.r2dbc; + +import com.clickhouse.client.ClickHouseFormat; +import com.clickhouse.client.ClickHouseRequest; +import com.clickhouse.client.ClickHouseResponse; +import com.clickhouse.client.config.ClickHouseClientOption; +import com.clickhouse.client.logging.Logger; +import com.clickhouse.client.logging.LoggerFactory; +import io.r2dbc.spi.Blob; +import io.r2dbc.spi.Clob; +import io.r2dbc.spi.Parameter; +import io.r2dbc.spi.Result; +import io.r2dbc.spi.Statement; +import reactor.core.publisher.Flux; +import reactor.core.publisher.Mono; + +import java.sql.Timestamp; +import java.time.LocalDateTime; +import java.util.List; +import java.util.NoSuchElementException; +import java.util.stream.Stream; + +public class ClickHouseStatement implements Statement { + + private static final Logger log = LoggerFactory.getLogger(ClickHouseStatement.class); + + private static final ClickHouseFormat PREFERRED_FORMAT = ClickHouseFormat.TabSeparatedWithNamesAndTypes; + private static final String NULL_VALUES_ARE_NOT_ALLOWED_AS_VALUE = "null values are not allowed as value."; + private static final String CLASS_TYPES_ARE_NOT_ALLOWED_AS_VALUE = "class types are not allowed as value."; + private static final String INVALID_PARAMETER_INDEX = "Invalid parameter index! Parameter index must be greater than 0."; + + private static final Object EXPLICITLY_SET_NULL_VALUE = new Object(); + public static final String NULL_VALUES_ARE_NOT_ALLOWED_AS_PARAMETER_NAME = "null values are not allowed as parameter name."; + public static final String GENERATED_VALUES_CAN_NOT_BE_RETURNED_FROM_CLICKHOUSE_DATABASE = "Generated values can not be returned from Clickhouse database."; + public static final String NON_EXISTING_IDENTIFIER_TEMPLATE = "non-existing identifier : %s"; + public static final String UNSUPPORTED_DATATYPE_BLOB = "Unsupported datatype: Blob"; + public static final String UNSUPPORTED_DATATYPE_CLOB = "Unsupported datatype: Clob"; + public static final String SQL_DOESN_T_HAVE_BINDING_PARAMETER_NAMES = "Sql doesn't have binding parameter names."; + + private final ClickHouseRequest request; + private final List namedParameters; + private final ClickHouseStatementBinding bindings; + private int fetchSize; + + public ClickHouseStatement(String sql, ClickHouseRequest request) { + this.request = request + .format(PREFERRED_FORMAT) + .query(sql); + namedParameters = request.getPreparedQuery().getParameters(); + bindings = new ClickHouseStatementBinding(namedParameters.size()); + } + + + @Override + public Statement add() { + bindings.add(); + return this; + } + + @Override + public Statement bind(int identifierIndex, Object o) { + if (o == null) { + throw new IllegalArgumentException(NULL_VALUES_ARE_NOT_ALLOWED_AS_VALUE); + } else if (o instanceof Class) { + throw new IllegalArgumentException(CLASS_TYPES_ARE_NOT_ALLOWED_AS_VALUE); + } + + if (identifierIndex < 0) { + throw new IllegalArgumentException(INVALID_PARAMETER_INDEX); + } + + bindings.addBinding(identifierIndex, safeValue(o)); + return this; + } + + private Object safeValue(Object o) { + if (o instanceof Blob) { + throw new IllegalArgumentException(UNSUPPORTED_DATATYPE_BLOB); + } else if (o instanceof Clob) { + throw new IllegalArgumentException(UNSUPPORTED_DATATYPE_CLOB); + } else if (o instanceof LocalDateTime) { + LocalDateTime dateTime = (LocalDateTime) o; + return (Timestamp.valueOf(dateTime).getTime() / 1000); + } else if (o instanceof Parameter) { + Object value = ((Parameter) o).getValue(); + if (value == null) + return EXPLICITLY_SET_NULL_VALUE; + return value; + } + return o; + } + + @Override + public Statement bind(String identifierName, Object o) { + if (o == null) { + throw new IllegalArgumentException(NULL_VALUES_ARE_NOT_ALLOWED_AS_VALUE); + } else if (o instanceof Class) { + throw new IllegalArgumentException(CLASS_TYPES_ARE_NOT_ALLOWED_AS_VALUE); + } else if (namedParameters.isEmpty()) { + throw new IllegalArgumentException(SQL_DOESN_T_HAVE_BINDING_PARAMETER_NAMES); + } + int index = namedParameters.indexOf(identifierName); + if (index < 0) { + throw new NoSuchElementException(String.format(NON_EXISTING_IDENTIFIER_TEMPLATE, identifierName)); + } + bindings.addBinding(index, safeValue(o)); + return this; + } + + @Override + public Statement bindNull(int identifierIndex, Class aClass) { + if (identifierIndex < 0) { + throw new IllegalArgumentException(INVALID_PARAMETER_INDEX); + } + bindings.addBinding(identifierIndex, EXPLICITLY_SET_NULL_VALUE); + return this; + } + + @Override + public Statement bindNull(String identifierName, Class aClass) { + if (identifierName == null) { + throw new IllegalArgumentException(NULL_VALUES_ARE_NOT_ALLOWED_AS_PARAMETER_NAME); + } + bindings.addBinding(namedParameters.indexOf(identifierName), EXPLICITLY_SET_NULL_VALUE); + return this; + } + + @Override + public Statement fetchSize(int rows) { + this.fetchSize = rows; + return this; + } + + @Override + public Flux execute() { + List boundList = bindings.getBoundList(); + if (fetchSize > 0) { + log.debug("setting fetch size {}", fetchSize); + request.option(ClickHouseClientOption.MAX_RESULT_ROWS, fetchSize); + } + if (boundList.isEmpty()) { + return Flux.from(Mono.fromFuture(request::execute) + .map(ClickHouseResult::new)); + } else { + Stream> monoStream = boundList.stream().map(binding -> { + for (int i = 0; i < binding.values.length; i++ ) { + if (binding.values[i] == EXPLICITLY_SET_NULL_VALUE) { + binding.values[i] = null; + } + } + request.params(binding.values); + return Mono.fromFuture(request::execute); + }); + return Flux.fromStream(monoStream) + .flatMap(Mono::flux) + .map(ClickHouseResult::new); + } + } + + @Override + public Statement returnGeneratedValues(String... columns) { + throw new UnsupportedOperationException(GENERATED_VALUES_CAN_NOT_BE_RETURNED_FROM_CLICKHOUSE_DATABASE); + } +} diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseStatementBinding.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseStatementBinding.java new file mode 100644 index 000000000..e202c388a --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseStatementBinding.java @@ -0,0 +1,57 @@ +package com.clickhouse.r2dbc; + +import java.util.ArrayList; +import java.util.List; + +class ClickHouseStatementBinding { + public static final String NOT_ALL_PARAMETERS_ARE_SET = "Not all parameters are set."; + List bindingList; + Binding current; + int size; + + ClickHouseStatementBinding(int size) { + this.size = size; + current = new Binding(size); + bindingList = new ArrayList<>(); + } + + void addBinding(int index, Object value) { + current.setParam(index, value); + } + + void add() { + if (current.isCompleted()) { + bindingList.add(current); + current = new Binding(size); + } + + } + + List getBoundList() { + List bindingList = (this.bindingList == null) ? new ArrayList<>() : this.bindingList; + if (current.values.length > 0 && current.isCompleted()) + bindingList.add(current); + return bindingList; + } + + public static class Binding { + + Object[] values; + + private Binding(int size) { + values = new Object[size]; + } + + private void setParam(int index, Object value) { + values[index] = value; + } + + private boolean isCompleted(){ + for (Object value: values) { + if (value == null) throw new IllegalStateException(NOT_ALL_PARAMETERS_ARE_SET); + } + return true; + } + } +} + diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnection.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnection.java new file mode 100644 index 000000000..ffc68acce --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnection.java @@ -0,0 +1,198 @@ +package com.clickhouse.r2dbc.connection; + +import com.clickhouse.client.ClickHouseClient; +import com.clickhouse.client.ClickHouseNode; +import com.clickhouse.client.ClickHouseNodeSelector; +import com.clickhouse.client.ClickHouseNodes; +import com.clickhouse.client.ClickHouseProtocol; +import com.clickhouse.client.ClickHouseRequest; +import com.clickhouse.client.config.ClickHouseClientOption; +import com.clickhouse.client.logging.Logger; +import com.clickhouse.client.logging.LoggerFactory; +import com.clickhouse.r2dbc.ClickHouseBatch; +import com.clickhouse.r2dbc.ClickHouseStatement; +import io.r2dbc.spi.Batch; +import io.r2dbc.spi.Connection; +import io.r2dbc.spi.ConnectionMetadata; +import io.r2dbc.spi.IsolationLevel; +import io.r2dbc.spi.Statement; +import io.r2dbc.spi.TransactionDefinition; +import io.r2dbc.spi.ValidationDepth; +import org.reactivestreams.Publisher; +import reactor.core.publisher.Mono; + +import java.time.Duration; + +import static reactor.core.publisher.Mono.just; + +public class ClickHouseConnection implements Connection { + + private static final Logger log = LoggerFactory.getLogger(ClickHouseConnection.class); + + public static final int DEFAULT_TIMEOUT_FOR_CONNECTION_HEALTH_CHECK = (Integer) ClickHouseClientOption.CONNECTION_TIMEOUT.getDefaultValue(); + final ClickHouseClient client; + final ClickHouseNode node; + private boolean closed = false; + + ClickHouseConnection(ClickHouseNodes nodes) { + ClickHouseNode node = nodes.apply(ClickHouseNodeSelector.EMPTY); + this.node = nodes.apply(ClickHouseNodeSelector.EMPTY); + this.client = ClickHouseClient.newInstance(node.getProtocol()); + } + + + /** + * Transactions are not supported so this is a no-op implementation, + */ + @Override + public Mono beginTransaction() { + log.debug("Clickhouse does not support transactions so skipping initialization of transaction."); + return Mono.empty(); + } + + /** + * Transactions are not supported so this is a no-op implementation, + */ + @Override + public Mono beginTransaction(TransactionDefinition transactionDefinition) { + log.debug("Clickhouse does not support transactions so skipping initialization of transaction."); + return Mono.empty(); + } + + @Override + public Publisher close() { + try { + client.close(); + closed = true; + return Mono.empty(); + } catch (Exception e) { + return Mono.error(e); + } + } + + /** + * Transactions are not supported so this is a no-op implementation, + */ + @Override + public Publisher commitTransaction() { + log.debug("Clickhouse does not support transactions so skipping commit of transaction."); + return Mono.empty(); + } + + /** + * Returns {@link ClickHouseBatch} for batching statements. + * @return Batch object + */ + @Override + public Batch createBatch() { + ClickHouseRequest req = client.connect(node); + if (isHttp()) { + req = req.set("send_progress_in_http_headers", 1); + } + req.option(ClickHouseClientOption.ASYNC, true); + return new ClickHouseBatch(req); + } + + /** + * Returns true since there is no transaction support. + * @return true + */ + @Override + public Publisher createSavepoint(String s) { + return Mono.empty(); + } + + @Override + public Statement createStatement(String sql) { + ClickHouseRequest req = client.connect(node); + if (isHttp()) { + req = req.set("send_progress_in_http_headers", 1); + } + req.option(ClickHouseClientOption.ASYNC, true); + return new ClickHouseStatement(sql, req); + } + + private boolean isHttp() { + return node.getProtocol() == ClickHouseProtocol.HTTP; + } + + /** + * Returns true since there is no transaction support. + * @return true + */ + @Override + public boolean isAutoCommit() { + return true; + } + + + @Override + public ConnectionMetadata getMetadata() { + return new ClickHouseConnectionMetadata(client, node); + } + + /** + * + * @return Always returns read committed. + */ + @Override + public IsolationLevel getTransactionIsolationLevel() { + return IsolationLevel.READ_COMMITTED; + } + + @Override + public Publisher releaseSavepoint(String s) { + return null; + } + + /** + * Transactions are not supported so this is a no-op implementation, + */ + @Override + public Publisher rollbackTransaction() { + log.debug("Clickhouse does not support transactions so skipping rollback of transaction."); + return Mono.empty(); + } + + @Override + public Publisher rollbackTransactionToSavepoint(String s) { + return null; + } + + /** + * Transactions are not supported so this is a no-op implementation, + */ + @Override + public Publisher setAutoCommit(boolean b) { + log.debug("Clickhouse does not support transactions so skipping setting of transaction auto commit."); + return Mono.empty(); + } + + @Override + public Publisher setLockWaitTimeout(Duration duration) { + return null; + } + + @Override + public Publisher setStatementTimeout(Duration duration) { + return null; + } + + /** + * Since transactions are not supported, this method will throw exception. + * @param isolationLevel isolation level for transaction + */ + @Override + public Mono setTransactionIsolationLevel(IsolationLevel isolationLevel) { + return Mono.error(new UnsupportedOperationException("Transaction isolation level can not be changed.")); + } + + @Override + public Publisher validate(ValidationDepth validationDepth) { + if (validationDepth == ValidationDepth.REMOTE) { + return closed ? just(false) : just(client.ping(node, DEFAULT_TIMEOUT_FOR_CONNECTION_HEALTH_CHECK)); + } else { // validationDepth.LOCAL + return just(client != null && !closed); + } + } +} diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactory.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactory.java new file mode 100644 index 000000000..ff3ee2854 --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactory.java @@ -0,0 +1,28 @@ +package com.clickhouse.r2dbc.connection; + + +import com.clickhouse.client.ClickHouseNodes; +import io.r2dbc.spi.Connection; +import io.r2dbc.spi.ConnectionFactory; +import io.r2dbc.spi.ConnectionFactoryMetadata; +import reactor.core.publisher.Mono; + +public class ClickHouseConnectionFactory implements ConnectionFactory { + + + private final ClickHouseNodes nodes; + + ClickHouseConnectionFactory(ClickHouseNodes nodes) { + this.nodes = nodes; + } + + @Override + public Mono create() { + return Mono.just(new ClickHouseConnection(nodes)); + } + + @Override + public ConnectionFactoryMetadata getMetadata() { + return ClickHouseConnectionFactoryMetadata.INSTANCE; + } +} diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryMetadata.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryMetadata.java new file mode 100644 index 000000000..c8835a0c1 --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryMetadata.java @@ -0,0 +1,13 @@ +package com.clickhouse.r2dbc.connection; + +import io.r2dbc.spi.ConnectionFactoryMetadata; + +public class ClickHouseConnectionFactoryMetadata implements ConnectionFactoryMetadata { + + static final ClickHouseConnectionFactoryMetadata INSTANCE = new ClickHouseConnectionFactoryMetadata(); + + @Override + public String getName() { + return "ClickHouse"; + } +} diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryProvider.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryProvider.java new file mode 100644 index 000000000..142554c69 --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryProvider.java @@ -0,0 +1,67 @@ +package com.clickhouse.r2dbc.connection; + +import com.clickhouse.client.ClickHouseNodes; +import io.r2dbc.spi.ConnectionFactory; +import io.r2dbc.spi.ConnectionFactoryOptions; +import io.r2dbc.spi.Option; + +import java.util.Arrays; +import java.util.List; +import java.util.stream.Collectors; + +import static io.r2dbc.spi.ConnectionFactoryOptions.*; + +public class ClickHouseConnectionFactoryProvider implements io.r2dbc.spi.ConnectionFactoryProvider { + + /** + * The name of the driver used for discovery, should not be changed. + */ + public static final String CLICKHOUSE_DRIVER = "clickhouse"; + + private static final List connQueryParams = Arrays.asList("auto_discovery", "node_discovery_interval", + "node_discovery_limit", "load_balancing_policy", "load_balancing_tags", "health_check_method", + "health_check_interval", "check_all_nodes", "node_check_interval", "node_group_size", "failover", + "retry"); + + @Override + public ConnectionFactory create(ConnectionFactoryOptions cfOpt) { + String hosts = getHosts(cfOpt); + String database = cfOpt.getValue(DATABASE).toString(); + String protocol = cfOpt.getValue(PROTOCOL).toString(); + if (cfOpt.getValue(USER) == null ) { + throw new IllegalArgumentException("User and password is mandatory."); + } + String username = cfOpt.getValue(USER).toString(); + String password = ""; + if (cfOpt.getValue(PASSWORD) != null) { + password = cfOpt.getValue(PASSWORD).toString(); + } + + StringBuilder urlBuilder = new StringBuilder(String.format("%s://%s/%s?user=%s&password=%s", protocol, hosts, database, username, password)); + String params = connQueryParams.stream().filter(queryParam -> cfOpt.getValue(Option.valueOf(queryParam)) != null) + .map(queryParam -> String.format("%s=%s", queryParam, cfOpt.getValue(Option.valueOf(queryParam)))) + .collect(Collectors.joining("%")); + urlBuilder.append(params.isEmpty() ? "" : ("&" + params)); + + ClickHouseNodes nodes = ClickHouseNodes.of(urlBuilder.toString()); + return new ClickHouseConnectionFactory(nodes); + } + + private String getHosts(ConnectionFactoryOptions cfOpt) { + String hosts = cfOpt.getValue(HOST).toString(); + if (!hosts.contains(",") && !hosts.contains(":")){ + return hosts + ":" + cfOpt.getValue(PORT); + } + return hosts; + } + + @Override + public boolean supports(ConnectionFactoryOptions connectionFactoryOptions) { + return connectionFactoryOptions.getValue(DRIVER).equals(CLICKHOUSE_DRIVER); + } + + @Override + public String getDriver() { + return CLICKHOUSE_DRIVER; + } +} diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionMetadata.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionMetadata.java new file mode 100644 index 000000000..727c06c70 --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionMetadata.java @@ -0,0 +1,47 @@ +package com.clickhouse.r2dbc.connection; + +import com.clickhouse.client.ClickHouseClient; +import com.clickhouse.client.ClickHouseNode; +import com.clickhouse.client.ClickHouseResponse; +import com.clickhouse.client.logging.Logger; +import com.clickhouse.client.logging.LoggerFactory; +import io.r2dbc.spi.ConnectionMetadata; + +public class ClickHouseConnectionMetadata implements ConnectionMetadata { + + private static final Logger log = LoggerFactory.getLogger(ClickHouseConnectionMetadata.class); + + final ClickHouseClient client; + final ClickHouseNode server; + + private final String serverVersion = null; + + ClickHouseConnectionMetadata(ClickHouseClient client, ClickHouseNode server) { + this.client = client; + this.server = server; + } + + @Override + public String getDatabaseProductName() { + return "Clickhouse"; + } + + /** + * Blocking operation. Queries server version by calling "SELECT version()" statement. + * @return server version + */ + @Override + public String getDatabaseVersion() { + if (serverVersion != null) { + return serverVersion; + } + try { + // blocking here + ClickHouseResponse resp = client.connect(server).query("SELECT version()").executeAndWait(); + return resp.records().iterator().next().getValue(0).asString(); + } catch (Exception e) { + log.error("While fetching server version, error occured.", e); + return null; + } + } +} diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/types/ClickHouseDataTypeWrapper.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/types/ClickHouseDataTypeWrapper.java new file mode 100644 index 000000000..dc324319d --- /dev/null +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/types/ClickHouseDataTypeWrapper.java @@ -0,0 +1,28 @@ +package com.clickhouse.r2dbc.types; + +import com.clickhouse.client.ClickHouseDataType; +import io.r2dbc.spi.Type; + + +public class ClickHouseDataTypeWrapper implements Type { + ClickHouseDataType dType; + + private ClickHouseDataTypeWrapper(ClickHouseDataType dType){ + this.dType = dType; + } + + public static ClickHouseDataTypeWrapper of(ClickHouseDataType dType) { + return new ClickHouseDataTypeWrapper(dType); + } + + + @Override + public Class getJavaType() { + return dType.getObjectClass(); + } + + @Override + public String getName() { + return dType.name(); + } +} diff --git a/clickhouse-r2dbc/src/main/resources/META-INF/services/io.r2dbc.spi.ConnectionFactoryProvider b/clickhouse-r2dbc/src/main/resources/META-INF/services/io.r2dbc.spi.ConnectionFactoryProvider new file mode 100644 index 000000000..448a87cc4 --- /dev/null +++ b/clickhouse-r2dbc/src/main/resources/META-INF/services/io.r2dbc.spi.ConnectionFactoryProvider @@ -0,0 +1 @@ +com.clickhouse.r2dbc.connection.ClickHouseConnectionFactoryProvider diff --git a/clickhouse-r2dbc/src/test/java/com/clickhouse/r2dbc/spi/test/R2DBCTestKitImplTest.java b/clickhouse-r2dbc/src/test/java/com/clickhouse/r2dbc/spi/test/R2DBCTestKitImplTest.java new file mode 100644 index 000000000..d31cda77f --- /dev/null +++ b/clickhouse-r2dbc/src/test/java/com/clickhouse/r2dbc/spi/test/R2DBCTestKitImplTest.java @@ -0,0 +1,302 @@ +package com.clickhouse.r2dbc.spi.test; + +import com.clickhouse.client.ClickHouseException; +import com.clickhouse.client.ClickHouseProtocol; +import com.clickhouse.client.ClickHouseServerForTest; +import com.clickhouse.jdbc.ClickHouseDriver; +import com.zaxxer.hikari.HikariDataSource; +import io.r2dbc.spi.Blob; +import io.r2dbc.spi.Clob; +import io.r2dbc.spi.Connection; +import io.r2dbc.spi.ConnectionFactories; +import io.r2dbc.spi.ConnectionFactory; +import io.r2dbc.spi.Statement; +import io.r2dbc.spi.test.TestKit; +import org.junit.jupiter.api.BeforeAll; +import org.junit.jupiter.api.Disabled; +import org.junit.jupiter.api.Test; +import org.springframework.jdbc.core.JdbcOperations; +import org.springframework.jdbc.core.JdbcTemplate; +import reactor.core.publisher.Flux; +import reactor.core.publisher.Mono; +import reactor.test.StepVerifier; + +import java.nio.ByteBuffer; +import java.sql.Driver; +import java.sql.DriverManager; +import java.sql.SQLException; +import java.time.Duration; +import java.time.ZoneId; +import java.util.Optional; +import java.util.TimeZone; + +import static com.clickhouse.client.ClickHouseServerForTest.getClickHouseAddress; +import static java.lang.String.format; +import static org.junit.jupiter.api.Assertions.assertThrows; + + +public class R2DBCTestKitImplTest implements TestKit { + + private static final String DATABASE = "default"; + private static final String USER = "default"; + public static final String PASSWORD = ""; + + static ConnectionFactory connectionFactory; + static JdbcTemplate jdbcTemplate; + + @BeforeAll + public static void setup() throws Exception { + ClickHouseServerForTest.beforeSuite(); + connectionFactory = ConnectionFactories.get( + format("r2dbc:clickhouse:http://%s:%s@%s/%s?falan=filan#tag1", USER, PASSWORD, + getClickHouseAddress(ClickHouseProtocol.HTTP, false), DATABASE)); + jdbcTemplate = jdbcTemplate(null); + } + + @Override + public ConnectionFactory getConnectionFactory() { + return connectionFactory; + } + + @Override + public String getPlaceholder(int i) { + return ":param" + i; + } + + @Override + public String getIdentifier(int i) { + return "param" + i; + } + + @Override + public JdbcOperations getJdbcOperations() { + return jdbcTemplate; + } + + private static JdbcTemplate jdbcTemplate(String database) throws SQLException { + HikariDataSource source = new HikariDataSource(); + + Driver driver = new ClickHouseDriver(); + DriverManager.registerDriver(driver); + if (database == null) { + source.setJdbcUrl(format("jdbc:clickhouse://%s", getClickHouseAddress(ClickHouseProtocol.HTTP, false))); + } else { + source.setJdbcUrl(format("jdbc:clickhouse://%s/%s", getClickHouseAddress(ClickHouseProtocol.HTTP, false), DATABASE)); + } + + source.setUsername(USER); + source.setPassword(Optional.ofNullable(PASSWORD) + .map(Object::toString).orElse(null)); + source.setMaximumPoolSize(1); + source.setConnectionTimeout(Optional.ofNullable(Duration.ofSeconds(5)) + .map(Duration::toMillis).orElse(0L)); + + ZoneId zoneId = ZoneId.systemDefault(); + source.addDataSourceProperty("serverTimezone", TimeZone.getTimeZone(zoneId).getID()); + + return new JdbcTemplate(source); + } + + @Override + @Test + public void blobInsert() { + Flux.usingWhen(getConnectionFactory().create(), + connection -> { + + Statement statement = connection.createStatement(expand(TestStatement.INSERT_VALUE_PLACEHOLDER, getPlaceholder(0))); + assertThrows(IllegalArgumentException.class, () -> statement.bind(0, Blob.from(Mono.just(ByteBuffer.wrap("Unsupported type".getBytes())))), "bind(0, Blob) should fail"); + return Mono.empty(); + }, + Connection::close) + .as(StepVerifier::create) + .verifyComplete(); + } + + @Override + @Test + public void clobInsert() { + Flux.usingWhen(getConnectionFactory().create(), + connection -> { + + Statement statement = connection.createStatement(expand(TestStatement.INSERT_VALUE_PLACEHOLDER, getPlaceholder(0))); + assertThrows(IllegalArgumentException.class, () -> statement.bind(0, Clob.from(Mono.just("Unsupported type"))), "bind(0, Clob) should fail"); + return Mono.empty(); + }, + Connection::close) + .as(StepVerifier::create) + .verifyComplete(); + } + + @Override + @Disabled + public void blobSelect() { + // not supported + } + + @Override + @Disabled + public void clobSelect() { + // not supported + } + + @Override + @Test + public void columnMetadata() { + getJdbcOperations().execute(expand(TestStatement.INSERT_TWO_COLUMNS)); + + Flux.usingWhen(getConnectionFactory().create(), + connection -> Flux.from(connection + + .createStatement(expand(TestStatement.SELECT_VALUE_TWO_COLUMNS)) + .execute()), + Connection::close) + .as(StepVerifier::create) + .expectErrorMatches(ClickHouseException.class::isInstance) + .verify(); + } + + @Override + @Test + // TODO: check if it is doable. + public void compoundStatement() { + //compound statements are not supported by clickhouse. + getJdbcOperations().execute(expand(TestStatement.INSERT_VALUE100)); + + Flux.usingWhen(getConnectionFactory().create(), + connection -> Flux.from(connection + + .createStatement(expand(TestStatement.SELECT_VALUE_BATCH)) + .execute()), + Connection::close) + .as(StepVerifier::create) + .expectErrorMatches(ClickHouseException.class::isInstance) + .verify(); + } + + @Override + @Test + public void duplicateColumnNames() { + getJdbcOperations().execute(expand(TestStatement.INSERT_TWO_COLUMNS)); + + Flux.usingWhen(getConnectionFactory().create(), + connection -> Flux.from(connection + + .createStatement(expand(TestStatement.SELECT_VALUE_TWO_COLUMNS)) + .execute()), + Connection::close) + + .as(StepVerifier::create) + .expectErrorMatches(ClickHouseException.class::isInstance) + .verify(); + } + + @Override + @Test + public void returnGeneratedValues() { + getJdbcOperations().execute(expand(TestStatement.DROP_TABLE)); + getJdbcOperations().execute(getCreateTableWithAutogeneratedKey()); + Flux.usingWhen(getConnectionFactory().create(), + connection -> { + Statement statement = connection.createStatement(getInsertIntoWithAutogeneratedKey()); + + statement.returnGeneratedValues(); + + return Flux.from(statement + .execute()) + .flatMap(it -> it.map((row, rowMetadata) -> row.get(0))); + }, + Connection::close) + .as(StepVerifier::create) + .expectErrorMatches(UnsupportedOperationException.class::isInstance) + .verify(); + } + + @Override + @Test + public void returnGeneratedValuesFails() { + + Flux.usingWhen(getConnectionFactory().create(), + connection -> { + Statement statement = connection.createStatement(expand(TestStatement.INSERT_VALUE100)); + + assertThrows(UnsupportedOperationException.class, () -> statement.returnGeneratedValues((String[]) null)); + return Mono.empty(); + }, + Connection::close) + .as(StepVerifier::create) + .verifyComplete(); + } + + @Override + @Test + @Disabled + public void transactionRollback() { + // since there is not transaction support, this test case is disabled. + } + + @Override + @Test + @Disabled + public void sameAutoCommitLeavesTransactionUnchanged() { + // since there is not transaction support, this test case is disabled. + } + + @Override + @Test + @Disabled + public void savePoint() { + + } + + @Override + @Test + @Disabled + public void savePointStartsTransaction() { + + } + + @Override + public String expand(TestStatement statement, Object... args) { + try { + String sql = ClickHouseTestStatement.get(statement).getSql(); + return String.format(sql, args); + } catch (IllegalArgumentException e) { + return String.format(statement.getSql(), args); + } + } + + + private enum ClickHouseTestStatement { + CREATE_TABLE(TestStatement.CREATE_TABLE, "CREATE TABLE test ( test_value INTEGER ) ENGINE = Memory"), + CREATE_TABLE_TWO_COLUMNS(TestStatement.CREATE_TABLE_TWO_COLUMNS, "CREATE TABLE test_two_column ( col1 INTEGER, col2 VARCHAR(100) ) ENGINE = Memory"), + CREATE_BLOB_TABLE(TestStatement.CREATE_BLOB_TABLE, "CREATE TABLE blob_test ( test_value %s ) ENGINE = Memory"), + CREATE_CLOB_TABLE(TestStatement.CREATE_CLOB_TABLE, "CREATE TABLE clob_test ( test_value %s ) ENGINE = Memory"), + CREATE_TABLE_AUTOGENERATED_KEY(TestStatement.CREATE_TABLE_AUTOGENERATED_KEY, "CREATE TABLE test ( id DATE DEFAULT toDate(now()) , test_value INTEGER ) ENGINE = Memory"), + INSERT_VALUE_AUTOGENERATED_KEY(TestStatement.INSERT_VALUE_AUTOGENERATED_KEY, "INSERT INTO test(test_value) VALUES(100)"); + + + ClickHouseTestStatement(TestStatement testStatement, String sql) { + this.testStatementToBeOverwridden = testStatement; + this.sql = sql; + } + + TestStatement testStatementToBeOverwridden; + String sql; + + static ClickHouseTestStatement get(TestStatement testStatement) { + for (ClickHouseTestStatement cts : values()) { + if (cts.getTestStatementToBeOverwridden() == testStatement) + return cts; + } + throw new IllegalArgumentException("Teststatement is not found."); + } + + public String getSql() { + return sql; + } + + public TestStatement getTestStatementToBeOverwridden() { + return testStatementToBeOverwridden; + } + } +} diff --git a/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.d/docker_related_config.xml b/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.d/docker_related_config.xml new file mode 100644 index 000000000..3025dc269 --- /dev/null +++ b/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.d/docker_related_config.xml @@ -0,0 +1,12 @@ + + + :: + 0.0.0.0 + 1 + + + diff --git a/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.xml b/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.xml new file mode 100644 index 000000000..7e103bd6b --- /dev/null +++ b/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.xml @@ -0,0 +1,1294 @@ + + + + + + trace + /var/log/clickhouse-server/clickhouse-server.log + /var/log/clickhouse-server/clickhouse-server.err.log + + 1000M + 10 + + + + + + + + + + + + + + + + + + 8123 + + + 9000 + + + 9004 + + + 9005 + + + + + + + + + + + + 9009 + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + 4096 + + + 3 + + + 9100 + + false + + + /path/to/ssl_cert_file + /path/to/ssl_key_file + + + false + + + /path/to/ssl_ca_cert_file + + + deflate + + + medium + + + -1 + -1 + + + false + + + + + + + /etc/clickhouse-server/server.crt + /etc/clickhouse-server/server.key + + /etc/clickhouse-server/dhparam.pem + none + true + true + sslv2,sslv3 + true + + + + true + true + sslv2,sslv3 + true + + + + RejectCertificateHandler + + + + + + + + + 100 + + + 0 + + + + 10000 + + + + + + 0.9 + + + 4194304 + + + 0 + + + + + + 8589934592 + + + 5368709120 + + + + 1000 + + + 134217728 + + + 10000 + + + /var/lib/clickhouse/ + + + /var/lib/clickhouse/tmp/ + + + + + + /var/lib/clickhouse/user_files/ + + + + + + + + + + + + + users.xml + + + + /var/lib/clickhouse/access/ + + + + + + + default + + + + + + + + + + + + default + + + + + + + + + true + + + false + + ' | sed -e 's|.*>\(.*\)<.*|\1|') + wget https://github.com/ClickHouse/clickhouse-jdbc-bridge/releases/download/v$PKG_VER/clickhouse-jdbc-bridge_$PKG_VER-1_all.deb + apt install --no-install-recommends -f ./clickhouse-jdbc-bridge_$PKG_VER-1_all.deb + clickhouse-jdbc-bridge & + + * [CentOS/RHEL] + export MVN_URL=https://repo1.maven.org/maven2/ru/yandex/clickhouse/clickhouse-jdbc-bridge + export PKG_VER=$(curl -sL $MVN_URL/maven-metadata.xml | grep '' | sed -e 's|.*>\(.*\)<.*|\1|') + wget https://github.com/ClickHouse/clickhouse-jdbc-bridge/releases/download/v$PKG_VER/clickhouse-jdbc-bridge-$PKG_VER-1.noarch.rpm + yum localinstall -y clickhouse-jdbc-bridge-$PKG_VER-1.noarch.rpm + clickhouse-jdbc-bridge & + + Please refer to https://github.com/ClickHouse/clickhouse-jdbc-bridge#usage for more information. + ]]> + + + + + + + + + + + + + + + + localhost + 9000 + + + + + + + + false + + 127.0.0.1 + 9000 + + + 127.0.0.2 + 9000 + + + 127.0.0.3 + 9000 + + + + + + + + localhost + 9000 + + + + + localhost + 9000 + + + + + + + 127.0.0.1 + 9000 + + + + + 127.0.0.2 + 9000 + + + + + + true + + 127.0.0.1 + 9000 + + + + true + + 127.0.0.2 + 9000 + + + + + + + localhost + 9440 + 1 + + + + + + + localhost + 9000 + + + + + localhost + 1 + + + + + + + + + + + + + + + + + + + + + + + + 3600 + + + + 3600 + + + 60 + + + + + + + + + + + + + system + query_log
+ + toYYYYMM(event_date) + + + + + + 7500 +
+ + + + system + trace_log
+ + toYYYYMM(event_date) + 7500 +
+ + + + system + query_thread_log
+ toYYYYMM(event_date) + 7500 +
+ + + + system + query_views_log
+ toYYYYMM(event_date) + 7500 +
+ + + + system + part_log
+ toYYYYMM(event_date) + 7500 +
+ + + + + + system + metric_log
+ 7500 + 1000 +
+ + + + system + asynchronous_metric_log
+ + 7000 +
+ + + + + + engine MergeTree + partition by toYYYYMM(finish_date) + order by (finish_date, finish_time_us, trace_id) + + system + opentelemetry_span_log
+ 7500 +
+ + + + + system + crash_log
+ + + 1000 +
+ + + + system + session_log
+ + toYYYYMM(event_date) + 7500 +
+ + + + + + + + + + + + + + + + + + *_dictionary.xml + + + *_function.xml + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + /clickhouse/task_queue/ddl + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + click_cost + any + + 0 + 3600 + + + 86400 + 60 + + + + max + + 0 + 60 + + + 3600 + 300 + + + 86400 + 3600 + + + + + + /var/lib/clickhouse/format_schemas/ + + + + + hide encrypt/decrypt arguments + ((?:aes_)?(?:encrypt|decrypt)(?:_mysql)?)\s*\(\s*(?:'(?:\\'|.)+'|.*?)\s*\) + + \1(???) + + + + + + + + + + false + + false + + + https://6f33034cfe684dd7a3ab9875e57b1c8d@o388870.ingest.sentry.io/5226277 + + + + + + + +
diff --git a/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/docker_related_config.xml b/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/docker_related_config.xml new file mode 100644 index 000000000..3025dc269 --- /dev/null +++ b/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/docker_related_config.xml @@ -0,0 +1,12 @@ + + + :: + 0.0.0.0 + 1 + + + diff --git a/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/users.xml b/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/users.xml new file mode 100644 index 000000000..fd5fe4145 --- /dev/null +++ b/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/users.xml @@ -0,0 +1,123 @@ + + + + + + + + + + 10000000000 + + + random + + + + + 1 + + + + + + + + + + + + + ::/0 + + + + default + + + default + + + + + + + + + + + + + + 3600 + + + 0 + 0 + 0 + 0 + 0 + + + + diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/README.md b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/README.md new file mode 100644 index 000000000..2a1d58768 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/README.md @@ -0,0 +1,11 @@ +#clickhouse-r2dbc-spring-webflux-sample + +This is a sample rest api which will insert clicks and get the list of clicks per day. + +In order to run the application; +- Go clickhouse-jdbc/examples/clickhouse-r2dbc-samples/misc/docker and run; +``` docker-compose up -d ``` +- Execute the table creation sql at clickhouse-jdbc/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/init.sql . +- Import the postman export clickhouse-jdbc/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/postman . +- Run the application by using Application.java . +- Create some clicks by postman and list daily clicks per domain and path. \ No newline at end of file diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/pom.xml b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/pom.xml new file mode 100644 index 000000000..f1c91e008 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/pom.xml @@ -0,0 +1,50 @@ + + + + com.clickhouse + clickhouse-r2dbc-samples + 1.0.0-SNAPSHOT + + 4.0.0 + + clickhouse-r2dbc-spring-webflux-sample + + + 1.8 + 1.8 + 2.7.1 + + + + + org.springframework.boot + spring-boot-starter-webflux + ${spring-boot-starter.version} + + + org.slf4j + slf4j-api + 2.0.0-alpha0 + + + org.slf4j + slf4j-log4j12 + 2.0.0-alpha0 + runtime + + + com.clickhouse + clickhouse-r2dbc + 0.3.3-SNAPSHOT + + + com.fasterxml.jackson.datatype + jackson-datatype-jsr310 + 2.13.3 + + + + + \ No newline at end of file diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/Application.java b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/Application.java new file mode 100644 index 000000000..142338a58 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/Application.java @@ -0,0 +1,17 @@ +package com.clickhouse.r2dbc.spring.webflux.sample; + +import org.springframework.boot.SpringApplication; +import org.springframework.boot.autoconfigure.SpringBootApplication; +import org.springframework.boot.autoconfigure.data.r2dbc.R2dbcDataAutoConfiguration; +import org.springframework.boot.autoconfigure.r2dbc.R2dbcAutoConfiguration; +import org.springframework.web.reactive.config.EnableWebFlux; + +@SpringBootApplication(exclude = { R2dbcAutoConfiguration.class, + R2dbcDataAutoConfiguration.class}) +@EnableWebFlux +public class Application { + + public static void main(String[] args) { + SpringApplication.run(Application.class, args); + } +} diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/config/R2DBCConfig.java b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/config/R2DBCConfig.java new file mode 100644 index 000000000..0fbff97b9 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/config/R2DBCConfig.java @@ -0,0 +1,34 @@ +package com.clickhouse.r2dbc.spring.webflux.sample.config; + +import io.r2dbc.spi.ConnectionFactory; +import org.springframework.beans.factory.annotation.Value; +import org.springframework.context.annotation.Bean; +import org.springframework.context.annotation.Configuration; + +import static io.r2dbc.spi.ConnectionFactories.get; +import static java.lang.String.format; + +@Configuration +public class R2DBCConfig { + + @Value("${clickhouse.host:localhost}") + private String host; + + @Value("${clickhouse.port:8123}") + private String port; + + @Value("${clickhouse.database:clickdb}") + private String database; + + @Value("${clickhouse.user:default}") + private String user; + + @Value("${clickhouse.password:''}") + private String password; + + @Bean + public ConnectionFactory connectionFactory() { + return get(format("r2dbc:clickhouse:http://%s:%s@%s:%d/%s", user, password, + host, Integer.parseInt(port), database)); + } +} diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/controller/ClickController.java b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/controller/ClickController.java new file mode 100644 index 000000000..ea79b3327 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/controller/ClickController.java @@ -0,0 +1,46 @@ +package com.clickhouse.r2dbc.spring.webflux.sample.controller; + +import com.clickhouse.r2dbc.spring.webflux.sample.model.Click; +import com.clickhouse.r2dbc.spring.webflux.sample.model.ClickStats; +import com.clickhouse.r2dbc.spring.webflux.sample.repository.ClickRepository; +import org.reactivestreams.Publisher; +import org.springframework.beans.factory.annotation.Autowired; +import org.springframework.http.HttpStatus; +import org.springframework.http.ResponseEntity; +import org.springframework.web.bind.annotation.GetMapping; +import org.springframework.web.bind.annotation.PathVariable; +import org.springframework.web.bind.annotation.PostMapping; +import org.springframework.web.bind.annotation.RequestBody; +import org.springframework.web.bind.annotation.RequestMapping; +import org.springframework.web.bind.annotation.ResponseStatus; +import org.springframework.web.bind.annotation.RestController; +import reactor.core.publisher.Flux; +import reactor.core.publisher.Mono; + +import java.time.LocalDateTime; +import java.util.List; +import java.util.stream.Collectors; +import java.util.stream.IntStream; +import java.util.stream.Stream; + +@RestController +@RequestMapping("/clicks") +public class ClickController { + + @Autowired + ClickRepository clickRepository; + + @GetMapping("/{domain}") + public Publisher> getEmployeeById(@PathVariable("domain") String domain) { + return Flux.from(clickRepository.getStatsByDomain(domain).collect(Collectors.toList())); + } + + @PostMapping + @ResponseStatus(HttpStatus.CREATED) + public Mono add(@RequestBody Click click){ + return Mono.from(clickRepository.add(click)); + + + } + +} diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/Click.java b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/Click.java new file mode 100644 index 000000000..c160dba89 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/Click.java @@ -0,0 +1,24 @@ +package com.clickhouse.r2dbc.spring.webflux.sample.model; + + +public class Click { + + String domain; + String path; + + public String getDomain() { + return domain; + } + + public String getPath() { + return path; + } + + public void setDomain(String domain) { + this.domain = domain; + } + + public void setPath(String path) { + this.path = path; + } +} diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/ClickStats.java b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/ClickStats.java new file mode 100644 index 000000000..5dd280be9 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/ClickStats.java @@ -0,0 +1,53 @@ +package com.clickhouse.r2dbc.spring.webflux.sample.model; + + +import com.fasterxml.jackson.annotation.JsonFormat; + +import java.time.LocalDate; + +public class ClickStats { + private String domain; + private String path; + @JsonFormat(pattern = "yyyy-MM-dd") + private LocalDate cdate; + private long count; + + public ClickStats(String domain, String path, LocalDate date, long count) { + this.domain = domain; + this.path = path; + this.cdate = date; + this.count = count; + } + + public String getDomain() { + return domain; + } + + public String getPath() { + return path; + } + + public void setDomain(String domain) { + this.domain = domain; + } + + public void setPath(String path) { + this.path = path; + } + + public long getCount() { + return count; + } + + public void setCount(long count) { + this.count = count; + } + + public LocalDate getCdate() { + return cdate; + } + + public void setCdate(LocalDate cdate) { + this.cdate = cdate; + } +} diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/repository/ClickRepository.java b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/repository/ClickRepository.java new file mode 100644 index 000000000..1e8ecc100 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/repository/ClickRepository.java @@ -0,0 +1,47 @@ +package com.clickhouse.r2dbc.spring.webflux.sample.repository; + +import com.clickhouse.r2dbc.spring.webflux.sample.model.Click; +import com.clickhouse.r2dbc.spring.webflux.sample.model.ClickStats; +import io.r2dbc.spi.Connection; +import io.r2dbc.spi.ConnectionFactory; +import io.r2dbc.spi.Result; +import org.reactivestreams.Publisher; +import org.springframework.beans.factory.annotation.Autowired; +import org.springframework.stereotype.Repository; +import reactor.core.publisher.Flux; +import reactor.core.publisher.Mono; + +import java.time.LocalDate; +import java.time.LocalDateTime; + + + +@Repository +public class ClickRepository { + + @Autowired + ConnectionFactory connectionFactory; + + public Flux getStatsByDomain(String domain){ + return Mono.from(connectionFactory.create()) + .flatMapMany(conn -> conn.createStatement("select domain, path, toDate(cdate) as d, count(1) as count from clickdb.clicks where domain = :domain group by domain, path, d") + .bind("domain", domain) + .execute()) + .flatMap(result -> result.map((row, rowMetadata) -> new ClickStats(row + .get("domain", String.class), row.get("path", String.class), row.get("d", LocalDate.class), row.get("count", Long.class)))); + } + + public Mono add(Click click){ + return Mono.from(connectionFactory.create()) + .flatMapMany(conn -> execute(click, conn)).then(); + } + + private Publisher execute(Click click, Connection conn) { + return conn.createStatement("insert into clickdb.clicks values (:domain, :path, :cdate, :count)") + .bind("domain", click.getDomain()) + .bind("path", click.getPath()) + .bind("cdate", LocalDateTime.now()) + .bind("count", 1).execute(); + } + +} diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/application.yaml b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/application.yaml new file mode 100644 index 000000000..cd9a08f04 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/application.yaml @@ -0,0 +1,7 @@ +clickhouse: + host: localhost + port: 8123 + database: clickdb + user: default + password: "" +debug: true \ No newline at end of file diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/init.sql b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/init.sql new file mode 100644 index 000000000..75d8664a9 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/init.sql @@ -0,0 +1,11 @@ +create database clickdb; + +create table if not exists clickdb.clicks +( + domain String, + path String, + cdate DateTime, + count UInt64 +) +engine = SummingMergeTree(count) +order by (domain, path, cdate); \ No newline at end of file diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/log4j.properties b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/log4j.properties new file mode 100644 index 000000000..2ad8a90bb --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/log4j.properties @@ -0,0 +1,4 @@ +log4j.rootCategory=DEBUG, stdout +log4j.appender.stdout=org.apache.log4j.ConsoleAppender +log4j.appender.stdout.layout=org.apache.log4j.PatternLayout +log4j.appender.stdout.layout.ConversionPattern=%d{yy-MM-dd HH:mm:ss:SSS} %5p %t %c{2}:%L - %m%n \ No newline at end of file diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/postman/clickhouse-r2dbc-sample.postman_collection.json b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/postman/clickhouse-r2dbc-sample.postman_collection.json new file mode 100644 index 000000000..6fed7ad0a --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/postman/clickhouse-r2dbc-sample.postman_collection.json @@ -0,0 +1,57 @@ +{ + "info": { + "_postman_id": "42544c65-9bda-4155-b00c-68bf62de19c2", + "name": "clickhouse-r2dbc-sample", + "schema": "https://schema.getpostman.com/json/collection/v2.1.0/collection.json" + }, + "item": [ + { + "name": "List Clicks", + "request": { + "method": "GET", + "header": [], + "url": { + "raw": "http://localhost:8080/clicks/google.com", + "protocol": "http", + "host": [ + "localhost" + ], + "port": "8080", + "path": [ + "clicks", + "google.com" + ] + } + }, + "response": [] + }, + { + "name": "Create Clicks", + "request": { + "method": "POST", + "header": [], + "body": { + "mode": "raw", + "raw": "{\n \"domain\" : \"google.com\",\n \"path\" : \"/mail\"\n}", + "options": { + "raw": { + "language": "json" + } + } + }, + "url": { + "raw": "http://localhost:8080/clicks", + "protocol": "http", + "host": [ + "localhost" + ], + "port": "8080", + "path": [ + "clicks" + ] + } + }, + "response": [] + } + ] +} \ No newline at end of file diff --git a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.d/docker_related_config.xml b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.d/docker_related_config.xml new file mode 100644 index 000000000..3025dc269 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.d/docker_related_config.xml @@ -0,0 +1,12 @@ + + + :: + 0.0.0.0 + 1 + + + diff --git a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.xml b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.xml new file mode 100644 index 000000000..7e103bd6b --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.xml @@ -0,0 +1,1294 @@ + + + + + + trace + /var/log/clickhouse-server/clickhouse-server.log + /var/log/clickhouse-server/clickhouse-server.err.log + + 1000M + 10 + + + + + + + + + + + + + + + + + + 8123 + + + 9000 + + + 9004 + + + 9005 + + + + + + + + + + + + 9009 + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + 4096 + + + 3 + + + 9100 + + false + + + /path/to/ssl_cert_file + /path/to/ssl_key_file + + + false + + + /path/to/ssl_ca_cert_file + + + deflate + + + medium + + + -1 + -1 + + + false + + + + + + + /etc/clickhouse-server/server.crt + /etc/clickhouse-server/server.key + + /etc/clickhouse-server/dhparam.pem + none + true + true + sslv2,sslv3 + true + + + + true + true + sslv2,sslv3 + true + + + + RejectCertificateHandler + + + + + + + + + 100 + + + 0 + + + + 10000 + + + + + + 0.9 + + + 4194304 + + + 0 + + + + + + 8589934592 + + + 5368709120 + + + + 1000 + + + 134217728 + + + 10000 + + + /var/lib/clickhouse/ + + + /var/lib/clickhouse/tmp/ + + + + + + /var/lib/clickhouse/user_files/ + + + + + + + + + + + + + users.xml + + + + /var/lib/clickhouse/access/ + + + + + + + default + + + + + + + + + + + + default + + + + + + + + + true + + + false + + ' | sed -e 's|.*>\(.*\)<.*|\1|') + wget https://github.com/ClickHouse/clickhouse-jdbc-bridge/releases/download/v$PKG_VER/clickhouse-jdbc-bridge_$PKG_VER-1_all.deb + apt install --no-install-recommends -f ./clickhouse-jdbc-bridge_$PKG_VER-1_all.deb + clickhouse-jdbc-bridge & + + * [CentOS/RHEL] + export MVN_URL=https://repo1.maven.org/maven2/ru/yandex/clickhouse/clickhouse-jdbc-bridge + export PKG_VER=$(curl -sL $MVN_URL/maven-metadata.xml | grep '' | sed -e 's|.*>\(.*\)<.*|\1|') + wget https://github.com/ClickHouse/clickhouse-jdbc-bridge/releases/download/v$PKG_VER/clickhouse-jdbc-bridge-$PKG_VER-1.noarch.rpm + yum localinstall -y clickhouse-jdbc-bridge-$PKG_VER-1.noarch.rpm + clickhouse-jdbc-bridge & + + Please refer to https://github.com/ClickHouse/clickhouse-jdbc-bridge#usage for more information. + ]]> + + + + + + + + + + + + + + + + localhost + 9000 + + + + + + + + false + + 127.0.0.1 + 9000 + + + 127.0.0.2 + 9000 + + + 127.0.0.3 + 9000 + + + + + + + + localhost + 9000 + + + + + localhost + 9000 + + + + + + + 127.0.0.1 + 9000 + + + + + 127.0.0.2 + 9000 + + + + + + true + + 127.0.0.1 + 9000 + + + + true + + 127.0.0.2 + 9000 + + + + + + + localhost + 9440 + 1 + + + + + + + localhost + 9000 + + + + + localhost + 1 + + + + + + + + + + + + + + + + + + + + + + + + 3600 + + + + 3600 + + + 60 + + + + + + + + + + + + + system + query_log
+ + toYYYYMM(event_date) + + + + + + 7500 +
+ + + + system + trace_log
+ + toYYYYMM(event_date) + 7500 +
+ + + + system + query_thread_log
+ toYYYYMM(event_date) + 7500 +
+ + + + system + query_views_log
+ toYYYYMM(event_date) + 7500 +
+ + + + system + part_log
+ toYYYYMM(event_date) + 7500 +
+ + + + + + system + metric_log
+ 7500 + 1000 +
+ + + + system + asynchronous_metric_log
+ + 7000 +
+ + + + + + engine MergeTree + partition by toYYYYMM(finish_date) + order by (finish_date, finish_time_us, trace_id) + + system + opentelemetry_span_log
+ 7500 +
+ + + + + system + crash_log
+ + + 1000 +
+ + + + system + session_log
+ + toYYYYMM(event_date) + 7500 +
+ + + + + + + + + + + + + + + + + + *_dictionary.xml + + + *_function.xml + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + /clickhouse/task_queue/ddl + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + click_cost + any + + 0 + 3600 + + + 86400 + 60 + + + + max + + 0 + 60 + + + 3600 + 300 + + + 86400 + 3600 + + + + + + /var/lib/clickhouse/format_schemas/ + + + + + hide encrypt/decrypt arguments + ((?:aes_)?(?:encrypt|decrypt)(?:_mysql)?)\s*\(\s*(?:'(?:\\'|.)+'|.*?)\s*\) + + \1(???) + + + + + + + + + + false + + false + + + https://6f33034cfe684dd7a3ab9875e57b1c8d@o388870.ingest.sentry.io/5226277 + + + + + + + +
diff --git a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/docker_related_config.xml b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/docker_related_config.xml new file mode 100644 index 000000000..3025dc269 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/docker_related_config.xml @@ -0,0 +1,12 @@ + + + :: + 0.0.0.0 + 1 + + + diff --git a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/users.xml b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/users.xml new file mode 100644 index 000000000..fd5fe4145 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/users.xml @@ -0,0 +1,123 @@ + + + + + + + + + + 10000000000 + + + random + + + + + 1 + + + + + + + + + + + + + ::/0 + + + + default + + + default + + + + + + + + + + + + + + 3600 + + + 0 + 0 + 0 + 0 + 0 + + + + diff --git a/examples/clickhouse-r2dbc-samples/misc/docker/compose.yaml b/examples/clickhouse-r2dbc-samples/misc/docker/compose.yaml new file mode 100644 index 000000000..48672b736 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/misc/docker/compose.yaml @@ -0,0 +1,10 @@ +services: + db: + image: yandex/clickhouse-server + ports: + - 8123:8123 + - 9009:9009 + - 9100:9100 + - 9000:9000 + volumes: + - ./clickhouse-docker-mount:/etc/clickhouse-server \ No newline at end of file diff --git a/examples/clickhouse-r2dbc-samples/pom.xml b/examples/clickhouse-r2dbc-samples/pom.xml new file mode 100644 index 000000000..5081d7670 --- /dev/null +++ b/examples/clickhouse-r2dbc-samples/pom.xml @@ -0,0 +1,15 @@ + + + 4.0.0 + com.clickhouse + clickhouse-r2dbc-samples + 1.0.0-SNAPSHOT + pom + + + clickhouse-r2dbc-spring-webflux-sample + + + \ No newline at end of file diff --git a/pom.xml b/pom.xml index 2390a6d85..d23785043 100644 --- a/pom.xml +++ b/pom.xml @@ -43,6 +43,7 @@ clickhouse-sql-parser clickhouse-jdbc clickhouse-benchmark + clickhouse-r2dbc @@ -98,6 +99,7 @@ 2.33.2 1.17.3 7.5 + 3.12.0 3.0.7 8.0.30 @@ -311,6 +313,14 @@ + + + io.projectreactor + reactor-bom + 2020.0.16 + pom + import + From dc23b90f1e3364dcc83a8c4e039400164edd7b89 Mon Sep 17 00:00:00 2001 From: Zhichun Wu Date: Sun, 11 Sep 2022 10:07:11 +0800 Subject: [PATCH 2/2] Fit and finish --- clickhouse-r2dbc/pom.xml | 49 +- .../com/clickhouse/r2dbc/ClickHouseBatch.java | 5 +- .../clickhouse/r2dbc/ClickHouseResult.java | 1 + .../connection/ClickHouseConnection.java | 6 +- .../ClickHouseConnectionFactoryProvider.java | 85 +- .../ClickHouseConnectionMetadata.java | 27 +- .../types/ClickHouseDataTypeWrapper.java | 4 +- .../r2dbc/spi/test/R2DBCTestKitImplTest.java | 16 +- .../test/resources/simplelogger.properties | 7 + .../config.d/docker_related_config.xml | 12 - .../docker/clickhouse-docker-mount/config.xml | 1294 ----------------- .../docker_related_config.xml | 12 - .../docker/clickhouse-docker-mount/users.xml | 123 -- .../README.md | 0 .../pom.xml | 0 .../spring/webflux/sample/Application.java | 0 .../webflux/sample/config/R2DBCConfig.java | 0 .../sample/controller/ClickController.java | 0 .../spring/webflux/sample/model/Click.java | 0 .../webflux/sample/model/ClickStats.java | 0 .../sample/repository/ClickRepository.java | 0 .../src/main/resources/application.yaml | 0 .../src/main/resources/init.sql | 0 .../src/main/resources/log4j.properties | 0 ...house-r2dbc-sample.postman_collection.json | 0 .../config.d/docker_related_config.xml | 0 .../clickhouse-docker-mount/config.xml | 0 .../docker_related_config.xml | 0 .../docker}/clickhouse-docker-mount/users.xml | 0 .../misc/docker/compose.yaml | 0 .../pom.xml | 0 31 files changed, 138 insertions(+), 1503 deletions(-) create mode 100644 clickhouse-r2dbc/src/test/resources/simplelogger.properties delete mode 100644 examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.d/docker_related_config.xml delete mode 100644 examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.xml delete mode 100644 examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/docker_related_config.xml delete mode 100644 examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/users.xml rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/README.md (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/pom.xml (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/Application.java (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/config/R2DBCConfig.java (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/controller/ClickController.java (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/Click.java (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/ClickStats.java (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/repository/ClickRepository.java (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/application.yaml (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/init.sql (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/log4j.properties (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/postman/clickhouse-r2dbc-sample.postman_collection.json (100%) rename {clickhouse-r2dbc/src/test/resources => examples/r2dbc/misc/docker}/clickhouse-docker-mount/config.d/docker_related_config.xml (100%) rename {clickhouse-r2dbc/src/test/resources => examples/r2dbc/misc/docker}/clickhouse-docker-mount/config.xml (100%) rename {clickhouse-r2dbc/src/test/resources => examples/r2dbc/misc/docker}/clickhouse-docker-mount/docker_related_config.xml (100%) rename {clickhouse-r2dbc/src/test/resources => examples/r2dbc/misc/docker}/clickhouse-docker-mount/users.xml (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/misc/docker/compose.yaml (100%) rename examples/{clickhouse-r2dbc-samples => r2dbc}/pom.xml (100%) diff --git a/clickhouse-r2dbc/pom.xml b/clickhouse-r2dbc/pom.xml index 49285bbdd..04dd2ef7e 100644 --- a/clickhouse-r2dbc/pom.xml +++ b/clickhouse-r2dbc/pom.xml @@ -1,7 +1,5 @@ - + clickhouse-java com.clickhouse @@ -11,8 +9,13 @@ clickhouse-r2dbc + ClickHouse R2DBC Driver + R2DBC driver for ClickHouse + https://github.com/ClickHouse/clickhouse-jdbc/tree/master/clickhouse-r2dbc + 4.0.3 + 5.9.0 1.8 1.8 1.0.0.RELEASE @@ -39,7 +42,7 @@ com.clickhouse - clickhouse-client + clickhouse-cli-client ${project.version} @@ -52,16 +55,12 @@ clickhouse-http-client ${project.version} - - com.clickhouse - clickhouse-jdbc - ${project.version} - org.apache.commons commons-lang3 ${commons-lang3.version} + org.testcontainers testcontainers @@ -74,6 +73,12 @@ test-jar test + + com.clickhouse + clickhouse-jdbc + ${project.version} + test + com.zaxxer HikariCP @@ -89,7 +94,12 @@ org.junit.jupiter junit-jupiter-api - 5.8.2 + ${junit.version} + test + + + org.slf4j + slf4j-simple test @@ -98,13 +108,24 @@ org.apache.maven.plugins - maven-surefire-plugin - ${surefire-plugin.version} + maven-failsafe-plugin - ${excludedGroups} - ${skipUTs} + **/*.java + ${skipTests} + ${skipITs} + true false + + + run-integration-tests + integration-test + + integration-test + verify + + + org.apache.maven.plugins diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseBatch.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseBatch.java index c8d1344d8..7314e66f2 100644 --- a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseBatch.java +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseBatch.java @@ -15,10 +15,11 @@ public class ClickHouseBatch implements Batch { private static final ClickHouseFormat PREFERRED_FORMAT = ClickHouseFormat.TabSeparatedWithNamesAndTypes; private ClickHouseRequest request; - List sqlList = new ArrayList<>(); + final List sqlList; - public ClickHouseBatch(ClickHouseRequest request) { + public ClickHouseBatch(ClickHouseRequest request) { this.request = request; + this.sqlList = new ArrayList<>(); } @Override diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseResult.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseResult.java index 0df8734c2..54dc80f12 100644 --- a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseResult.java +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/ClickHouseResult.java @@ -47,6 +47,7 @@ public class ClickHouseResult implements Result { /** * Returns updated count(written rows from summary of {@link ClickHouseResponse}).Important! if writtenRows is greater than MAX_INT then it will return MAX_INT. + * * @return updated count */ @Override diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnection.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnection.java index ffc68acce..9fb68968e 100644 --- a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnection.java +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnection.java @@ -35,7 +35,6 @@ public class ClickHouseConnection implements Connection { private boolean closed = false; ClickHouseConnection(ClickHouseNodes nodes) { - ClickHouseNode node = nodes.apply(ClickHouseNodeSelector.EMPTY); this.node = nodes.apply(ClickHouseNodeSelector.EMPTY); this.client = ClickHouseClient.newInstance(node.getProtocol()); } @@ -81,6 +80,7 @@ public Publisher commitTransaction() { /** * Returns {@link ClickHouseBatch} for batching statements. + * * @return Batch object */ @Override @@ -95,6 +95,7 @@ public Batch createBatch() { /** * Returns true since there is no transaction support. + * * @return true */ @Override @@ -118,6 +119,7 @@ private boolean isHttp() { /** * Returns true since there is no transaction support. + * * @return true */ @Override @@ -132,6 +134,7 @@ public ConnectionMetadata getMetadata() { } /** + * Returns transaction isolation level. * * @return Always returns read committed. */ @@ -180,6 +183,7 @@ public Publisher setStatementTimeout(Duration duration) { /** * Since transactions are not supported, this method will throw exception. + * * @param isolationLevel isolation level for transaction */ @Override diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryProvider.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryProvider.java index 142554c69..9faa3955b 100644 --- a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryProvider.java +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionFactoryProvider.java @@ -1,13 +1,24 @@ package com.clickhouse.r2dbc.connection; +import com.clickhouse.client.ClickHouseClient; import com.clickhouse.client.ClickHouseNodes; +import com.clickhouse.client.ClickHouseProtocol; +import com.clickhouse.client.ClickHouseUtils; +import com.clickhouse.client.config.ClickHouseDefaults; +import com.clickhouse.client.config.ClickHouseOption; + import io.r2dbc.spi.ConnectionFactory; import io.r2dbc.spi.ConnectionFactoryOptions; import io.r2dbc.spi.Option; -import java.util.Arrays; +import java.io.Serializable; +import java.util.ArrayList; +import java.util.Collections; +import java.util.LinkedHashSet; import java.util.List; -import java.util.stream.Collectors; +import java.util.Locale; +import java.util.ServiceLoader; +import java.util.Set; import static io.r2dbc.spi.ConnectionFactoryOptions.*; @@ -18,43 +29,63 @@ public class ClickHouseConnectionFactoryProvider implements io.r2dbc.spi.Connect */ public static final String CLICKHOUSE_DRIVER = "clickhouse"; - private static final List connQueryParams = Arrays.asList("auto_discovery", "node_discovery_interval", - "node_discovery_limit", "load_balancing_policy", "load_balancing_tags", "health_check_method", - "health_check_interval", "check_all_nodes", "node_check_interval", "node_group_size", "failover", - "retry"); + private static final List> connQueryParams; - @Override - public ConnectionFactory create(ConnectionFactoryOptions cfOpt) { - String hosts = getHosts(cfOpt); - String database = cfOpt.getValue(DATABASE).toString(); - String protocol = cfOpt.getValue(PROTOCOL).toString(); - if (cfOpt.getValue(USER) == null ) { - throw new IllegalArgumentException("User and password is mandatory."); - } - String username = cfOpt.getValue(USER).toString(); - String password = ""; - if (cfOpt.getValue(PASSWORD) != null) { - password = cfOpt.getValue(PASSWORD).toString(); + static { + Set> allOptions = new LinkedHashSet<>(); + try { + for (ClickHouseClient client : ServiceLoader.load(ClickHouseClient.class, ClickHouseConnectionFactoryProvider.class.getClassLoader())) { + for (ClickHouseOption option : client.getOptionClass().getEnumConstants()) { + allOptions.add(Option.valueOf(option.getKey())); + } + } + } catch (Exception e) { + // ignore } + connQueryParams = Collections.unmodifiableList(new ArrayList<>(allOptions)); + } - StringBuilder urlBuilder = new StringBuilder(String.format("%s://%s/%s?user=%s&password=%s", protocol, hosts, database, username, password)); - String params = connQueryParams.stream().filter(queryParam -> cfOpt.getValue(Option.valueOf(queryParam)) != null) - .map(queryParam -> String.format("%s=%s", queryParam, cfOpt.getValue(Option.valueOf(queryParam)))) - .collect(Collectors.joining("%")); - urlBuilder.append(params.isEmpty() ? "" : ("&" + params)); - - ClickHouseNodes nodes = ClickHouseNodes.of(urlBuilder.toString()); - return new ClickHouseConnectionFactory(nodes); + private String getOptionValueAsString(ConnectionFactoryOptions cfOpt, Option option, Serializable defaultValue) { + Object value = cfOpt.getValue(option); + return value != null ? value.toString() : defaultValue.toString(); } private String getHosts(ConnectionFactoryOptions cfOpt) { - String hosts = cfOpt.getValue(HOST).toString(); + String hosts = getOptionValueAsString(cfOpt, HOST, ClickHouseDefaults.HOST.getEffectiveDefaultValue()); if (!hosts.contains(",") && !hosts.contains(":")){ return hosts + ":" + cfOpt.getValue(PORT); } return hosts; } + @Override + public ConnectionFactory create(ConnectionFactoryOptions cfOpt) { + String hosts = getHosts(cfOpt); + String database = getOptionValueAsString(cfOpt, DATABASE, ""); + String protocol = getOptionValueAsString(cfOpt, PROTOCOL, ClickHouseProtocol.HTTP.name()).toLowerCase(Locale.ROOT); + if (Boolean.parseBoolean(getOptionValueAsString(cfOpt, SSL, "false"))) { + protocol += "s"; + } + + StringBuilder urlBuilder = new StringBuilder(); + urlBuilder.append(protocol).append("://").append(hosts).append('/'); + if (!database.isEmpty()) { + urlBuilder.append(database); + } + String user = getOptionValueAsString(cfOpt, USER, ""); + String password = getOptionValueAsString(cfOpt, PASSWORD, ""); + urlBuilder.append("?user=").append(ClickHouseUtils.encode(user)).append("&password=").append(ClickHouseUtils.encode(password)); + for (Option option : connQueryParams) { + Object value = cfOpt.getValue(option); + if (value != null) { + urlBuilder.append('&').append(option.name()).append('=').append(ClickHouseUtils.encode(cfOpt.getValue(option).toString())); + } + } + + ClickHouseNodes nodes = ClickHouseNodes.of(urlBuilder.toString()); + return new ClickHouseConnectionFactory(nodes); + } + @Override public boolean supports(ConnectionFactoryOptions connectionFactoryOptions) { return connectionFactoryOptions.getValue(DRIVER).equals(CLICKHOUSE_DRIVER); diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionMetadata.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionMetadata.java index 727c06c70..6fb435ed9 100644 --- a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionMetadata.java +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/connection/ClickHouseConnectionMetadata.java @@ -1,5 +1,7 @@ package com.clickhouse.r2dbc.connection; +import java.util.concurrent.atomic.AtomicReference; + import com.clickhouse.client.ClickHouseClient; import com.clickhouse.client.ClickHouseNode; import com.clickhouse.client.ClickHouseResponse; @@ -14,7 +16,7 @@ public class ClickHouseConnectionMetadata implements ConnectionMetadata { final ClickHouseClient client; final ClickHouseNode server; - private final String serverVersion = null; + private final AtomicReference serverVersion = new AtomicReference<>(""); ClickHouseConnectionMetadata(ClickHouseClient client, ClickHouseNode server) { this.client = client; @@ -28,20 +30,23 @@ public String getDatabaseProductName() { /** * Blocking operation. Queries server version by calling "SELECT version()" statement. - * @return server version + * + * @return non-null server version */ @Override public String getDatabaseVersion() { - if (serverVersion != null) { - return serverVersion; - } - try { + String version = serverVersion.get(); + if (version.isEmpty()) { // blocking here - ClickHouseResponse resp = client.connect(server).query("SELECT version()").executeAndWait(); - return resp.records().iterator().next().getValue(0).asString(); - } catch (Exception e) { - log.error("While fetching server version, error occured.", e); - return null; + try (ClickHouseResponse resp = client.connect(server).query("SELECT version()").executeAndWait()) { + version = resp.firstRecord().getValue(0).asString(); + if (!serverVersion.compareAndSet("", version)) { + return serverVersion.get(); + } + } catch (Exception e) { + log.error("While fetching server version, error occured.", e); + } } + return version; } } diff --git a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/types/ClickHouseDataTypeWrapper.java b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/types/ClickHouseDataTypeWrapper.java index dc324319d..e0a14914f 100644 --- a/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/types/ClickHouseDataTypeWrapper.java +++ b/clickhouse-r2dbc/src/main/java/com/clickhouse/r2dbc/types/ClickHouseDataTypeWrapper.java @@ -5,7 +5,7 @@ public class ClickHouseDataTypeWrapper implements Type { - ClickHouseDataType dType; + final ClickHouseDataType dType; private ClickHouseDataTypeWrapper(ClickHouseDataType dType){ this.dType = dType; @@ -17,7 +17,7 @@ public static ClickHouseDataTypeWrapper of(ClickHouseDataType dType) { @Override - public Class getJavaType() { + public Class getJavaType() { return dType.getObjectClass(); } diff --git a/clickhouse-r2dbc/src/test/java/com/clickhouse/r2dbc/spi/test/R2DBCTestKitImplTest.java b/clickhouse-r2dbc/src/test/java/com/clickhouse/r2dbc/spi/test/R2DBCTestKitImplTest.java index d31cda77f..29ba250b1 100644 --- a/clickhouse-r2dbc/src/test/java/com/clickhouse/r2dbc/spi/test/R2DBCTestKitImplTest.java +++ b/clickhouse-r2dbc/src/test/java/com/clickhouse/r2dbc/spi/test/R2DBCTestKitImplTest.java @@ -39,7 +39,12 @@ public class R2DBCTestKitImplTest implements TestKit { private static final String DATABASE = "default"; private static final String USER = "default"; - public static final String PASSWORD = ""; + private static final String PASSWORD = ""; + + private static final String CUSTOM_PROTOCOL_NAME = System.getProperty("protocol", "http").toUpperCase(); + private static final ClickHouseProtocol DEFAULT_PROTOCOL = ClickHouseProtocol + .valueOf(CUSTOM_PROTOCOL_NAME.startsWith("HTTP") ? "HTTP" : CUSTOM_PROTOCOL_NAME); + private static final String EXTRA_PARAM = "HTTP2".equals(CUSTOM_PROTOCOL_NAME) ? "http_connection_provider=HTTP_CLIENT" : ""; static ConnectionFactory connectionFactory; static JdbcTemplate jdbcTemplate; @@ -47,9 +52,10 @@ public class R2DBCTestKitImplTest implements TestKit { @BeforeAll public static void setup() throws Exception { ClickHouseServerForTest.beforeSuite(); + connectionFactory = ConnectionFactories.get( - format("r2dbc:clickhouse:http://%s:%s@%s/%s?falan=filan#tag1", USER, PASSWORD, - getClickHouseAddress(ClickHouseProtocol.HTTP, false), DATABASE)); + format("r2dbc:clickhouse:%s://%s:%s@%s/%s?falan=filan&%s#tag1", DEFAULT_PROTOCOL, USER, PASSWORD, + getClickHouseAddress(DEFAULT_PROTOCOL, false), DATABASE, EXTRA_PARAM)); jdbcTemplate = jdbcTemplate(null); } @@ -79,9 +85,9 @@ private static JdbcTemplate jdbcTemplate(String database) throws SQLException { Driver driver = new ClickHouseDriver(); DriverManager.registerDriver(driver); if (database == null) { - source.setJdbcUrl(format("jdbc:clickhouse://%s", getClickHouseAddress(ClickHouseProtocol.HTTP, false))); + source.setJdbcUrl(format("jdbc:clickhouse:%s://%s?%s", DEFAULT_PROTOCOL, getClickHouseAddress(DEFAULT_PROTOCOL, false), EXTRA_PARAM)); } else { - source.setJdbcUrl(format("jdbc:clickhouse://%s/%s", getClickHouseAddress(ClickHouseProtocol.HTTP, false), DATABASE)); + source.setJdbcUrl(format("jdbc:clickhouse:%s://%s/%s?%s", DEFAULT_PROTOCOL, getClickHouseAddress(DEFAULT_PROTOCOL, false), DATABASE, EXTRA_PARAM)); } source.setUsername(USER); diff --git a/clickhouse-r2dbc/src/test/resources/simplelogger.properties b/clickhouse-r2dbc/src/test/resources/simplelogger.properties new file mode 100644 index 000000000..7589a0e57 --- /dev/null +++ b/clickhouse-r2dbc/src/test/resources/simplelogger.properties @@ -0,0 +1,7 @@ +org.slf4j.simpleLogger.defaultLogLevel=info +org.slf4j.simpleLogger.log.com.clickhouse.client=debug +org.slf4j.simpleLogger.showDateTime=true +org.slf4j.simpleLogger.dateTimeFormat=yyyy-MM-dd HH:mm:ss:SSS Z +org.slf4j.simpleLogger.showThreadName=true +org.slf4j.simpleLogger.showLogName=true +org.slf4j.simpleLogger.showShortLogName=true diff --git a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.d/docker_related_config.xml b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.d/docker_related_config.xml deleted file mode 100644 index 3025dc269..000000000 --- a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.d/docker_related_config.xml +++ /dev/null @@ -1,12 +0,0 @@ - - - :: - 0.0.0.0 - 1 - - - diff --git a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.xml b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.xml deleted file mode 100644 index 7e103bd6b..000000000 --- a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/config.xml +++ /dev/null @@ -1,1294 +0,0 @@ - - - - - - trace - /var/log/clickhouse-server/clickhouse-server.log - /var/log/clickhouse-server/clickhouse-server.err.log - - 1000M - 10 - - - - - - - - - - - - - - - - - - 8123 - - - 9000 - - - 9004 - - - 9005 - - - - - - - - - - - - 9009 - - - - - - - - - - - - - - - - - - - - - - - - - - - - - - 4096 - - - 3 - - - 9100 - - false - - - /path/to/ssl_cert_file - /path/to/ssl_key_file - - - false - - - /path/to/ssl_ca_cert_file - - - deflate - - - medium - - - -1 - -1 - - - false - - - - - - - /etc/clickhouse-server/server.crt - /etc/clickhouse-server/server.key - - /etc/clickhouse-server/dhparam.pem - none - true - true - sslv2,sslv3 - true - - - - true - true - sslv2,sslv3 - true - - - - RejectCertificateHandler - - - - - - - - - 100 - - - 0 - - - - 10000 - - - - - - 0.9 - - - 4194304 - - - 0 - - - - - - 8589934592 - - - 5368709120 - - - - 1000 - - - 134217728 - - - 10000 - - - /var/lib/clickhouse/ - - - /var/lib/clickhouse/tmp/ - - - - - - /var/lib/clickhouse/user_files/ - - - - - - - - - - - - - users.xml - - - - /var/lib/clickhouse/access/ - - - - - - - default - - - - - - - - - - - - default - - - - - - - - - true - - - false - - ' | sed -e 's|.*>\(.*\)<.*|\1|') - wget https://github.com/ClickHouse/clickhouse-jdbc-bridge/releases/download/v$PKG_VER/clickhouse-jdbc-bridge_$PKG_VER-1_all.deb - apt install --no-install-recommends -f ./clickhouse-jdbc-bridge_$PKG_VER-1_all.deb - clickhouse-jdbc-bridge & - - * [CentOS/RHEL] - export MVN_URL=https://repo1.maven.org/maven2/ru/yandex/clickhouse/clickhouse-jdbc-bridge - export PKG_VER=$(curl -sL $MVN_URL/maven-metadata.xml | grep '' | sed -e 's|.*>\(.*\)<.*|\1|') - wget https://github.com/ClickHouse/clickhouse-jdbc-bridge/releases/download/v$PKG_VER/clickhouse-jdbc-bridge-$PKG_VER-1.noarch.rpm - yum localinstall -y clickhouse-jdbc-bridge-$PKG_VER-1.noarch.rpm - clickhouse-jdbc-bridge & - - Please refer to https://github.com/ClickHouse/clickhouse-jdbc-bridge#usage for more information. - ]]> - - - - - - - - - - - - - - - - localhost - 9000 - - - - - - - - false - - 127.0.0.1 - 9000 - - - 127.0.0.2 - 9000 - - - 127.0.0.3 - 9000 - - - - - - - - localhost - 9000 - - - - - localhost - 9000 - - - - - - - 127.0.0.1 - 9000 - - - - - 127.0.0.2 - 9000 - - - - - - true - - 127.0.0.1 - 9000 - - - - true - - 127.0.0.2 - 9000 - - - - - - - localhost - 9440 - 1 - - - - - - - localhost - 9000 - - - - - localhost - 1 - - - - - - - - - - - - - - - - - - - - - - - - 3600 - - - - 3600 - - - 60 - - - - - - - - - - - - - system - query_log
- - toYYYYMM(event_date) - - - - - - 7500 -
- - - - system - trace_log
- - toYYYYMM(event_date) - 7500 -
- - - - system - query_thread_log
- toYYYYMM(event_date) - 7500 -
- - - - system - query_views_log
- toYYYYMM(event_date) - 7500 -
- - - - system - part_log
- toYYYYMM(event_date) - 7500 -
- - - - - - system - metric_log
- 7500 - 1000 -
- - - - system - asynchronous_metric_log
- - 7000 -
- - - - - - engine MergeTree - partition by toYYYYMM(finish_date) - order by (finish_date, finish_time_us, trace_id) - - system - opentelemetry_span_log
- 7500 -
- - - - - system - crash_log
- - - 1000 -
- - - - system - session_log
- - toYYYYMM(event_date) - 7500 -
- - - - - - - - - - - - - - - - - - *_dictionary.xml - - - *_function.xml - - - - - - - - - - - - - - - - - - - - - - - - - - - - - - - - /clickhouse/task_queue/ddl - - - - - - - - - - - - - - - - - - - - - - - - - - - - - - click_cost - any - - 0 - 3600 - - - 86400 - 60 - - - - max - - 0 - 60 - - - 3600 - 300 - - - 86400 - 3600 - - - - - - /var/lib/clickhouse/format_schemas/ - - - - - hide encrypt/decrypt arguments - ((?:aes_)?(?:encrypt|decrypt)(?:_mysql)?)\s*\(\s*(?:'(?:\\'|.)+'|.*?)\s*\) - - \1(???) - - - - - - - - - - false - - false - - - https://6f33034cfe684dd7a3ab9875e57b1c8d@o388870.ingest.sentry.io/5226277 - - - - - - - -
diff --git a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/docker_related_config.xml b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/docker_related_config.xml deleted file mode 100644 index 3025dc269..000000000 --- a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/docker_related_config.xml +++ /dev/null @@ -1,12 +0,0 @@ - - - :: - 0.0.0.0 - 1 - - - diff --git a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/users.xml b/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/users.xml deleted file mode 100644 index fd5fe4145..000000000 --- a/examples/clickhouse-r2dbc-samples/misc/docker/clickhouse-docker-mount/users.xml +++ /dev/null @@ -1,123 +0,0 @@ - - - - - - - - - - 10000000000 - - - random - - - - - 1 - - - - - - - - - - - - - ::/0 - - - - default - - - default - - - - - - - - - - - - - - 3600 - - - 0 - 0 - 0 - 0 - 0 - - - - diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/README.md b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/README.md similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/README.md rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/README.md diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/pom.xml b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/pom.xml similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/pom.xml rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/pom.xml diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/Application.java b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/Application.java similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/Application.java rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/Application.java diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/config/R2DBCConfig.java b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/config/R2DBCConfig.java similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/config/R2DBCConfig.java rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/config/R2DBCConfig.java diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/controller/ClickController.java b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/controller/ClickController.java similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/controller/ClickController.java rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/controller/ClickController.java diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/Click.java b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/Click.java similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/Click.java rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/Click.java diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/ClickStats.java b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/ClickStats.java similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/ClickStats.java rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/model/ClickStats.java diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/repository/ClickRepository.java b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/repository/ClickRepository.java similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/repository/ClickRepository.java rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/java/com/clickhouse/r2dbc/spring/webflux/sample/repository/ClickRepository.java diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/application.yaml b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/application.yaml similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/application.yaml rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/application.yaml diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/init.sql b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/init.sql similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/init.sql rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/init.sql diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/log4j.properties b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/log4j.properties similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/log4j.properties rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/log4j.properties diff --git a/examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/postman/clickhouse-r2dbc-sample.postman_collection.json b/examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/postman/clickhouse-r2dbc-sample.postman_collection.json similarity index 100% rename from examples/clickhouse-r2dbc-samples/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/postman/clickhouse-r2dbc-sample.postman_collection.json rename to examples/r2dbc/clickhouse-r2dbc-spring-webflux-sample/src/main/resources/postman/clickhouse-r2dbc-sample.postman_collection.json diff --git a/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.d/docker_related_config.xml b/examples/r2dbc/misc/docker/clickhouse-docker-mount/config.d/docker_related_config.xml similarity index 100% rename from clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.d/docker_related_config.xml rename to examples/r2dbc/misc/docker/clickhouse-docker-mount/config.d/docker_related_config.xml diff --git a/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.xml b/examples/r2dbc/misc/docker/clickhouse-docker-mount/config.xml similarity index 100% rename from clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/config.xml rename to examples/r2dbc/misc/docker/clickhouse-docker-mount/config.xml diff --git a/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/docker_related_config.xml b/examples/r2dbc/misc/docker/clickhouse-docker-mount/docker_related_config.xml similarity index 100% rename from clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/docker_related_config.xml rename to examples/r2dbc/misc/docker/clickhouse-docker-mount/docker_related_config.xml diff --git a/clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/users.xml b/examples/r2dbc/misc/docker/clickhouse-docker-mount/users.xml similarity index 100% rename from clickhouse-r2dbc/src/test/resources/clickhouse-docker-mount/users.xml rename to examples/r2dbc/misc/docker/clickhouse-docker-mount/users.xml diff --git a/examples/clickhouse-r2dbc-samples/misc/docker/compose.yaml b/examples/r2dbc/misc/docker/compose.yaml similarity index 100% rename from examples/clickhouse-r2dbc-samples/misc/docker/compose.yaml rename to examples/r2dbc/misc/docker/compose.yaml diff --git a/examples/clickhouse-r2dbc-samples/pom.xml b/examples/r2dbc/pom.xml similarity index 100% rename from examples/clickhouse-r2dbc-samples/pom.xml rename to examples/r2dbc/pom.xml