http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/0a1b1b7a/modules/core/src/main/java/org/apache/ignite/internal/client/router/impl/GridTcpRouterImpl.java
----------------------------------------------------------------------
diff --cc 
modules/core/src/main/java/org/apache/ignite/internal/client/router/impl/GridTcpRouterImpl.java
index acfd924,0000000..cd4c607
mode 100644,000000..100644
--- 
a/modules/core/src/main/java/org/apache/ignite/internal/client/router/impl/GridTcpRouterImpl.java
+++ 
b/modules/core/src/main/java/org/apache/ignite/internal/client/router/impl/GridTcpRouterImpl.java
@@@ -1,348 -1,0 +1,348 @@@
 +/*
 + * 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.ignite.internal.client.router.impl;
 +
 +import org.apache.ignite.*;
 +import org.apache.ignite.internal.client.*;
 +import org.apache.ignite.internal.client.router.*;
 +import org.apache.ignite.internal.client.ssl.*;
 +import org.apache.ignite.internal.processors.rest.client.message.*;
 +import org.apache.ignite.internal.util.nio.*;
 +import org.apache.ignite.internal.util.nio.ssl.*;
 +import org.apache.ignite.internal.util.typedef.internal.*;
 +import org.apache.ignite.lifecycle.*;
 +import org.apache.ignite.logger.java.*;
 +import org.jetbrains.annotations.*;
 +
 +import javax.management.*;
 +import javax.net.ssl.*;
 +import java.lang.management.*;
 +import java.lang.reflect.*;
 +import java.net.*;
 +import java.nio.*;
 +import java.util.*;
 +
 +/**
 + * Wrapper class for router process.
 + */
 +public class GridTcpRouterImpl implements GridTcpRouter, GridTcpRouterMBean, 
LifecycleAware {
 +    /** */
-     private static final String ENT_NIO_LSNR_CLS = 
"org.gridgain.client.router.impl.GridTcpRouterNioListenerEntImpl";
++    private static final String ENT_NIO_LSNR_CLS = 
"org.apache.ignite.client.router.impl.GridTcpRouterNioListenerEntImpl";
 +
 +    /** Id. */
 +    private final UUID id = UUID.randomUUID();
 +
 +    /** Configuration. */
 +    private final GridTcpRouterConfiguration cfg;
 +
 +    /** Logger. */
 +    private final IgniteLogger log;
 +
 +    /** Server. */
 +    private GridNioServer<GridClientMessage> srv;
 +
 +    /** Client. */
 +    private GridRouterClientImpl client;
 +
 +    /** MBean name. */
 +    private ObjectName mbeanName;
 +
 +    /** NIO parser. */
 +    private volatile GridTcpRouterNioParser parser;
 +
 +    /** Host where server was actually bound. */
 +    private volatile String bindHost;
 +
 +    /** Port where server was actually bound. */
 +    private volatile int bindPort;
 +
 +    /**
 +     * Creates new router instance.
 +     *
 +     * @param cfg Router configuration.
 +     */
 +    public GridTcpRouterImpl(GridTcpRouterConfiguration cfg) {
 +        this.cfg = cfg;
 +
 +        log = cfg.getLogger() != null ?
-             cfg.getLogger().getLogger(getClass()) : new 
IgniteJavaLogger().getLogger(getClass());
++            cfg.getLogger().getLogger(getClass()) : new 
JavaLogger().getLogger(getClass());
 +    }
 +
 +    /**
 +     * Starts router.
 +     *
 +     * @throws IgniteException If failed.
 +     */
 +    @Override public void start() throws IgniteException {
 +        try {
 +            client = createClient(cfg);
 +        }
 +        catch (GridClientException e) {
 +            throw new IgniteException("Failed to initialise embedded 
client.", e);
 +        }
 +
 +        GridNioServerListener<GridClientMessage> lsnr;
 +
 +        try {
 +            Class<?> cls = Class.forName(ENT_NIO_LSNR_CLS);
 +
 +            Constructor<?> cons = 
cls.getDeclaredConstructor(IgniteLogger.class, GridRouterClientImpl.class);
 +
 +            cons.setAccessible(true);
 +
 +            lsnr = 
(GridNioServerListener<GridClientMessage>)cons.newInstance(log, client);
 +        }
 +        catch (ClassNotFoundException ignored) {
 +            lsnr = new GridTcpRouterNioListenerOsImpl(log, client);
 +        }
 +        catch (NoSuchMethodException | IllegalAccessException | 
InstantiationException | InvocationTargetException e) {
 +            throw new IgniteException("Failed to create NIO listener.", e);
 +        }
 +
 +        parser = new GridTcpRouterNioParser();
 +
 +        final InetAddress hostAddr;
 +
 +        try {
 +            hostAddr = InetAddress.getByName(cfg.getHost());
 +        }
 +        catch (UnknownHostException e) {
 +            throw new IgniteException("Failed to resolve grid address for 
configured host: " + cfg.getHost(), e);
 +        }
 +
 +        SSLContext sslCtx;
 +
 +        try {
 +            GridSslContextFactory sslCtxFactory = cfg.getSslContextFactory();
 +
 +            sslCtx = sslCtxFactory == null ? null : 
sslCtxFactory.createSslContext();
 +        }
 +        catch (SSLException e) {
 +            throw new IgniteException("Failed to create SSL context.", e);
 +        }
 +
 +        for (int port = cfg.getPort(), last = port + cfg.getPortRange(); port 
<= last; port++) {
 +            if (startTcpServer(hostAddr, port, lsnr, parser, cfg.isNoDelay(), 
sslCtx, cfg.isSslClientAuth(),
 +                cfg.isSslClientAuth())) {
 +                if (log.isInfoEnabled())
 +                    log.info("TCP router successfully started for endpoint: " 
+ hostAddr.getHostAddress() + ":" + port);
 +
 +                bindPort = port;
 +                bindHost = hostAddr.getHostName();
 +
 +                break;
 +            }
 +            else
 +                U.warn(log, "TCP REST router failed to start on endpoint: " + 
hostAddr.getHostAddress() + ":" + port +
 +                    ". Will try next port within allowed port range.");
 +        }
 +
 +        if (bindPort == 0)
 +            throw new IgniteException("Failed to bind TCP router server 
(possibly all ports in range " +
 +                "are in use) [firstPort=" + cfg.getPort() + ", lastPort=" + 
(cfg.getPort() + cfg.getPortRange()) +
 +                ", addr=" + hostAddr + ']');
 +
 +        try {
 +            ObjectName objName = U.registerMBean(
 +                ManagementFactory.getPlatformMBeanServer(),
 +                "Router",
 +                "TCP Router " + id,
 +                getClass().getSimpleName(),
 +                this,
 +                GridTcpRouterMBean.class);
 +
 +            if (log.isDebugEnabled())
 +                log.debug("Registered MBean: " + objName);
 +
 +            mbeanName = objName;
 +        }
 +        catch (JMException e) {
 +            U.error(log, "Failed to register MBean.", e);
 +        }
 +    }
 +
 +    /**
 +     * Stops this router.
 +     */
 +    @Override public void stop() {
 +        if (srv != null)
 +            srv.stop();
 +
 +        if (client != null)
 +            client.stop(true);
 +
 +        if (mbeanName != null)
 +            try {
 +                
ManagementFactory.getPlatformMBeanServer().unregisterMBean(mbeanName);
 +
 +                if (log.isDebugEnabled())
 +                    log.debug("Unregistered MBean: " + mbeanName);
 +            }
 +            catch (JMException e) {
 +                U.error(log, "Failed to unregister MBean.", e);
 +            }
 +
 +        if (log.isInfoEnabled())
 +            log.info("TCP router successfully stopped.");
 +    }
 +
 +    /**
 +     * Tries to start server with given parameters.
 +     *
 +     * @param hostAddr Host on which server should be bound.
 +     * @param port Port on which server should be bound.
 +     * @param lsnr Server message listener.
 +     * @param parser Server message parser.
 +     * @param tcpNoDelay Flag indicating whether TCP_NODELAY flag should be 
set for accepted connections.
 +     * @param sslCtx SSL context in case if SSL is enabled.
 +     * @param wantClientAuth Whether client will be requested for 
authentication.
 +     * @param needClientAuth Whether client is required to be authenticated.
 +     * @return {@code True} if server successfully started, {@code false} if 
port is used and
 +     *      server was unable to start.
 +     */
 +    private boolean startTcpServer(InetAddress hostAddr, int port, 
GridNioServerListener<GridClientMessage> lsnr,
 +        GridNioParser parser, boolean tcpNoDelay, @Nullable SSLContext 
sslCtx, boolean wantClientAuth,
 +        boolean needClientAuth) {
 +        try {
 +            GridNioFilter codec = new GridNioCodecFilter(parser, log, false);
 +
 +            // This name is required to be unique in order to avoid 
collisions with
 +            // ThreadWorkerGroups running in the same JVM by other 
routers/nodes.
 +            String gridName = "router-" + id;
 +
 +            GridNioFilter[] filters;
 +
 +            if (sslCtx != null) {
 +                GridNioSslFilter sslFilter = new GridNioSslFilter(sslCtx, 
log);
 +
 +                sslFilter.wantClientAuth(wantClientAuth);
 +
 +                sslFilter.needClientAuth(needClientAuth);
 +
 +                filters = new GridNioFilter[] { codec, sslFilter };
 +            }
 +            else
 +                filters = new GridNioFilter[] { codec };
 +
 +            srv = GridNioServer.<GridClientMessage>builder()
 +                .address(hostAddr)
 +                .port(port)
 +                .listener(lsnr)
 +                .logger(log)
 +                .selectorCount(Runtime.getRuntime().availableProcessors())
 +                .gridName(gridName)
 +                .tcpNoDelay(tcpNoDelay)
 +                .directBuffer(false)
 +                .byteOrder(ByteOrder.nativeOrder())
 +                .socketSendBufferSize(0)
 +                .socketReceiveBufferSize(0)
 +                .sendQueueLimit(0)
 +                .filters(filters)
 +                .idleTimeout(cfg.getIdleTimeout())
 +                .build();
 +
 +            srv.start();
 +
 +            return true;
 +        }
 +        catch (IgniteCheckedException e) {
 +            if (log.isDebugEnabled())
 +                log.debug("Failed to start TCP router protocol on port " + 
port + ": " + e.getMessage());
 +
 +            srv = null;
 +
 +            return false;
 +        }
 +    }
 +
 +    /**
 +     * Creates a client for forwarding messages to the grid.
 +     *
 +     * @param routerCfg Router configuration.
 +     * @return Client instance.
 +     * @throws GridClientException If client creation failed.
 +     */
 +    private GridRouterClientImpl createClient(GridTcpRouterConfiguration 
routerCfg) throws GridClientException {
 +        UUID clientId = UUID.randomUUID();
 +
 +        return new GridRouterClientImpl(clientId, routerCfg);
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public String getHost() {
 +        return bindHost;
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public int getPort() {
 +        return bindPort;
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public boolean isSslEnabled() {
 +        return cfg.getSslContextFactory() != null;
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public boolean isSslClientAuth() {
 +        return cfg.isSslClientAuth();
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public Collection<String> getServers() {
 +        return cfg.getServers();
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public UUID id() {
 +        return id;
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public GridTcpRouterConfiguration configuration() {
 +        return cfg;
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public long getReceivedCount() {
 +        return parser != null ? parser.getReceivedCount() : 0;
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public long getSendCount() {
 +        return parser != null ? parser.getSendCount() : 0;
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public int hashCode() {
 +        return id.hashCode();
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public boolean equals(Object o) {
 +        if (o == this)
 +            return true;
 +
 +        if (o == null || getClass() != o.getClass())
 +            return false;
 +
 +        GridTcpRouterImpl that = (GridTcpRouterImpl)o;
 +
 +        return id.equals(that.id);
 +    }
 +}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/0a1b1b7a/modules/core/src/main/java/org/apache/ignite/internal/client/router/impl/GridTcpRouterNioParser.java
----------------------------------------------------------------------
diff --cc 
modules/core/src/main/java/org/apache/ignite/internal/client/router/impl/GridTcpRouterNioParser.java
index 35625b1,0000000..5111ffc
mode 100644,000000..100644
--- 
a/modules/core/src/main/java/org/apache/ignite/internal/client/router/impl/GridTcpRouterNioParser.java
+++ 
b/modules/core/src/main/java/org/apache/ignite/internal/client/router/impl/GridTcpRouterNioParser.java
@@@ -1,109 -1,0 +1,115 @@@
 +/*
 + * 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.ignite.internal.client.router.impl;
 +
 +import org.apache.ignite.*;
 +import org.apache.ignite.internal.client.marshaller.*;
 +import org.apache.ignite.internal.processors.rest.client.message.*;
 +import org.apache.ignite.internal.processors.rest.protocols.tcp.*;
 +import org.apache.ignite.internal.util.nio.*;
 +import org.apache.ignite.internal.util.typedef.internal.*;
 +
 +import java.io.*;
 +import java.nio.*;
 +
 +import static 
org.apache.ignite.internal.processors.rest.protocols.tcp.GridMemcachedMessage.*;
 +
 +/**
 + *
 + */
 +class GridTcpRouterNioParser extends GridTcpRestParser {
 +    /** Number of received messages. */
 +    private volatile long rcvCnt;
 +
 +    /** Number of sent messages. */
 +    private volatile long sndCnt;
 +
++    /**
++     */
++    public GridTcpRouterNioParser() {
++        super(false);
++    }
++
 +    /** {@inheritDoc} */
 +    @Override protected GridClientMessage parseClientMessage(GridNioSession 
ses, ParserState state) {
 +        rcvCnt++;
 +
 +        return new GridRouterRequest(
 +            state.buffer().toByteArray(),
 +            state.header().reqId(),
 +            state.header().clientId(),
 +            state.header().destinationId());
 +    }
 +
 +    /** {@inheritDoc} */
 +    @Override public ByteBuffer encode(GridNioSession ses, Object msg) throws 
IOException, IgniteCheckedException {
 +        sndCnt++;
 +
 +        if (msg instanceof GridRouterResponse) {
 +            GridRouterResponse resp = (GridRouterResponse)msg;
 +
 +            ByteBuffer res = ByteBuffer.allocate(resp.body().length + 45);
 +
 +            res.put(IGNITE_REQ_FLAG);
 +            res.putInt(resp.body().length + 40);
 +            res.putLong(resp.requestId());
 +            res.put(U.uuidToBytes(resp.clientId()));
 +            res.put(U.uuidToBytes(resp.destinationId()));
 +            res.put(resp.body());
 +
 +            res.flip();
 +
 +            return res;
 +        }
 +        else if (msg instanceof GridClientResponse) {
 +            GridClientMarshaller marsh = marshaller(ses);
 +
 +            GridClientMessage clientMsg = (GridClientMessage)msg;
 +
 +            ByteBuffer res = marsh.marshal(msg, 45);
 +
 +            ByteBuffer slice = res.slice();
 +
 +            slice.put(IGNITE_REQ_FLAG);
 +            slice.putInt(res.remaining() - 5);
 +            slice.putLong(clientMsg.requestId());
 +            slice.put(U.uuidToBytes(clientMsg.clientId()));
 +            slice.put(U.uuidToBytes(clientMsg.destinationId()));
 +
 +            return res;
 +        }
 +        else if (msg instanceof GridClientPingPacket || msg instanceof 
GridClientHandshakeResponse)
 +            return super.encode(ses, msg);
 +        else
 +            throw new IgniteCheckedException("Unsupported message: " + msg);
 +    }
 +
 +    /**
 +     * @return Number of received messages.
 +     */
 +    public long getReceivedCount() {
 +        return rcvCnt;
 +    }
 +
 +    /**
 +     * @return Number of sent messages.
 +     */
 +    public long getSendCount() {
 +        return sndCnt;
 +    }
 +}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/0a1b1b7a/modules/core/src/main/java/org/apache/ignite/internal/jdbc/JdbcConnection.java
----------------------------------------------------------------------
diff --cc 
modules/core/src/main/java/org/apache/ignite/internal/jdbc/JdbcConnection.java
index 0000000,0977600..6e73b1f
mode 000000,100644..100644
--- 
a/modules/core/src/main/java/org/apache/ignite/internal/jdbc/JdbcConnection.java
+++ 
b/modules/core/src/main/java/org/apache/ignite/internal/jdbc/JdbcConnection.java
@@@ -1,0 -1,547 +1,547 @@@
+ /*
+  * 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.ignite.internal.jdbc;
+ 
 -import org.apache.ignite.client.*;
++import org.apache.ignite.internal.client.*;
+ 
+ import java.sql.*;
+ import java.util.*;
+ import java.util.concurrent.*;
+ 
+ import static java.sql.ResultSet.*;
+ import static java.util.concurrent.TimeUnit.*;
+ import static org.apache.ignite.IgniteJdbcDriver.*;
+ 
+ /**
+  * JDBC connection implementation.
+  */
+ public class JdbcConnection implements Connection {
+     /** Validation task name. */
+     private static final String VALID_TASK_NAME =
+         
"org.apache.ignite.internal.processors.cache.query.jdbc.GridCacheQueryJdbcValidationTask";
+ 
+     /** Ignite client. */
+     private final GridClient client;
+ 
+     /** Cache name. */
+     private String cacheName;
+ 
+     /** Closed flag. */
+     private boolean closed;
+ 
+     /** URL. */
+     private String url;
+ 
+     /** Node ID. */
+     private UUID nodeId;
+ 
+     /** Timeout. */
+     private int timeout;
+ 
+     /**
+      * Creates new connection.
+      *
+      * @param url Connection URL.
+      * @param props Additional properties.
+      * @throws SQLException In case Ignite client failed to start.
+      */
+     public JdbcConnection(String url, Properties props) throws SQLException {
+         assert url != null;
+         assert props != null;
+ 
+         this.url = url;
+         cacheName = props.getProperty(PROP_CACHE);
+ 
+         String nodeIdProp = props.getProperty(PROP_NODE_ID);
+ 
+         if (nodeIdProp != null)
+             nodeId = UUID.fromString(nodeIdProp);
+ 
+         try {
+             GridClientConfiguration cfg = new GridClientConfiguration(props);
+ 
+             cfg.setServers(Collections.singleton(props.getProperty(PROP_HOST) 
+ ":" + props.getProperty(PROP_PORT)));
+ 
+             // Disable all fetching and caching for metadata.
+             cfg.setEnableMetricsCache(false);
+             cfg.setEnableAttributesCache(false);
+             cfg.setAutoFetchMetrics(false);
+             cfg.setAutoFetchAttributes(false);
+ 
+             client = GridClientFactory.start(cfg);
+         }
+         catch (GridClientException e) {
+             throw new SQLException("Failed to start Ignite client.", e);
+         }
+ 
+         if (!isValid(2))
+             throw new SQLException("Client is invalid. Probably cache name is 
wrong.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Statement createStatement() throws SQLException {
+         return createStatement(TYPE_FORWARD_ONLY, CONCUR_READ_ONLY, 
HOLD_CURSORS_OVER_COMMIT);
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public PreparedStatement prepareStatement(String sql) throws 
SQLException {
+         ensureNotClosed();
+ 
+         return prepareStatement(sql, TYPE_FORWARD_ONLY, CONCUR_READ_ONLY, 
HOLD_CURSORS_OVER_COMMIT);
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public CallableStatement prepareCall(String sql) throws 
SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Callable functions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String nativeSQL(String sql) throws SQLException {
+         ensureNotClosed();
+ 
+         return sql;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void setAutoCommit(boolean autoCommit) throws 
SQLException {
+         ensureNotClosed();
+ 
+         if (!autoCommit)
+             throw new SQLFeatureNotSupportedException("Transactions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean getAutoCommit() throws SQLException {
+         ensureNotClosed();
+ 
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void commit() throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Transactions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void rollback() throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Transactions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void close() throws SQLException {
+         if (closed)
+             return;
+ 
+         closed = true;
+ 
+         GridClientFactory.stop(client.id(), false);
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean isClosed() throws SQLException {
+         return closed;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public DatabaseMetaData getMetaData() throws SQLException {
+         ensureNotClosed();
+ 
+         return new JdbcDatabaseMetadata(this);
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void setReadOnly(boolean readOnly) throws SQLException {
+         ensureNotClosed();
+ 
+         if (!readOnly)
+             throw new SQLFeatureNotSupportedException("Updates are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean isReadOnly() throws SQLException {
+         ensureNotClosed();
+ 
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void setCatalog(String catalog) throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Catalogs are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getCatalog() throws SQLException {
+         ensureNotClosed();
+ 
+         return null;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void setTransactionIsolation(int level) throws 
SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Transactions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getTransactionIsolation() throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Transactions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public SQLWarning getWarnings() throws SQLException {
+         ensureNotClosed();
+ 
+         return null;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void clearWarnings() throws SQLException {
+         ensureNotClosed();
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Statement createStatement(int resSetType, int 
resSetConcurrency) throws SQLException {
+         return createStatement(resSetType, resSetConcurrency, 
HOLD_CURSORS_OVER_COMMIT);
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public PreparedStatement prepareStatement(String sql, int 
resSetType,
+         int resSetConcurrency) throws SQLException {
+         ensureNotClosed();
+ 
+         return prepareStatement(sql, resSetType, resSetConcurrency, 
HOLD_CURSORS_OVER_COMMIT);
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public CallableStatement prepareCall(String sql, int resSetType,
+         int resSetConcurrency) throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Callable functions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Map<String, Class<?>> getTypeMap() throws SQLException {
+         throw new SQLFeatureNotSupportedException("Types mapping is not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void setTypeMap(Map<String, Class<?>> map) throws 
SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Types mapping is not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void setHoldability(int holdability) throws SQLException 
{
+         ensureNotClosed();
+ 
+         if (holdability != HOLD_CURSORS_OVER_COMMIT)
+             throw new SQLFeatureNotSupportedException("Invalid holdability 
(transactions are not supported).");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getHoldability() throws SQLException {
+         ensureNotClosed();
+ 
+         return HOLD_CURSORS_OVER_COMMIT;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Savepoint setSavepoint() throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Transactions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Savepoint setSavepoint(String name) throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Transactions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void rollback(Savepoint savepoint) throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Transactions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void releaseSavepoint(Savepoint savepoint) throws 
SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Transactions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Statement createStatement(int resSetType, int 
resSetConcurrency,
+         int resSetHoldability) throws SQLException {
+         ensureNotClosed();
+ 
+         if (resSetType != TYPE_FORWARD_ONLY)
+             throw new SQLFeatureNotSupportedException("Invalid result set 
type (only forward is supported.)");
+ 
+         if (resSetConcurrency != CONCUR_READ_ONLY)
+             throw new SQLFeatureNotSupportedException("Invalid concurrency 
(updates are not supported).");
+ 
+         if (resSetHoldability != HOLD_CURSORS_OVER_COMMIT)
+             throw new SQLFeatureNotSupportedException("Invalid holdability 
(transactions are not supported).");
+ 
+         JdbcStatement stmt = new JdbcStatement(this);
+ 
+         if (timeout > 0)
+             stmt.timeout(timeout);
+ 
+         return stmt;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public PreparedStatement prepareStatement(String sql, int 
resSetType, int resSetConcurrency,
+         int resSetHoldability) throws SQLException {
+         ensureNotClosed();
+ 
+         if (resSetType != TYPE_FORWARD_ONLY)
+             throw new SQLFeatureNotSupportedException("Invalid result set 
type (only forward is supported.)");
+ 
+         if (resSetConcurrency != CONCUR_READ_ONLY)
+             throw new SQLFeatureNotSupportedException("Invalid concurrency 
(updates are not supported).");
+ 
+         if (resSetHoldability != HOLD_CURSORS_OVER_COMMIT)
+             throw new SQLFeatureNotSupportedException("Invalid holdability 
(transactions are not supported).");
+ 
+         JdbcPreparedStatement stmt = new JdbcPreparedStatement(this, sql);
+ 
+         if (timeout > 0)
+             stmt.timeout(timeout);
+ 
+         return stmt;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public CallableStatement prepareCall(String sql, int 
resSetType, int resSetConcurrency,
+         int resSetHoldability) throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Callable functions are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public PreparedStatement prepareStatement(String sql, int 
autoGeneratedKeys) throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Updates are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public PreparedStatement prepareStatement(String sql, int[] 
colIndexes) throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Updates are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public PreparedStatement prepareStatement(String sql, String[] 
colNames) throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("Updates are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Clob createClob() throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("SQL-specific types are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Blob createBlob() throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("SQL-specific types are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public NClob createNClob() throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("SQL-specific types are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public SQLXML createSQLXML() throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("SQL-specific types are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean isValid(int timeout) throws SQLException {
+         ensureNotClosed();
+ 
+         if (timeout < 0)
+             throw new SQLException("Invalid timeout: " + timeout);
+ 
+         try {
+             return client.compute().<Boolean>executeAsync(VALID_TASK_NAME, 
cacheName).get(timeout, SECONDS);
+         }
+         catch (GridClientDisconnectedException | 
GridClientFutureTimeoutException e) {
+             throw new SQLException("Failed to establish connection.", e);
+         }
+         catch (GridClientException ignored) {
+             return false;
+         }
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void setClientInfo(String name, String val) throws 
SQLClientInfoException {
+         throw new UnsupportedOperationException("Client info is not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void setClientInfo(Properties props) throws 
SQLClientInfoException {
+         throw new UnsupportedOperationException("Client info is not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getClientInfo(String name) throws SQLException {
+         ensureNotClosed();
+ 
+         return null;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Properties getClientInfo() throws SQLException {
+         ensureNotClosed();
+ 
+         return new Properties();
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Array createArrayOf(String typeName, Object[] elements) 
throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("SQL-specific types are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Struct createStruct(String typeName, Object[] attrs) 
throws SQLException {
+         ensureNotClosed();
+ 
+         throw new SQLFeatureNotSupportedException("SQL-specific types are not 
supported.");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public <T> T unwrap(Class<T> iface) throws SQLException {
+         if (!isWrapperFor(iface))
+             throw new SQLException("Connection is not a wrapper for " + 
iface.getName());
+ 
+         return (T)this;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean isWrapperFor(Class<?> iface) throws SQLException 
{
+         return iface != null && iface == Connection.class;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void setSchema(String schema) throws SQLException {
+         cacheName = schema;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getSchema() throws SQLException {
+         return cacheName;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void abort(Executor executor) throws SQLException {
+         close();
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public void setNetworkTimeout(Executor executor, int ms) throws 
SQLException {
+         if (ms < 0)
+             throw new IllegalArgumentException("Timeout is below zero: " + 
ms);
+ 
+         timeout = ms;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getNetworkTimeout() throws SQLException {
+         return timeout;
+     }
+ 
+     /**
+      * @return Ignite client.
+      */
+     GridClient client() {
+         return client;
+     }
+ 
+     /**
+      * @return Cache name.
+      */
+     String cacheName() {
+         return cacheName;
+     }
+ 
+     /**
+      * @return URL.
+      */
+     String url() {
+         return url;
+     }
+ 
+     /**
+      * @return Node ID.
+      */
+     UUID nodeId() {
+         return nodeId;
+     }
+ 
+     /**
+      * Ensures that connection is not closed.
+      *
+      * @throws SQLException If connection is closed.
+      */
+     private void ensureNotClosed() throws SQLException {
+         if (closed)
+             throw new SQLException("Connection is closed.");
+     }
+ 
+     /**
+      * @return Internal statement.
+      * @throws SQLException In case of error.
+      */
+     JdbcStatement createStatement0() throws SQLException {
+         return (JdbcStatement)createStatement();
+     }
+ }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/0a1b1b7a/modules/core/src/main/java/org/apache/ignite/internal/jdbc/JdbcDatabaseMetadata.java
----------------------------------------------------------------------
diff --cc 
modules/core/src/main/java/org/apache/ignite/internal/jdbc/JdbcDatabaseMetadata.java
index 0000000,c5d9079..d13fe04
mode 000000,100644..100644
--- 
a/modules/core/src/main/java/org/apache/ignite/internal/jdbc/JdbcDatabaseMetadata.java
+++ 
b/modules/core/src/main/java/org/apache/ignite/internal/jdbc/JdbcDatabaseMetadata.java
@@@ -1,0 -1,1313 +1,1313 @@@
+ /*
+  * 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.ignite.internal.jdbc;
+ 
 -import org.apache.ignite.client.*;
++import org.apache.ignite.internal.client.*;
+ import org.apache.ignite.internal.util.typedef.internal.*;
+ 
+ import java.sql.*;
+ import java.util.*;
+ 
+ import static java.sql.Connection.*;
+ import static java.sql.ResultSet.*;
+ import static java.sql.RowIdLifetime.*;
+ 
+ /**
+  * JDBC database metadata implementation.
+  */
+ @SuppressWarnings("RedundantCast")
+ public class JdbcDatabaseMetadata implements DatabaseMetaData {
+     /** Task name. */
+     private static final String TASK_NAME =
+         
"org.apache.ignite.internal.processors.cache.query.jdbc.GridCacheQueryJdbcMetadataTask";
+ 
+     /** Connection. */
+     private final JdbcConnection conn;
+ 
+     /** Metadata. */
+     private Map<String, Map<String, Map<String, String>>> meta;
+ 
+     /** Index info. */
+     private Collection<List<Object>> indexes;
+ 
+     /**
+      * @param conn Connection.
+      */
+     JdbcDatabaseMetadata(JdbcConnection conn) {
+         this.conn = conn;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean allProceduresAreCallable() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean allTablesAreSelectable() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getURL() throws SQLException {
+         return conn.url();
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getUserName() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean isReadOnly() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean nullsAreSortedHigh() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean nullsAreSortedLow() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean nullsAreSortedAtStart() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean nullsAreSortedAtEnd() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getDatabaseProductName() throws SQLException {
+         return "Ignite Cache";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getDatabaseProductVersion() throws SQLException {
+         return "4.1.0";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getDriverName() throws SQLException {
+         return "Ignite JDBC Driver";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getDriverVersion() throws SQLException {
+         return "1.0";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getDriverMajorVersion() {
+         return 1;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getDriverMinorVersion() {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean usesLocalFiles() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean usesLocalFilePerTable() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsMixedCaseIdentifiers() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean storesUpperCaseIdentifiers() throws SQLException 
{
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean storesLowerCaseIdentifiers() throws SQLException 
{
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean storesMixedCaseIdentifiers() throws SQLException 
{
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsMixedCaseQuotedIdentifiers() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean storesUpperCaseQuotedIdentifiers() throws 
SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean storesLowerCaseQuotedIdentifiers() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean storesMixedCaseQuotedIdentifiers() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getIdentifierQuoteString() throws SQLException {
+         return " ";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getSQLKeywords() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getNumericFunctions() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getStringFunctions() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getSystemFunctions() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getTimeDateFunctions() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getSearchStringEscape() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getExtraNameCharacters() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsAlterTableWithAddColumn() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsAlterTableWithDropColumn() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsColumnAliasing() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean nullPlusNonNullIsNull() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsConvert() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsConvert(int fromType, int toType) throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsTableCorrelationNames() throws 
SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsDifferentTableCorrelationNames() throws 
SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsExpressionsInOrderBy() throws 
SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsOrderByUnrelated() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsGroupBy() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsGroupByUnrelated() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsGroupByBeyondSelect() throws 
SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsLikeEscapeClause() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsMultipleResultSets() throws SQLException 
{
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsMultipleTransactions() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsNonNullableColumns() throws SQLException 
{
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsMinimumSQLGrammar() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsCoreSQLGrammar() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsExtendedSQLGrammar() throws SQLException 
{
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsANSI92EntryLevelSQL() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsANSI92IntermediateSQL() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsANSI92FullSQL() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsIntegrityEnhancementFacility() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsOuterJoins() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsFullOuterJoins() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsLimitedOuterJoins() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getSchemaTerm() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getProcedureTerm() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getCatalogTerm() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean isCatalogAtStart() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public String getCatalogSeparator() throws SQLException {
+         return "";
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsSchemasInDataManipulation() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsSchemasInProcedureCalls() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsSchemasInTableDefinitions() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsSchemasInIndexDefinitions() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsSchemasInPrivilegeDefinitions() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsCatalogsInDataManipulation() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsCatalogsInProcedureCalls() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsCatalogsInTableDefinitions() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsCatalogsInIndexDefinitions() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsCatalogsInPrivilegeDefinitions() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsPositionedDelete() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsPositionedUpdate() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsSelectForUpdate() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsStoredProcedures() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsSubqueriesInComparisons() throws 
SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsSubqueriesInExists() throws SQLException 
{
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsSubqueriesInIns() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsSubqueriesInQuantifieds() throws 
SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsCorrelatedSubqueries() throws 
SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsUnion() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsUnionAll() throws SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsOpenCursorsAcrossCommit() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsOpenCursorsAcrossRollback() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsOpenStatementsAcrossCommit() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsOpenStatementsAcrossRollback() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxBinaryLiteralLength() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxCharLiteralLength() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxColumnNameLength() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxColumnsInGroupBy() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxColumnsInIndex() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxColumnsInOrderBy() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxColumnsInSelect() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxColumnsInTable() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxConnections() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxCursorNameLength() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxIndexLength() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxSchemaNameLength() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxProcedureNameLength() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxCatalogNameLength() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxRowSize() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean doesMaxRowSizeIncludeBlobs() throws SQLException 
{
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxStatementLength() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxStatements() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxTableNameLength() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxTablesInSelect() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getMaxUserNameLength() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getDefaultTransactionIsolation() throws SQLException 
{
+         return TRANSACTION_NONE;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsTransactions() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsTransactionIsolationLevel(int level) 
throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean 
supportsDataDefinitionAndDataManipulationTransactions() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsDataManipulationTransactionsOnly() 
throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean dataDefinitionCausesTransactionCommit() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean dataDefinitionIgnoredInTransactions() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getProcedures(String catalog, String 
schemaPtrn,
+         String procedureNamePtrn) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Arrays.asList("PROCEDURE_CAT", "PROCEDURE_SCHEM", 
"PROCEDURE_NAME",
+                 "REMARKS", "PROCEDURE_TYPE", "SPECIFIC_NAME"),
+             Arrays.<String>asList(String.class.getName(), 
String.class.getName(), String.class.getName(),
+                 String.class.getName(), Short.class.getName(), 
String.class.getName()),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getProcedureColumns(String catalog, String 
schemaPtrn, String procedureNamePtrn,
+         String colNamePtrn) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Arrays.asList("PROCEDURE_CAT", "PROCEDURE_SCHEM", 
"PROCEDURE_NAME",
+                 "COLUMN_NAME", "COLUMN_TYPE", "DATA_TYPE", "TYPE_NAME", 
"PRECISION",
+                 "LENGTH", "SCALE", "RADIX", "NULLABLE", "REMARKS", 
"COLUMN_DEF",
+                 "SQL_DATA_TYPE", "SQL_DATETIME_SUB", "CHAR_OCTET_LENGTH",
+                 "ORDINAL_POSITION", "IS_NULLABLE", "SPECIFIC_NAME"),
+             Arrays.<String>asList(String.class.getName(), 
String.class.getName(), String.class.getName(),
+                 String.class.getName(), Short.class.getName(), 
Integer.class.getName(), String.class.getName(),
+                 Integer.class.getName(), Integer.class.getName(), 
Short.class.getName(), Short.class.getName(),
+                 Short.class.getName(), String.class.getName(), 
String.class.getName(), Integer.class.getName(),
+                 Integer.class.getName(), Integer.class.getName(), 
Integer.class.getName(), String.class.getName(),
+                 String.class.getName()),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getTables(String catalog, String schemaPtrn, 
String tblNamePtrn,
+         String[] tblTypes) throws SQLException {
+         updateMetaData();
+ 
+         List<List<Object>> rows = new LinkedList<>();
+ 
+         if (tblTypes == null || Arrays.asList(tblTypes).contains("TABLE"))
+             for (Map.Entry<String, Map<String, Map<String, String>>> schema : 
meta.entrySet())
+                 if (matches(schema.getKey(), schemaPtrn))
+                     for (String tbl : schema.getValue().keySet())
+                         if (matches(tbl, tblNamePtrn))
+                             rows.add(tableRow(schema.getKey(), tbl));
+ 
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Arrays.asList("TABLE_CAT", "TABLE_SCHEM", "TABLE_NAME", 
"TABLE_TYPE", "REMARKS", "TYPE_CAT",
+                 "TYPE_SCHEM", "TYPE_NAME", "SELF_REFERENCING_COL_NAME", 
"REF_GENERATION"),
+             Arrays.<String>asList(String.class.getName(), 
String.class.getName(), String.class.getName(),
+                 String.class.getName(), String.class.getName(), 
String.class.getName(), String.class.getName(),
+                 String.class.getName(), String.class.getName(), 
String.class.getName()),
+             rows
+         );
+     }
+ 
+     /**
+      * @param schema Schema name.
+      * @param tbl Table name.
+      * @return Table metadata row.
+      */
+     private List<Object> tableRow(String schema, String tbl) {
+         List<Object> row = new ArrayList<>(10);
+ 
+         row.add((String)null);
+         row.add(schema);
+         row.add(tbl.toUpperCase());
+         row.add("TABLE");
+         row.add((String)null);
+         row.add((String)null);
+         row.add((String)null);
+         row.add((String)null);
+         row.add((String)null);
+         row.add((String)null);
+ 
+         return row;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getSchemas() throws SQLException {
+         return getSchemas(null, "%");
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getCatalogs() throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Arrays.asList("TABLE_CAT"),
+             Arrays.<String>asList(String.class.getName()),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getTableTypes() throws SQLException {
+         return new JdbcResultSet(conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.singletonList("TABLE_TYPE"),
+             Collections.<String>singletonList(String.class.getName()),
+             
Collections.singletonList(Collections.<Object>singletonList("TABLE")));
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getColumns(String catalog, String schemaPtrn, 
String tblNamePtrn,
+         String colNamePtrn) throws SQLException {
+         updateMetaData();
+ 
+         List<List<Object>> rows = new LinkedList<>();
+ 
+         int cnt = 0;
+ 
+         for (Map.Entry<String, Map<String, Map<String, String>>> schema : 
meta.entrySet())
+             if (matches(schema.getKey(), schemaPtrn))
+                 for (Map.Entry<String, Map<String, String>> tbl : 
schema.getValue().entrySet())
+                     if (matches(tbl.getKey(), tblNamePtrn))
+                         for (Map.Entry<String, String> col : 
tbl.getValue().entrySet())
+                             rows.add(columnRow(schema.getKey(), tbl.getKey(), 
col.getKey(),
+                                 JdbcUtils.type(col.getValue()), 
JdbcUtils.typeName(col.getValue()),
+                                 JdbcUtils.nullable(col.getKey(), 
col.getValue()), ++cnt));
+ 
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Arrays.asList("TABLE_CAT", "TABLE_SCHEM", "TABLE_NAME", 
"COLUMN_NAME", "DATA_TYPE",
+                 "TYPE_NAME", "COLUMN_SIZE", "DECIMAL_DIGITS", 
"NUM_PREC_RADIX", "NULLABLE",
+                 "REMARKS", "COLUMN_DEF", "CHAR_OCTET_LENGTH", 
"ORDINAL_POSITION", "IS_NULLABLE",
+                 "SCOPE_CATLOG", "SCOPE_SCHEMA", "SCOPE_TABLE", 
"SOURCE_DATA_TYPE", "IS_AUTOINCREMENT"),
+             Arrays.<String>asList(String.class.getName(), 
String.class.getName(), String.class.getName(),
+                 String.class.getName(), Integer.class.getName(), 
String.class.getName(), Integer.class.getName(),
+                 Integer.class.getName(), Integer.class.getName(), 
Integer.class.getName(), String.class.getName(),
+                 String.class.getName(), Integer.class.getName(), 
Integer.class.getName(), String.class.getName(),
+                 String.class.getName(), String.class.getName(), 
String.class.getName(), Short.class.getName(),
+                 String.class.getName()),
+             rows
+         );
+     }
+ 
+     /**
+      * @param schema Schema name.
+      * @param tbl Table name.
+      * @param col Column name.
+      * @param type Type.
+      * @param typeName Type name.
+      * @param nullable Nullable flag.
+      * @param pos Ordinal position.
+      * @return Column metadata row.
+      */
+     private List<Object> columnRow(String schema, String tbl, String col, int 
type, String typeName,
+         boolean nullable, int pos) {
+         List<Object> row = new ArrayList<>(20);
+ 
+         row.add((String)null);
+         row.add(schema);
+         row.add(tbl);
+         row.add(col);
+         row.add(type);
+         row.add(typeName);
+         row.add((Integer)null);
+         row.add((Integer)null);
+         row.add(10);
+         row.add(nullable ? columnNullable : columnNoNulls);
+         row.add((String)null);
+         row.add((String)null);
+         row.add(Integer.MAX_VALUE);
+         row.add(pos);
+         row.add("YES");
+         row.add((String)null);
+         row.add((String)null);
+         row.add((String)null);
+         row.add((Short)null);
+         row.add("NO");
+ 
+         return row;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getColumnPrivileges(String catalog, String 
schema, String tbl,
+         String colNamePtrn) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getTablePrivileges(String catalog, String 
schemaPtrn,
+         String tblNamePtrn) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getBestRowIdentifier(String catalog, String 
schema, String tbl, int scope,
+         boolean nullable) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getVersionColumns(String catalog, String 
schema, String tbl) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getPrimaryKeys(String catalog, String schema, 
String tbl) throws SQLException {
+         updateMetaData();
+ 
+         List<List<Object>> rows = new LinkedList<>();
+ 
+         for (Map.Entry<String, Map<String, Map<String, String>>> s : 
meta.entrySet())
+             if (schema == null || schema.toUpperCase().equals(s.getKey()))
+                 for (Map.Entry<String, Map<String, String>> t : 
s.getValue().entrySet())
+                     if (tbl == null || tbl.toUpperCase().equals(t.getKey()))
+                         rows.add(Arrays.<Object>asList((String)null, 
s.getKey().toUpperCase(),
+                             t.getKey().toUpperCase(), "_KEY", 1, "_KEY"));
+ 
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Arrays.asList("TABLE_CAT", "TABLE_SCHEM", "TABLE_NAME", 
"COLUMN_NAME", "KEY_SEQ", "PK_NAME"),
+             Arrays.asList(String.class.getName(), String.class.getName(), 
String.class.getName(),
+                 String.class.getName(), Short.class.getName(), 
String.class.getName()),
+             rows
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getImportedKeys(String catalog, String schema, 
String tbl) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getExportedKeys(String catalog, String schema, 
String tbl) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getCrossReference(String parentCatalog, String 
parentSchema, String parentTbl,
+         String foreignCatalog, String foreignSchema, String foreignTbl) 
throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getTypeInfo() throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getIndexInfo(String catalog, String schema, 
String tbl, boolean unique,
+         boolean approximate) throws SQLException {
+         Collection<List<Object>> rows = new ArrayList<>(indexes.size());
+ 
+         for (List<Object> idx : indexes) {
+             String idxSchema = (String)idx.get(0);
+             String idxTbl = (String)idx.get(1);
+ 
+             if ((schema == null || schema.equals(idxSchema)) && (tbl == null 
|| tbl.equals(idxTbl))) {
+                 List<Object> row = new ArrayList<>(13);
+ 
+                 row.add((String)null);
+                 row.add(idxSchema);
+                 row.add(idxTbl);
+                 row.add((Boolean)idx.get(2));
+                 row.add((String)null);
+                 row.add((String)idx.get(3));
+                 row.add((int)tableIndexOther);
+                 row.add((Integer)idx.get(4));
+                 row.add((String)idx.get(5));
+                 row.add((Boolean)idx.get(6) ? "D" : "A");
+                 row.add(0);
+                 row.add(0);
+                 row.add((String)null);
+ 
+                 rows.add(row);
+             }
+         }
+ 
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Arrays.asList("TABLE_CAT", "TABLE_SCHEM", "TABLE_NAME", 
"NON_UNIQUE", "INDEX_QUALIFIER",
+                 "INDEX_NAME", "TYPE", "ORDINAL_POSITION", "COLUMN_NAME", 
"ASC_OR_DESC", "CARDINALITY",
+                 "PAGES", "FILTER_CONDITION"),
+             Arrays.asList(String.class.getName(), String.class.getName(), 
String.class.getName(),
+                 Boolean.class.getName(), String.class.getName(), 
String.class.getName(), Short.class.getName(),
+                 Short.class.getName(), String.class.getName(), 
String.class.getName(), Integer.class.getName(),
+                 Integer.class.getName(), String.class.getName()),
+             rows
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsResultSetType(int type) throws 
SQLException {
+         return true;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsResultSetConcurrency(int type, int 
concurrency) throws SQLException {
+         return concurrency == CONCUR_READ_ONLY;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean ownUpdatesAreVisible(int type) throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean ownDeletesAreVisible(int type) throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean ownInsertsAreVisible(int type) throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean othersUpdatesAreVisible(int type) throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean othersDeletesAreVisible(int type) throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean othersInsertsAreVisible(int type) throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean updatesAreDetected(int type) throws SQLException 
{
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean deletesAreDetected(int type) throws SQLException 
{
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean insertsAreDetected(int type) throws SQLException 
{
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsBatchUpdates() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getUDTs(String catalog, String schemaPtrn, 
String typeNamePtrn,
+         int[] types) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public Connection getConnection() throws SQLException {
+         return conn;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsSavepoints() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsNamedParameters() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsMultipleOpenResults() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsGetGeneratedKeys() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getSuperTypes(String catalog, String 
schemaPtrn,
+         String typeNamePtrn) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getSuperTables(String catalog, String 
schemaPtrn,
+         String tblNamePtrn) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getAttributes(String catalog, String 
schemaPtrn, String typeNamePtrn,
+         String attributeNamePtrn) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsResultSetHoldability(int holdability) 
throws SQLException {
+         return holdability == HOLD_CURSORS_OVER_COMMIT;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getResultSetHoldability() throws SQLException {
+         return HOLD_CURSORS_OVER_COMMIT;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getDatabaseMajorVersion() throws SQLException {
+         return 1;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getDatabaseMinorVersion() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getJDBCMajorVersion() throws SQLException {
+         return 1;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getJDBCMinorVersion() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public int getSQLStateType() throws SQLException {
+         return 0;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean locatorsUpdateCopy() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsStatementPooling() throws SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public RowIdLifetime getRowIdLifetime() throws SQLException {
+         return ROWID_UNSUPPORTED;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getSchemas(String catalog, String schemaPtrn) 
throws SQLException {
+         updateMetaData();
+ 
+         List<List<Object>> rows = new ArrayList<>(meta.size());
+ 
+         for (String schema : meta.keySet())
+             if (matches(schema, schemaPtrn))
+                 rows.add(Arrays.<Object>asList(schema, (String)null));
+ 
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Arrays.asList("TABLE_SCHEM", "TABLE_CATALOG"),
+             Arrays.<String>asList(String.class.getName(), 
String.class.getName()),
+             rows
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean supportsStoredFunctionsUsingCallSyntax() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean autoCommitFailureClosesAllResultSets() throws 
SQLException {
+         return false;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getClientInfoProperties() throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getFunctions(String catalog, String schemaPtrn,
+         String functionNamePtrn) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Arrays.asList("FUNCTION_CAT", "FUNCTION_SCHEM", "FUNCTION_NAME",
+                 "REMARKS", "FUNCTION_TYPE", "SPECIFIC_NAME"),
+             Arrays.<String>asList(String.class.getName(), 
String.class.getName(), String.class.getName(),
+                 String.class.getName(), Short.class.getName(), 
String.class.getName()),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getFunctionColumns(String catalog, String 
schemaPtrn, String functionNamePtrn,
+         String colNamePtrn) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Arrays.asList("FUNCTION_CAT", "FUNCTION_SCHEM", "FUNCTION_NAME",
+                 "COLUMN_NAME", "COLUMN_TYPE", "DATA_TYPE", "TYPE_NAME", 
"PRECISION",
+                 "LENGTH", "SCALE", "RADIX", "NULLABLE", "REMARKS", 
"CHAR_OCTET_LENGTH",
+                 "ORDINAL_POSITION", "IS_NULLABLE", "SPECIFIC_NAME"),
+             Arrays.<String>asList(String.class.getName(), 
String.class.getName(), String.class.getName(),
+                 String.class.getName(), Short.class.getName(), 
Integer.class.getName(), String.class.getName(),
+                 Integer.class.getName(), Integer.class.getName(), 
Short.class.getName(), Short.class.getName(),
+                 Short.class.getName(), String.class.getName(), 
Integer.class.getName(), Integer.class.getName(),
+                 String.class.getName(), String.class.getName()),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public <T> T unwrap(Class<T> iface) throws SQLException {
+         if (!isWrapperFor(iface))
+             throw new SQLException("Database meta data is not a wrapper for " 
+ iface.getName());
+ 
+         return (T)this;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean isWrapperFor(Class<?> iface) throws SQLException 
{
+         return iface != null && iface == DatabaseMetaData.class;
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public ResultSet getPseudoColumns(String catalog, String 
schemaPtrn, String tblNamePtrn,
+         String colNamePtrn) throws SQLException {
+         return new JdbcResultSet(
+             conn.createStatement0(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<String>emptyList(),
+             Collections.<List<Object>>emptyList()
+         );
+     }
+ 
+     /** {@inheritDoc} */
+     @Override public boolean generatedKeyAlwaysReturned() throws SQLException 
{
+         return false;
+     }
+ 
+     /**
+      * Updates meta data.
+      *
+      * @throws SQLException In case of error.
+      */
+     private void updateMetaData() throws SQLException {
+         if (conn.isClosed())
+             throw new SQLException("Connection is closed.");
+ 
+         try {
+             byte[] packet = conn.client().compute().execute(TASK_NAME, 
conn.cacheName());
+ 
+             byte status = packet[0];
+             byte[] data = new byte[packet.length - 1];
+ 
+             U.arrayCopy(packet, 1, data, 0, data.length);
+ 
+             if (status == 1)
+                 throw JdbcUtils.unmarshalError(data);
+             else {
+                 List<Object> res = JdbcUtils.unmarshal(data);
+ 
+                 meta = (Map<String, Map<String, Map<String, 
String>>>)res.get(0);
+                 indexes = (Collection<List<Object>>)res.get(1);
+             }
+         }
+         catch (GridClientException e) {
+             throw new SQLException("Failed to get meta data from Ignite.", e);
+         }
+     }
+ 
+     /**
+      * Checks whether string matches SQL pattern.
+      *
+      * @param str String.
+      * @param ptrn Pattern.
+      * @return Whether string matches pattern.
+      */
+     private boolean matches(String str, String ptrn) {
+         return str != null && (ptrn == null ||
+             str.toUpperCase().matches(ptrn.toUpperCase().replace("%", 
".*").replace("_", ".")));
+     }
+ }

Reply via email to