diff --git a/connectors/flink/src/main/scala/io/delta/flink/internal/KernelSnapshotDelegator.scala b/connectors/flink/src/main/scala/io/delta/flink/internal/KernelSnapshotDelegator.scala index cd41349277..761ee57e59 100644 --- a/connectors/flink/src/main/scala/io/delta/flink/internal/KernelSnapshotDelegator.scala +++ b/connectors/flink/src/main/scala/io/delta/flink/internal/KernelSnapshotDelegator.scala @@ -18,9 +18,11 @@ package io.delta.standalone.internal +import io.delta.kernel.defaults.engine.DefaultEngine import io.delta.kernel.engine.Engine import org.apache.hadoop.conf.Configuration import org.apache.hadoop.fs.Path + import io.delta.kernel.internal.{SnapshotImpl => SnapshotImplKernel} import io.delta.standalone.DeltaScan import io.delta.standalone.actions.{AddFile => AddFileJ, Metadata => MetadataJ} @@ -63,6 +65,18 @@ class KernelSnapshotDelegator( engine: Engine) extends SnapshotImpl(hadoopConf, path, -1, LogSegment.empty(path), -1, standaloneDeltaLog, -1) { + // For backward compatibility + def this( + kernelSnapshot: SnapshotImplKernel, + kernelSnapshotWrapper: KernelSnapshotWrapper, + hadoopConf: Configuration, + path: Path, + version: Long, + kernelDeltaLog: KernelDeltaLogDelegator, + standaloneDeltaLog: DeltaLogImpl) = + this(kernelSnapshot, kernelSnapshotWrapper, hadoopConf, path, version, kernelDeltaLog, + standaloneDeltaLog, DefaultEngine.create(hadoopConf)) + lazy val standaloneSnapshot: SnapshotImpl = standaloneDeltaLog.getSnapshotForVersionAsOf(getVersion()) /**