Skip to content
Open
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -395,33 +395,38 @@ public void testInsertMetricsOperationId() throws Exception {
@Test(groups = { "integration" })
public void testInsertSettingsAddDatabase() throws Exception {
final String tableName = "insert_settings_database_test";
final String new_database = "new_database";
final String createDatabaseSQL = "CREATE DATABASE " + new_database;
final String new_database = client.getDefaultDatabase() + "_new_database";
final String createDatabaseSQL = "CREATE DATABASE IF NOT EXISTS " + new_database;
final String createTableSQL = "CREATE TABLE " + new_database + "." + tableName +
" (Id UInt32, event_ts Timestamp, name String, p1 Int64, p2 String) ENGINE = MergeTree() ORDER BY ()";
final String dropDatabaseSQL = "DROP DATABASE IF EXISTS " + new_database;

client.execute(dropDatabaseSQL).get(EXECUTE_CMD_TIMEOUT, TimeUnit.SECONDS);
client.execute(createDatabaseSQL).get(EXECUTE_CMD_TIMEOUT, TimeUnit.SECONDS);
client.execute(createTableSQL).get(EXECUTE_CMD_TIMEOUT, TimeUnit.SECONDS);
try {
client.execute(dropDatabaseSQL).get(EXECUTE_CMD_TIMEOUT, TimeUnit.SECONDS);
client.execute(createDatabaseSQL).get(EXECUTE_CMD_TIMEOUT, TimeUnit.SECONDS);
client.execute(createTableSQL).get(EXECUTE_CMD_TIMEOUT, TimeUnit.SECONDS);

InsertSettings insertSettings = settings.setInputStreamCopyBufferSize(8198 * 2)
.setDeduplicationToken(RandomStringUtils.randomAlphabetic(36))
.setQueryId(String.valueOf(UUID.randomUUID()))
.setDatabase(new_database);
InsertSettings insertSettings = settings.setInputStreamCopyBufferSize(8198 * 2)
.setDeduplicationToken(RandomStringUtils.randomAlphabetic(36))
.setQueryId(String.valueOf(UUID.randomUUID()))
.setDatabase(new_database);

ByteArrayOutputStream data = new ByteArrayOutputStream();
PrintWriter writer = new PrintWriter(data);
for (int i = 0; i < 1000; i++) {
writer.printf("%d\t%s\t%s\t%d\t%s\n", i, "2021-01-01 00:00:00", "name" + i, i, "p2");
}
writer.flush();
InsertResponse response = client.insert(tableName, new ByteArrayInputStream(data.toByteArray()),
ClickHouseFormat.TSV, insertSettings).get(EXECUTE_CMD_TIMEOUT, TimeUnit.SECONDS);
assertEquals((int)response.getWrittenRows(), 1000 );
ByteArrayOutputStream data = new ByteArrayOutputStream();
PrintWriter writer = new PrintWriter(data);
final int rows = 1000;
for (int i = 0; i < rows; i++) {
writer.printf("%d\t%s\t%s\t%d\t%s\n", i, "2021-01-01 00:00:00", "name" + i, i, "p2");
}
writer.flush();
InsertResponse response = client.insert(tableName, new ByteArrayInputStream(data.toByteArray()),
ClickHouseFormat.TSV, insertSettings).get(EXECUTE_CMD_TIMEOUT, TimeUnit.SECONDS);
assertEquals((int) response.getWrittenRows(), rows);

List<GenericRecord> records = client.queryAll("SELECT * FROM " + new_database + "." + tableName);
assertEquals(records.size(), 1000);
List<GenericRecord> records = client.queryAll("SELECT * FROM " + new_database + "." + tableName);
assertEquals(records.size(), rows);
} finally {
client.execute(dropDatabaseSQL);
}
}

@Test(groups = {"integration"}, dataProviderClass = InsertTests.class, dataProvider = "logCommentDataProvider")
Expand Down
20 changes: 12 additions & 8 deletions jdbc-v2/src/test/java/com/clickhouse/jdbc/ConnectionTest.java
Original file line number Diff line number Diff line change
Expand Up @@ -630,23 +630,23 @@ public void abortTest() throws SQLException {
}

@Test(groups = {"integration"})
public void testNetworkTimeout() throws Exception {
public void testNetworkTimeoutCausesConnectionClosure() throws Exception {
try (Connection conn = this.getJdbcConnection()) {
Assert.assertThrows(SQLException.class, () -> conn.setNetworkTimeout(null, 1000));
Assert.assertThrows(SQLException.class, () -> conn.setNetworkTimeout(Executors.newSingleThreadExecutor(), -1));

int timeout = 10;
int timeout = (int) TimeUnit.SECONDS.toMillis(1);
ExecutorService executorService = Executors.newSingleThreadExecutor();
conn.setNetworkTimeout(executorService, timeout);
conn.setNetworkTimeout(executorService, timeout); // timeout in ms
Assert.assertEquals(conn.getNetworkTimeout(), timeout);
Statement stmt = conn.createStatement();
try (ResultSet rs = stmt.executeQuery("SELECT sleepEachRow(1) FROM system.numbers LIMIT 2")) {
fail("Exception expected");
} catch (Exception e) {
executorService.shutdown();
executorService.awaitTermination(20, TimeUnit.SECONDS);
executorService.awaitTermination(5, TimeUnit.SECONDS);
Assert.assertTrue(conn.isClosed());
Assert.assertFalse(conn.isValid(5000));
Assert.assertFalse(conn.isValid(5)); // timeout in seconds
conn.close();

}
Expand All @@ -657,22 +657,26 @@ public void testNetworkTimeout() throws Exception {
Assert.assertTrue(e.getMessage().contains("closed"));
}
}
}

@Test(groups = {"integration"})
public void testHandlingTimeoutWithoutTimeoutIsSet() throws Exception {
int timeout = (int) TimeUnit.SECONDS.toMillis(1);
Properties connConfig = new Properties();
connConfig.setProperty(ClientConfigProperties.SOCKET_OPERATION_TIMEOUT.getKey(), "10");
connConfig.setProperty(ClientConfigProperties.SOCKET_OPERATION_TIMEOUT.getKey(), String.valueOf(timeout)); // in ms
try (Connection conn = getJdbcConnection(connConfig)) {
Statement stmt = conn.createStatement();
try (ResultSet rs = stmt.executeQuery("SELECT sleepEachRow(1) FROM system.numbers LIMIT 2")) {
fail("Exception expected");
} catch (Exception e) {
Assert.assertFalse(conn.isClosed());
Assert.assertTrue(conn.isValid(5000));
Assert.assertTrue(conn.isValid(5)); // in seconds
}
}

try (Connection conn = getJdbcConnection(connConfig)) {
ExecutorService executorService = Executors.newSingleThreadExecutor();
conn.setNetworkTimeout(executorService, 10);
conn.setNetworkTimeout(executorService, timeout); // ms
try (Statement stmt1 = conn.createStatement(); Statement stmt2 = conn.createStatement()) {
ScheduledExecutorService stmtExecutor = Executors.newScheduledThreadPool(2);
long t1 = System.currentTimeMillis();
Expand Down