mirror of https://github.com/apache/druid.git
Merge pull request #1665 from gianm/consolidate-sql-retrying
Consolidate SQL retrying by moving logic into the connectors.
This commit is contained in:
commit
2c8b3901be
|
@ -104,9 +104,8 @@ public interface MetadataStorageActionHandler<EntryType, StatusType, LogType, Lo
|
|||
* Remove the lock with the given lock id.
|
||||
*
|
||||
* @param lockId lock id
|
||||
* @return true if the lock was removed, false if the given lock id did not exist
|
||||
*/
|
||||
public boolean removeLock(long lockId);
|
||||
public void removeLock(long lockId);
|
||||
|
||||
/**
|
||||
* Add a log to the entry with the given id.
|
||||
|
|
|
@ -77,11 +77,11 @@ public class MySQLConnector extends SQLMetadataConnector
|
|||
{
|
||||
// ensure database defaults to utf8, otherwise bail
|
||||
boolean isUtf8 = handle
|
||||
.createQuery("SHOW VARIABLES where variable_name = 'character_set_database' and value = 'utf8'")
|
||||
.list()
|
||||
.size() == 1;
|
||||
.createQuery("SHOW VARIABLES where variable_name = 'character_set_database' and value = 'utf8'")
|
||||
.list()
|
||||
.size() == 1;
|
||||
|
||||
if(!isUtf8) {
|
||||
if (!isUtf8) {
|
||||
throw new ISE(
|
||||
"Database default character set is not UTF-8." + System.lineSeparator()
|
||||
+ " Druid requires its MySQL database to be created using UTF-8 as default character set."
|
||||
|
@ -97,11 +97,10 @@ public class MySQLConnector extends SQLMetadataConnector
|
|||
}
|
||||
|
||||
@Override
|
||||
protected boolean isTransientException(Throwable e)
|
||||
protected boolean connectorIsTransientException(Throwable e)
|
||||
{
|
||||
return e instanceof MySQLTransientException
|
||||
|| (e instanceof SQLException && ((SQLException) e).getErrorCode() == 1317)
|
||||
;
|
||||
|| (e instanceof SQLException && ((SQLException) e).getErrorCode() == 1317 /* ER_QUERY_INTERRUPTED */);
|
||||
}
|
||||
|
||||
@Override
|
||||
|
|
|
@ -118,7 +118,7 @@ public class PostgreSQLConnector extends SQLMetadataConnector
|
|||
public DBI getDBI() { return dbi; }
|
||||
|
||||
@Override
|
||||
protected boolean isTransientException(Throwable e)
|
||||
protected boolean connectorIsTransientException(Throwable e)
|
||||
{
|
||||
if(e instanceof SQLException) {
|
||||
final String sqlState = ((SQLException) e).getSQLState();
|
||||
|
|
|
@ -17,9 +17,12 @@
|
|||
|
||||
package io.druid.metadata;
|
||||
|
||||
import com.google.common.base.Predicate;
|
||||
import com.google.common.base.Supplier;
|
||||
import com.google.common.base.Throwables;
|
||||
import com.google.common.collect.ImmutableList;
|
||||
import com.metamx.common.ISE;
|
||||
import com.metamx.common.RetryUtils;
|
||||
import com.metamx.common.logger.Logger;
|
||||
import org.apache.commons.dbcp2.BasicDataSource;
|
||||
import org.skife.jdbi.v2.Batch;
|
||||
|
@ -28,11 +31,17 @@ import org.skife.jdbi.v2.Handle;
|
|||
import org.skife.jdbi.v2.IDBI;
|
||||
import org.skife.jdbi.v2.TransactionCallback;
|
||||
import org.skife.jdbi.v2.TransactionStatus;
|
||||
import org.skife.jdbi.v2.exceptions.DBIException;
|
||||
import org.skife.jdbi.v2.exceptions.UnableToObtainConnectionException;
|
||||
import org.skife.jdbi.v2.tweak.HandleCallback;
|
||||
import org.skife.jdbi.v2.util.ByteArrayMapper;
|
||||
import org.skife.jdbi.v2.util.IntegerMapper;
|
||||
|
||||
import java.sql.SQLException;
|
||||
import java.sql.SQLRecoverableException;
|
||||
import java.sql.SQLTransientException;
|
||||
import java.util.List;
|
||||
import java.util.concurrent.Callable;
|
||||
|
||||
public abstract class SQLMetadataConnector implements MetadataStorageConnector
|
||||
{
|
||||
|
@ -41,32 +50,43 @@ public abstract class SQLMetadataConnector implements MetadataStorageConnector
|
|||
|
||||
private final Supplier<MetadataStorageConnectorConfig> config;
|
||||
private final Supplier<MetadataStorageTablesConfig> tablesConfigSupplier;
|
||||
private final Predicate<Throwable> shouldRetry;
|
||||
|
||||
public SQLMetadataConnector(Supplier<MetadataStorageConnectorConfig> config,
|
||||
Supplier<MetadataStorageTablesConfig> tablesConfigSupplier
|
||||
public SQLMetadataConnector(
|
||||
Supplier<MetadataStorageConnectorConfig> config,
|
||||
Supplier<MetadataStorageTablesConfig> tablesConfigSupplier
|
||||
)
|
||||
{
|
||||
this.config = config;
|
||||
this.tablesConfigSupplier = tablesConfigSupplier;
|
||||
this.shouldRetry = new Predicate<Throwable>()
|
||||
{
|
||||
@Override
|
||||
public boolean apply(Throwable e)
|
||||
{
|
||||
return isTransientException(e);
|
||||
}
|
||||
};
|
||||
}
|
||||
|
||||
/**
|
||||
* SQL type to use for payload data (e.g. JSON blobs).
|
||||
* Must be a binary type, which values can be accessed using ResultSet.getBytes()
|
||||
*
|
||||
* <p/>
|
||||
* The resulting string will be interpolated into the table creation statement, e.g.
|
||||
* <code>CREATE TABLE druid_table ( payload <type> NOT NULL, ... )</code>
|
||||
*
|
||||
* @return String representing the SQL type
|
||||
*/
|
||||
protected String getPayloadType() {
|
||||
protected String getPayloadType()
|
||||
{
|
||||
return PAYLOAD_TYPE;
|
||||
}
|
||||
|
||||
/**
|
||||
* Auto-incrementing SQL type to use for IDs
|
||||
* Must be an integer type, which values will be automatically set by the database
|
||||
*
|
||||
* <p/>
|
||||
* The resulting string will be interpolated into the table creation statement, e.g.
|
||||
* <code>CREATE TABLE druid_table ( id <type> NOT NULL, ... )</code>
|
||||
*
|
||||
|
@ -78,7 +98,56 @@ public abstract class SQLMetadataConnector implements MetadataStorageConnector
|
|||
|
||||
public abstract boolean tableExists(Handle handle, final String tableName);
|
||||
|
||||
protected boolean isTransientException(Throwable e) {
|
||||
public <T> T retryWithHandle(final HandleCallback<T> callback)
|
||||
{
|
||||
final Callable<T> call = new Callable<T>()
|
||||
{
|
||||
@Override
|
||||
public T call() throws Exception
|
||||
{
|
||||
return getDBI().withHandle(callback);
|
||||
}
|
||||
};
|
||||
final int maxTries = 10;
|
||||
try {
|
||||
return RetryUtils.retry(call, shouldRetry, maxTries);
|
||||
}
|
||||
catch (Exception e) {
|
||||
throw Throwables.propagate(e);
|
||||
}
|
||||
}
|
||||
|
||||
public <T> T retryTransaction(final TransactionCallback<T> callback)
|
||||
{
|
||||
final Callable<T> call = new Callable<T>()
|
||||
{
|
||||
@Override
|
||||
public T call() throws Exception
|
||||
{
|
||||
return getDBI().inTransaction(callback);
|
||||
}
|
||||
};
|
||||
final int maxTries = 10;
|
||||
try {
|
||||
return RetryUtils.retry(call, shouldRetry, maxTries);
|
||||
}
|
||||
catch (Exception e) {
|
||||
throw Throwables.propagate(e);
|
||||
}
|
||||
}
|
||||
|
||||
public final boolean isTransientException(Throwable e)
|
||||
{
|
||||
return e != null && (e instanceof SQLTransientException
|
||||
|| e instanceof SQLRecoverableException
|
||||
|| e instanceof UnableToObtainConnectionException
|
||||
|| connectorIsTransientException(e)
|
||||
|| (e instanceof SQLException && isTransientException(e.getCause()))
|
||||
|| (e instanceof DBIException && isTransientException(e.getCause())));
|
||||
}
|
||||
|
||||
protected boolean connectorIsTransientException(Throwable e)
|
||||
{
|
||||
return false;
|
||||
}
|
||||
|
||||
|
@ -94,7 +163,7 @@ public abstract class SQLMetadataConnector implements MetadataStorageConnector
|
|||
if (!tableExists(handle, tableName)) {
|
||||
log.info("Creating table[%s]", tableName);
|
||||
final Batch batch = handle.createBatch();
|
||||
for(String s : sql) {
|
||||
for (String s : sql) {
|
||||
batch.add(s);
|
||||
}
|
||||
batch.execute();
|
||||
|
|
|
@ -20,12 +20,10 @@ package io.druid.metadata;
|
|||
import com.fasterxml.jackson.core.type.TypeReference;
|
||||
import com.fasterxml.jackson.databind.ObjectMapper;
|
||||
import com.google.common.base.Optional;
|
||||
import com.google.common.base.Predicate;
|
||||
import com.google.common.base.Throwables;
|
||||
import com.google.common.collect.Lists;
|
||||
import com.google.common.collect.Maps;
|
||||
import com.metamx.common.Pair;
|
||||
import com.metamx.common.RetryUtils;
|
||||
import com.metamx.common.StringUtils;
|
||||
import com.metamx.emitter.EmittingLogger;
|
||||
import org.joda.time.DateTime;
|
||||
|
@ -34,9 +32,7 @@ import org.skife.jdbi.v2.Folder3;
|
|||
import org.skife.jdbi.v2.Handle;
|
||||
import org.skife.jdbi.v2.StatementContext;
|
||||
import org.skife.jdbi.v2.exceptions.CallbackFailedException;
|
||||
import org.skife.jdbi.v2.exceptions.DBIException;
|
||||
import org.skife.jdbi.v2.exceptions.StatementException;
|
||||
import org.skife.jdbi.v2.exceptions.UnableToObtainConnectionException;
|
||||
import org.skife.jdbi.v2.tweak.HandleCallback;
|
||||
import org.skife.jdbi.v2.tweak.ResultSetMapper;
|
||||
import org.skife.jdbi.v2.util.ByteArrayMapper;
|
||||
|
@ -44,11 +40,8 @@ import org.skife.jdbi.v2.util.ByteArrayMapper;
|
|||
import java.io.IOException;
|
||||
import java.sql.ResultSet;
|
||||
import java.sql.SQLException;
|
||||
import java.sql.SQLRecoverableException;
|
||||
import java.sql.SQLTransientException;
|
||||
import java.util.List;
|
||||
import java.util.Map;
|
||||
import java.util.concurrent.Callable;
|
||||
|
||||
public class SQLMetadataStorageActionHandler<EntryType, StatusType, LogType, LockType>
|
||||
implements MetadataStorageActionHandler<EntryType, StatusType, LogType, LockType>
|
||||
|
@ -99,7 +92,7 @@ public class SQLMetadataStorageActionHandler<EntryType, StatusType, LogType, Loc
|
|||
) throws EntryExistsException
|
||||
{
|
||||
try {
|
||||
retryingHandle(
|
||||
connector.retryWithHandle(
|
||||
new HandleCallback<Void>()
|
||||
{
|
||||
@Override
|
||||
|
@ -122,7 +115,8 @@ public class SQLMetadataStorageActionHandler<EntryType, StatusType, LogType, Loc
|
|||
}
|
||||
}
|
||||
);
|
||||
} catch(Exception e) {
|
||||
}
|
||||
catch (Exception e) {
|
||||
final boolean isStatementException = e instanceof StatementException ||
|
||||
(e instanceof CallbackFailedException
|
||||
&& e.getCause() instanceof StatementException);
|
||||
|
@ -136,30 +130,30 @@ public class SQLMetadataStorageActionHandler<EntryType, StatusType, LogType, Loc
|
|||
|
||||
public boolean setStatus(final String entryId, final boolean active, final StatusType status)
|
||||
{
|
||||
return retryingHandle(
|
||||
new HandleCallback<Boolean>()
|
||||
{
|
||||
@Override
|
||||
public Boolean withHandle(Handle handle) throws Exception
|
||||
{
|
||||
return handle.createStatement(
|
||||
String.format(
|
||||
"UPDATE %s SET active = :active, status_payload = :status_payload WHERE id = :id AND active = TRUE",
|
||||
entryTable
|
||||
)
|
||||
)
|
||||
.bind("id", entryId)
|
||||
.bind("active", active)
|
||||
.bind("status_payload", jsonMapper.writeValueAsBytes(status))
|
||||
.execute() == 1;
|
||||
}
|
||||
}
|
||||
);
|
||||
return connector.retryWithHandle(
|
||||
new HandleCallback<Boolean>()
|
||||
{
|
||||
@Override
|
||||
public Boolean withHandle(Handle handle) throws Exception
|
||||
{
|
||||
return handle.createStatement(
|
||||
String.format(
|
||||
"UPDATE %s SET active = :active, status_payload = :status_payload WHERE id = :id AND active = TRUE",
|
||||
entryTable
|
||||
)
|
||||
)
|
||||
.bind("id", entryId)
|
||||
.bind("active", active)
|
||||
.bind("status_payload", jsonMapper.writeValueAsBytes(status))
|
||||
.execute() == 1;
|
||||
}
|
||||
}
|
||||
);
|
||||
}
|
||||
|
||||
public Optional<EntryType> getEntry(final String entryId)
|
||||
{
|
||||
return retryingHandle(
|
||||
return connector.retryWithHandle(
|
||||
new HandleCallback<Optional<EntryType>>()
|
||||
{
|
||||
@Override
|
||||
|
@ -183,7 +177,7 @@ public class SQLMetadataStorageActionHandler<EntryType, StatusType, LogType, Loc
|
|||
|
||||
public Optional<StatusType> getStatus(final String entryId)
|
||||
{
|
||||
return retryingHandle(
|
||||
return connector.retryWithHandle(
|
||||
new HandleCallback<Optional<StatusType>>()
|
||||
{
|
||||
@Override
|
||||
|
@ -206,306 +200,265 @@ public class SQLMetadataStorageActionHandler<EntryType, StatusType, LogType, Loc
|
|||
|
||||
public List<Pair<EntryType, StatusType>> getActiveEntriesWithStatus()
|
||||
{
|
||||
return retryingHandle(
|
||||
new HandleCallback<List<Pair<EntryType, StatusType>>>()
|
||||
{
|
||||
@Override
|
||||
public List<Pair<EntryType, StatusType>> withHandle(Handle handle) throws Exception
|
||||
return connector.retryWithHandle(
|
||||
new HandleCallback<List<Pair<EntryType, StatusType>>>()
|
||||
{
|
||||
return handle
|
||||
.createQuery(
|
||||
String.format(
|
||||
"SELECT id, payload, status_payload FROM %s WHERE active = TRUE ORDER BY created_date",
|
||||
entryTable
|
||||
)
|
||||
)
|
||||
.map(
|
||||
new ResultSetMapper<Pair<EntryType, StatusType>>()
|
||||
{
|
||||
@Override
|
||||
public Pair<EntryType, StatusType> map(int index, ResultSet r, StatementContext ctx)
|
||||
throws SQLException
|
||||
@Override
|
||||
public List<Pair<EntryType, StatusType>> withHandle(Handle handle) throws Exception
|
||||
{
|
||||
return handle
|
||||
.createQuery(
|
||||
String.format(
|
||||
"SELECT id, payload, status_payload FROM %s WHERE active = TRUE ORDER BY created_date",
|
||||
entryTable
|
||||
)
|
||||
)
|
||||
.map(
|
||||
new ResultSetMapper<Pair<EntryType, StatusType>>()
|
||||
{
|
||||
try {
|
||||
return Pair.of(
|
||||
jsonMapper.<EntryType>readValue(
|
||||
r.getBytes("payload"),
|
||||
entryType
|
||||
),
|
||||
jsonMapper.<StatusType>readValue(
|
||||
r.getBytes("status_payload"),
|
||||
statusType
|
||||
)
|
||||
);
|
||||
}
|
||||
catch (IOException e) {
|
||||
log.makeAlert(e, "Failed to parse entry payload").addData("entry", r.getString("id")).emit();
|
||||
throw new SQLException(e);
|
||||
@Override
|
||||
public Pair<EntryType, StatusType> map(int index, ResultSet r, StatementContext ctx)
|
||||
throws SQLException
|
||||
{
|
||||
try {
|
||||
return Pair.of(
|
||||
jsonMapper.<EntryType>readValue(
|
||||
r.getBytes("payload"),
|
||||
entryType
|
||||
),
|
||||
jsonMapper.<StatusType>readValue(
|
||||
r.getBytes("status_payload"),
|
||||
statusType
|
||||
)
|
||||
);
|
||||
}
|
||||
catch (IOException e) {
|
||||
log.makeAlert(e, "Failed to parse entry payload").addData("entry", r.getString("id")).emit();
|
||||
throw new SQLException(e);
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
).list();
|
||||
).list();
|
||||
}
|
||||
}
|
||||
}
|
||||
);
|
||||
|
||||
}
|
||||
|
||||
public List<StatusType> getInactiveStatusesSince(final DateTime timestamp)
|
||||
{
|
||||
return retryingHandle(
|
||||
new HandleCallback<List<StatusType>>()
|
||||
{
|
||||
@Override
|
||||
public List<StatusType> withHandle(Handle handle) throws Exception
|
||||
return connector.retryWithHandle(
|
||||
new HandleCallback<List<StatusType>>()
|
||||
{
|
||||
return handle
|
||||
.createQuery(
|
||||
String.format(
|
||||
"SELECT id, status_payload FROM %s WHERE active = FALSE AND created_date >= :start ORDER BY created_date DESC",
|
||||
entryTable
|
||||
)
|
||||
).bind("start", timestamp.toString())
|
||||
.map(
|
||||
new ResultSetMapper<StatusType>()
|
||||
{
|
||||
@Override
|
||||
public StatusType map(int index, ResultSet r, StatementContext ctx) throws SQLException
|
||||
@Override
|
||||
public List<StatusType> withHandle(Handle handle) throws Exception
|
||||
{
|
||||
return handle
|
||||
.createQuery(
|
||||
String.format(
|
||||
"SELECT id, status_payload FROM %s WHERE active = FALSE AND created_date >= :start ORDER BY created_date DESC",
|
||||
entryTable
|
||||
)
|
||||
).bind("start", timestamp.toString())
|
||||
.map(
|
||||
new ResultSetMapper<StatusType>()
|
||||
{
|
||||
try {
|
||||
return jsonMapper.readValue(
|
||||
r.getBytes("status_payload"),
|
||||
statusType
|
||||
);
|
||||
}
|
||||
catch (IOException e) {
|
||||
log.makeAlert(e, "Failed to parse status payload")
|
||||
.addData("entry", r.getString("id"))
|
||||
.emit();
|
||||
throw new SQLException(e);
|
||||
@Override
|
||||
public StatusType map(int index, ResultSet r, StatementContext ctx) throws SQLException
|
||||
{
|
||||
try {
|
||||
return jsonMapper.readValue(
|
||||
r.getBytes("status_payload"),
|
||||
statusType
|
||||
);
|
||||
}
|
||||
catch (IOException e) {
|
||||
log.makeAlert(e, "Failed to parse status payload")
|
||||
.addData("entry", r.getString("id"))
|
||||
.emit();
|
||||
throw new SQLException(e);
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
).list();
|
||||
).list();
|
||||
}
|
||||
}
|
||||
}
|
||||
);
|
||||
}
|
||||
|
||||
public boolean addLock(final String entryId, final LockType lock)
|
||||
{
|
||||
return retryingHandle(
|
||||
new HandleCallback<Boolean>()
|
||||
{
|
||||
@Override
|
||||
public Boolean withHandle(Handle handle) throws Exception
|
||||
return connector.retryWithHandle(
|
||||
new HandleCallback<Boolean>()
|
||||
{
|
||||
return handle.createStatement(
|
||||
String.format(
|
||||
"INSERT INTO %1$s (%2$s_id, lock_payload) VALUES (:entryId, :payload)",
|
||||
lockTable, entryTypeName
|
||||
)
|
||||
)
|
||||
.bind("entryId", entryId)
|
||||
.bind("payload", jsonMapper.writeValueAsBytes(lock))
|
||||
.execute() == 1;
|
||||
@Override
|
||||
public Boolean withHandle(Handle handle) throws Exception
|
||||
{
|
||||
return handle.createStatement(
|
||||
String.format(
|
||||
"INSERT INTO %1$s (%2$s_id, lock_payload) VALUES (:entryId, :payload)",
|
||||
lockTable, entryTypeName
|
||||
)
|
||||
)
|
||||
.bind("entryId", entryId)
|
||||
.bind("payload", jsonMapper.writeValueAsBytes(lock))
|
||||
.execute() == 1;
|
||||
}
|
||||
}
|
||||
}
|
||||
);
|
||||
}
|
||||
|
||||
public boolean removeLock(final long lockId)
|
||||
public void removeLock(final long lockId)
|
||||
{
|
||||
return retryingHandle(
|
||||
new HandleCallback<Boolean>()
|
||||
{
|
||||
@Override
|
||||
public Boolean withHandle(Handle handle) throws Exception
|
||||
connector.retryWithHandle(
|
||||
new HandleCallback<Void>()
|
||||
{
|
||||
return handle.createStatement(
|
||||
String.format(
|
||||
"DELETE FROM %s WHERE id = :id",
|
||||
lockTable
|
||||
)
|
||||
)
|
||||
.bind("id", lockId)
|
||||
.execute() == 1;
|
||||
@Override
|
||||
public Void withHandle(Handle handle) throws Exception
|
||||
{
|
||||
handle.createStatement(String.format("DELETE FROM %s WHERE id = :id", lockTable))
|
||||
.bind("id", lockId)
|
||||
.execute();
|
||||
|
||||
return null;
|
||||
}
|
||||
}
|
||||
}
|
||||
);
|
||||
}
|
||||
|
||||
public boolean addLog(final String entryId, final LogType log)
|
||||
{
|
||||
return retryingHandle(
|
||||
new HandleCallback<Boolean>()
|
||||
{
|
||||
@Override
|
||||
public Boolean withHandle(Handle handle) throws Exception
|
||||
return connector.retryWithHandle(
|
||||
new HandleCallback<Boolean>()
|
||||
{
|
||||
return handle.createStatement(
|
||||
String.format(
|
||||
"INSERT INTO %1$s (%2$s_id, log_payload) VALUES (:entryId, :payload)",
|
||||
logTable, entryTypeName
|
||||
)
|
||||
)
|
||||
.bind("entryId", entryId)
|
||||
.bind("payload", jsonMapper.writeValueAsBytes(log))
|
||||
.execute() == 1;
|
||||
@Override
|
||||
public Boolean withHandle(Handle handle) throws Exception
|
||||
{
|
||||
return handle.createStatement(
|
||||
String.format(
|
||||
"INSERT INTO %1$s (%2$s_id, log_payload) VALUES (:entryId, :payload)",
|
||||
logTable, entryTypeName
|
||||
)
|
||||
)
|
||||
.bind("entryId", entryId)
|
||||
.bind("payload", jsonMapper.writeValueAsBytes(log))
|
||||
.execute() == 1;
|
||||
}
|
||||
}
|
||||
}
|
||||
);
|
||||
}
|
||||
|
||||
public List<LogType> getLogs(final String entryId)
|
||||
{
|
||||
return retryingHandle(
|
||||
new HandleCallback<List<LogType>>()
|
||||
{
|
||||
@Override
|
||||
public List<LogType> withHandle(Handle handle) throws Exception
|
||||
return connector.retryWithHandle(
|
||||
new HandleCallback<List<LogType>>()
|
||||
{
|
||||
return handle
|
||||
.createQuery(
|
||||
String.format(
|
||||
"SELECT log_payload FROM %1$s WHERE %2$s_id = :entryId",
|
||||
logTable, entryTypeName
|
||||
)
|
||||
)
|
||||
.bind("entryId", entryId)
|
||||
.map(ByteArrayMapper.FIRST)
|
||||
.fold(
|
||||
Lists.<LogType>newLinkedList(),
|
||||
new Folder3<List<LogType>, byte[]>()
|
||||
{
|
||||
@Override
|
||||
public List<LogType> fold(
|
||||
List<LogType> list, byte[] bytes, FoldController control, StatementContext ctx
|
||||
) throws SQLException
|
||||
@Override
|
||||
public List<LogType> withHandle(Handle handle) throws Exception
|
||||
{
|
||||
return handle
|
||||
.createQuery(
|
||||
String.format(
|
||||
"SELECT log_payload FROM %1$s WHERE %2$s_id = :entryId",
|
||||
logTable, entryTypeName
|
||||
)
|
||||
)
|
||||
.bind("entryId", entryId)
|
||||
.map(ByteArrayMapper.FIRST)
|
||||
.fold(
|
||||
Lists.<LogType>newLinkedList(),
|
||||
new Folder3<List<LogType>, byte[]>()
|
||||
{
|
||||
try {
|
||||
list.add(
|
||||
jsonMapper.<LogType>readValue(
|
||||
bytes, logType
|
||||
)
|
||||
);
|
||||
return list;
|
||||
}
|
||||
catch (IOException e) {
|
||||
log.makeAlert(e, "Failed to deserialize log")
|
||||
.addData("entryId", entryId)
|
||||
.addData("payload", StringUtils.fromUtf8(bytes))
|
||||
.emit();
|
||||
throw new SQLException(e);
|
||||
@Override
|
||||
public List<LogType> fold(
|
||||
List<LogType> list, byte[] bytes, FoldController control, StatementContext ctx
|
||||
) throws SQLException
|
||||
{
|
||||
try {
|
||||
list.add(
|
||||
jsonMapper.<LogType>readValue(
|
||||
bytes, logType
|
||||
)
|
||||
);
|
||||
return list;
|
||||
}
|
||||
catch (IOException e) {
|
||||
log.makeAlert(e, "Failed to deserialize log")
|
||||
.addData("entryId", entryId)
|
||||
.addData("payload", StringUtils.fromUtf8(bytes))
|
||||
.emit();
|
||||
throw new SQLException(e);
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
);
|
||||
);
|
||||
}
|
||||
}
|
||||
}
|
||||
);
|
||||
}
|
||||
|
||||
public Map<Long, LockType> getLocks(final String entryId)
|
||||
{
|
||||
return retryingHandle(
|
||||
new HandleCallback<Map<Long, LockType>>()
|
||||
{
|
||||
@Override
|
||||
public Map<Long, LockType> withHandle(Handle handle) throws Exception
|
||||
return connector.retryWithHandle(
|
||||
new HandleCallback<Map<Long, LockType>>()
|
||||
{
|
||||
return handle.createQuery(
|
||||
String.format(
|
||||
"SELECT id, lock_payload FROM %1$s WHERE %2$s_id = :entryId",
|
||||
lockTable, entryTypeName
|
||||
)
|
||||
)
|
||||
.bind("entryId", entryId)
|
||||
.map(
|
||||
new ResultSetMapper<Pair<Long, LockType>>()
|
||||
{
|
||||
@Override
|
||||
public Pair<Long, LockType> map(int index, ResultSet r, StatementContext ctx)
|
||||
throws SQLException
|
||||
{
|
||||
try {
|
||||
return Pair.of(
|
||||
r.getLong("id"),
|
||||
jsonMapper.<LockType>readValue(
|
||||
r.getBytes("lock_payload"),
|
||||
lockType
|
||||
)
|
||||
);
|
||||
}
|
||||
catch (IOException e) {
|
||||
log.makeAlert(e, "Failed to deserialize " + lockType.getType())
|
||||
.addData("id", r.getLong("id"))
|
||||
.addData(
|
||||
"lockPayload", StringUtils.fromUtf8(r.getBytes("lock_payload"))
|
||||
)
|
||||
.emit();
|
||||
throw new SQLException(e);
|
||||
}
|
||||
}
|
||||
}
|
||||
)
|
||||
.fold(
|
||||
Maps.<Long, LockType>newLinkedHashMap(),
|
||||
new Folder3<Map<Long, LockType>, Pair<Long, LockType>>()
|
||||
{
|
||||
@Override
|
||||
public Map<Long, LockType> fold(
|
||||
Map<Long, LockType> accumulator,
|
||||
Pair<Long, LockType> lock,
|
||||
FoldController control,
|
||||
StatementContext ctx
|
||||
) throws SQLException
|
||||
{
|
||||
accumulator.put(lock.lhs, lock.rhs);
|
||||
return accumulator;
|
||||
}
|
||||
}
|
||||
);
|
||||
@Override
|
||||
public Map<Long, LockType> withHandle(Handle handle) throws Exception
|
||||
{
|
||||
return handle.createQuery(
|
||||
String.format(
|
||||
"SELECT id, lock_payload FROM %1$s WHERE %2$s_id = :entryId",
|
||||
lockTable, entryTypeName
|
||||
)
|
||||
)
|
||||
.bind("entryId", entryId)
|
||||
.map(
|
||||
new ResultSetMapper<Pair<Long, LockType>>()
|
||||
{
|
||||
@Override
|
||||
public Pair<Long, LockType> map(int index, ResultSet r, StatementContext ctx)
|
||||
throws SQLException
|
||||
{
|
||||
try {
|
||||
return Pair.of(
|
||||
r.getLong("id"),
|
||||
jsonMapper.<LockType>readValue(
|
||||
r.getBytes("lock_payload"),
|
||||
lockType
|
||||
)
|
||||
);
|
||||
}
|
||||
catch (IOException e) {
|
||||
log.makeAlert(e, "Failed to deserialize " + lockType.getType())
|
||||
.addData("id", r.getLong("id"))
|
||||
.addData(
|
||||
"lockPayload", StringUtils.fromUtf8(r.getBytes("lock_payload"))
|
||||
)
|
||||
.emit();
|
||||
throw new SQLException(e);
|
||||
}
|
||||
}
|
||||
}
|
||||
)
|
||||
.fold(
|
||||
Maps.<Long, LockType>newLinkedHashMap(),
|
||||
new Folder3<Map<Long, LockType>, Pair<Long, LockType>>()
|
||||
{
|
||||
@Override
|
||||
public Map<Long, LockType> fold(
|
||||
Map<Long, LockType> accumulator,
|
||||
Pair<Long, LockType> lock,
|
||||
FoldController control,
|
||||
StatementContext ctx
|
||||
) throws SQLException
|
||||
{
|
||||
accumulator.put(lock.lhs, lock.rhs);
|
||||
return accumulator;
|
||||
}
|
||||
}
|
||||
);
|
||||
}
|
||||
}
|
||||
}
|
||||
);
|
||||
}
|
||||
|
||||
private <T> T retryingHandle(final HandleCallback<T> callback)
|
||||
{
|
||||
final Callable<T> call = new Callable<T>()
|
||||
{
|
||||
@Override
|
||||
public T call() throws Exception
|
||||
{
|
||||
return connector.getDBI().withHandle(callback);
|
||||
}
|
||||
};
|
||||
final Predicate<Throwable> shouldRetry = new Predicate<Throwable>()
|
||||
{
|
||||
@Override
|
||||
public boolean apply(Throwable e)
|
||||
{
|
||||
return shouldRetryException(e);
|
||||
}
|
||||
};
|
||||
final int maxTries = 10;
|
||||
try {
|
||||
return RetryUtils.retry(call, shouldRetry, maxTries);
|
||||
}
|
||||
catch (Exception e) {
|
||||
throw Throwables.propagate(e);
|
||||
}
|
||||
}
|
||||
|
||||
protected boolean shouldRetryException(final Throwable e)
|
||||
{
|
||||
return e != null && (e instanceof SQLTransientException
|
||||
|| connector.isTransientException(e)
|
||||
|| e instanceof SQLRecoverableException
|
||||
|| e instanceof UnableToObtainConnectionException
|
||||
|| (e instanceof SQLException && shouldRetryException(e.getCause()))
|
||||
|| (e instanceof DBIException && shouldRetryException(e.getCause())));
|
||||
}
|
||||
|
||||
}
|
||||
|
|
|
@ -40,11 +40,12 @@ public class SQLMetadataStorageActionHandlerTest
|
|||
private static final ObjectMapper jsonMapper = new DefaultObjectMapper();
|
||||
private TestDerbyConnector connector;
|
||||
private MetadataStorageTablesConfig tablesConfig = MetadataStorageTablesConfig.fromBase("test");
|
||||
private SQLMetadataStorageActionHandler<Map<String, Integer>,Map<String, Integer>,Map<String, String>,Map<String, Integer>> handler;
|
||||
private SQLMetadataStorageActionHandler<Map<String, Integer>, Map<String, Integer>, Map<String, String>, Map<String, Integer>> handler;
|
||||
|
||||
@Before
|
||||
public void setUp() throws Exception {
|
||||
MetadataStorageConnectorConfig config = new MetadataStorageConnectorConfig();
|
||||
public void setUp() throws Exception
|
||||
{
|
||||
MetadataStorageConnectorConfig config = new MetadataStorageConnectorConfig();
|
||||
|
||||
connector = new TestDerbyConnector(
|
||||
Suppliers.ofInstance(config),
|
||||
|
@ -67,38 +68,48 @@ public class SQLMetadataStorageActionHandlerTest
|
|||
jsonMapper,
|
||||
new MetadataStorageActionHandlerTypes<Map<String, Integer>, Map<String, Integer>, Map<String, String>, Map<String, Integer>>()
|
||||
{
|
||||
@Override
|
||||
public TypeReference<Map<String, Integer>> getEntryType()
|
||||
{
|
||||
return new TypeReference<Map<String, Integer>>() {};
|
||||
}
|
||||
@Override
|
||||
public TypeReference<Map<String, Integer>> getEntryType()
|
||||
{
|
||||
return new TypeReference<Map<String, Integer>>()
|
||||
{
|
||||
};
|
||||
}
|
||||
|
||||
@Override
|
||||
public TypeReference<Map<String, Integer>> getStatusType()
|
||||
{
|
||||
return new TypeReference<Map<String, Integer>>() {};
|
||||
}
|
||||
@Override
|
||||
public TypeReference<Map<String, Integer>> getStatusType()
|
||||
{
|
||||
return new TypeReference<Map<String, Integer>>()
|
||||
{
|
||||
};
|
||||
}
|
||||
|
||||
@Override
|
||||
public TypeReference<Map<String, String>> getLogType()
|
||||
{
|
||||
return new TypeReference<Map<String, String>>() {};
|
||||
}
|
||||
@Override
|
||||
public TypeReference<Map<String, String>> getLogType()
|
||||
{
|
||||
return new TypeReference<Map<String, String>>()
|
||||
{
|
||||
};
|
||||
}
|
||||
|
||||
@Override
|
||||
public TypeReference<Map<String, Integer>> getLockType()
|
||||
{
|
||||
return new TypeReference<Map<String, Integer>>() {};
|
||||
}
|
||||
},
|
||||
@Override
|
||||
public TypeReference<Map<String, Integer>> getLockType()
|
||||
{
|
||||
return new TypeReference<Map<String, Integer>>()
|
||||
{
|
||||
};
|
||||
}
|
||||
},
|
||||
entryType,
|
||||
entryTable,
|
||||
logTable,
|
||||
lockTable);
|
||||
lockTable
|
||||
);
|
||||
}
|
||||
|
||||
@After
|
||||
public void tearDown() {
|
||||
public void tearDown()
|
||||
{
|
||||
connector.tearDown();
|
||||
}
|
||||
|
||||
|
@ -240,7 +251,7 @@ public class SQLMetadataStorageActionHandlerTest
|
|||
);
|
||||
|
||||
long lockId = locks.keySet().iterator().next();
|
||||
Assert.assertTrue(handler.removeLock(lockId));
|
||||
handler.removeLock(lockId);
|
||||
locks.remove(lockId);
|
||||
|
||||
final Map<Long, Map<String, Integer>> updated = handler.getLocks(entryId);
|
||||
|
|
Loading…
Reference in New Issue