Skip to content
Merged
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
4 changes: 2 additions & 2 deletions common/src/main/java/io/druid/guice/DruidSecondaryModule.java
Original file line number Diff line number Diff line change
Expand Up @@ -72,14 +72,14 @@ public void configure(Binder binder)
}

@Provides @LazySingleton @Json
public ObjectMapper getJsonMapper(final Injector injector)
public final ObjectMapper getJsonMapper(final Injector injector)
{
setupJackson(injector, jsonMapper);
return jsonMapper;
}

@Provides @LazySingleton @Smile
public ObjectMapper getSmileMapper(Injector injector)
public final ObjectMapper getSmileMapper(Injector injector)
{
setupJackson(injector, smileMapper);
return smileMapper;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -40,7 +40,7 @@ public void configure(Binder binder)
}

@Provides @ManageLifecycle
public ConfigManager getConfigManager(
public final ConfigManager getConfigManager(
final MetadataStorageConnector dbConnector,
final Supplier<MetadataStorageTablesConfig> dbTables,
final Supplier<ConfigManagerConfig> config,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -98,7 +98,7 @@ public void configure(Binder binder)

@Provides
@LazySingleton
public CloudBlobClient getCloudBlobClient(final AzureAccountConfig config)
public final CloudBlobClient getCloudBlobClient(final AzureAccountConfig config)
throws URISyntaxException, InvalidKeyException
{
CloudStorageAccount account = CloudStorageAccount.parse(
Expand All @@ -115,7 +115,7 @@ public CloudBlobClient getCloudBlobClient(final AzureAccountConfig config)

@Provides
@LazySingleton
public AzureStorage getAzureStorageContainer(
public final AzureStorage getAzureStorageContainer(
final CloudBlobClient cloudBlobClient
)
{
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -71,7 +71,15 @@ public void setupModule(SetupContext setupContext)
@Provides
@Named("renameKafkaProperties")
@LazySingleton
public Properties getProperties(
public final Properties getProperties(
@Json ObjectMapper mapper,
Properties systemProperties
){
return getPropertiesInner(mapper, systemProperties);
}

// Protected for use in unit tests
protected Properties getPropertiesInner(
@Json ObjectMapper mapper,
Properties systemProperties
)
Expand All @@ -98,7 +106,7 @@ public Properties getProperties(

@Provides
@LazySingleton
public KafkaExtractionNamespaceFactory factoryFactory(
public final KafkaExtractionNamespaceFactory factoryFactory(
KafkaExtractionManager kafkaManager
)
{
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -31,9 +31,6 @@
import com.google.inject.Module;
import com.google.inject.Provider;
import com.google.inject.TypeLiteral;
import com.google.inject.multibindings.MapBinder;
import com.google.inject.multibindings.Multibinder;
import com.google.inject.name.Named;
import com.google.inject.name.Names;
import com.metamx.common.ISE;
import com.metamx.common.StringUtils;
Expand All @@ -58,10 +55,6 @@
import org.I0Itec.zkclient.ZkClient;
import org.apache.commons.io.FileUtils;
import org.apache.curator.test.TestingServer;
import org.apache.log4j.BasicConfigurator;
import org.apache.log4j.ConsoleAppender;
import org.apache.log4j.Level;
import org.apache.log4j.PatternLayout;
import org.apache.zookeeper.CreateMode;
import org.joda.time.DateTime;
import org.junit.AfterClass;
Expand Down Expand Up @@ -97,16 +90,18 @@ public class TestKafkaExtractionCluster
private static Injector injector;



public static class KafkaFactoryProvider implements Provider<ExtractionNamespaceFunctionFactory<?>>
{
private final KafkaExtractionManager kafkaExtractionManager;

@Inject
public KafkaFactoryProvider(
KafkaExtractionManager kafkaExtractionManager
){
)
{
this.kafkaExtractionManager = kafkaExtractionManager;
}

@Override
public ExtractionNamespaceFunctionFactory<?> get()
{
Expand Down Expand Up @@ -222,6 +217,10 @@ public void sleep(long ms)
}

System.setProperty("druid.extensions.searchCurrentClassloader", "false");
final Properties consumerProperties = new Properties(kafkaProperties);
consumerProperties.put("zookeeper.connect", zkTestServer.getConnectString() + "/kafka");
consumerProperties.put("zookeeper.session.timeout.ms", "10000");
consumerProperties.put("zookeeper.sync.time.ms", "200");

injector = Initialization.makeInjectorWithModules(
GuiceInjectors.makeStartupInjectorWithModules(
Expand All @@ -239,15 +238,11 @@ public void configure(Binder binder)
new KafkaExtractionNamespaceModule()
{
@Override
public Properties getProperties(
protected Properties getPropertiesInner(
@Json ObjectMapper mapper,
Properties systemProperties
)
{
final Properties consumerProperties = new Properties(kafkaProperties);
consumerProperties.put("zookeeper.connect", zkTestServer.getConnectString() + "/kafka");
consumerProperties.put("zookeeper.session.timeout.ms", "10000");
consumerProperties.put("zookeeper.sync.time.ms", "200");
return consumerProperties;
}
}
Expand Down Expand Up @@ -287,7 +282,8 @@ public static void closeStatic() throws IOException
if (zkClient.exists("/kafka")) {
try {
zkClient.deleteRecursive("/kafka");
}catch(org.I0Itec.zkclient.exception.ZkException ex){
}
catch (org.I0Itec.zkclient.exception.ZkException ex) {
log.warn(ex, "error deleting /kafka zk node");
}
}
Expand All @@ -296,12 +292,13 @@ public static void closeStatic() throws IOException
if (null != zkTestServer) {
zkTestServer.stop();
}
if(tmpDir.exists()){
if (tmpDir.exists()) {
FileUtils.deleteDirectory(tmpDir);
}
}

private static final Properties makeProducerProperties(){
private static final Properties makeProducerProperties()
{
final Properties kafkaProducerProperties = new Properties();
kafkaProducerProperties.putAll(kafkaProperties);
kafkaProducerProperties.put(
Expand All @@ -327,7 +324,16 @@ public void testSimpleRename() throws InterruptedException
final Producer<byte[], byte[]> producer = new Producer<byte[], byte[]>(new ProducerConfig(kafkaProducerProperties));
try {
checkServer();
final ConcurrentMap<String, Function<String, String>> fnFn = injector.getInstance(Key.get(new TypeLiteral<ConcurrentMap<String, Function<String, String>>>(){}, Names.named("namespaceExtractionFunctionCache")));
final ConcurrentMap<String, Function<String, String>> fnFn = injector.getInstance(
Key.get(
new TypeLiteral<ConcurrentMap<String, Function<String, String>>>()
{
},
Names.named(
"namespaceExtractionFunctionCache"
)
)
);
KafkaExtractionNamespace extractionNamespace = new KafkaExtractionNamespace(topicName, namespace);

Assert.assertEquals(null, fnFn.get(extractionNamespace.getNamespace()).apply("foo"));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -164,22 +164,22 @@ public void configure(Binder binder)
@Provides
@Named("namespaceVersionMap")
@LazySingleton
public ConcurrentMap<String, String> getVersionMap()
public final ConcurrentMap<String, String> getVersionMap()
{
return new ConcurrentHashMap<>();
}

@Provides
@Named("namespaceExtractionFunctionCache")
public ConcurrentMap<String, Function<String, String>> getFnCache()
public final ConcurrentMap<String, Function<String, String>> getFnCache()
{
return fnCache;
}

@Provides
@Named("dimExtractionNamespace")
@LazySingleton
public Function<String, Function<String, String>> getFunctionMaker(
public final Function<String, Function<String, String>> getFunctionMaker(
@Named("namespaceExtractionFunctionCache")
final ConcurrentMap<String, Function<String, String>> fnCache
)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -95,14 +95,14 @@ public void configure(Binder binder)

@Provides
@LazySingleton
public AWSCredentialsProvider getAWSCredentialsProvider(final AWSCredentialsConfig config)
public final AWSCredentialsProvider getAWSCredentialsProvider(final AWSCredentialsConfig config)
{
return AWSCredentialsUtils.defaultAWSCredentialsProviderChain(config);
}

@Provides
@LazySingleton
public RestS3Service getRestS3Service(AWSCredentialsProvider provider)
public final RestS3Service getRestS3Service(AWSCredentialsProvider provider)
{
if(provider.getCredentials() instanceof com.amazonaws.auth.AWSSessionCredentials) {
return new RestS3Service(new AWSSessionCredentialsAdapter(provider));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -44,7 +44,7 @@ public void configure(Binder binder)

@Provides
@LazySingleton
public ServiceEmitter getServiceEmitter(Supplier<LoggingEmitterConfig> config, ObjectMapper jsonMapper)
public final ServiceEmitter getServiceEmitter(Supplier<LoggingEmitterConfig> config, ObjectMapper jsonMapper)
{
return new ServiceEmitter("", "", new LoggingEmitter(config.get(), jsonMapper));
}
Expand Down
9 changes: 5 additions & 4 deletions pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -69,11 +69,12 @@
<apache.curator.version>2.8.0</apache.curator.version>
<jetty.version>9.2.5.v20141112</jetty.version>
<jersey.version>1.19</jersey.version>
<druid.api.version>0.3.9</druid.api.version>
<druid.api.version>0.3.10</druid.api.version>
<jackson.version>2.4.4</jackson.version>
<log4j.version>2.2</log4j.version>
<slf4j.version>1.7.10</slf4j.version>
<hadoop.compile.version>2.3.0</hadoop.compile.version>
<guice.version>4.0</guice.version>
</properties>

<modules>
Expand Down Expand Up @@ -235,17 +236,17 @@
<dependency>
<groupId>com.google.inject</groupId>
<artifactId>guice</artifactId>
<version>4.0-beta</version>
<version>${guice.version}</version>
</dependency>
<dependency>
<groupId>com.google.inject.extensions</groupId>
<artifactId>guice-servlet</artifactId>
<version>4.0-beta</version>
<version>${guice.version}</version>
</dependency>
<dependency>
<groupId>com.google.inject.extensions</groupId>
<artifactId>guice-multibindings</artifactId>
<version>4.0-beta</version>
<version>${guice.version}</version>
</dependency>
<dependency>
<groupId>com.ibm.icu</groupId>
Expand Down
2 changes: 1 addition & 1 deletion processing/src/main/java/io/druid/guice/ConfigModule.java
Original file line number Diff line number Diff line change
Expand Up @@ -39,7 +39,7 @@ public void configure(Binder binder)
}

@Provides @LazySingleton
public ConfigurationObjectFactory makeFactory(Properties props)
public final ConfigurationObjectFactory makeFactory(Properties props)
{
return Config.createFactory(props);
}
Expand Down
4 changes: 2 additions & 2 deletions processing/src/main/java/io/druid/jackson/JacksonModule.java
Original file line number Diff line number Diff line change
Expand Up @@ -39,13 +39,13 @@ public void configure(Binder binder)
}

@Provides @LazySingleton @Json
public ObjectMapper jsonMapper()
public final ObjectMapper jsonMapper()
{
return new DefaultObjectMapper();
}

@Provides @LazySingleton @Smile
public ObjectMapper smileMapper()
public final ObjectMapper smileMapper()
{
final SmileFactory smileFactory = new SmileFactory();
smileFactory.configure(SmileGenerator.Feature.ENCODE_BINARY_AS_7BIT, false);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -52,7 +52,7 @@ public void configure(Binder binder)

@Provides
@LazySingleton
public ObjectMapper jsonMapper()
public final ObjectMapper jsonMapper()
{
return new DefaultObjectMapper();
}
Expand Down
2 changes: 1 addition & 1 deletion server/src/main/java/io/druid/curator/CuratorModule.java
Original file line number Diff line number Diff line change
Expand Up @@ -45,7 +45,7 @@ public void configure(Binder binder)
}

@Provides @LazySingleton
public CuratorFramework makeCurator(CuratorConfig config, Lifecycle lifecycle) throws IOException
public final CuratorFramework makeCurator(CuratorConfig config, Lifecycle lifecycle) throws IOException
{
final CuratorFramework framework =
CuratorFrameworkFactory.builder()
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -145,7 +145,7 @@ public void configure(Binder binder)
@Provides
@LazySingleton
@Named(NAME)
public CuratorServiceAnnouncer getServiceAnnouncer(
public final CuratorServiceAnnouncer getServiceAnnouncer(
final CuratorServiceAnnouncer announcer,
final Injector injector,
final Set<KeyHolder<DruidNode>> nodesToAnnounce,
Expand Down Expand Up @@ -190,7 +190,7 @@ public void stop()

@Provides
@LazySingleton
public ServiceDiscovery<Void> getServiceDiscovery(
public final ServiceDiscovery<Void> getServiceDiscovery(
CuratorFramework curator,
CuratorDiscoveryConfig config,
Lifecycle lifecycle
Expand Down Expand Up @@ -233,7 +233,7 @@ public void stop()

@Provides
@LazySingleton
public ServerDiscoveryFactory getServerDiscoveryFactory(
public final ServerDiscoveryFactory getServerDiscoveryFactory(
ServiceDiscovery<Void> serviceDiscovery
)
{
Expand Down
4 changes: 2 additions & 2 deletions server/src/main/java/io/druid/guice/AWSModule.java
Original file line number Diff line number Diff line change
Expand Up @@ -38,14 +38,14 @@ public void configure(Binder binder)

@Provides
@LazySingleton
public AWSCredentialsProvider getAWSCredentialsProvider(final AWSCredentialsConfig config)
public final AWSCredentialsProvider getAWSCredentialsProvider(final AWSCredentialsConfig config)
{
return AWSCredentialsUtils.defaultAWSCredentialsProviderChain(config);
}

@Provides
@LazySingleton
public AmazonEC2 getEc2Client(AWSCredentialsProvider credentials)
public final AmazonEC2 getEc2Client(AWSCredentialsProvider credentials)
{
return new AmazonEC2Client(credentials);
}
Expand Down
2 changes: 1 addition & 1 deletion server/src/main/java/io/druid/guice/AnnouncerModule.java
Original file line number Diff line number Diff line change
Expand Up @@ -45,7 +45,7 @@ public void configure(Binder binder)

@Provides
@ManageLifecycle
public Announcer getAnnouncer(CuratorFramework curator)
public final Announcer getAnnouncer(CuratorFramework curator)
{
return new Announcer(curator, Execs.singleThreaded("Announcer-%s"));
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -60,7 +60,7 @@ public void configure(Binder binder)
@Provides
@BackgroundCaching
@LazySingleton
public ExecutorService getBackgroundExecutorService(
public final ExecutorService getBackgroundExecutorService(
CacheConfig cacheConfig
)
{
Expand All @@ -81,7 +81,7 @@ public ExecutorService getBackgroundExecutorService(
@Provides
@Processing
@ManageLifecycle
public ExecutorService getProcessingExecutorService(
public final ExecutorService getProcessingExecutorService(
ExecutorServiceConfig config,
ServiceEmitter emitter,
Lifecycle lifecycle
Expand All @@ -100,7 +100,7 @@ public ExecutorService getProcessingExecutorService(
@Provides
@LazySingleton
@Global
public StupidPool<ByteBuffer> getIntermediateResultsPool(DruidProcessingConfig config)
public final StupidPool<ByteBuffer> getIntermediateResultsPool(DruidProcessingConfig config)
{
try {
long maxDirectMemory = VMUtils.getMaxDirectMemory();
Expand Down
Loading