You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pinot.apache.org by xi...@apache.org on 2021/05/13 21:59:39 UTC

[incubator-pinot] branch master updated: Core Pinot Environment Provider Implementation Logic to fetch Failure… (#6842)

This is an automated email from the ASF dual-hosted git repository.

xiangfu pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-pinot.git


The following commit(s) were added to refs/heads/master by this push:
     new 8102e43  Core Pinot Environment Provider Implementation Logic to fetch Failure… (#6842)
8102e43 is described below

commit 8102e4361291860c7dff3d4558a969eed0b44d2d
Author: Jay Desai <ja...@gmail.com>
AuthorDate: Thu May 13 14:59:18 2021 -0700

    Core Pinot Environment Provider Implementation Logic to fetch Failure… (#6842)
    
    * Core Pinot Environment Provider Implementation Logic to fetch Failure Domain Information for the Server Instance
    
    * Add logic for fetching custom environment configs containing failure domain info and update instance configs in zookeeper
    
    * Add null check for overidden server configs
    
    * Remove unnecessary flag checks and restructure the code
    
    * Resolve integration test null pointer failure
    
    * Remove redundant instance config supply to PinotEnvironmentProvider and remove map keyset iterations to fetch provider info
    
    * Enhance log messages and make cosmetic changes to thrown exception type
    
    * Remove redundant map assignment and key removal
    
    * Remove generic getEnvironment method and replace with environment property specific interface method
    
    Co-authored-by: Jay Desai <ja...@jadesai-mn2.linkedin.biz>
---
 pinot-distribution/pinot-assembly.xml              |   6 +
 .../pinot-environment/pinot-azure/pom.xml          |  53 +++++++
 .../plugin/provider/AzureEnvironmentProvider.java  | 164 +++++++++++++++++++++
 .../provider/AzureEnvironmentProviderTest.java     | 147 ++++++++++++++++++
 .../mock-imds-response-without-computenode.json    | 118 +++++++++++++++
 .../mock-imds-response-without-faultDomain.json    | 118 +++++++++++++++
 .../src/test/resources/mock-imds-response.json     | 118 +++++++++++++++
 pinot-plugins/pinot-environment/pom.xml            |  55 +++++++
 pinot-plugins/pom.xml                              |   1 +
 .../server/starter/helix/HelixServerStarter.java   |  51 +++++++
 .../PinotEnvironmentProvider.java                  |  42 ++++++
 .../PinotEnvironmentProviderFactory.java           |  93 ++++++++++++
 .../apache/pinot/spi/utils/CommonConstants.java    |   8 +
 .../PinotEnvironmentProviderFactoryTest.java       |  68 +++++++++
 14 files changed, 1042 insertions(+)

diff --git a/pinot-distribution/pinot-assembly.xml b/pinot-distribution/pinot-assembly.xml
index e117b48..1bd2031 100644
--- a/pinot-distribution/pinot-assembly.xml
+++ b/pinot-distribution/pinot-assembly.xml
@@ -88,6 +88,12 @@
       <destName>plugins/pinot-file-system/pinot-s3/pinot-s3-${project.version}-shaded.jar</destName>
     </file>
     <!-- End Include Pinot File System Plugins-->
+    <!-- Start Include Pinot Environment Plugins-->
+    <file>
+      <source>${pinot.root}/pinot-plugins/pinot-environment/pinot-azure/target/pinot-azure-${project.version}-shaded.jar</source>
+      <destName>plugins/pinot-environment/pinot-azure/pinot-azure-${project.version}-shaded.jar</destName>
+    </file>
+    <!-- End Include Pinot Environment Plugins-->
     <!-- Start Include Pinot Input Format Plugins-->
     <file>
       <source>${pinot.root}/pinot-plugins/pinot-input-format/pinot-avro/target/pinot-avro-${project.version}-shaded.jar</source>
diff --git a/pinot-plugins/pinot-environment/pinot-azure/pom.xml b/pinot-plugins/pinot-environment/pinot-azure/pom.xml
new file mode 100644
index 0000000..d660c8b
--- /dev/null
+++ b/pinot-plugins/pinot-environment/pinot-azure/pom.xml
@@ -0,0 +1,53 @@
+<?xml version="1.0" encoding="UTF-8"?>
+<!--
+
+    Licensed to the Apache Software Foundation (ASF) under one
+    or more contributor license agreements.  See the NOTICE file
+    distributed with this work for additional information
+    regarding copyright ownership.  The ASF licenses this file
+    to you under the Apache License, Version 2.0 (the
+    "License"); you may not use this file except in compliance
+    with the License.  You may obtain a copy of the License at
+
+      http://www.apache.org/licenses/LICENSE-2.0
+
+    Unless required by applicable law or agreed to in writing,
+    software distributed under the License is distributed on an
+    "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+    KIND, either express or implied.  See the License for the
+    specific language governing permissions and limitations
+    under the License.
+
+-->
+<project xmlns="http://maven.apache.org/POM/4.0.0"
+         xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
+         xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/xsd/maven-4.0.0.xsd">
+    <modelVersion>4.0.0</modelVersion>
+    <parent>
+        <artifactId>pinot-environment</artifactId>
+        <groupId>org.apache.pinot</groupId>
+        <version>0.8.0-SNAPSHOT</version>
+        <relativePath>..</relativePath>
+    </parent>
+    <artifactId>pinot-azure</artifactId>
+    <name>Pinot Azure Environment</name>
+    <url>https://pinot.apache.org/</url>
+    <properties>
+        <pinot.root>${basedir}/../../..</pinot.root>
+        <phase.prop>package</phase.prop>
+    </properties>
+    <dependencies>
+        <dependency>
+            <groupId>org.apache.pinot</groupId>
+            <artifactId>pinot-spi</artifactId>
+            <version>${project.version}</version>
+            <scope>compile</scope>
+        </dependency>
+        <dependency>
+            <groupId>org.apache.pinot</groupId>
+            <artifactId>pinot-common</artifactId>
+            <version>${project.version}</version>
+            <scope>compile</scope>
+        </dependency>
+    </dependencies>
+</project>
diff --git a/pinot-plugins/pinot-environment/pinot-azure/src/main/java/org/apache/pinot/plugin/provider/AzureEnvironmentProvider.java b/pinot-plugins/pinot-environment/pinot-azure/src/main/java/org/apache/pinot/plugin/provider/AzureEnvironmentProvider.java
new file mode 100644
index 0000000..c710ae8
--- /dev/null
+++ b/pinot-plugins/pinot-environment/pinot-azure/src/main/java/org/apache/pinot/plugin/provider/AzureEnvironmentProvider.java
@@ -0,0 +1,164 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.pinot.plugin.provider;
+
+import com.fasterxml.jackson.databind.JsonNode;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import com.google.common.annotations.VisibleForTesting;
+import com.google.common.base.Preconditions;
+import java.io.IOException;
+import java.io.InterruptedIOException;
+import java.net.UnknownHostException;
+import javax.net.ssl.SSLException;
+import org.apache.commons.lang.StringUtils;
+import org.apache.http.HttpEntityEnclosingRequest;
+import org.apache.http.HttpStatus;
+import org.apache.http.StatusLine;
+import org.apache.http.client.HttpRequestRetryHandler;
+import org.apache.http.client.config.RequestConfig;
+import org.apache.http.client.methods.CloseableHttpResponse;
+import org.apache.http.client.methods.HttpGet;
+import org.apache.http.client.protocol.HttpClientContext;
+import org.apache.http.impl.client.CloseableHttpClient;
+import org.apache.http.impl.client.HttpClients;
+import org.apache.http.util.EntityUtils;
+import org.apache.pinot.spi.env.PinotConfiguration;
+import org.apache.pinot.spi.environmentprovider.PinotEnvironmentProvider;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+
+/**
+ * Azure Environment Provider used to retrieve azure cloud specific instance configuration.
+ */
+public class AzureEnvironmentProvider implements PinotEnvironmentProvider {
+
+  private static final Logger LOGGER = LoggerFactory.getLogger(AzureEnvironmentProvider.class);
+
+  protected static final String MAX_RETRY = "maxRetry";
+  protected static final String IMDS_ENDPOINT = "imdsEndpoint";
+  protected static final String CONNECTION_TIMEOUT = "connectionTimeout";
+  protected static final String REQUEST_TIMEOUT = "requestTimeout";
+  private static final String COMPUTE = "compute";
+  private static final String METADATA = "Metadata";
+  private static final String PLATFORM_FAULT_DOMAIN = "platformFaultDomain";
+  private int _maxRetry;
+  private String _imdsEndpoint;
+  private CloseableHttpClient _closeableHttpClient;
+
+  public AzureEnvironmentProvider() {
+  }
+
+  public void init(PinotConfiguration pinotConfiguration) {
+    Preconditions.checkArgument(0 < Integer.parseInt(pinotConfiguration.getProperty(MAX_RETRY)),
+         "[AzureEnvironmentProvider]: " + MAX_RETRY + " cannot be less than or equal to 0");
+    Preconditions.checkArgument(!StringUtils.isBlank(pinotConfiguration.getProperty(IMDS_ENDPOINT)),
+        "[AzureEnvironmentProvider]: " + IMDS_ENDPOINT + " should not be null or empty");
+
+    _maxRetry = Integer.parseInt(pinotConfiguration.getProperty(MAX_RETRY));
+    _imdsEndpoint = pinotConfiguration.getProperty(IMDS_ENDPOINT);
+    int connectionTimeout = Integer.parseInt(pinotConfiguration.getProperty(CONNECTION_TIMEOUT));
+    int requestTimeout = Integer.parseInt(pinotConfiguration.getProperty(REQUEST_TIMEOUT));
+
+    final RequestConfig requestConfig = RequestConfig.custom()
+        .setConnectTimeout(connectionTimeout)
+        .setConnectionRequestTimeout(requestTimeout)
+        .build();
+
+    final HttpRequestRetryHandler httpRequestRetryHandler = (iOException, executionCount, httpContext) ->
+        !(executionCount >= _maxRetry
+            || iOException instanceof InterruptedIOException
+            || iOException instanceof UnknownHostException
+            || iOException instanceof SSLException
+            || HttpClientContext.adapt(httpContext).getRequest() instanceof HttpEntityEnclosingRequest);
+
+    _closeableHttpClient =
+        HttpClients.custom().setDefaultRequestConfig(requestConfig).setRetryHandler(httpRequestRetryHandler).build();
+  }
+
+  // Constructor for test purposes.
+  @VisibleForTesting
+  public AzureEnvironmentProvider(int maxRetry, String imdsEndpoint, CloseableHttpClient closeableHttpClient) {
+    _maxRetry = maxRetry;
+    _imdsEndpoint = imdsEndpoint;
+    _closeableHttpClient = Preconditions.checkNotNull(closeableHttpClient,
+        "[AzureEnvironmentProvider]: Closeable Http Client cannot be null");
+  }
+
+  /**
+   *
+   * Utility used to query the azure instance metadata service (Azure IMDS) to fetch the failure domain information,
+   * used at HelixServerStarter startup to update the instance configs.
+   * @return failure domain information
+   */
+  @VisibleForTesting
+  @Override
+  public String getFailureDomain() {
+    final String responsePayload = getAzureInstanceMetadata();
+
+    // For a sample response payload,
+    // check https://docs.microsoft.com/en-us/azure/virtual-machines/windows/instance-metadata-service?tabs=linux
+    final ObjectMapper objectMapper = new ObjectMapper();
+    try {
+      final JsonNode jsonNode = objectMapper.readTree(responsePayload);
+      final JsonNode computeNode = jsonNode.path(COMPUTE);
+
+      if (computeNode.isMissingNode()) {
+        throw new RuntimeException(
+            "[AzureEnvironmentProvider]: Compute node is missing in the payload. Cannot retrieve failure domain information");
+      }
+      final JsonNode platformFailureDomainNode = computeNode.path(PLATFORM_FAULT_DOMAIN);
+      if (platformFailureDomainNode.isMissingNode() || !platformFailureDomainNode.isTextual()) {
+        throw new RuntimeException("[AzureEnvironmentProvider]: Json node platformFaultDomain is missing or is invalid."
+            + " No failure domain information retrieved for given server instance");
+      }
+      return platformFailureDomainNode.textValue();
+    } catch (IOException ex) {
+      throw new RuntimeException(
+          String.format("[AzureEnvironmentProvider]: Errors when parsing response payload from Azure Instance Metadata Service: %s",
+              responsePayload), ex);
+    }
+  }
+
+  // Utility used to construct the HTTP Request and fetch corresponding response entity.
+  @VisibleForTesting
+  private String getAzureInstanceMetadata() {
+    HttpGet httpGet = new HttpGet(_imdsEndpoint);
+    httpGet.setHeader(METADATA, Boolean.TRUE.toString());
+
+    try {
+      final CloseableHttpResponse closeableHttpResponse = _closeableHttpClient.execute(httpGet);
+      if (closeableHttpResponse == null) {
+        throw new RuntimeException("[AzureEnvironmentProvider]: Response is null. Please verify the imds endpoint");
+      }
+      final StatusLine statusLine = closeableHttpResponse.getStatusLine();
+      final int statusCode = statusLine.getStatusCode();
+      if (statusCode != HttpStatus.SC_OK) {
+        final String errorMsg = String.format(
+            "[AzureEnvironmentProvider]: Failed to retrieve azure instance metadata. Response Status code: %s", statusCode);
+        throw new RuntimeException(errorMsg);
+      }
+      return EntityUtils.toString(closeableHttpResponse.getEntity());
+    } catch (IOException ex) {
+      throw new RuntimeException(
+          String.format("[AzureEnvironmentProvider]: Failed to retrieve metadata from Azure Instance Metadata Service %s",
+              _imdsEndpoint), ex);
+    }
+  }
+}
diff --git a/pinot-plugins/pinot-environment/pinot-azure/src/test/java/org/apache/pinot/plugin/provider/AzureEnvironmentProviderTest.java b/pinot-plugins/pinot-environment/pinot-azure/src/test/java/org/apache/pinot/plugin/provider/AzureEnvironmentProviderTest.java
new file mode 100644
index 0000000..b0113e3
--- /dev/null
+++ b/pinot-plugins/pinot-environment/pinot-azure/src/test/java/org/apache/pinot/plugin/provider/AzureEnvironmentProviderTest.java
@@ -0,0 +1,147 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.pinot.plugin.provider;
+
+import java.io.IOException;
+import java.util.HashMap;
+import java.util.Map;
+import org.apache.http.HttpEntity;
+import org.apache.http.HttpStatus;
+import org.apache.http.StatusLine;
+import org.apache.http.client.methods.CloseableHttpResponse;
+import org.apache.http.client.methods.HttpGet;
+import org.apache.http.impl.client.CloseableHttpClient;
+import org.apache.pinot.spi.env.PinotConfiguration;
+import org.mockito.Mock;
+import org.testng.Assert;
+import org.testng.annotations.BeforeMethod;
+import org.testng.annotations.Test;
+
+import static org.apache.http.HttpStatus.*;
+import static org.apache.pinot.plugin.provider.AzureEnvironmentProvider.*;
+import static org.mockito.Mockito.*;
+import static org.mockito.MockitoAnnotations.*;
+
+/**
+ * Unit test for {@link AzureEnvironmentProviderTest}
+ */
+public class AzureEnvironmentProviderTest {
+  private final static String IMDS_RESPONSE_FILE = "mock-imds-response.json";
+  private final static String IMDS_RESPONSE_WITHOUT_COMPUTE_INFO = "mock-imds-response-without-computenode.json";
+  private final static String IMDS_RESPONSE_WITHOUT_FAULT_DOMAIN_INFO = "mock-imds-response-without-faultDomain.json";
+  private final static String IMDS_ENDPOINT_VALUE = "http://169.254.169.254/metadata/instance?api-version=2020-09-01";
+
+  @Mock
+  private CloseableHttpClient _mockHttpClient;
+  @Mock
+  private CloseableHttpResponse _mockHttpResponse;
+  @Mock
+  private StatusLine _mockStatusLine;
+  @Mock
+  private HttpEntity _mockHttpEntity;
+
+  private AzureEnvironmentProvider _azureEnvironmentProvider;
+
+  private AzureEnvironmentProvider _azureEnvironmentProviderWithParams;
+
+  PinotConfiguration _pinotConfiguration;
+
+  @BeforeMethod
+  public void init() {
+    initMocks(this);
+    _pinotConfiguration = new PinotConfiguration(new HashMap<>());
+    _azureEnvironmentProvider = new AzureEnvironmentProvider();
+    _azureEnvironmentProviderWithParams = new AzureEnvironmentProvider(3, IMDS_ENDPOINT_VALUE, _mockHttpClient);
+  }
+
+  @Test
+  public void testFailureDomainRetrieval() throws IOException {
+    mockUtil();
+    when(_mockHttpEntity.getContent()).thenReturn(getClass().getClassLoader().getResourceAsStream(IMDS_RESPONSE_FILE));
+    String failureDomain = _azureEnvironmentProviderWithParams.getFailureDomain();
+    Assert.assertEquals(failureDomain, "36");
+    verify(_mockHttpClient, times(1)).execute(any(HttpGet.class));
+    verify(_mockHttpResponse, times(1)).getStatusLine();
+    verify(_mockStatusLine, times(1)).getStatusCode();
+    verify(_mockHttpResponse, times(1)).getEntity();
+    verifyNoMoreInteractions(_mockHttpClient, _mockHttpResponse, _mockStatusLine);
+  }
+
+  @Test(expectedExceptions = IllegalArgumentException.class,
+      expectedExceptionsMessageRegExp = "\\[AzureEnvironmentProvider\\]: imdsEndpoint should not be null or empty")
+  public void testInvalidIMDSEndpoint() {
+    Map<String, Object> map = _pinotConfiguration.toMap();
+    map.put(MAX_RETRY, "3");
+    map.put(IMDS_ENDPOINT, "");
+    PinotConfiguration pinotConfiguration = new PinotConfiguration(map);
+    _azureEnvironmentProvider.init(pinotConfiguration);
+  }
+
+  @Test(expectedExceptions = IllegalArgumentException.class,
+        expectedExceptionsMessageRegExp = "\\[AzureEnvironmentProvider\\]: maxRetry cannot be less than or equal to 0")
+  public void testInvalidRetryCount() {
+    Map<String, Object> map = _pinotConfiguration.toMap();
+    map.put(MAX_RETRY, "0");
+    PinotConfiguration pinotConfiguration = new PinotConfiguration(map);
+    _azureEnvironmentProvider.init(pinotConfiguration);
+  }
+
+  @Test(expectedExceptions = NullPointerException.class,
+      expectedExceptionsMessageRegExp = "\\[AzureEnvironmentProvider\\]: Closeable Http Client cannot be null")
+  public void testInvalidHttpClient() {
+    new AzureEnvironmentProvider(3, IMDS_ENDPOINT_VALUE, null);
+  }
+
+  @Test(expectedExceptions = RuntimeException.class,
+      expectedExceptionsMessageRegExp = "\\[AzureEnvironmentProvider\\]: Compute node is missing in the payload. "
+          + "Cannot retrieve failure domain information")
+  public void testMissingComputeNodeResponse() throws IOException {
+    mockUtil();
+    when(_mockHttpEntity.getContent())
+        .thenReturn(getClass().getClassLoader().getResourceAsStream(IMDS_RESPONSE_WITHOUT_COMPUTE_INFO));
+    _azureEnvironmentProviderWithParams.getFailureDomain();
+  }
+
+  @Test(expectedExceptions = RuntimeException.class,
+      expectedExceptionsMessageRegExp = "\\[AzureEnvironmentProvider\\]: Json node platformFaultDomain is missing or is invalid."
+          + " No failure domain information retrieved for given server instance")
+  public void testMissingFaultDomainResponse() throws IOException {
+    mockUtil();
+    when(_mockHttpEntity.getContent())
+        .thenReturn(getClass().getClassLoader().getResourceAsStream(IMDS_RESPONSE_WITHOUT_FAULT_DOMAIN_INFO));
+    _azureEnvironmentProviderWithParams.getFailureDomain();
+  }
+
+  @Test(expectedExceptions = RuntimeException.class,
+      expectedExceptionsMessageRegExp = "\\[AzureEnvironmentProvider\\]: Failed to retrieve azure instance metadata."
+          + " Response Status code: " + SC_NOT_FOUND)
+  public void testIMDSCallFailure() throws IOException {
+    mockUtil();
+    when(_mockStatusLine.getStatusCode()).thenReturn(SC_NOT_FOUND);
+    _azureEnvironmentProviderWithParams.getFailureDomain();
+  }
+
+  // Mock Response utility method
+  private void mockUtil() throws IOException {
+    when(_mockHttpClient.execute(any(HttpGet.class))).thenReturn(_mockHttpResponse);
+    when(_mockHttpResponse.getStatusLine()).thenReturn(_mockStatusLine);
+    when(_mockStatusLine.getStatusCode()).thenReturn(HttpStatus.SC_OK);
+    when(_mockHttpResponse.getEntity()).thenReturn(_mockHttpEntity);
+  }
+}
diff --git a/pinot-plugins/pinot-environment/pinot-azure/src/test/resources/mock-imds-response-without-computenode.json b/pinot-plugins/pinot-environment/pinot-azure/src/test/resources/mock-imds-response-without-computenode.json
new file mode 100644
index 0000000..f916b74
--- /dev/null
+++ b/pinot-plugins/pinot-environment/pinot-azure/src/test/resources/mock-imds-response-without-computenode.json
@@ -0,0 +1,118 @@
+{
+  "computer": {
+    "azEnvironment": "AZUREPUBLICCLOUD",
+    "isHostCompatibilityLayerVm": "true",
+    "licenseType":  "Windows_Client",
+    "location": "westus",
+    "name": "examplevmname",
+    "offer": "Windows",
+    "osProfile": {
+      "adminUsername": "admin",
+      "computerName": "examplevmname",
+      "disablePasswordAuthentication": "true"
+    },
+    "osType": "linux",
+    "placementGroupId": "f67c14ab-e92c-408c-ae2d-da15866ec79a",
+    "plan": {
+      "name": "planName",
+      "product": "planProduct",
+      "publisher": "planPublisher"
+    },
+    "platformFaultDomain": "36",
+    "platformUpdateDomain": "42",
+    "publicKeys": [{
+      "keyData": "ssh-rsa 0",
+      "path": "/home/user/.ssh/authorized_keys0"
+    },
+      {
+        "keyData": "ssh-rsa 1",
+        "path": "/home/user/.ssh/authorized_keys1"
+      }
+    ],
+    "publisher": "RDFE-Test-Microsoft-Windows-Server-Group",
+    "resourceGroupName": "macikgo-test-may-23",
+    "resourceId": "/subscriptions/xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx/resourceGroups/macikgo-test-may-23/providers/Microsoft.Compute/virtualMachines/examplevmname",
+    "securityProfile": {
+      "secureBootEnabled": "true",
+      "virtualTpmEnabled": "false"
+    },
+    "sku": "Windows-Server-2012-R2-Datacenter",
+    "storageProfile": {
+      "dataDisks": [{
+        "caching": "None",
+        "createOption": "Empty",
+        "diskSizeGB": "1024",
+        "image": {
+          "uri": ""
+        },
+        "lun": "0",
+        "managedDisk": {
+          "id": "/subscriptions/xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx/resourceGroups/macikgo-test-may-23/providers/Microsoft.Compute/disks/exampledatadiskname",
+          "storageAccountType": "Standard_LRS"
+        },
+        "name": "exampledatadiskname",
+        "vhd": {
+          "uri": ""
+        },
+        "writeAcceleratorEnabled": "false"
+      }],
+      "imageReference": {
+        "id": "",
+        "offer": "UbuntuServer",
+        "publisher": "Canonical",
+        "sku": "16.04.0-LTS",
+        "version": "latest"
+      },
+      "osDisk": {
+        "caching": "ReadWrite",
+        "createOption": "FromImage",
+        "diskSizeGB": "30",
+        "diffDiskSettings": {
+          "option": "Local"
+        },
+        "encryptionSettings": {
+          "enabled": "false"
+        },
+        "image": {
+          "uri": ""
+        },
+        "managedDisk": {
+          "id": "/subscriptions/xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx/resourceGroups/macikgo-test-may-23/providers/Microsoft.Compute/disks/exampleosdiskname",
+          "storageAccountType": "Standard_LRS"
+        },
+        "name": "exampleosdiskname",
+        "osType": "Linux",
+        "vhd": {
+          "uri": ""
+        },
+        "writeAcceleratorEnabled": "false"
+      }
+    },
+    "subscriptionId": "xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx",
+    "tags": "baz:bash;foo:bar",
+    "version": "15.05.22",
+    "vmId": "02aab8a4-74ef-476e-8182-f6d2ba4166a6",
+    "vmScaleSetName": "crpteste9vflji9",
+    "vmSize": "Standard_A3",
+    "zone": ""
+  },
+  "network": {
+    "interface": [{
+      "ipv4": {
+        "ipAddress": [{
+          "privateIpAddress": "10.144.133.132",
+          "publicIpAddress": ""
+        }],
+        "subnet": [{
+          "address": "10.144.133.128",
+          "prefix": "26"
+        }]
+      },
+      "ipv6": {
+        "ipAddress": [
+        ]
+      },
+      "macAddress": "0011AAFFBB22"
+    }]
+  }
+}
diff --git a/pinot-plugins/pinot-environment/pinot-azure/src/test/resources/mock-imds-response-without-faultDomain.json b/pinot-plugins/pinot-environment/pinot-azure/src/test/resources/mock-imds-response-without-faultDomain.json
new file mode 100644
index 0000000..e82b270
--- /dev/null
+++ b/pinot-plugins/pinot-environment/pinot-azure/src/test/resources/mock-imds-response-without-faultDomain.json
@@ -0,0 +1,118 @@
+{
+  "compute": {
+    "azEnvironment": "AZUREPUBLICCLOUD",
+    "isHostCompatibilityLayerVm": "true",
+    "licenseType":  "Windows_Client",
+    "location": "westus",
+    "name": "examplevmname",
+    "offer": "Windows",
+    "osProfile": {
+      "adminUsername": "admin",
+      "computerName": "examplevmname",
+      "disablePasswordAuthentication": "true"
+    },
+    "osType": "linux",
+    "placementGroupId": "f67c14ab-e92c-408c-ae2d-da15866ec79a",
+    "plan": {
+      "name": "planName",
+      "product": "planProduct",
+      "publisher": "planPublisher"
+    },
+    "platformFailureDomain": "36",
+    "platformUpdateDomain": "42",
+    "publicKeys": [{
+      "keyData": "ssh-rsa 0",
+      "path": "/home/user/.ssh/authorized_keys0"
+    },
+      {
+        "keyData": "ssh-rsa 1",
+        "path": "/home/user/.ssh/authorized_keys1"
+      }
+    ],
+    "publisher": "RDFE-Test-Microsoft-Windows-Server-Group",
+    "resourceGroupName": "macikgo-test-may-23",
+    "resourceId": "/subscriptions/xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx/resourceGroups/macikgo-test-may-23/providers/Microsoft.Compute/virtualMachines/examplevmname",
+    "securityProfile": {
+      "secureBootEnabled": "true",
+      "virtualTpmEnabled": "false"
+    },
+    "sku": "Windows-Server-2012-R2-Datacenter",
+    "storageProfile": {
+      "dataDisks": [{
+        "caching": "None",
+        "createOption": "Empty",
+        "diskSizeGB": "1024",
+        "image": {
+          "uri": ""
+        },
+        "lun": "0",
+        "managedDisk": {
+          "id": "/subscriptions/xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx/resourceGroups/macikgo-test-may-23/providers/Microsoft.Compute/disks/exampledatadiskname",
+          "storageAccountType": "Standard_LRS"
+        },
+        "name": "exampledatadiskname",
+        "vhd": {
+          "uri": ""
+        },
+        "writeAcceleratorEnabled": "false"
+      }],
+      "imageReference": {
+        "id": "",
+        "offer": "UbuntuServer",
+        "publisher": "Canonical",
+        "sku": "16.04.0-LTS",
+        "version": "latest"
+      },
+      "osDisk": {
+        "caching": "ReadWrite",
+        "createOption": "FromImage",
+        "diskSizeGB": "30",
+        "diffDiskSettings": {
+          "option": "Local"
+        },
+        "encryptionSettings": {
+          "enabled": "false"
+        },
+        "image": {
+          "uri": ""
+        },
+        "managedDisk": {
+          "id": "/subscriptions/xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx/resourceGroups/macikgo-test-may-23/providers/Microsoft.Compute/disks/exampleosdiskname",
+          "storageAccountType": "Standard_LRS"
+        },
+        "name": "exampleosdiskname",
+        "osType": "Linux",
+        "vhd": {
+          "uri": ""
+        },
+        "writeAcceleratorEnabled": "false"
+      }
+    },
+    "subscriptionId": "xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx",
+    "tags": "baz:bash;foo:bar",
+    "version": "15.05.22",
+    "vmId": "02aab8a4-74ef-476e-8182-f6d2ba4166a6",
+    "vmScaleSetName": "crpteste9vflji9",
+    "vmSize": "Standard_A3",
+    "zone": ""
+  },
+  "network": {
+    "interface": [{
+      "ipv4": {
+        "ipAddress": [{
+          "privateIpAddress": "10.144.133.132",
+          "publicIpAddress": ""
+        }],
+        "subnet": [{
+          "address": "10.144.133.128",
+          "prefix": "26"
+        }]
+      },
+      "ipv6": {
+        "ipAddress": [
+        ]
+      },
+      "macAddress": "0011AAFFBB22"
+    }]
+  }
+}
diff --git a/pinot-plugins/pinot-environment/pinot-azure/src/test/resources/mock-imds-response.json b/pinot-plugins/pinot-environment/pinot-azure/src/test/resources/mock-imds-response.json
new file mode 100644
index 0000000..b5a932e
--- /dev/null
+++ b/pinot-plugins/pinot-environment/pinot-azure/src/test/resources/mock-imds-response.json
@@ -0,0 +1,118 @@
+{
+  "compute": {
+    "azEnvironment": "AZUREPUBLICCLOUD",
+    "isHostCompatibilityLayerVm": "true",
+    "licenseType":  "Windows_Client",
+    "location": "westus",
+    "name": "examplevmname",
+    "offer": "Windows",
+    "osProfile": {
+      "adminUsername": "admin",
+      "computerName": "examplevmname",
+      "disablePasswordAuthentication": "true"
+    },
+    "osType": "linux",
+    "placementGroupId": "f67c14ab-e92c-408c-ae2d-da15866ec79a",
+    "plan": {
+      "name": "planName",
+      "product": "planProduct",
+      "publisher": "planPublisher"
+    },
+    "platformFaultDomain": "36",
+    "platformUpdateDomain": "42",
+    "publicKeys": [{
+      "keyData": "ssh-rsa 0",
+      "path": "/home/user/.ssh/authorized_keys0"
+    },
+      {
+        "keyData": "ssh-rsa 1",
+        "path": "/home/user/.ssh/authorized_keys1"
+      }
+    ],
+    "publisher": "RDFE-Test-Microsoft-Windows-Server-Group",
+    "resourceGroupName": "macikgo-test-may-23",
+    "resourceId": "/subscriptions/xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx/resourceGroups/macikgo-test-may-23/providers/Microsoft.Compute/virtualMachines/examplevmname",
+    "securityProfile": {
+      "secureBootEnabled": "true",
+      "virtualTpmEnabled": "false"
+    },
+    "sku": "Windows-Server-2012-R2-Datacenter",
+    "storageProfile": {
+      "dataDisks": [{
+        "caching": "None",
+        "createOption": "Empty",
+        "diskSizeGB": "1024",
+        "image": {
+          "uri": ""
+        },
+        "lun": "0",
+        "managedDisk": {
+          "id": "/subscriptions/xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx/resourceGroups/macikgo-test-may-23/providers/Microsoft.Compute/disks/exampledatadiskname",
+          "storageAccountType": "Standard_LRS"
+        },
+        "name": "exampledatadiskname",
+        "vhd": {
+          "uri": ""
+        },
+        "writeAcceleratorEnabled": "false"
+      }],
+      "imageReference": {
+        "id": "",
+        "offer": "UbuntuServer",
+        "publisher": "Canonical",
+        "sku": "16.04.0-LTS",
+        "version": "latest"
+      },
+      "osDisk": {
+        "caching": "ReadWrite",
+        "createOption": "FromImage",
+        "diskSizeGB": "30",
+        "diffDiskSettings": {
+          "option": "Local"
+        },
+        "encryptionSettings": {
+          "enabled": "false"
+        },
+        "image": {
+          "uri": ""
+        },
+        "managedDisk": {
+          "id": "/subscriptions/xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx/resourceGroups/macikgo-test-may-23/providers/Microsoft.Compute/disks/exampleosdiskname",
+          "storageAccountType": "Standard_LRS"
+        },
+        "name": "exampleosdiskname",
+        "osType": "Linux",
+        "vhd": {
+          "uri": ""
+        },
+        "writeAcceleratorEnabled": "false"
+      }
+    },
+    "subscriptionId": "xxxxx-xxxx-xxxx-xxxx-xxxxxxxxxxx",
+    "tags": "baz:bash;foo:bar",
+    "version": "15.05.22",
+    "vmId": "02aab8a4-74ef-476e-8182-f6d2ba4166a6",
+    "vmScaleSetName": "crpteste9vflji9",
+    "vmSize": "Standard_A3",
+    "zone": ""
+  },
+  "network": {
+    "interface": [{
+      "ipv4": {
+        "ipAddress": [{
+          "privateIpAddress": "10.144.133.132",
+          "publicIpAddress": ""
+        }],
+        "subnet": [{
+          "address": "10.144.133.128",
+          "prefix": "26"
+        }]
+      },
+      "ipv6": {
+        "ipAddress": [
+        ]
+      },
+      "macAddress": "0011AAFFBB22"
+    }]
+  }
+}
diff --git a/pinot-plugins/pinot-environment/pom.xml b/pinot-plugins/pinot-environment/pom.xml
new file mode 100644
index 0000000..fd0c55a
--- /dev/null
+++ b/pinot-plugins/pinot-environment/pom.xml
@@ -0,0 +1,55 @@
+<?xml version="1.0"?>
+<!--
+
+    Licensed to the Apache Software Foundation (ASF) under one
+    or more contributor license agreements.  See the NOTICE file
+    distributed with this work for additional information
+    regarding copyright ownership.  The ASF licenses this file
+    to you under the Apache License, Version 2.0 (the
+    "License"); you may not use this file except in compliance
+    with the License.  You may obtain a copy of the License at
+
+      http://www.apache.org/licenses/LICENSE-2.0
+
+    Unless required by applicable law or agreed to in writing,
+    software distributed under the License is distributed on an
+    "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+    KIND, either express or implied.  See the License for the
+    specific language governing permissions and limitations
+    under the License.
+
+-->
+<project xmlns="http://maven.apache.org/POM/4.0.0"
+         xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
+         xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/xsd/maven-4.0.0.xsd">
+  <modelVersion>4.0.0</modelVersion>
+  <parent>
+    <artifactId>pinot-plugins</artifactId>
+    <groupId>org.apache.pinot</groupId>
+    <version>0.8.0-SNAPSHOT</version>
+    <relativePath>..</relativePath>
+  </parent>
+
+  <artifactId>pinot-environment</artifactId>
+  <packaging>pom</packaging>
+  <name>Pluggable Pinot Environment Provider </name>
+  <url>https://pinot.apache.org/</url>
+  <properties>
+    <pinot.root>${basedir}/../..</pinot.root>
+    <plugin.type>pinot-environment</plugin.type>
+  </properties>
+
+  <modules>
+    <module>pinot-azure</module>
+  </modules>
+
+  <dependencies>
+    <!-- test -->
+    <dependency>
+      <groupId>org.mockito</groupId>
+      <artifactId>mockito-core</artifactId>
+      <scope>test</scope>
+    </dependency>
+  </dependencies>
+
+</project>
diff --git a/pinot-plugins/pom.xml b/pinot-plugins/pom.xml
index 4dec684..a38b3f0 100644
--- a/pinot-plugins/pom.xml
+++ b/pinot-plugins/pom.xml
@@ -48,6 +48,7 @@
     <module>pinot-metrics</module>
     <module>pinot-segment-writer</module>
     <module>pinot-segment-uploader</module>
+    <module>pinot-environment</module>
   </modules>
 
   <dependencies>
diff --git a/pinot-server/src/main/java/org/apache/pinot/server/starter/helix/HelixServerStarter.java b/pinot-server/src/main/java/org/apache/pinot/server/starter/helix/HelixServerStarter.java
index 920b055..ecb01fd 100644
--- a/pinot-server/src/main/java/org/apache/pinot/server/starter/helix/HelixServerStarter.java
+++ b/pinot-server/src/main/java/org/apache/pinot/server/starter/helix/HelixServerStarter.java
@@ -30,6 +30,7 @@ import java.util.List;
 import java.util.Map;
 import java.util.Optional;
 import java.util.Set;
+import javax.annotation.Nullable;
 import org.apache.commons.lang3.StringUtils;
 import org.apache.helix.HelixAdmin;
 import org.apache.helix.HelixDataAccessor;
@@ -69,6 +70,8 @@ import org.apache.pinot.server.realtime.ServerSegmentCompletionProtocolHandler;
 import org.apache.pinot.server.starter.ServerInstance;
 import org.apache.pinot.server.starter.ServerQueriesDisabledTracker;
 import org.apache.pinot.spi.env.PinotConfiguration;
+import org.apache.pinot.spi.environmentprovider.PinotEnvironmentProvider;
+import org.apache.pinot.spi.environmentprovider.PinotEnvironmentProviderFactory;
 import org.apache.pinot.spi.filesystem.PinotFSFactory;
 import org.apache.pinot.spi.plugin.PluginManager;
 import org.apache.pinot.spi.services.ServiceRole;
@@ -121,6 +124,7 @@ public class HelixServerStarter implements ServiceStartable {
   private AdminApiApplication _adminApiApplication;
   private ServerQueriesDisabledTracker _serverQueriesDisabledTracker;
   private RealtimeLuceneIndexRefreshState _realtimeLuceneIndexRefreshState;
+  private PinotEnvironmentProvider _pinotEnvironmentProvider;
 
   public HelixServerStarter(String helixClusterName, String zkAddress, PinotConfiguration serverConf)
       throws Exception {
@@ -146,6 +150,9 @@ public class HelixServerStarter implements ServiceStartable {
         new HelixConfigScopeBuilder(ConfigScopeProperty.PARTICIPANT, _helixClusterName).forParticipant(_instanceId)
             .build();
 
+    // Initialize Pinot Environment Provider
+    _pinotEnvironmentProvider = initializePinotEnvironmentProvider();
+
     // Enable/disable thread CPU time measurement through instance config.
     ThreadTimer.setThreadCpuTimeMeasurementEnabled(_serverConf
         .getProperty(Server.CONFIG_OF_ENABLE_THREAD_CPU_TIME_MEASUREMENT,
@@ -158,6 +165,32 @@ public class HelixServerStarter implements ServiceStartable {
   }
 
   /**
+   *  Invoke pinot environment provider factory's init method to register the environment provider &
+   *  return the instantiated environment provider.
+   */
+  @Nullable
+  private PinotEnvironmentProvider initializePinotEnvironmentProvider() {
+    PinotConfiguration environmentProviderConfigs = _serverConf.subset(Server.PREFIX_OF_CONFIG_OF_ENVIRONMENT_PROVIDER_FACTORY);
+    if (environmentProviderConfigs.toMap().isEmpty()) {
+      LOGGER.info("No environment provider config values provided for server property: {}",
+          Server.PREFIX_OF_CONFIG_OF_ENVIRONMENT_PROVIDER_FACTORY);
+      return null;
+    }
+
+    // Invoke pinot environment provider factory's init method
+    PinotEnvironmentProviderFactory.init(environmentProviderConfigs);
+
+    String environmentProviderClassName = _serverConf.getProperty(Server.ENVIRONMENT_PROVIDER_CLASS_NAME);
+    if (environmentProviderClassName == null) {
+      LOGGER.info("No className value provided for property: {}", Server.ENVIRONMENT_PROVIDER_CLASS_NAME);
+      return null;
+    }
+
+    // Fetch environment provider instance
+    return PinotEnvironmentProviderFactory.getEnvironmentProvider(environmentProviderClassName.toLowerCase());
+  }
+
+  /**
    * Fetches the resources to monitor and registers the {@link org.apache.pinot.common.utils.ServiceStatus.ServiceStatusCallback}s
    */
   private void registerServiceStatusHandler() {
@@ -247,6 +280,24 @@ public class HelixServerStarter implements ServiceStartable {
       needToUpdateInstanceConfig = true;
     }
 
+    // Update instance config with environment properties
+    if (_pinotEnvironmentProvider != null) {
+      // Retrieve failure domain information and add to the environment properties map
+      String failureDomain = _pinotEnvironmentProvider.getFailureDomain();
+      Map<String, String> environmentProperties = new HashMap<>();
+      environmentProperties.put(CommonConstants.INSTANCE_FAILURE_DOMAIN, failureDomain);
+
+      // Fetch existing environment properties map from instance configs
+      Map<String, String> existingEnvironmentConfigsMap = instanceConfig.getRecord().getMapField(
+          CommonConstants.ENVIRONMENT_IDENTIFIER);
+
+      if (existingEnvironmentConfigsMap != null && !existingEnvironmentConfigsMap.equals(environmentProperties)) {
+        instanceConfig.getRecord().setMapField(CommonConstants.ENVIRONMENT_IDENTIFIER, environmentProperties);
+        LOGGER.info("Adding environment properties: {} for instance: {}", environmentProperties, _instanceId);
+        needToUpdateInstanceConfig = true;
+      }
+    }
+
     if (needToUpdateInstanceConfig) {
       LOGGER.info("Updating instance config for instance: {} with instance tags: {}, host: {}, port: {}", _instanceId,
           instanceTags, host, port);
diff --git a/pinot-spi/src/main/java/org/apache/pinot/spi/environmentprovider/PinotEnvironmentProvider.java b/pinot-spi/src/main/java/org/apache/pinot/spi/environmentprovider/PinotEnvironmentProvider.java
new file mode 100644
index 0000000..72e2d3a
--- /dev/null
+++ b/pinot-spi/src/main/java/org/apache/pinot/spi/environmentprovider/PinotEnvironmentProvider.java
@@ -0,0 +1,42 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.pinot.spi.environmentprovider;
+
+import org.apache.pinot.spi.env.PinotConfiguration;
+import org.apache.pinot.spi.utils.CommonConstants;
+
+/**
+ *  Environment Provider interface implemented by different cloud providers to customize
+ *  the base pinot configuration to add environment variables & instance specific configuration
+ */
+public interface PinotEnvironmentProvider {
+
+  /**
+   * Initializes the configurations specific to an environment provider.
+   */
+   void init(PinotConfiguration pinotConfiguration);
+
+  /**
+   * Method to retrieve failure domain information for a pinot instance.
+   * @return failure domain information
+   */
+   default String getFailureDomain() {
+     return CommonConstants.DEFAULT_FAILURE_DOMAIN;
+   }
+}
diff --git a/pinot-spi/src/main/java/org/apache/pinot/spi/environmentprovider/PinotEnvironmentProviderFactory.java b/pinot-spi/src/main/java/org/apache/pinot/spi/environmentprovider/PinotEnvironmentProviderFactory.java
new file mode 100644
index 0000000..47b8c37
--- /dev/null
+++ b/pinot-spi/src/main/java/org/apache/pinot/spi/environmentprovider/PinotEnvironmentProviderFactory.java
@@ -0,0 +1,93 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.pinot.spi.environmentprovider;
+
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Iterables;
+import java.util.List;
+import org.apache.pinot.spi.env.PinotConfiguration;
+import org.apache.pinot.spi.plugin.PluginManager;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * This factory class initializes the PinotEnvironmentProvider class.
+ * It creates a PinotEnvironment object based on the URI found.
+ */
+public class PinotEnvironmentProviderFactory {
+  private final static PinotEnvironmentProviderFactory INSTANCE = new PinotEnvironmentProviderFactory();
+
+  private static final Logger LOGGER = LoggerFactory.getLogger(PinotEnvironmentProviderFactory.class);
+
+  private static final String CLASS = "class";
+  private PinotEnvironmentProvider pinotEnvironmentProvider;
+
+  private PinotEnvironmentProviderFactory() {
+  }
+
+  public static PinotEnvironmentProviderFactory getInstance( ) {
+    return INSTANCE;
+  }
+
+  public static void init(PinotConfiguration environmentProviderFactoryConfig) {
+    getInstance().initInternal(environmentProviderFactoryConfig);
+  }
+
+  public static PinotEnvironmentProvider getEnvironmentProvider(String environment) {
+   return getInstance().getEnvironmentProviderInternal(environment);
+  }
+
+  private void initInternal(PinotConfiguration environmentProviderFactoryConfig) {
+    // Get environment and it's respective class
+    PinotConfiguration environmentConfiguration = environmentProviderFactoryConfig.subset(CLASS);
+    List<String> environments = environmentConfiguration.getKeys();
+
+    if (environments.isEmpty()) {
+      LOGGER.info("Did not find any environment provider classes in the configuration");
+      return;
+    }
+
+    String environment = Iterables.getOnlyElement(environments);
+    String environmentProviderClassName = environmentConfiguration.getProperty(environment);
+    PinotConfiguration environmentProviderConfiguration = environmentProviderFactoryConfig.subset(environment);
+    LOGGER.info("Got environment {}, initializing class {}", environment, environmentProviderClassName);
+    register(environment, environmentProviderClassName, environmentProviderConfiguration);
+  }
+
+  // Utility to invoke the cloud specific environment provider.
+  private PinotEnvironmentProvider getEnvironmentProviderInternal(String environment) {
+    Preconditions.checkState(pinotEnvironmentProvider != null,
+        "PinotEnvironmentProvider for environment: %s has not been initialized", environment);
+    return pinotEnvironmentProvider;
+  }
+
+  private void register(String environment, String environmentProviderClassName,
+      PinotConfiguration environmentProviderConfiguration) {
+    try {
+      LOGGER.info("Initializing PinotEnvironmentProvider for environment {}, classname {}",
+          environment, environmentProviderClassName);
+      pinotEnvironmentProvider = PluginManager.get().createInstance(environmentProviderClassName);
+      pinotEnvironmentProvider.init(environmentProviderConfiguration);
+    } catch (Exception ex) {
+      LOGGER.error("Could not instantiate environment provider for class {} with environment {}",
+          environmentProviderClassName, environment, ex);
+      throw new RuntimeException(ex);
+    }
+  }
+}
diff --git a/pinot-spi/src/main/java/org/apache/pinot/spi/utils/CommonConstants.java b/pinot-spi/src/main/java/org/apache/pinot/spi/utils/CommonConstants.java
index 534a3a0..4e8d69a 100644
--- a/pinot-spi/src/main/java/org/apache/pinot/spi/utils/CommonConstants.java
+++ b/pinot-spi/src/main/java/org/apache/pinot/spi/utils/CommonConstants.java
@@ -23,6 +23,10 @@ import java.io.File;
 
 public class CommonConstants {
 
+  public static final String ENVIRONMENT_IDENTIFIER = "environment";
+  public static final String INSTANCE_FAILURE_DOMAIN = "failureDomain";
+  public static final String DEFAULT_FAILURE_DOMAIN = "No such domain";
+
   public static final String PREFIX_OF_SSL_SUBSET = "ssl";
   public static final String HTTP_PROTOCOL = "http";
   public static final String HTTPS_PROTOCOL = "https";
@@ -344,6 +348,10 @@ public class CommonConstants {
 
     public static final String CONFIG_OF_CURRENT_DATA_TABLE_VERSION = "pinot.server.instance.currentDataTableVersion";
     public static final int DEFAULT_CURRENT_DATA_TABLE_VERSION = 3;
+
+    // Environment Provider Configs
+    public static final String PREFIX_OF_CONFIG_OF_ENVIRONMENT_PROVIDER_FACTORY = "pinot.server.environmentProvider.factory";
+    public static final String ENVIRONMENT_PROVIDER_CLASS_NAME = "pinot.server.environmentProvider.className";
   }
 
   public static class Controller {
diff --git a/pinot-spi/src/test/java/org/apache/pinot/spi/environmentprovider/PinotEnvironmentProviderFactoryTest.java b/pinot-spi/src/test/java/org/apache/pinot/spi/environmentprovider/PinotEnvironmentProviderFactoryTest.java
new file mode 100644
index 0000000..c6b54fb
--- /dev/null
+++ b/pinot-spi/src/test/java/org/apache/pinot/spi/environmentprovider/PinotEnvironmentProviderFactoryTest.java
@@ -0,0 +1,68 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.pinot.spi.environmentprovider;
+
+import java.util.HashMap;
+import java.util.Map;
+import org.apache.pinot.spi.env.PinotConfiguration;
+import org.testng.Assert;
+import org.testng.annotations.Test;
+
+public class PinotEnvironmentProviderFactoryTest {
+
+  @Test
+  public void testCustomPinotEnvironmentProviderFactory() {
+    Map<String, Object> properties = new HashMap<>();
+    properties.put("class.test", PinotEnvironmentProviderFactoryTest.TestEnvironmentProvider.class.getName());
+    properties.put("test.maxRetry", "3");
+    properties.put("test.connectionTimeout", "100");
+    properties.put("test.requestTimeout", "100");
+    PinotEnvironmentProviderFactory.init(new PinotConfiguration(properties));
+
+    PinotEnvironmentProvider testPinotEnvironment = PinotEnvironmentProviderFactory.getEnvironmentProvider("test");
+    Assert.assertTrue(testPinotEnvironment instanceof PinotEnvironmentProviderFactoryTest.TestEnvironmentProvider);
+    Assert.assertEquals(((PinotEnvironmentProviderFactoryTest.TestEnvironmentProvider)
+        testPinotEnvironment).getInitCalled(), 1);
+    Assert.assertEquals(((PinotEnvironmentProviderFactoryTest.TestEnvironmentProvider)
+        testPinotEnvironment).getConfiguration().getProperty("maxRetry"), "3");
+    Assert.assertEquals(((PinotEnvironmentProviderFactoryTest.TestEnvironmentProvider)
+        testPinotEnvironment).getConfiguration().getProperty("connectionTimeout"), "100");
+    Assert.assertEquals(((PinotEnvironmentProviderFactoryTest.TestEnvironmentProvider)
+        testPinotEnvironment).getConfiguration().getProperty("requestTimeout"), "100");
+  }
+
+  public static class TestEnvironmentProvider implements PinotEnvironmentProvider {
+    public int initCalled = 0;
+    private PinotConfiguration _configuration;
+
+    public int getInitCalled() {
+      return initCalled;
+    }
+
+    @Override
+    public void init(PinotConfiguration configuration) {
+      _configuration = configuration;
+      initCalled++;
+    }
+
+    public PinotConfiguration getConfiguration() {
+      return _configuration;
+    }
+  }
+}

---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org